Skip to content

Commit

Permalink
Add planner tests for SetFlatteningOptimizer
Browse files Browse the repository at this point in the history
  • Loading branch information
rschlussel-zz authored and martint committed Dec 9, 2016
1 parent ca5b866 commit d5b7093
Show file tree
Hide file tree
Showing 3 changed files with 162 additions and 0 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -67,6 +67,18 @@ protected void assertPlan(String sql, LogicalPlanner.Stage stage, PlanMatchPatte
});
}

protected void assertPlanWithOptimizers(String sql, PlanMatchPattern pattern, List<PlanOptimizer> optimizers)
{
queryRunner.inTransaction(transactionSession -> {
FeaturesConfig featuresConfig = new FeaturesConfig()
.setDistributedIndexJoinsEnabled(false)
.setOptimizeHashGeneration(true);
Plan actualPlan = queryRunner.createPlan(transactionSession, sql, featuresConfig, optimizers, LogicalPlanner.Stage.OPTIMIZED);
PlanAssert.assertPlan(transactionSession, queryRunner.getMetadata(), actualPlan, pattern);
return null;
});
}

protected void assertMinimallyOptimizedPlan(@Language("SQL") String sql, PlanMatchPattern pattern)
{
LocalQueryRunner queryRunner = getQueryRunner();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,14 +21,17 @@
import com.facebook.presto.sql.planner.Symbol;
import com.facebook.presto.sql.planner.plan.AggregationNode;
import com.facebook.presto.sql.planner.plan.ApplyNode;
import com.facebook.presto.sql.planner.plan.ExceptNode;
import com.facebook.presto.sql.planner.plan.FilterNode;
import com.facebook.presto.sql.planner.plan.GroupIdNode;
import com.facebook.presto.sql.planner.plan.IntersectNode;
import com.facebook.presto.sql.planner.plan.JoinNode;
import com.facebook.presto.sql.planner.plan.OutputNode;
import com.facebook.presto.sql.planner.plan.PlanNode;
import com.facebook.presto.sql.planner.plan.ProjectNode;
import com.facebook.presto.sql.planner.plan.SemiJoinNode;
import com.facebook.presto.sql.planner.plan.TableScanNode;
import com.facebook.presto.sql.planner.plan.UnionNode;
import com.facebook.presto.sql.planner.plan.ValuesNode;
import com.facebook.presto.sql.planner.plan.WindowNode;
import com.facebook.presto.sql.tree.Expression;
Expand Down Expand Up @@ -203,6 +206,21 @@ public static PlanMatchPattern join(JoinNode.Type joinType, List<ExpectedValuePr
expectedFilter.map(predicate -> rewriteQualifiedNamesToSymbolReferences(new SqlParser().createExpression(predicate)))));
}

public static PlanMatchPattern union(PlanMatchPattern... sources)
{
return node(UnionNode.class, sources);
}

public static PlanMatchPattern intersect(PlanMatchPattern... sources)
{
return node(IntersectNode.class, sources);
}

public static PlanMatchPattern except(PlanMatchPattern... sources)
{
return node(ExceptNode.class, sources);
}

public static ExpectedValueProvider<JoinNode.EquiJoinClause> equiJoinClause(String left, String right)
{
return new EquiJoinClauseProvider(new SymbolAlias(left), new SymbolAlias(right));
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,132 @@
/*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package com.facebook.presto.sql.planner.optimizations;

import com.facebook.presto.sql.planner.assertions.BasePlanTest;
import com.facebook.presto.sql.planner.assertions.PlanMatchPattern;
import com.google.common.collect.ImmutableList;
import org.testng.annotations.Test;

import java.util.List;

import static com.facebook.presto.sql.planner.assertions.PlanMatchPattern.anyTree;
import static com.facebook.presto.sql.planner.assertions.PlanMatchPattern.except;
import static com.facebook.presto.sql.planner.assertions.PlanMatchPattern.intersect;
import static com.facebook.presto.sql.planner.assertions.PlanMatchPattern.tableScan;
import static com.facebook.presto.sql.planner.assertions.PlanMatchPattern.union;

public class TestSetFlatteningOptimizer
extends BasePlanTest
{
@Test
public void testFlattensUnion()
{
assertPlan(
"(SELECT * FROM nation UNION SELECT * FROM nation)" +
"UNION (SELECT * FROM nation UNION SELECT * FROM nation)",
anyTree(
union(
tableScan("nation"),
tableScan("nation"),
tableScan("nation"),
tableScan("nation"))));
}

@Test
public void testFlattensUnionAll()
{
assertPlan(
"(SELECT * FROM nation UNION ALL SELECT * FROM nation)" +
"UNION ALL (SELECT * FROM nation UNION ALL SELECT * FROM nation)",
anyTree(
union(
tableScan("nation"),
tableScan("nation"),
tableScan("nation"),
tableScan("nation"))));
}

@Test
public void testFlattensUnionAndUnionAllWhenAllowed()
{
assertPlan(
"SELECT * FROM nation " +
"UNION ALL (SELECT * FROM nation " +
"UNION (SELECT * FROM nation UNION ALL select * FROM nation))",
anyTree(
union(
tableScan("nation"),
anyTree(
union(
tableScan("nation"),
tableScan("nation"),
tableScan("nation"))))));
}

@Test
public void testFlattensIntersect()
{
assertPlan(
"(SELECT * FROM nation INTERSECT SELECT * FROM nation)" +
"INTERSECT (SELECT * FROM nation INTERSECT SELECT * FROM nation)",
anyTree(
intersect(
tableScan("nation"),
tableScan("nation"),
tableScan("nation"),
tableScan("nation"))));
}

@Test
public void testFlattensOnlyFirstInputOfExcept()
{
assertPlan(
"(SELECT * FROM nation EXCEPT SELECT * FROM nation)" +
"EXCEPT (SELECT * FROM nation EXCEPT SELECT * FROM nation)",
anyTree(
except(
tableScan("nation"),
tableScan("nation"),
except(
tableScan("nation"),
tableScan("nation")))));
}

@Test
public void testDoesNotFlattenDifferentSetOperations()
{
assertPlan(
"(SELECT * FROM nation EXCEPT SELECT * FROM nation)" +
"UNION (SELECT * FROM nation INTERSECT SELECT * FROM nation)",
anyTree(
union(
except(
tableScan("nation"),
tableScan("nation")),
intersect(
tableScan("nation"),
tableScan("nation")))));
}

public void assertPlan(String sql, PlanMatchPattern pattern)
{
List<PlanOptimizer> optimizers = ImmutableList.of(
new UnaliasSymbolReferences(),
new PruneUnreferencedOutputs(),
new PruneIdentityProjections(),
new SetFlatteningOptimizer());
assertPlanWithOptimizers(sql, pattern, optimizers);
}
}

0 comments on commit d5b7093

Please sign in to comment.