From 7c18b68dedd22f57f015ef44fb02577199dc9882 Mon Sep 17 00:00:00 2001 From: seawinde Date: Sun, 10 Dec 2023 16:57:39 +0800 Subject: [PATCH] [feature](nereids) Support outer join rewrite by materialized view and add some regression test --- .../java/org/apache/doris/mtmv/MVCache.java | 2 +- .../jobs/joinorder/hypergraph/HyperGraph.java | 21 +- .../apache/doris/nereids/rules/RuleSet.java | 4 + ...AbstractMaterializedViewAggregateRule.java | 286 ++++++++++++++++- .../mv/AbstractMaterializedViewJoinRule.java | 24 +- .../mv/AbstractMaterializedViewRule.java | 90 ++++-- .../mv/InitMaterializationContextHook.java | 19 +- .../mv/LogicalCompatibilityContext.java | 30 +- .../mv/MaterializationContext.java | 19 +- .../mv/MaterializedViewAggregateRule.java | 15 +- .../MaterializedViewProjectAggregateRule.java | 46 +++ .../mv/MaterializedViewProjectJoinRule.java | 2 +- .../rules/exploration/mv/StructInfo.java | 101 ++++-- .../mv/mapping/ExpressionIndexMapping.java | 48 --- .../mv/mapping/ExpressionMapping.java | 40 ++- .../rules/exploration/mv/mapping/Mapping.java | 8 + .../apache/doris/nereids/trees/TreeNode.java | 17 ++ .../functions/agg/AggregateFunction.java | 4 + .../trees/expressions/functions/agg/Sum.java | 5 + .../plans/commands/info/CreateMTMVInfo.java | 2 +- .../trees/plans/logical/LogicalAggregate.java | 1 - .../visitor/ExpressionLineageReplacer.java | 44 ++- .../doris/nereids/util/ExpressionUtils.java | 66 ++-- .../org/apache/doris/qe/StmtExecutor.java | 4 - .../mv/MaterializedViewUtilsTest.java | 32 +- .../mv/aggregate_with_roll_up.out | 15 + .../mv/aggregate_without_roll_up.out | 31 ++ .../data/nereids_rules_p0/mv/inner_join.out | 44 ++- .../data/nereids_rules_p0/mv/outer_join.out | 69 +++++ .../mv/aggregate_with_roll_up.groovy | 237 ++++++++++++++ .../mv/aggregate_without_roll_up.groovy | 281 +++++++++++++++++ .../nereids_rules_p0/mv/inner_join.groovy | 74 ++++- .../nereids_rules_p0/mv/outer_join.groovy | 288 ++++++++++++++++++ 33 files changed, 1748 insertions(+), 221 deletions(-) create mode 100644 fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewProjectAggregateRule.java delete mode 100644 fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/ExpressionIndexMapping.java create mode 100644 regression-test/data/nereids_rules_p0/mv/aggregate_with_roll_up.out create mode 100644 regression-test/data/nereids_rules_p0/mv/aggregate_without_roll_up.out create mode 100644 regression-test/data/nereids_rules_p0/mv/outer_join.out create mode 100644 regression-test/suites/nereids_rules_p0/mv/aggregate_with_roll_up.groovy create mode 100644 regression-test/suites/nereids_rules_p0/mv/aggregate_without_roll_up.groovy create mode 100644 regression-test/suites/nereids_rules_p0/mv/outer_join.groovy diff --git a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MVCache.java b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MVCache.java index b5cf92f87e4e73..1ee1d4c2f98940 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MVCache.java +++ b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MVCache.java @@ -74,7 +74,7 @@ public static MVCache from(MTMV mtmv, ConnectContext connectContext) { ? (Plan) ((LogicalResultSink) mvRewrittenPlan).child() : mvRewrittenPlan; // use rewritten plan output expression currently, if expression rewrite fail, // consider to use the analyzed plan for output expressions only - List mvOutputExpressions = mvRewrittenPlan.getExpressions().stream() + List mvOutputExpressions = mvPlan.getExpressions().stream() .map(NamedExpression.class::cast) .collect(Collectors.toList()); return new MVCache(mvPlan, mvOutputExpressions); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/HyperGraph.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/HyperGraph.java index 7b721a40993ddf..8c678c0d2c9a2e 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/HyperGraph.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/HyperGraph.java @@ -82,6 +82,10 @@ public List getJoinEdges() { return joinEdges; } + public List getFilterEdges() { + return filterEdges; + } + public List getNodes() { return nodes; } @@ -589,7 +593,7 @@ public int edgeSize() { * * @param viewHG the compared hyper graph * @return null represents not compatible, or return some expression which can - * be pull up from this hyper graph + * be pull up from this hyper graph */ public @Nullable List isLogicCompatible(HyperGraph viewHG, LogicalCompatibilityContext ctx) { Map queryToView = constructEdgeMap(viewHG, ctx.getQueryToViewEdgeExpressionMapping()); @@ -661,14 +665,15 @@ private boolean compareJoinEdge(JoinEdge t, JoinEdge o, Map no long tRight = t.getRightExtendedNodes(); long oLeft = o.getLeftExtendedNodes(); long oRight = o.getRightExtendedNodes(); - if (!t.getJoinType().equals(o.getJoinType())) { - if (!t.getJoinType().swap().equals(o.getJoinType())) { - return false; - } - oRight = o.getLeftExtendedNodes(); - oLeft = o.getRightExtendedNodes(); + if (!t.getJoinType().equals(o.getJoinType()) && !t.getJoinType().swap().equals(o.getJoinType())) { + return false; + } + boolean matched = false; + if (t.getJoinType().swap().equals(o.getJoinType())) { + matched |= compareNodeMap(tRight, oLeft, nodeMap) && compareNodeMap(tLeft, oRight, nodeMap); } - return compareNodeMap(tLeft, oLeft, nodeMap) && compareNodeMap(tRight, oRight, nodeMap); + matched |= compareNodeMap(tLeft, oLeft, nodeMap) && compareNodeMap(tRight, oRight, nodeMap); + return matched; } private boolean compareNodeMap(long bitmap1, long bitmap2, Map nodeIDMap) { diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java index ed2bc775e259c8..a937472c862fbc 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java @@ -40,6 +40,8 @@ import org.apache.doris.nereids.rules.exploration.join.PushDownProjectThroughSemiJoin; import org.apache.doris.nereids.rules.exploration.join.SemiJoinSemiJoinTranspose; import org.apache.doris.nereids.rules.exploration.join.SemiJoinSemiJoinTransposeProject; +import org.apache.doris.nereids.rules.exploration.mv.MaterializedViewAggregateRule; +import org.apache.doris.nereids.rules.exploration.mv.MaterializedViewProjectAggregateRule; import org.apache.doris.nereids.rules.exploration.mv.MaterializedViewProjectJoinRule; import org.apache.doris.nereids.rules.implementation.AggregateStrategies; import org.apache.doris.nereids.rules.implementation.LogicalAssertNumRowsToPhysicalAssertNumRows; @@ -223,6 +225,8 @@ public class RuleSet { public static final List MATERIALIZED_VIEW_RULES = planRuleFactories() .add(MaterializedViewProjectJoinRule.INSTANCE) + .add(MaterializedViewAggregateRule.INSTANCE) + .add(MaterializedViewProjectAggregateRule.INSTANCE) .build(); public List getDPHypReorderRules() { diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java index a9a8b754d33648..0a5d3f0948c873 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java @@ -17,9 +17,293 @@ package org.apache.doris.nereids.rules.exploration.mv; +import org.apache.doris.common.Pair; +import org.apache.doris.nereids.jobs.joinorder.hypergraph.HyperGraph; +import org.apache.doris.nereids.jobs.joinorder.hypergraph.edge.JoinEdge; +import org.apache.doris.nereids.jobs.joinorder.hypergraph.node.AbstractNode; +import org.apache.doris.nereids.jobs.joinorder.hypergraph.node.StructInfoNode; +import org.apache.doris.nereids.rules.exploration.mv.StructInfo.PlanSplitContext; +import org.apache.doris.nereids.rules.exploration.mv.mapping.ExpressionMapping; +import org.apache.doris.nereids.rules.exploration.mv.mapping.SlotMapping; +import org.apache.doris.nereids.trees.expressions.ExprId; +import org.apache.doris.nereids.trees.expressions.Expression; +import org.apache.doris.nereids.trees.expressions.NamedExpression; +import org.apache.doris.nereids.trees.expressions.Slot; +import org.apache.doris.nereids.trees.expressions.functions.agg.AggregateFunction; +import org.apache.doris.nereids.trees.expressions.functions.agg.Sum; +import org.apache.doris.nereids.trees.plans.Plan; +import org.apache.doris.nereids.trees.plans.logical.LogicalAggregate; +import org.apache.doris.nereids.trees.plans.logical.LogicalProject; +import org.apache.doris.nereids.util.ExpressionUtils; + +import com.google.common.collect.HashMultimap; +import com.google.common.collect.Multimap; +import com.google.common.collect.Sets; + +import java.util.ArrayList; +import java.util.HashSet; +import java.util.List; +import java.util.Map; +import java.util.Set; +import java.util.stream.Collectors; + /** * AbstractMaterializedViewAggregateRule * This is responsible for common aggregate rewriting - * */ + */ public abstract class AbstractMaterializedViewAggregateRule extends AbstractMaterializedViewRule { + + @Override + protected Plan rewriteQueryByView(MatchMode matchMode, + StructInfo queryStructInfo, + StructInfo viewStructInfo, + SlotMapping queryToViewSlotMapping, + Plan tempRewritedPlan, + MaterializationContext materializationContext) { + // get view and query aggregate and top plan correspondingly + Pair> viewTopPlanAndAggPair = splitToTopPlanAndAggregate(viewStructInfo); + if (viewTopPlanAndAggPair == null) { + return null; + } + Pair> queryTopPlanAndAggPair = splitToTopPlanAndAggregate(queryStructInfo); + if (queryTopPlanAndAggPair == null) { + return null; + } + // Firstly, handle query group by expression rewrite + LogicalAggregate queryAggregate = queryTopPlanAndAggPair.value(); + Plan queryTopPlan = queryTopPlanAndAggPair.key(); + // query and view have the same dimension, try to rewrite rewrittenQueryGroupExpr + LogicalAggregate viewAggregate = viewTopPlanAndAggPair.value(); + Plan viewTopPlan = viewTopPlanAndAggPair.key(); + boolean needRollUp = + queryAggregate.getGroupByExpressions().size() != viewAggregate.getGroupByExpressions().size(); + if (queryAggregate.getGroupByExpressions().size() == viewAggregate.getGroupByExpressions().size()) { + List queryGroupShuttledExpression = ExpressionUtils.shuttleExpressionWithLineage( + queryAggregate.getGroupByExpressions(), queryTopPlan); + List viewGroupShuttledExpression = ExpressionUtils.shuttleExpressionWithLineage( + viewAggregate.getGroupByExpressions(), viewTopPlan) + .stream() + .map(expr -> ExpressionUtils.replace(expr, queryToViewSlotMapping.inverse().toSlotReferenceMap())) + .collect(Collectors.toList()); + needRollUp = !queryGroupShuttledExpression.equals(viewGroupShuttledExpression); + } + if (!needRollUp) { + List rewrittenQueryGroupExpr = rewriteExpression(queryTopPlan.getExpressions(), + queryTopPlan, + materializationContext.getMvExprToMvScanExprMapping(), + queryToViewSlotMapping, + true); + if (rewrittenQueryGroupExpr.isEmpty()) { + // can not rewrite, bail out. + return null; + } + return new LogicalProject<>( + rewrittenQueryGroupExpr.stream().map(NamedExpression.class::cast).collect(Collectors.toList()), + tempRewritedPlan); + } + // the dimension in query and view are different, try to roll up + // Split query aggregate to group expression and agg function + // Firstly, find the query top output rewrite function expr list which only use query aggregate function, + // This will be used to roll up + if (viewAggregate.getOutputExpressions().stream().anyMatch( + viewExpr -> viewExpr.anyMatch(expr -> expr instanceof AggregateFunction + && ((AggregateFunction) expr).isDistinct()))) { + // if mv aggregate function contains distinct, can not roll up, bail out. + return null; + } + // split the query top plan expressions to group expressions and functions, if can not, bail out. + Pair, Set> queryGroupAndFunctionPair + = topPlanSplitToGroupAndFunction(queryTopPlanAndAggPair); + if (queryGroupAndFunctionPair == null) { + return null; + } + // Secondly, try to roll up the agg functions + // this map will be used to rewrite expression + Multimap needRollupExprMap = HashMultimap.create(); + Multimap groupRewrittenExprMap = HashMultimap.create(); + Map mvExprToMvScanExprQueryBased = + materializationContext.getMvExprToMvScanExprMapping().keyPermute( + queryToViewSlotMapping.inverse()).flattenMap().get(0); + + Set queryTopPlanFunctionSet = queryGroupAndFunctionPair.value(); + // try to rewrite, contains both roll up aggregate functions and aggregate group expression + List finalAggregateExpressions = new ArrayList<>(); + List finalGroupExpressions = new ArrayList<>(); + for (Expression topExpression : queryTopPlan.getExpressions()) { + // is agg function, try to roll up and rewrite + if (queryTopPlanFunctionSet.contains(topExpression)) { + Expression needRollupShuttledExpr = ExpressionUtils.shuttleExpressionWithLineage( + topExpression, + queryTopPlan); + if (!mvExprToMvScanExprQueryBased.containsKey(needRollupShuttledExpr)) { + // function can not rewrite by view + return null; + } + // try to roll up + AggregateFunction needRollupAggFunction = (AggregateFunction) topExpression.firstMatch( + expr -> expr instanceof AggregateFunction); + AggregateFunction rollupAggregateFunction = rollup(needRollupAggFunction, + mvExprToMvScanExprQueryBased.get(needRollupShuttledExpr)); + if (rollupAggregateFunction == null) { + return null; + } + // key is query need roll up expr, value is mv scan based roll up expr + needRollupExprMap.put(needRollupShuttledExpr, rollupAggregateFunction); + // rewrite query function expression by mv expression + Expression rewrittenFunctionExpression = rewriteExpression(topExpression, + queryTopPlan, + new ExpressionMapping(needRollupExprMap), + queryToViewSlotMapping, + false); + if (rewrittenFunctionExpression == null) { + return null; + } + finalAggregateExpressions.add((NamedExpression) rewrittenFunctionExpression); + } else { + // try to rewrite group expression + Expression queryGroupShuttledExpr = + ExpressionUtils.shuttleExpressionWithLineage(topExpression, queryTopPlan); + if (!mvExprToMvScanExprQueryBased.containsKey(queryGroupShuttledExpr)) { + // group expr can not rewrite by view + return null; + } + groupRewrittenExprMap.put(queryGroupShuttledExpr, + mvExprToMvScanExprQueryBased.get(queryGroupShuttledExpr)); + // rewrite query group expression by mv expression + Expression rewrittenGroupExpression = rewriteExpression( + topExpression, + queryTopPlan, + new ExpressionMapping(groupRewrittenExprMap), + queryToViewSlotMapping, + true); + if (rewrittenGroupExpression == null) { + return null; + } + finalAggregateExpressions.add((NamedExpression) rewrittenGroupExpression); + finalGroupExpressions.add(rewrittenGroupExpression); + } + } + // add project to guarantee group by column ref is slot reference, + // this is necessary because physical createHash will need slotReference later + List copiedFinalGroupExpressions = new ArrayList<>(finalGroupExpressions); + List projectsUnderAggregate = copiedFinalGroupExpressions.stream() + .map(NamedExpression.class::cast) + .collect(Collectors.toList()); + projectsUnderAggregate.addAll(tempRewritedPlan.getOutput()); + LogicalProject mvProject = new LogicalProject<>(projectsUnderAggregate, tempRewritedPlan); + // add agg rewrite + Map projectOutPutExprIdMap = mvProject.getOutput().stream() + .distinct() + .collect(Collectors.toMap(NamedExpression::getExprId, slot -> slot)); + // make the expressions to re reference project output + finalGroupExpressions = finalGroupExpressions.stream() + .map(expr -> { + ExprId exprId = ((NamedExpression) expr).getExprId(); + if (projectOutPutExprIdMap.containsKey(exprId)) { + return projectOutPutExprIdMap.get(exprId); + } + return (NamedExpression) expr; + }) + .collect(Collectors.toList()); + finalAggregateExpressions = finalAggregateExpressions.stream() + .map(expr -> { + ExprId exprId = expr.getExprId(); + if (projectOutPutExprIdMap.containsKey(exprId)) { + return projectOutPutExprIdMap.get(exprId); + } + return expr; + }) + .collect(Collectors.toList()); + LogicalAggregate rewrittenAggregate = new LogicalAggregate(finalGroupExpressions, + finalAggregateExpressions, mvProject); + // record the group id in materializationContext, and when rewrite again in + // the same group, bail out quickly. + if (queryStructInfo.getOriginalPlan().getGroupExpression().isPresent()) { + materializationContext.addMatchedGroup( + queryStructInfo.getOriginalPlan().getGroupExpression().get().getOwnerGroup().getGroupId()); + } + return rewrittenAggregate; + } + + // only support sum roll up, support other agg functions later. + private AggregateFunction rollup(AggregateFunction originFunction, + Expression mappedExpression) { + Class rollupAggregateFunction = originFunction.getRollup(); + if (rollupAggregateFunction == null) { + return null; + } + if (Sum.class.isAssignableFrom(rollupAggregateFunction)) { + return new Sum(originFunction.isDistinct(), mappedExpression); + } + // can rollup return null + return null; + } + + private Pair, Set> topPlanSplitToGroupAndFunction( + Pair> topPlanAndAggPair) { + + LogicalAggregate queryAggregate = topPlanAndAggPair.value(); + Set queryAggGroupSet = new HashSet<>(queryAggregate.getGroupByExpressions()); + Set queryAggFunctionSet = queryAggregate.getOutputExpressions().stream() + .filter(expr -> !queryAggGroupSet.contains(expr)) + .collect(Collectors.toSet()); + + Plan queryTopPlan = topPlanAndAggPair.key(); + Set topGroupByExpressions = new HashSet<>(); + Set topFunctionExpressions = new HashSet<>(); + queryTopPlan.getExpressions().forEach( + expression -> { + if (expression.anyMatch(expr -> expr instanceof NamedExpression + && queryAggFunctionSet.contains((NamedExpression) expr))) { + topFunctionExpressions.add(expression); + } else { + topGroupByExpressions.add(expression); + } + }); + // only support to reference the aggregate function directly in top, will support expression later. + if (topFunctionExpressions.stream().anyMatch( + topAggFunc -> !(topAggFunc instanceof NamedExpression) && (!queryAggFunctionSet.contains(topAggFunc) + || !queryAggFunctionSet.contains(topAggFunc.child(0))))) { + return null; + } + return Pair.of(topGroupByExpressions, topFunctionExpressions); + } + + private Pair> splitToTopPlanAndAggregate(StructInfo structInfo) { + Plan topPlan = structInfo.getTopPlan(); + PlanSplitContext splitContext = new PlanSplitContext(Sets.newHashSet(LogicalAggregate.class)); + topPlan.accept(StructInfo.PLAN_SPLITTER, splitContext); + if (!(splitContext.getBottomPlan() instanceof LogicalAggregate)) { + return null; + } else { + return Pair.of(topPlan, (LogicalAggregate) splitContext.getBottomPlan()); + } + } + + // Check Aggregate is simple or not and check join is whether valid or not. + // Support join's input can not contain aggregate Only support project, filter, join, logical relation node and + // join condition should be slot reference equals currently + @Override + protected boolean checkPattern(StructInfo structInfo) { + + Plan topPlan = structInfo.getTopPlan(); + Boolean valid = topPlan.accept(StructInfo.AGGREGATE_PATTERN_CHECKER, null); + if (!valid) { + return false; + } + HyperGraph hyperGraph = structInfo.getHyperGraph(); + for (AbstractNode node : hyperGraph.getNodes()) { + StructInfoNode structInfoNode = (StructInfoNode) node; + if (!structInfoNode.getPlan().accept(StructInfo.JOIN_PATTERN_CHECKER, + SUPPORTED_JOIN_TYPE_SET)) { + return false; + } + for (JoinEdge edge : hyperGraph.getJoinEdges()) { + if (!edge.getJoin().accept(StructInfo.JOIN_PATTERN_CHECKER, SUPPORTED_JOIN_TYPE_SET)) { + return false; + } + } + } + return true; + } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java index c1873d09e481ee..aad1ffa529c5a1 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java @@ -24,14 +24,9 @@ import org.apache.doris.nereids.rules.exploration.mv.mapping.SlotMapping; import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.expressions.NamedExpression; -import org.apache.doris.nereids.trees.plans.JoinType; import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.trees.plans.logical.LogicalProject; -import org.apache.doris.nereids.util.ExpressionUtils; -import com.google.common.collect.Sets; - -import java.util.HashSet; import java.util.List; import java.util.stream.Collectors; @@ -40,28 +35,23 @@ * This is responsible for common join rewriting */ public abstract class AbstractMaterializedViewJoinRule extends AbstractMaterializedViewRule { - private static final HashSet SUPPORTED_JOIN_TYPE_SET = - Sets.newHashSet(JoinType.INNER_JOIN, JoinType.LEFT_OUTER_JOIN); - @Override protected Plan rewriteQueryByView(MatchMode matchMode, StructInfo queryStructInfo, StructInfo viewStructInfo, - SlotMapping queryToViewSlotMappings, + SlotMapping queryToViewSlotMapping, Plan tempRewritedPlan, MaterializationContext materializationContext) { - - List queryShuttleExpression = ExpressionUtils.shuttleExpressionWithLineage( - queryStructInfo.getExpressions(), - queryStructInfo.getOriginalPlan()); // Rewrite top projects, represent the query projects by view List expressionsRewritten = rewriteExpression( - queryShuttleExpression, - materializationContext.getViewExpressionIndexMapping(), - queryToViewSlotMappings + queryStructInfo.getExpressions(), + queryStructInfo.getOriginalPlan(), + materializationContext.getMvExprToMvScanExprMapping(), + queryToViewSlotMapping, + true ); // Can not rewrite, bail out - if (expressionsRewritten == null + if (expressionsRewritten.isEmpty() || expressionsRewritten.stream().anyMatch(expr -> !(expr instanceof NamedExpression))) { return null; } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java index c265b1de3e44e3..3139d98f90651c 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java @@ -24,6 +24,7 @@ import org.apache.doris.nereids.rules.exploration.mv.mapping.ExpressionMapping; import org.apache.doris.nereids.rules.exploration.mv.mapping.RelationMapping; import org.apache.doris.nereids.rules.exploration.mv.mapping.SlotMapping; +import org.apache.doris.nereids.trees.expressions.Alias; import org.apache.doris.nereids.trees.expressions.EqualTo; import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.expressions.NamedExpression; @@ -31,11 +32,13 @@ import org.apache.doris.nereids.trees.expressions.SlotReference; import org.apache.doris.nereids.trees.expressions.literal.BooleanLiteral; import org.apache.doris.nereids.trees.expressions.literal.Literal; +import org.apache.doris.nereids.trees.plans.JoinType; import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.trees.plans.algebra.CatalogRelation; import org.apache.doris.nereids.trees.plans.logical.LogicalFilter; import org.apache.doris.nereids.util.ExpressionUtils; +import com.google.common.collect.ImmutableList; import com.google.common.collect.Sets; import java.util.ArrayList; @@ -51,6 +54,9 @@ */ public abstract class AbstractMaterializedViewRule { + public static final HashSet SUPPORTED_JOIN_TYPE_SET = + Sets.newHashSet(JoinType.INNER_JOIN, JoinType.LEFT_OUTER_JOIN); + /** * The abstract template method for query rewrite, it contains the main logic and different query * pattern should override the sub logic. @@ -105,9 +111,15 @@ protected List rewrite(Plan queryPlan, CascadesContext cascadesContext) { LogicalCompatibilityContext.from(queryToViewTableMapping, queryToViewSlotMapping, queryStructInfo, viewStructInfo); // todo outer join compatibility check - if (StructInfo.isGraphLogicalEquals(queryStructInfo, viewStructInfo, compatibilityContext) == null) { + List pulledUpExpressions = StructInfo.isGraphLogicalEquals(queryStructInfo, viewStructInfo, + compatibilityContext); + if (pulledUpExpressions == null) { continue; } + // set pulled up expression to queryStructInfo predicates and update related predicates + if (!pulledUpExpressions.isEmpty()) { + queryStructInfo.addPredicates(pulledUpExpressions); + } SplitPredicate compensatePredicates = predicatesCompensate(queryStructInfo, viewStructInfo, queryToViewSlotMapping); // Can not compensate, bail out @@ -122,8 +134,10 @@ protected List rewrite(Plan queryPlan, CascadesContext cascadesContext) { // Try to rewrite compensate predicates by using mv scan List rewriteCompensatePredicates = rewriteExpression( compensatePredicates.toList(), - materializationContext.getViewExpressionIndexMapping(), - queryToViewSlotMapping); + queryPlan, + materializationContext.getMvExprToMvScanExprMapping(), + queryToViewSlotMapping, + true); if (rewriteCompensatePredicates.isEmpty()) { continue; } @@ -151,22 +165,30 @@ protected List rewrite(Plan queryPlan, CascadesContext cascadesContext) { protected Plan rewriteQueryByView(MatchMode matchMode, StructInfo queryStructInfo, StructInfo viewStructInfo, - SlotMapping queryToViewSlotMappings, + SlotMapping queryToViewSlotMapping, Plan tempRewritedPlan, MaterializationContext materializationContext) { return tempRewritedPlan; } /** - * Use target output expression to represent the source expression + * Use target expression to represent the source expression. Visit the source expression, + * try to replace the source expression with target expression in targetExpressionMapping, if found then + * replace the source expression by target expression mapping value. + * Note: make the target expression map key to source based according to targetExpressionNeedSourceBased, + * if targetExpressionNeedSourceBased is true, we should make it source based. + * the key expression in targetExpressionMapping should be shuttled. with the method + * ExpressionUtils.shuttleExpressionWithLineage. */ protected List rewriteExpression( List sourceExpressionsToWrite, - ExpressionMapping mvExprToMvScanExprMapping, - SlotMapping sourceToTargetMapping) { - // Firstly, rewrite the target plan output expression using query with inverse mapping - // then try to use the mv expression to represent the query. if any of source expressions - // can not be represented by mv, return null + Plan sourcePlan, + ExpressionMapping targetExpressionMapping, + SlotMapping sourceToTargetMapping, + boolean targetExpressionNeedSourceBased) { + // Firstly, rewrite the target expression using source with inverse mapping + // then try to use the target expression to represent the query. if any of source expressions + // can not be represented by target expressions, return null. // // example as following: // source target @@ -176,36 +198,58 @@ protected List rewriteExpression( // transform source to: // project(slot 2, 1) // target - // generate mvSql to mvScan mvExprToMvScanExprMapping, and change mv sql expression to query based - ExpressionMapping mvExprToMvScanExprMappingKeySourceBased = - mvExprToMvScanExprMapping.keyPermute(sourceToTargetMapping.inverse()); - List> flattenExpressionMapping = - mvExprToMvScanExprMappingKeySourceBased.flattenMap(); - // view to view scan expression is 1:1 so get first element - Map mvSqlToMvScanMappingQueryBased = - flattenExpressionMapping.get(0); + // generate target to target replacement expression mapping, and change target expression to source based + List sourceShuttledExpressions = + ExpressionUtils.shuttleExpressionWithLineage(sourceExpressionsToWrite, sourcePlan); + ExpressionMapping expressionMappingKeySourceBased = targetExpressionNeedSourceBased + ? targetExpressionMapping.keyPermute(sourceToTargetMapping.inverse()) : targetExpressionMapping; + // target to target replacement expression mapping, because mv is 1:1 so get first element + List> flattenExpressionMap = + expressionMappingKeySourceBased.flattenMap(); + Map targetToTargetReplacementMapping = flattenExpressionMap.get(0); List rewrittenExpressions = new ArrayList<>(); - for (Expression expressionToRewrite : sourceExpressionsToWrite) { + for (int index = 0; index < sourceShuttledExpressions.size(); index++) { + Expression expressionToRewrite = sourceShuttledExpressions.get(index); if (expressionToRewrite instanceof Literal) { rewrittenExpressions.add(expressionToRewrite); continue; } final Set slotsToRewrite = expressionToRewrite.collectToSet(expression -> expression instanceof Slot); - boolean wiAlias = expressionToRewrite instanceof NamedExpression; Expression replacedExpression = ExpressionUtils.replace(expressionToRewrite, - mvSqlToMvScanMappingQueryBased, - wiAlias); + targetToTargetReplacementMapping); if (replacedExpression.anyMatch(slotsToRewrite::contains)) { // if contains any slot to rewrite, which means can not be rewritten by target, bail out - return null; + return ImmutableList.of(); + } + Expression sourceExpression = sourceExpressionsToWrite.get(index); + if (sourceExpression instanceof NamedExpression) { + NamedExpression sourceNamedExpression = (NamedExpression) sourceExpression; + replacedExpression = new Alias(sourceNamedExpression.getExprId(), replacedExpression, + sourceNamedExpression.getName()); } rewrittenExpressions.add(replacedExpression); } return rewrittenExpressions; } + protected Expression rewriteExpression( + Expression sourceExpressionsToWrite, + Plan sourcePlan, + ExpressionMapping targetExpressionMapping, + SlotMapping sourceToTargetMapping, + boolean targetExpressionNeedSourceBased) { + List expressionToRewrite = new ArrayList<>(); + expressionToRewrite.add(sourceExpressionsToWrite); + List rewrittenExpressions = rewriteExpression(expressionToRewrite, sourcePlan, + targetExpressionMapping, sourceToTargetMapping, targetExpressionNeedSourceBased); + if (rewrittenExpressions.isEmpty()) { + return null; + } + return rewrittenExpressions.get(0); + } + /** * Compensate mv predicates by query predicates, compensate predicate result is query based. * Such as a > 5 in mv, and a > 10 in query, the compensatory predicate is a > 10. diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/InitMaterializationContextHook.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/InitMaterializationContextHook.java index 24dad225e999f1..85a10be2ae23f9 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/InitMaterializationContextHook.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/InitMaterializationContextHook.java @@ -30,6 +30,7 @@ import org.apache.doris.nereids.PlannerHook; import org.apache.doris.nereids.trees.expressions.NamedExpression; import org.apache.doris.nereids.trees.plans.Plan; +import org.apache.doris.nereids.trees.plans.PreAggStatus; import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan; import org.apache.doris.nereids.trees.plans.logical.LogicalProject; import org.apache.doris.nereids.trees.plans.visitor.TableCollector; @@ -60,6 +61,9 @@ public void afterAnalyze(NereidsPlanner planner) { private void initMaterializationContext(CascadesContext cascadesContext) { + if (!cascadesContext.getConnectContext().getSessionVariable().isEnableMaterializedViewRewrite()) { + return; + } Plan rewritePlan = cascadesContext.getRewritePlan(); TableCollectorContext collectorContext = new TableCollectorContext(Sets.newHashSet()); rewritePlan.accept(TableCollector.INSTANCE, collectorContext); @@ -90,19 +94,22 @@ private void initMaterializationContext(CascadesContext cascadesContext) { .getDbOrMetaException(mvBaseTableInfo.getDbId()) .getTableOrMetaException(mvBaseTableInfo.getTableId(), TableType.MATERIALIZED_VIEW); - String qualifiedName = materializedView.getQualifiedName(); // generate outside, maybe add partition filter in the future - Plan mvScan = new LogicalOlapScan(cascadesContext.getStatementContext().getNextRelationId(), + LogicalOlapScan mvScan = new LogicalOlapScan( + cascadesContext.getStatementContext().getNextRelationId(), (OlapTable) materializedView, - ImmutableList.of(qualifiedName), - Lists.newArrayList(materializedView.getId()), + ImmutableList.of(materializedView.getQualifiedDbName()), + // this must be empty, or it will be used to sample + Lists.newArrayList(), Lists.newArrayList(), Optional.empty()); + mvScan = mvScan.withMaterializedIndexSelected(PreAggStatus.on(), materializedView.getBaseIndexId()); List mvProjects = mvScan.getOutput().stream().map(NamedExpression.class::cast) .collect(Collectors.toList()); - mvScan = new LogicalProject(mvProjects, mvScan); + // todo should force keep consistency to mv sql plan output + Plan projectScan = new LogicalProject(mvProjects, mvScan); cascadesContext.addMaterializationContext( - MaterializationContext.fromMaterializedView(materializedView, mvScan, cascadesContext)); + MaterializationContext.fromMaterializedView(materializedView, projectScan, cascadesContext)); } catch (MetaNotFoundException metaNotFoundException) { LOG.error(mvBaseTableInfo.toString() + " can not find corresponding materialized view."); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java index 2632b7e4c5d15a..947f117acff907 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java @@ -21,8 +21,11 @@ import org.apache.doris.nereids.rules.exploration.mv.mapping.Mapping.MappedRelation; import org.apache.doris.nereids.rules.exploration.mv.mapping.RelationMapping; import org.apache.doris.nereids.rules.exploration.mv.mapping.SlotMapping; +import org.apache.doris.nereids.trees.expressions.EqualTo; import org.apache.doris.nereids.trees.expressions.Expression; +import org.apache.doris.nereids.trees.expressions.NamedExpression; import org.apache.doris.nereids.trees.expressions.SlotReference; +import org.apache.doris.nereids.trees.expressions.visitor.DefaultExpressionRewriter; import org.apache.doris.nereids.trees.plans.RelationId; import org.apache.doris.nereids.util.ExpressionUtils; @@ -92,16 +95,39 @@ public static LogicalCompatibilityContext from(RelationMapping relationMapping, final Map viewEdgeToConjunctsMapQueryBased = new HashMap<>(); viewShuttledExprToExprMap.forEach((shuttledExpr, expr) -> { viewEdgeToConjunctsMapQueryBased.put( - ExpressionUtils.replace(shuttledExpr, viewToQuerySlotMapping), + orderSlotAsc(ExpressionUtils.replace(shuttledExpr, viewToQuerySlotMapping)), expr); }); BiMap queryToViewEdgeMapping = HashBiMap.create(); queryShuttledExprToExprMap.forEach((exprSet, edge) -> { - Expression viewExpr = viewEdgeToConjunctsMapQueryBased.get(exprSet); + Expression viewExpr = viewEdgeToConjunctsMapQueryBased.get(orderSlotAsc(exprSet)); if (viewExpr != null) { queryToViewEdgeMapping.put(edge, viewExpr); } }); return new LogicalCompatibilityContext(queryToViewNodeMapping, queryToViewEdgeMapping); } + + private static Expression orderSlotAsc(Expression expression) { + return expression.accept(ExpressionSlotOrder.INSTANCE, null); + } + + private static final class ExpressionSlotOrder extends DefaultExpressionRewriter { + public static final ExpressionSlotOrder INSTANCE = new ExpressionSlotOrder(); + + @Override + public Expression visitEqualTo(EqualTo equalTo, Void context) { + if (!(equalTo.getArgument(0) instanceof NamedExpression) + || !(equalTo.getArgument(1) instanceof NamedExpression)) { + return equalTo; + } + NamedExpression left = (NamedExpression) equalTo.getArgument(0); + NamedExpression right = (NamedExpression) equalTo.getArgument(1); + if (right.getExprId().asInt() < left.getExprId().asInt()) { + return new EqualTo(right, left); + } else { + return equalTo; + } + } + } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java index 336c627da66574..3e1fe99c9c84e0 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java @@ -23,7 +23,6 @@ import org.apache.doris.nereids.CascadesContext; import org.apache.doris.nereids.memo.GroupId; import org.apache.doris.nereids.rules.exploration.mv.mapping.ExpressionMapping; -import org.apache.doris.nereids.trees.expressions.NamedExpression; import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.util.ExpressionUtils; @@ -32,7 +31,6 @@ import java.util.HashSet; import java.util.List; import java.util.Set; -import java.util.stream.Collectors; /** * Maintain the context for query rewrite by materialized view @@ -47,7 +45,7 @@ public class MaterializationContext { // Group ids that are rewritten by this mv to reduce rewrite times private final Set matchedGroups = new HashSet<>(); // generate form mv scan plan - private ExpressionMapping viewExpressionMapping; + private ExpressionMapping mvExprToMvScanExprMapping; /** * MaterializationContext, this contains necessary info for query rewriting by mv @@ -67,14 +65,11 @@ public MaterializationContext(MTMV mtmv, mvCache = MVCache.from(mtmv, cascadesContext.getConnectContext()); mtmv.setMvCache(mvCache); } - List mvOutputExpressions = mvCache.getMvOutputExpressions(); // mv output expression shuttle, this will be used to expression rewrite - mvOutputExpressions = - ExpressionUtils.shuttleExpressionWithLineage(mvOutputExpressions, mvCache.getLogicalPlan()).stream() - .map(NamedExpression.class::cast) - .collect(Collectors.toList()); - this.viewExpressionMapping = ExpressionMapping.generate( - mvOutputExpressions, + this.mvExprToMvScanExprMapping = ExpressionMapping.generate( + ExpressionUtils.shuttleExpressionWithLineage( + mvCache.getMvOutputExpressions(), + mvCache.getLogicalPlan()), mvScanPlan.getExpressions()); } @@ -106,8 +101,8 @@ public List getBaseViews() { return baseViews; } - public ExpressionMapping getViewExpressionIndexMapping() { - return viewExpressionMapping; + public ExpressionMapping getMvExprToMvScanExprMapping() { + return mvExprToMvScanExprMapping; } /** diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewAggregateRule.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewAggregateRule.java index ce9c208e5f5d16..3bd0b1080cc9fe 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewAggregateRule.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewAggregateRule.java @@ -18,7 +18,13 @@ package org.apache.doris.nereids.rules.exploration.mv; import org.apache.doris.nereids.rules.Rule; +import org.apache.doris.nereids.rules.RulePromise; +import org.apache.doris.nereids.rules.RuleType; import org.apache.doris.nereids.rules.rewrite.RewriteRuleFactory; +import org.apache.doris.nereids.trees.plans.Plan; +import org.apache.doris.nereids.trees.plans.logical.LogicalAggregate; + +import com.google.common.collect.ImmutableList; import java.util.List; @@ -26,8 +32,15 @@ * This is responsible for aggregate rewriting according to different pattern * */ public class MaterializedViewAggregateRule extends AbstractMaterializedViewAggregateRule implements RewriteRuleFactory { + + public static final MaterializedViewAggregateRule INSTANCE = new MaterializedViewAggregateRule(); + @Override public List buildRules() { - return null; + return ImmutableList.of( + logicalAggregate(any()).thenApplyMulti(ctx -> { + LogicalAggregate root = ctx.root; + return rewrite(root, ctx.cascadesContext); + }).toRule(RuleType.MATERIALIZED_VIEW_ONLY_AGGREGATE, RulePromise.EXPLORE)); } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewProjectAggregateRule.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewProjectAggregateRule.java new file mode 100644 index 00000000000000..e9a31f45535c84 --- /dev/null +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewProjectAggregateRule.java @@ -0,0 +1,46 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you 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 org.apache.doris.nereids.rules.exploration.mv; + +import org.apache.doris.nereids.rules.Rule; +import org.apache.doris.nereids.rules.RulePromise; +import org.apache.doris.nereids.rules.RuleType; +import org.apache.doris.nereids.rules.rewrite.RewriteRuleFactory; +import org.apache.doris.nereids.trees.plans.Plan; +import org.apache.doris.nereids.trees.plans.logical.LogicalAggregate; +import org.apache.doris.nereids.trees.plans.logical.LogicalProject; + +import com.google.common.collect.ImmutableList; + +import java.util.List; + +/**MaterializedViewProjectAggregateRule*/ +public class MaterializedViewProjectAggregateRule extends AbstractMaterializedViewAggregateRule implements + RewriteRuleFactory { + + public static final MaterializedViewProjectAggregateRule INSTANCE = new MaterializedViewProjectAggregateRule(); + + @Override + public List buildRules() { + return ImmutableList.of( + logicalProject(logicalAggregate(any())).thenApplyMulti(ctx -> { + LogicalProject> root = ctx.root; + return rewrite(root, ctx.cascadesContext); + }).toRule(RuleType.MATERIALIZED_VIEW_PROJECT_AGGREGATE, RulePromise.EXPLORE)); + } +} diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewProjectJoinRule.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewProjectJoinRule.java index 92f102dc1dece3..456d1ce24a0519 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewProjectJoinRule.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewProjectJoinRule.java @@ -42,6 +42,6 @@ public List buildRules() { logicalProject(logicalJoin(any(), any())).thenApplyMulti(ctx -> { LogicalProject> root = ctx.root; return rewrite(root, ctx.cascadesContext); - }).toRule(RuleType.MATERIALIZED_VIEW_ONLY_JOIN, RulePromise.EXPLORE)); + }).toRule(RuleType.MATERIALIZED_VIEW_PROJECT_JOIN, RulePromise.EXPLORE)); } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/StructInfo.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/StructInfo.java index f889058fa21d5b..ba6c0f64bb3233 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/StructInfo.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/StructInfo.java @@ -24,7 +24,7 @@ import org.apache.doris.nereids.trees.expressions.EqualTo; import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.expressions.SlotReference; -import org.apache.doris.nereids.trees.expressions.literal.BooleanLiteral; +import org.apache.doris.nereids.trees.expressions.literal.Literal; import org.apache.doris.nereids.trees.plans.JoinType; import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.trees.plans.RelationId; @@ -32,13 +32,14 @@ import org.apache.doris.nereids.trees.plans.algebra.Filter; import org.apache.doris.nereids.trees.plans.algebra.Join; import org.apache.doris.nereids.trees.plans.algebra.Project; +import org.apache.doris.nereids.trees.plans.logical.LogicalAggregate; import org.apache.doris.nereids.trees.plans.logical.LogicalFilter; import org.apache.doris.nereids.trees.plans.logical.LogicalJoin; import org.apache.doris.nereids.trees.plans.logical.LogicalProject; +import org.apache.doris.nereids.trees.plans.logical.LogicalRepeat; import org.apache.doris.nereids.trees.plans.visitor.DefaultPlanVisitor; import org.apache.doris.nereids.util.ExpressionUtils; -import com.google.common.collect.ImmutableList; import com.google.common.collect.Lists; import com.google.common.collect.Sets; @@ -47,15 +48,17 @@ import java.util.HashSet; import java.util.List; import java.util.Map; +import java.util.Optional; import java.util.Set; import java.util.stream.Collectors; import javax.annotation.Nullable; /** - * StructInfo + * StructInfo for plan, this contains necessary info for query rewrite by materialized view */ public class StructInfo { public static final JoinPatternChecker JOIN_PATTERN_CHECKER = new JoinPatternChecker(); + public static final AggregatePatternChecker AGGREGATE_PATTERN_CHECKER = new AggregatePatternChecker(); // struct info splitter public static final PlanSplitter PLAN_SPLITTER = new PlanSplitter(); private static final RelationCollector RELATION_COLLECTOR = new RelationCollector(); @@ -72,7 +75,9 @@ public class StructInfo { private final List relations = new ArrayList<>(); // this is for LogicalCompatibilityContext later private final Map relationIdStructInfoNodeMap = new HashMap<>(); + // this recorde the predicates which can pull up, not shuttled private Predicates predicates; + // split predicates is shuttled private SplitPredicate splitPredicate; private EquivalenceClass equivalenceClass; // this is for LogicalCompatibilityContext later @@ -87,20 +92,28 @@ private StructInfo(Plan originalPlan, @Nullable Plan topPlan, @Nullable Plan bot } private void init() { - + // split the top plan to two parts by join node if (topPlan == null || bottomPlan == null) { PlanSplitContext planSplitContext = new PlanSplitContext(Sets.newHashSet(LogicalJoin.class)); originalPlan.accept(PLAN_SPLITTER, planSplitContext); this.bottomPlan = planSplitContext.getBottomPlan(); this.topPlan = planSplitContext.getTopPlan(); } + collectStructInfoFromGraph(); + initPredicates(); + predicatesDerive(); + } - this.predicates = Predicates.of(); - // Collect predicate from join condition in hyper graph + public void addPredicates(List canPulledUpExpressions) { + canPulledUpExpressions.forEach(this.predicates::addPredicate); + predicatesDerive(); + } + + private void collectStructInfoFromGraph() { + // Collect expression from join condition in hyper graph this.hyperGraph.getJoinEdges().forEach(edge -> { List hashJoinConjuncts = edge.getHashJoinConjuncts(); hashJoinConjuncts.forEach(conjunctExpr -> { - predicates.addPredicate(conjunctExpr); // shuttle expression in edge for LogicalCompatibilityContext later shuttledHashConjunctsToConjunctsMap.put( ExpressionUtils.shuttleExpressionWithLineage( @@ -115,8 +128,7 @@ private void init() { if (!this.isValid()) { return; } - - // Collect predicate from filter node in hyper graph + // Collect relations from hyper graph which in the bottom plan this.hyperGraph.getNodes().forEach(node -> { // plan relation collector and set to map Plan nodePlan = node.getPlan(); @@ -125,28 +137,40 @@ private void init() { this.relations.addAll(nodeRelations); // every node should only have one relation, this is for LogicalCompatibilityContext relationIdStructInfoNodeMap.put(nodeRelations.get(0).getRelationId(), (StructInfoNode) node); - - // if inner join add where condition - Set predicates = new HashSet<>(); - nodePlan.accept(PREDICATE_COLLECTOR, predicates); - predicates.forEach(this.predicates::addPredicate); }); + // Collect expression from where in hyper graph + this.hyperGraph.getFilterEdges().forEach(filterEdge -> { + List filterExpressions = filterEdge.getExpressions(); + filterExpressions.forEach(predicate -> { + // this is used for LogicalCompatibilityContext + ExpressionUtils.extractConjunction(predicate).forEach(expr -> + shuttledHashConjunctsToConjunctsMap.put( + ExpressionUtils.shuttleExpressionWithLineage(predicate, topPlan), predicate)); + }); + }); + } - // TODO Collect predicate from top plan not in hyper graph, should optimize, twice now + private void initPredicates() { + // Collect predicate from top plan which not in hyper graph + this.predicates = Predicates.of(); Set topPlanPredicates = new HashSet<>(); topPlan.accept(PREDICATE_COLLECTOR, topPlanPredicates); topPlanPredicates.forEach(this.predicates::addPredicate); + } + // derive some useful predicate by predicates + private void predicatesDerive() { // construct equivalenceClass according to equals predicates - this.equivalenceClass = new EquivalenceClass(); List shuttledExpression = ExpressionUtils.shuttleExpressionWithLineage( this.predicates.getPulledUpPredicates(), originalPlan).stream() .map(Expression.class::cast) .collect(Collectors.toList()); SplitPredicate splitPredicate = Predicates.splitPredicates(ExpressionUtils.and(shuttledExpression)); this.splitPredicate = splitPredicate; + + this.equivalenceClass = new EquivalenceClass(); for (Expression expression : ExpressionUtils.extractConjunction(splitPredicate.getEqualPredicate())) { - if (expression instanceof BooleanLiteral && ((BooleanLiteral) expression).getValue()) { + if (expression instanceof Literal) { continue; } if (expression instanceof EqualTo) { @@ -166,6 +190,7 @@ public static List of(Plan originalPlan) { // TODO only consider the inner join currently, Should support outer join // Split plan by the boundary which contains multi child PlanSplitContext planSplitContext = new PlanSplitContext(Sets.newHashSet(LogicalJoin.class)); + // if single table without join, the bottom is originalPlan.accept(PLAN_SPLITTER, planSplitContext); List structInfos = HyperGraph.toStructInfo(planSplitContext.getBottomPlan()); @@ -240,9 +265,7 @@ public List getExpressions() { */ public static @Nullable List isGraphLogicalEquals(StructInfo queryStructInfo, StructInfo viewStructInfo, LogicalCompatibilityContext compatibilityContext) { - // TODO: open it after supporting filter - // return queryStructInfo.hyperGraph.isLogicCompatible(viewStructInfo.hyperGraph, compatibilityContext); - return ImmutableList.of(); + return queryStructInfo.hyperGraph.isLogicCompatible(viewStructInfo.hyperGraph, compatibilityContext); } private static class RelationCollector extends DefaultPlanVisitor> { @@ -258,8 +281,14 @@ public Void visit(Plan plan, List collectedRelations) { private static class PredicateCollector extends DefaultPlanVisitor> { @Override public Void visit(Plan plan, Set predicates) { + // Just collect the filter in top plan, if meet other node except project and filter, return + if (!(plan instanceof LogicalProject) + && !(plan instanceof LogicalFilter) + && !(plan instanceof LogicalAggregate)) { + return null; + } if (plan instanceof LogicalFilter) { - predicates.add(((LogicalFilter) plan).getPredicate()); + predicates.addAll(ExpressionUtils.extractConjunction(((LogicalFilter) plan).getPredicate())); } return super.visit(plan, predicates); } @@ -267,7 +296,7 @@ public Void visit(Plan plan, Set predicates) { /** * Split the plan into bottom and up, the boundary is given by context, - * the bottom contains the boundary. + * the bottom contains the boundary, and top plan doesn't contain the boundary. */ public static class PlanSplitter extends DefaultPlanVisitor { @Override @@ -275,6 +304,10 @@ public Void visit(Plan plan, PlanSplitContext context) { if (context.getTopPlan() == null) { context.setTopPlan(plan); } + if (plan.children().isEmpty() && context.getBottomPlan() == null) { + context.setBottomPlan(plan); + return null; + } if (context.isBoundary(plan)) { context.setBottomPlan(plan); return null; @@ -349,4 +382,28 @@ public Boolean visit(Plan plan, Set requiredJoinType) { return true; } } + + /** + * AggregatePatternChecker + */ + public static class AggregatePatternChecker extends DefaultPlanVisitor { + @Override + public Boolean visit(Plan plan, Void context) { + if (plan instanceof LogicalAggregate) { + LogicalAggregate aggregate = (LogicalAggregate) plan; + Optional> sourceRepeat = aggregate.getSourceRepeat(); + if (sourceRepeat.isPresent()) { + return false; + } + super.visit(aggregate, context); + return true; + } + if (plan instanceof LogicalProject) { + super.visit(plan, context); + return true; + } + super.visit(plan, context); + return false; + } + } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/ExpressionIndexMapping.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/ExpressionIndexMapping.java deleted file mode 100644 index f63017633a82aa..00000000000000 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/ExpressionIndexMapping.java +++ /dev/null @@ -1,48 +0,0 @@ -// Licensed to the Apache Software Foundation (ASF) under one -// or more contributor license agreements. See the NOTICE file -// distributed with this work for additional information -// regarding copyright ownership. The ASF licenses this file -// to you 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 org.apache.doris.nereids.rules.exploration.mv.mapping; - -import org.apache.doris.nereids.trees.expressions.Expression; - -import com.google.common.collect.ArrayListMultimap; -import com.google.common.collect.Multimap; - -import java.util.List; - -/** - * Expression and it's index mapping - */ -public class ExpressionIndexMapping extends Mapping { - private final Multimap expressionIndexMapping; - - public ExpressionIndexMapping(Multimap expressionIndexMapping) { - this.expressionIndexMapping = expressionIndexMapping; - } - - public Multimap getExpressionIndexMapping() { - return expressionIndexMapping; - } - - public static ExpressionIndexMapping generate(List expressions) { - Multimap expressionIndexMapping = ArrayListMultimap.create(); - for (int i = 0; i < expressions.size(); i++) { - expressionIndexMapping.put(expressions.get(i), i); - } - return new ExpressionIndexMapping(expressionIndexMapping); - } -} diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/ExpressionMapping.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/ExpressionMapping.java index 7c1f06746cdb3c..2f1ed230145f35 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/ExpressionMapping.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/ExpressionMapping.java @@ -22,6 +22,7 @@ import org.apache.doris.nereids.util.ExpressionUtils; import com.google.common.collect.ArrayListMultimap; +import com.google.common.collect.ImmutableMultimap; import com.google.common.collect.Lists; import com.google.common.collect.Multimap; @@ -30,25 +31,26 @@ import java.util.HashMap; import java.util.List; import java.util.Map; +import java.util.Map.Entry; /** * Expression mapping, maybe one expression map to multi expression */ public class ExpressionMapping extends Mapping { - private final Multimap expressionMapping; + private final Multimap expressionMapping; - public ExpressionMapping(Multimap expressionMapping) { + public ExpressionMapping(Multimap expressionMapping) { this.expressionMapping = expressionMapping; } - public Multimap getExpressionMapping() { + public Multimap getExpressionMapping() { return expressionMapping; } /** * ExpressionMapping flatten */ - public List> flattenMap() { + public List> flattenMap() { List>> tmpExpressionPairs = new ArrayList<>(this.expressionMapping.size()); Map> expressionMappingMap = expressionMapping.asMap(); @@ -62,7 +64,7 @@ public ExpressionMapping(Multimap ex } List>> cartesianExpressionMap = Lists.cartesianProduct(tmpExpressionPairs); - final List> flattenedMap = new ArrayList<>(); + final List> flattenedMap = new ArrayList<>(); for (List> listPair : cartesianExpressionMap) { final Map expressionMap = new HashMap<>(); listPair.forEach(pair -> expressionMap.put(pair.key(), pair.value())); @@ -71,7 +73,8 @@ public ExpressionMapping(Multimap ex return flattenedMap; } - /**Permute the key of expression mapping. this is useful for expression rewrite, if permute key to query based + /** + * Permute the key of expression mapping. this is useful for expression rewrite, if permute key to query based * then when expression rewrite success, we can get the mv scan expression directly. */ public ExpressionMapping keyPermute(SlotMapping slotMapping) { @@ -86,7 +89,9 @@ public ExpressionMapping keyPermute(SlotMapping slotMapping) { return new ExpressionMapping(permutedExpressionMapping); } - /**ExpressionMapping generate*/ + /** + * ExpressionMapping generate + */ public static ExpressionMapping generate( List sourceExpressions, List targetExpressions) { @@ -97,4 +102,25 @@ public static ExpressionMapping generate( } return new ExpressionMapping(expressionMultiMap); } + + @Override + public Mapping chainedFold(Mapping target) { + + ImmutableMultimap.Builder foldedMappingBuilder = + ImmutableMultimap.builder(); + + Multimap targetMapping + = ((ExpressionMapping) target).getExpressionMapping(); + for (Entry> exprMapping : + this.getExpressionMapping().asMap().entrySet()) { + Collection valueExpressions = exprMapping.getValue(); + valueExpressions.forEach(valueExpr -> { + if (targetMapping.containsKey(valueExpr)) { + targetMapping.get(valueExpr).forEach( + targetValue -> foldedMappingBuilder.put(exprMapping.getKey(), targetValue)); + } + }); + } + return new ExpressionMapping(foldedMappingBuilder.build()); + } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/Mapping.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/Mapping.java index 17a412dab10d39..3d9f95a5049dc0 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/Mapping.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/mapping/Mapping.java @@ -136,4 +136,12 @@ public int hashCode() { return Objects.hash(exprId); } } + + /** Chain fold tow mapping, such as this mapping is {[a -> b]}, the target mapping is + * {[b -> c]} after chain fold, this result will be {[a -> c]}, if the value side in this mapping + * can get the key in the target mapping, will lose the mapping + */ + protected Mapping chainedFold(Mapping target) { + return null; + } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/TreeNode.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/TreeNode.java index 694f0611567ec8..557ff43b51d994 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/TreeNode.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/TreeNode.java @@ -185,6 +185,23 @@ default boolean anyMatch(Predicate> predicate) { return false; } + /** + * iterate top down and test predicate if any matched. Top-down traverse implicitly. + * @param predicate predicate + * @return the first node which match the predicate + */ + default TreeNode firstMatch(Predicate> predicate) { + if (predicate.test(this)) { + return this; + } + for (NODE_TYPE child : children()) { + if (child.anyMatch(predicate)) { + return child; + } + } + return this; + } + /** * Collect the nodes that satisfied the predicate. */ diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/agg/AggregateFunction.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/agg/AggregateFunction.java index a7e523dfdb549e..61a589daba2781 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/agg/AggregateFunction.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/agg/AggregateFunction.java @@ -77,6 +77,10 @@ public boolean isDistinct() { return distinct; } + public Class getRollup() { + return null; + } + @Override public boolean equals(Object o) { if (this == o) { diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/agg/Sum.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/agg/Sum.java index b8ee59c81f059a..b99f836e09e8b2 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/agg/Sum.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/agg/Sum.java @@ -109,4 +109,9 @@ public R accept(ExpressionVisitor visitor, C context) { public List getSignatures() { return SIGNATURES; } + + @Override + public Class getRollup() { + return Sum.class; + } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java index 03c158f404fad7..4ea2a3b6982243 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java @@ -209,7 +209,7 @@ private void getColumns(Plan plan) { colNames.add(colName); } columns.add(new ColumnDefinition( - colName, slots.get(i).getDataType(), true, + colName, slots.get(i).getDataType(), slots.get(i).nullable(), CollectionUtils.isEmpty(simpleColumnDefinitions) ? null : simpleColumnDefinitions.get(i).getComment())); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalAggregate.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalAggregate.java index 53877ff091e43b..edf962838d3cfa 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalAggregate.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalAggregate.java @@ -210,7 +210,6 @@ public R accept(PlanVisitor visitor, C context) { @Override public List getExpressions() { return new ImmutableList.Builder() - .addAll(groupByExpressions) .addAll(outputExpressions) .build(); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/ExpressionLineageReplacer.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/ExpressionLineageReplacer.java index ca490b9ef1f27b..2a4b81a58285e5 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/ExpressionLineageReplacer.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/ExpressionLineageReplacer.java @@ -19,16 +19,16 @@ import org.apache.doris.catalog.TableIf.TableType; import org.apache.doris.nereids.trees.expressions.Alias; -import org.apache.doris.nereids.trees.expressions.ArrayItemReference.ArrayItemSlot; import org.apache.doris.nereids.trees.expressions.ExprId; import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.expressions.NamedExpression; -import org.apache.doris.nereids.trees.expressions.SlotReference; +import org.apache.doris.nereids.trees.expressions.Slot; import org.apache.doris.nereids.trees.expressions.visitor.DefaultExpressionRewriter; import org.apache.doris.nereids.trees.expressions.visitor.DefaultExpressionVisitor; import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.trees.plans.visitor.ExpressionLineageReplacer.ExpressionReplaceContext; +import java.util.ArrayList; import java.util.Collection; import java.util.List; import java.util.Map; @@ -67,14 +67,34 @@ public static class ExpressionReplacer extends DefaultExpressionRewriter exprIdExpressionMap) { if (exprIdExpressionMap.containsKey(namedExpression.getExprId())) { - return super.visit(exprIdExpressionMap.get(namedExpression.getExprId()), exprIdExpressionMap); + return visit(exprIdExpressionMap.get(namedExpression.getExprId()), exprIdExpressionMap); } - return super.visitNamedExpression(namedExpression, exprIdExpressionMap); + return visit(namedExpression, exprIdExpressionMap); + } + + @Override + public Expression visit(Expression expr, Map exprIdExpressionMap) { + if (expr instanceof NamedExpression + && expr.arity() == 0 + && exprIdExpressionMap.containsKey(((NamedExpression) expr).getExprId())) { + expr = exprIdExpressionMap.get(((NamedExpression) expr).getExprId()); + } + List newChildren = new ArrayList<>(expr.arity()); + boolean hasNewChildren = false; + for (Expression child : expr.children()) { + Expression newChild = child.accept(this, exprIdExpressionMap); + if (newChild != child) { + hasNewChildren = true; + } + newChildren.add(newChild); + } + return hasNewChildren ? expr.withChildren(newChildren) : expr; } } /** - * The Collector for target named expressions + * The Collector for target named expressions in the whole plan, and will be used to + * replace the target expression later * TODO Collect named expression by targetTypes, tableIdentifiers */ public static class NamedExpressionCollector @@ -83,15 +103,9 @@ public static class NamedExpressionCollector public static final NamedExpressionCollector INSTANCE = new NamedExpressionCollector(); @Override - public Void visitSlotReference(SlotReference slotReference, ExpressionReplaceContext context) { - context.getExprIdExpressionMap().put(slotReference.getExprId(), slotReference); - return super.visitSlotReference(slotReference, context); - } - - @Override - public Void visitArrayItemSlot(ArrayItemSlot arrayItemSlot, ExpressionReplaceContext context) { - context.getExprIdExpressionMap().put(arrayItemSlot.getExprId(), arrayItemSlot); - return super.visitArrayItemSlot(arrayItemSlot, context); + public Void visitSlot(Slot slot, ExpressionReplaceContext context) { + context.getExprIdExpressionMap().put(slot.getExprId(), slot); + return super.visit(slot, context); } @Override @@ -121,7 +135,7 @@ public ExpressionReplaceContext(List targetExpressions, this.targetExpressions = targetExpressions; this.targetTypes = targetTypes; this.tableIdentifiers = tableIdentifiers; - // collect only named expressions and replace them with linage identifier later + // collect the named expressions used in target expression and will be replaced later this.exprIdExpressionMap = targetExpressions.stream() .map(each -> each.collectToList(NamedExpression.class::isInstance)) .flatMap(Collection::stream) diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java index cbe1cf7172ba85..f080eace3971a6 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java @@ -207,6 +207,11 @@ public static Expression combine(Class type, Collection shuttleExpressionWithLineage(List expressions, Plan plan) { return shuttleExpressionWithLineage(expressions, plan, ImmutableSet.of(), ImmutableSet.of()); @@ -310,24 +315,7 @@ public static Optional extractSlotOrCastOnSlot(Expression expr) { * */ public static Expression replace(Expression expr, Map replaceMap) { - return expr.accept(ExpressionReplacer.INSTANCE, ExpressionReplacerContext.of(replaceMap, false)); - } - - /** - * Replace expression node in the expression tree by `replaceMap` in top-down manner. - * if replaced, create alias - * For example. - *
-     * input expression: a > 1
-     * replaceMap: a -> b + c
-     *
-     * output:
-     * ((b + c) as a) > 1
-     * 
- */ - public static Expression replace(Expression expr, Map replaceMap, - boolean withAlias) { - return expr.accept(ExpressionReplacer.INSTANCE, ExpressionReplacerContext.of(replaceMap, true)); + return expr.accept(ExpressionReplacer.INSTANCE, replaceMap); } /** @@ -335,8 +323,7 @@ public static Expression replace(Expression expr, Map replaceMap) { - Expression newExpr = expr.accept(ExpressionReplacer.INSTANCE, - ExpressionReplacerContext.of(replaceMap, false)); + Expression newExpr = expr.accept(ExpressionReplacer.INSTANCE, replaceMap); if (newExpr instanceof NamedExpression) { return (NamedExpression) newExpr; } else { @@ -366,54 +353,49 @@ public static List rewriteDownShortCircuit( } private static class ExpressionReplacer - extends DefaultExpressionRewriter { + extends DefaultExpressionRewriter> { public static final ExpressionReplacer INSTANCE = new ExpressionReplacer(); private ExpressionReplacer() { } @Override - public Expression visit(Expression expr, ExpressionReplacerContext replacerContext) { - Map replaceMap = replacerContext.getReplaceMap(); - boolean isContained = replaceMap.containsKey(expr); - if (!isContained) { - return super.visit(expr, replacerContext); - } - boolean withAlias = replacerContext.isWithAlias(); - if (!withAlias) { - return replaceMap.get(expr); - } else { + public Expression visit(Expression expr, Map replaceMap) { + if (replaceMap.containsKey(expr)) { Expression replacedExpression = replaceMap.get(expr); - if (replacedExpression instanceof SlotReference) { - replacedExpression = ((SlotReference) (replacedExpression)).withNullable(expr.nullable()); + if (replacedExpression instanceof SlotReference + && replacedExpression.nullable() != expr.nullable()) { + replacedExpression = ((SlotReference) replacedExpression).withNullable(expr.nullable()); } - return new Alias(((NamedExpression) expr).getExprId(), replacedExpression, - ((NamedExpression) expr).getName()); + return replacedExpression; } + return super.visit(expr, replaceMap); } } private static class ExpressionReplacerContext { private final Map replaceMap; - private final boolean withAlias; + // if the key of replaceMap is named expr and withAlias is true, we should + // add alias after replaced + private final boolean withAliasIfKeyNamed; private ExpressionReplacerContext(Map replaceMap, - boolean withAlias) { + boolean withAliasIfKeyNamed) { this.replaceMap = replaceMap; - this.withAlias = withAlias; + this.withAliasIfKeyNamed = withAliasIfKeyNamed; } public static ExpressionReplacerContext of(Map replaceMap, - boolean withAlias) { - return new ExpressionReplacerContext(replaceMap, withAlias); + boolean withAliasIfKeyNamed) { + return new ExpressionReplacerContext(replaceMap, withAliasIfKeyNamed); } public Map getReplaceMap() { return replaceMap; } - public boolean isWithAlias() { - return withAlias; + public boolean isWithAliasIfKeyNamed() { + return withAliasIfKeyNamed; } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/qe/StmtExecutor.java b/fe/fe-core/src/main/java/org/apache/doris/qe/StmtExecutor.java index d19f68a6caed68..3a3fe60042b27c 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/qe/StmtExecutor.java +++ b/fe/fe-core/src/main/java/org/apache/doris/qe/StmtExecutor.java @@ -122,7 +122,6 @@ import org.apache.doris.nereids.minidump.MinidumpUtils; import org.apache.doris.nereids.parser.NereidsParser; import org.apache.doris.nereids.rules.exploration.mv.InitMaterializationContextHook; -import org.apache.doris.nereids.stats.StatsErrorEstimator; import org.apache.doris.nereids.trees.plans.commands.BatchInsertIntoTableCommand; import org.apache.doris.nereids.trees.plans.commands.Command; import org.apache.doris.nereids.trees.plans.commands.CreateTableCommand; @@ -581,9 +580,6 @@ private void executeByNereids(TUniqueId queryId) throws Exception { } } else { context.getState().setIsQuery(true); - if (context.getSessionVariable().enableProfile) { - ConnectContext.get().setStatsErrorEstimator(new StatsErrorEstimator()); - } // create plan planner = new NereidsPlanner(statementContext); if (context.getSessionVariable().isEnableMaterializedViewRewrite()) { diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewUtilsTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewUtilsTest.java index f6dba53996235a..2e402cd5c7aa38 100644 --- a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewUtilsTest.java +++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewUtilsTest.java @@ -122,7 +122,7 @@ public void getRelatedTableInfoTestWithoutGroupTest() { } @Test - public void getRelatedTableInfoTestWithAliasAndGroupTest() { + public void getRelatedTableInfoTestWithSubqueryTest() { PlanChecker.from(connectContext) .checkExplain("SELECT l.L_SHIPDATE AS ship_data_alias, o.O_ORDERDATE, count(*) " + "FROM " @@ -147,6 +147,36 @@ public void getRelatedTableInfoTestWithAliasAndGroupTest() { }); } + @Test + public void getRelatedTableInfoTestWithAliasAndGroupTest() { + PlanChecker.from(connectContext) + .checkExplain("SELECT t1.L_SHIPDATE, t2.O_ORDERDATE, t1.L_QUANTITY, t2.O_ORDERSTATUS, " + + "count(distinct case when t1.L_SUPPKEY > 0 then t2.O_ORDERSTATUS else null end) as cnt_1 " + + "from " + + " (select * from " + + " lineitem " + + " where L_SHIPDATE in ('2017-01-30')) t1 " + + "left join " + + " (select * from " + + " orders " + + " where O_ORDERDATE in ('2017-01-30')) t2 " + + "on t1.L_ORDERKEY = t2.O_ORDERKEY " + + "group by " + + "t1.L_SHIPDATE, " + + "t2.O_ORDERDATE, " + + "t1.L_QUANTITY, " + + "t2.O_ORDERSTATUS;", + nereidsPlanner -> { + Plan rewrittenPlan = nereidsPlanner.getRewrittenPlan(); + Optional relatedTableInfo = + MaterializedViewUtils.getRelatedTableInfo("L_SHIPDATE", rewrittenPlan); + checkRelatedTableInfo(relatedTableInfo, + "lineitem", + "L_SHIPDATE", + true); + }); + } + @Test public void getRelatedTableInfoTestWithoutPartitionTest() { PlanChecker.from(connectContext) diff --git a/regression-test/data/nereids_rules_p0/mv/aggregate_with_roll_up.out b/regression-test/data/nereids_rules_p0/mv/aggregate_with_roll_up.out new file mode 100644 index 00000000000000..89514050c1092d --- /dev/null +++ b/regression-test/data/nereids_rules_p0/mv/aggregate_with_roll_up.out @@ -0,0 +1,15 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !query1_0_before -- +4 208.70 +6 109.20 + +-- !query1_0_after -- +4 208.70 +6 109.20 + +-- !query2_0_before -- +2 3 2023-12-11 109.20 + +-- !query2_0_after -- +2 3 2023-12-11 109.20 + diff --git a/regression-test/data/nereids_rules_p0/mv/aggregate_without_roll_up.out b/regression-test/data/nereids_rules_p0/mv/aggregate_without_roll_up.out new file mode 100644 index 00000000000000..8539b5446a1d13 --- /dev/null +++ b/regression-test/data/nereids_rules_p0/mv/aggregate_without_roll_up.out @@ -0,0 +1,31 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !query1_0_before -- +4 1 99.50 +4 2 109.20 +6 2 109.20 + +-- !query1_0_after -- +4 1 99.50 +4 2 109.20 +6 2 109.20 + +-- !query1_1_before -- +1 yy 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 +2 mi 1 0 0 0 0 0 0 0 0 0 0 0 0 0 1 +2 mm 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 + +-- !query1_1_after -- +1 yy 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 +2 mi 1 0 0 0 0 0 0 0 0 0 0 0 0 0 1 +2 mm 0 0 0 0 0 0 0 0 0 0 0 0 0 0 0 + +-- !query2_0_before -- +1 1 yy 0 0 0 0 0 0 0 0 0 0 0 0 0 +1 2 mi 1 0 0 0 0 0 0 0 0 0 0 0 0 +2 2 mm 0 0 0 0 0 0 0 0 0 0 0 0 0 + +-- !query2_0_after -- +1 1 yy 0 0 0 0 0 0 0 0 0 0 0 0 0 +1 2 mi 1 0 0 0 0 0 0 0 0 0 0 0 0 +2 2 mm 0 0 0 0 0 0 0 0 0 0 0 0 0 + diff --git a/regression-test/data/nereids_rules_p0/mv/inner_join.out b/regression-test/data/nereids_rules_p0/mv/inner_join.out index 999748537661f3..863d6765698c5c 100644 --- a/regression-test/data/nereids_rules_p0/mv/inner_join.out +++ b/regression-test/data/nereids_rules_p0/mv/inner_join.out @@ -1,9 +1,45 @@ -- This file is automatically generated. You should know what you did if you want to edit this --- !query1_0 -- +-- !query1_0_before -- +4 +6 --- !query1_1 -- +-- !query1_0_after -- +4 +6 --- !query1_2 -- +-- !query1_1_before -- +4 +4 +6 +6 --- !query1_3 -- +-- !query1_1_after -- +4 +4 +6 +6 + +-- !query1_2_before -- +4 +6 + +-- !query1_2_after -- +4 +6 + +-- !query1_3_before -- + +-- !query1_3_after -- + +-- !query1_4_before -- +1 +2 + +-- !query1_4_after -- +1 +2 + +-- !query10_0_before -- + +-- !query10_0_after -- diff --git a/regression-test/data/nereids_rules_p0/mv/outer_join.out b/regression-test/data/nereids_rules_p0/mv/outer_join.out new file mode 100644 index 00000000000000..9579dd54810014 --- /dev/null +++ b/regression-test/data/nereids_rules_p0/mv/outer_join.out @@ -0,0 +1,69 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !query1_0_before -- +4 +6 + +-- !query1_0_after -- +4 +6 + +-- !query1_1_before -- +4 +4 +6 +6 + +-- !query1_1_after -- +4 +4 +6 +6 + +-- !query1_2_before -- +4 +6 + +-- !query1_2_after -- +4 +6 + +-- !query1_3_before -- +1 +2 + +-- !query1_3_after -- +1 +2 + +-- !query2_0_before -- + +-- !query2_0_after -- + +-- !query2_1_before -- + +-- !query2_1_after -- + +-- !query2_2_before -- +4 +6 + +-- !query2_2_after -- +4 +6 + +-- !query2_3_before -- + +-- !query2_3_after -- + +-- !query2_4_before -- + +-- !query2_4_after -- + +-- !query2_5_before -- +4 +6 + +-- !query2_5_after -- +4 +6 + diff --git a/regression-test/suites/nereids_rules_p0/mv/aggregate_with_roll_up.groovy b/regression-test/suites/nereids_rules_p0/mv/aggregate_with_roll_up.groovy new file mode 100644 index 00000000000000..79a5fe2c1c801b --- /dev/null +++ b/regression-test/suites/nereids_rules_p0/mv/aggregate_with_roll_up.groovy @@ -0,0 +1,237 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you 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. + +suite("aggregate_with_roll_up") { + String db = context.config.getDbNameByFile(context.file) + sql "use ${db}" + sql "SET enable_nereids_planner=true" + sql "SET enable_fallback_to_original_planner=false" + sql "SET enable_materialized_view_rewrite=true" + sql "SET enable_nereids_timeout = false" + // tmp disable to rewrite, will be removed in the future + sql "SET disable_nereids_rules = 'INFER_PREDICATES, ELIMINATE_OUTER_JOIN'" + + sql """ + drop table if exists orders + """ + + sql """ + CREATE TABLE IF NOT EXISTS orders ( + O_ORDERKEY INTEGER NOT NULL, + O_CUSTKEY INTEGER NOT NULL, + O_ORDERSTATUS CHAR(1) NOT NULL, + O_TOTALPRICE DECIMALV3(15,2) NOT NULL, + O_ORDERDATE DATE NOT NULL, + O_ORDERPRIORITY CHAR(15) NOT NULL, + O_CLERK CHAR(15) NOT NULL, + O_SHIPPRIORITY INTEGER NOT NULL, + O_COMMENT VARCHAR(79) NOT NULL + ) + DUPLICATE KEY(O_ORDERKEY, O_CUSTKEY) + PARTITION BY RANGE(O_ORDERDATE) (PARTITION `day_2` VALUES LESS THAN ('2023-12-30')) + DISTRIBUTED BY HASH(O_ORDERKEY) BUCKETS 3 + PROPERTIES ( + "replication_num" = "1" + ) + """ + + sql """ + drop table if exists lineitem + """ + + sql""" + CREATE TABLE IF NOT EXISTS lineitem ( + L_ORDERKEY INTEGER NOT NULL, + L_PARTKEY INTEGER NOT NULL, + L_SUPPKEY INTEGER NOT NULL, + L_LINENUMBER INTEGER NOT NULL, + L_QUANTITY DECIMALV3(15,2) NOT NULL, + L_EXTENDEDPRICE DECIMALV3(15,2) NOT NULL, + L_DISCOUNT DECIMALV3(15,2) NOT NULL, + L_TAX DECIMALV3(15,2) NOT NULL, + L_RETURNFLAG CHAR(1) NOT NULL, + L_LINESTATUS CHAR(1) NOT NULL, + L_SHIPDATE DATE NOT NULL, + L_COMMITDATE DATE NOT NULL, + L_RECEIPTDATE DATE NOT NULL, + L_SHIPINSTRUCT CHAR(25) NOT NULL, + L_SHIPMODE CHAR(10) NOT NULL, + L_COMMENT VARCHAR(44) NOT NULL + ) + DUPLICATE KEY(L_ORDERKEY, L_PARTKEY, L_SUPPKEY, L_LINENUMBER) + PARTITION BY RANGE(L_SHIPDATE) (PARTITION `day_1` VALUES LESS THAN ('2023-12-30')) + DISTRIBUTED BY HASH(L_ORDERKEY) BUCKETS 3 + PROPERTIES ( + "replication_num" = "1" + ) + """ + + sql """ + drop table if exists partsupp + """ + + sql """ + CREATE TABLE IF NOT EXISTS partsupp ( + PS_PARTKEY INTEGER NOT NULL, + PS_SUPPKEY INTEGER NOT NULL, + PS_AVAILQTY INTEGER NOT NULL, + PS_SUPPLYCOST DECIMALV3(15,2) NOT NULL, + PS_COMMENT VARCHAR(199) NOT NULL + ) + DUPLICATE KEY(PS_PARTKEY, PS_SUPPKEY) + DISTRIBUTED BY HASH(PS_PARTKEY) BUCKETS 3 + PROPERTIES ( + "replication_num" = "1" + ) + """ + + sql """ insert into lineitem values (1, 2, 3, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-08', '2023-12-09', '2023-12-10', 'a', 'b', 'yyyyyyyyy'), + (2, 2, 3, 6, 7.5, 8.5, 9.5, 10.5, 'k', 'o', '2023-12-11', '2023-12-12', '2023-12-13', 'c', 'd', 'xxxxxxxxx');""" + + sql """ + insert into orders values (1, 1, 'ok', 99.5, '2023-12-08', 'a', 'b', 1, 'yy'), + (2, 2, 'ok', 109.2, '2023-12-11', 'c','d',2, 'mm'), + (1, 2, 'ok', 109.2, '2023-12-11', 'c','d',2, 'mi'); + """ + + sql """ + insert into partsupp values (2, 3, 9, 10.01, 'supply1'), + (2, 3, 10, 11.01, 'supply2'); + """ + + def check_rewrite = { mv_sql, query_sql, mv_name -> + + sql """DROP MATERIALIZED VIEW IF EXISTS ${mv_name}""" + sql""" + CREATE MATERIALIZED VIEW ${mv_name} + BUILD IMMEDIATE REFRESH COMPLETE ON MANUAL + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS ${mv_sql} + """ + + def job_name = getJobName(db, mv_name); + waitingMTMVTaskFinished(job_name) + explain { + sql("${query_sql}") + contains "(${mv_name})" + } + } + + def check_not_match = { mv_sql, query_sql, mv_name -> + + sql """DROP MATERIALIZED VIEW IF EXISTS ${mv_name}""" + sql""" + CREATE MATERIALIZED VIEW ${mv_name} + BUILD IMMEDIATE REFRESH COMPLETE ON MANUAL + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS ${mv_sql} + """ + + def job_name = getJobName(db, mv_name); + waitingMTMVTaskFinished(job_name) + explain { + sql("${query_sql}") + notContains "(${mv_name})" + } + } + + // select + from + inner join + group by + def mv1_0 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY, sum(O_TOTALPRICE) as sum_alias " + + "from lineitem " + + "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "group by lineitem.L_LINENUMBER, orders.O_CUSTKEY " + def query1_0 = "select lineitem.L_LINENUMBER, sum(O_TOTALPRICE) as sum_alias " + + "from lineitem " + + "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "group by lineitem.L_LINENUMBER" + order_qt_query1_0_before "${query1_0}" + check_rewrite(mv1_0, query1_0, "mv1_0") + order_qt_query1_0_after "${query1_0}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_0""" + + + // select case when + from + where + group by roll up + def mv1_1 = "select O_ORDERDATE, O_SHIPPRIORITY, O_COMMENT, " + + "bitmap_union(to_bitmap(case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (1, 3) then O_CUSTKEY else null end)) cnt_1, " + + "bitmap_union(to_bitmap(case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (2) then O_CUSTKEY else null end)) as cnt_2 " + + "from orders " + + "group by " + + "O_ORDERDATE, " + + "O_SHIPPRIORITY, " + + "O_COMMENT " + def query1_1 = "select O_SHIPPRIORITY, O_COMMENT, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (1, 3) then O_CUSTKEY else null end) as cnt_1, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (2) then O_CUSTKEY else null end) as cnt_2 " + + "from orders " + + "where O_ORDERDATE = '2023-12-09' " + + "group by " + + "O_SHIPPRIORITY, " + + "O_COMMENT " +// order_qt_query1_1_before "${query1_1}" +// unsupported, need to fix, because create materialized view contains bitmap which is not supported +// check_rewrite(mv1_1, query1_1, "mv1_1") +// order_qt_query1_1_after "${query1_1}" +// sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_1""" + + + def mv1_2 = "select O_ORDERDATE, O_SHIPPRIORITY, O_COMMENT, " + + "bitmap_union(to_bitmap(O_CUSTKEY)) as cnt_1, " + + "bitmap_union(to_bitmap(O_CUSTKEY)) as cnt_2 " + + "from orders " + + "group by " + + "O_ORDERDATE, " + + "O_SHIPPRIORITY, " + + "O_COMMENT " + def query1_2 = "select O_SHIPPRIORITY, O_COMMENT, " + + "count(distinct O_CUSTKEY) as cnt_1, " + + "count(distinct O_ORDERKEY) as cnt_2 " + + "from orders " + + "where O_ORDERDATE = '2023-12-09' " + + "group by " + + "O_SHIPPRIORITY, " + + "O_COMMENT " + // order_qt_query1_2_before "${query1_2}" + // unsupported, need to fix + // check_rewrite(mv1_2, query1_2, "mv1_2") + // order_qt_query1_2_after "${query1_2}" + // sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_2""" + + + + // select + from + where + outer join + group by roll up + def mv2_0 = "select L_SHIPDATE, O_ORDERDATE, L_PARTKEY, L_SUPPKEY, sum(O_TOTALPRICE) as sum_total " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY and L_SHIPDATE = O_ORDERDATE " + + "group by " + + "L_SHIPDATE, " + + "O_ORDERDATE, " + + "L_PARTKEY, " + + "L_SUPPKEY" + def query2_0 = "select t1.L_PARTKEY, t1.L_SUPPKEY, O_ORDERDATE, sum(O_TOTALPRICE) " + + "from (select * from lineitem where L_SHIPDATE = '2023-12-11') t1 " + + "left join orders on t1.L_ORDERKEY = orders.O_ORDERKEY and t1.L_SHIPDATE = O_ORDERDATE " + + "group by " + + "O_ORDERDATE, " + + "L_PARTKEY, " + + "L_SUPPKEY" + order_qt_query2_0_before "${query2_0}" + check_rewrite(mv2_0, query2_0, "mv2_0") + order_qt_query2_0_after "${query2_0}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_0""" +} diff --git a/regression-test/suites/nereids_rules_p0/mv/aggregate_without_roll_up.groovy b/regression-test/suites/nereids_rules_p0/mv/aggregate_without_roll_up.groovy new file mode 100644 index 00000000000000..7834741c06c50b --- /dev/null +++ b/regression-test/suites/nereids_rules_p0/mv/aggregate_without_roll_up.groovy @@ -0,0 +1,281 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you 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. + +suite("aggregate_without_roll_up") { + String db = context.config.getDbNameByFile(context.file) + sql "use ${db}" + sql "SET enable_nereids_planner=true" + sql "SET enable_fallback_to_original_planner=false" + sql "SET enable_materialized_view_rewrite=true" + sql "SET enable_nereids_timeout = false" + // tmp disable to rewrite, will be removed in the future + sql "SET disable_nereids_rules = 'INFER_PREDICATES, ELIMINATE_OUTER_JOIN'" + + sql """ + drop table if exists orders + """ + + sql """ + CREATE TABLE IF NOT EXISTS orders ( + O_ORDERKEY INTEGER NOT NULL, + O_CUSTKEY INTEGER NOT NULL, + O_ORDERSTATUS CHAR(1) NOT NULL, + O_TOTALPRICE DECIMALV3(15,2) NOT NULL, + O_ORDERDATE DATE NOT NULL, + O_ORDERPRIORITY CHAR(15) NOT NULL, + O_CLERK CHAR(15) NOT NULL, + O_SHIPPRIORITY INTEGER NOT NULL, + O_COMMENT VARCHAR(79) NOT NULL + ) + DUPLICATE KEY(O_ORDERKEY, O_CUSTKEY) + PARTITION BY RANGE(O_ORDERDATE) (PARTITION `day_2` VALUES LESS THAN ('2023-12-30')) + DISTRIBUTED BY HASH(O_ORDERKEY) BUCKETS 3 + PROPERTIES ( + "replication_num" = "1" + ) + """ + + sql """ + drop table if exists lineitem + """ + + sql""" + CREATE TABLE IF NOT EXISTS lineitem ( + L_ORDERKEY INTEGER NOT NULL, + L_PARTKEY INTEGER NOT NULL, + L_SUPPKEY INTEGER NOT NULL, + L_LINENUMBER INTEGER NOT NULL, + L_QUANTITY DECIMALV3(15,2) NOT NULL, + L_EXTENDEDPRICE DECIMALV3(15,2) NOT NULL, + L_DISCOUNT DECIMALV3(15,2) NOT NULL, + L_TAX DECIMALV3(15,2) NOT NULL, + L_RETURNFLAG CHAR(1) NOT NULL, + L_LINESTATUS CHAR(1) NOT NULL, + L_SHIPDATE DATE NOT NULL, + L_COMMITDATE DATE NOT NULL, + L_RECEIPTDATE DATE NOT NULL, + L_SHIPINSTRUCT CHAR(25) NOT NULL, + L_SHIPMODE CHAR(10) NOT NULL, + L_COMMENT VARCHAR(44) NOT NULL + ) + DUPLICATE KEY(L_ORDERKEY, L_PARTKEY, L_SUPPKEY, L_LINENUMBER) + PARTITION BY RANGE(L_SHIPDATE) (PARTITION `day_1` VALUES LESS THAN ('2023-12-30')) + DISTRIBUTED BY HASH(L_ORDERKEY) BUCKETS 3 + PROPERTIES ( + "replication_num" = "1" + ) + """ + + sql """ + drop table if exists partsupp + """ + + sql """ + CREATE TABLE IF NOT EXISTS partsupp ( + PS_PARTKEY INTEGER NOT NULL, + PS_SUPPKEY INTEGER NOT NULL, + PS_AVAILQTY INTEGER NOT NULL, + PS_SUPPLYCOST DECIMALV3(15,2) NOT NULL, + PS_COMMENT VARCHAR(199) NOT NULL + ) + DUPLICATE KEY(PS_PARTKEY, PS_SUPPKEY) + DISTRIBUTED BY HASH(PS_PARTKEY) BUCKETS 3 + PROPERTIES ( + "replication_num" = "1" + ) + """ + + sql """ insert into lineitem values (1, 2, 3, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-08', '2023-12-09', '2023-12-10', 'a', 'b', 'yyyyyyyyy'), + (2, 2, 3, 6, 7.5, 8.5, 9.5, 10.5, 'k', 'o', '2023-12-11', '2023-12-12', '2023-12-13', 'c', 'd', 'xxxxxxxxx');""" + + sql """ + insert into orders values (1, 1, 'ok', 99.5, '2023-12-08', 'a', 'b', 1, 'yy'), + (2, 2, 'ok', 109.2, '2023-12-11', 'c','d',2, 'mm'), + (1, 2, 'ok', 109.2, '2023-12-11', 'c','d',2, 'mi'); + """ + + sql """ + insert into partsupp values (2, 3, 9, 10.01, 'supply1'), + (2, 3, 10, 11.01, 'supply2'); + """ + + def check_rewrite = { mv_sql, query_sql, mv_name -> + + sql """DROP MATERIALIZED VIEW IF EXISTS ${mv_name}""" + sql""" + CREATE MATERIALIZED VIEW ${mv_name} + BUILD IMMEDIATE REFRESH COMPLETE ON MANUAL + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS ${mv_sql} + """ + + def job_name = getJobName(db, mv_name); + waitingMTMVTaskFinished(job_name) + explain { + sql("${query_sql}") + contains "(${mv_name})" + } + } + + def check_not_match = { mv_sql, query_sql, mv_name -> + + sql """DROP MATERIALIZED VIEW IF EXISTS ${mv_name}""" + sql""" + CREATE MATERIALIZED VIEW ${mv_name} + BUILD IMMEDIATE REFRESH COMPLETE ON MANUAL + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS ${mv_sql} + """ + + def job_name = getJobName(db, mv_name); + waitingMTMVTaskFinished(job_name) + explain { + sql("${query_sql}") + notContains "(${mv_name})" + } + } + + // select + from + inner join + group by + def mv1_0 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY, sum(O_TOTALPRICE) as sum_alias " + + "from lineitem " + + "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "group by lineitem.L_LINENUMBER, orders.O_CUSTKEY " + def query1_0 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY, sum(O_TOTALPRICE) as sum_alias " + + "from lineitem " + + "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "group by lineitem.L_LINENUMBER, orders.O_CUSTKEY " + order_qt_query1_0_before "${query1_0}" + check_rewrite(mv1_0, query1_0, "mv1_0") + order_qt_query1_0_after "${query1_0}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_0""" + + + // select case when + from + where + group by + def mv1_1 = "select O_SHIPPRIORITY, O_COMMENT, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (1, 3) then O_ORDERSTATUS else null end) as filter_cnt_1, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (2) then O_ORDERSTATUS else null end) as filter_cnt_2, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (3, 4) then O_ORDERSTATUS else null end) as filter_cnt_3, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (5, 6) then O_ORDERSTATUS else null end) as filter_cnt_4, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (2, 3) then O_ORDERSTATUS else null end) as filter_cnt_5, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (7, 9) then O_ORDERSTATUS else null end) as filter_cnt_6, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (8, 10) then O_ORDERSTATUS else null end) as filter_cnt_7, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (11, 13) then O_ORDERSTATUS else null end) as filter_cnt_8, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (12, 11) then O_ORDERSTATUS else null end) as filter_cnt_9, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (14, 15) then O_ORDERSTATUS else null end) as filter_cnt_10, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (11, 12) then O_ORDERSTATUS else null end) as filter_cnt_11, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (3, 6) then O_ORDERSTATUS else null end) as filter_cnt_12, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (16, 19) then O_ORDERSTATUS else null end) as filter_cnt_13, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (20, 3) then O_ORDERSTATUS else null end) as filter_cnt_14, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (15, 19) then O_ORDERSTATUS else null end) as filter_cnt_15, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (13, 21) then O_ORDERSTATUS else null end) as filter_cnt_16, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (14, 22) then O_ORDERSTATUS else null end) as filter_cnt_17, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (16, 25) then O_ORDERSTATUS else null end) as filter_cnt_18, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (19, 3) then O_ORDERSTATUS else null end) as filter_cnt_19, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (1, 20) then O_ORDERSTATUS else null end) as filter_cnt_20 " + + "from orders " + + "where O_ORDERDATE < '2023-12-30' and O_ORDERDATE > '2023-12-01'" + + "group by " + + "O_SHIPPRIORITY, " + + "O_COMMENT " + def query1_1 = "select O_SHIPPRIORITY, O_COMMENT, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (1, 3) then O_ORDERSTATUS else null end) as filter_cnt_1, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (2) then O_ORDERSTATUS else null end) as filter_cnt_2, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (3, 4) then O_ORDERSTATUS else null end) as filter_cnt_3, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (2, 3) then O_ORDERSTATUS else null end) as filter_cnt_5, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (7, 9) then O_ORDERSTATUS else null end) as filter_cnt_6, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (11, 13) then O_ORDERSTATUS else null end) as filter_cnt_8, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (12, 11) then O_ORDERSTATUS else null end) as filter_cnt_9, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (11, 12) then O_ORDERSTATUS else null end) as filter_cnt_11, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (3, 6) then O_ORDERSTATUS else null end) as filter_cnt_12, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (16, 19) then O_ORDERSTATUS else null end) as filter_cnt_13, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (15, 19) then O_ORDERSTATUS else null end) as filter_cnt_15, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (13, 21) then O_ORDERSTATUS else null end) as filter_cnt_16, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (16, 25) then O_ORDERSTATUS else null end) as filter_cnt_18, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (19, 3) then O_ORDERSTATUS else null end) as filter_cnt_19, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (1, 20) then O_ORDERSTATUS else null end) as filter_cnt_20 " + + "from orders " + + "where O_ORDERDATE < '2023-12-30' and O_ORDERDATE > '2023-12-01'" + + "group by " + + "O_SHIPPRIORITY, " + + "O_COMMENT " + order_qt_query1_1_before "${query1_1}" + check_rewrite(mv1_1, query1_1, "mv1_1") + order_qt_query1_1_after "${query1_1}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_1""" + + + // select case when + from + where + group by + def mv2_0 = "select L_ORDERKEY, O_SHIPPRIORITY, O_COMMENT, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (1, 3) then O_ORDERSTATUS else null end) as filter_cnt_1, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (2) then O_ORDERSTATUS else null end) as filter_cnt_2, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (3, 4) then O_ORDERSTATUS else null end) as filter_cnt_3, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (5, 6) then O_ORDERSTATUS else null end) as filter_cnt_4, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (2, 3) then O_ORDERSTATUS else null end) as filter_cnt_5, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (7, 9) then O_ORDERSTATUS else null end) as filter_cnt_6, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (8, 10) then O_ORDERSTATUS else null end) as filter_cnt_7, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (11, 13) then O_ORDERSTATUS else null end) as filter_cnt_8, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (12, 11) then O_ORDERSTATUS else null end) as filter_cnt_9, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (14, 15) then O_ORDERSTATUS else null end) as filter_cnt_10, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (11, 12) then O_ORDERSTATUS else null end) as filter_cnt_11, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (3, 6) then O_ORDERSTATUS else null end) as filter_cnt_12, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (16, 19) then O_ORDERSTATUS else null end) as filter_cnt_13, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (20, 3) then O_ORDERSTATUS else null end) as filter_cnt_14, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (15, 19) then O_ORDERSTATUS else null end) as filter_cnt_15, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (13, 21) then O_ORDERSTATUS else null end) as filter_cnt_16, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (14, 22) then O_ORDERSTATUS else null end) as filter_cnt_17, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (16, 25) then O_ORDERSTATUS else null end) as filter_cnt_18, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (19, 3) then O_ORDERSTATUS else null end) as filter_cnt_19, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (1, 20) then O_ORDERSTATUS else null end) as filter_cnt_20 " + + "from lineitem " + + "left join " + + "orders " + + "on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "where orders.O_ORDERDATE < '2023-12-30' and orders.O_ORDERDATE > '2023-12-01' " + + "group by " + + "lineitem.L_ORDERKEY, " + + "orders.O_SHIPPRIORITY, " + + "orders.O_COMMENT " + def query2_0 = "select L_ORDERKEY, O_SHIPPRIORITY, O_COMMENT, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (1, 3) then O_ORDERSTATUS else null end) as filter_cnt_1, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (2) then O_ORDERSTATUS else null end) as filter_cnt_2, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (3, 4) then O_ORDERSTATUS else null end) as filter_cnt_3, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (5, 6) then O_ORDERSTATUS else null end) as filter_cnt_4, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (2, 3) then O_ORDERSTATUS else null end) as filter_cnt_5, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (7, 9) then O_ORDERSTATUS else null end) as filter_cnt_6, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (8, 10) then O_ORDERSTATUS else null end) as filter_cnt_7, " + + "count(distinct case when O_SHIPPRIORITY > 4 and O_ORDERKEY IN (3, 6) then O_ORDERSTATUS else null end) as filter_cnt_12, " + + "count(distinct case when O_SHIPPRIORITY > 3 and O_ORDERKEY IN (16, 19) then O_ORDERSTATUS else null end) as filter_cnt_13, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (20, 3) then O_ORDERSTATUS else null end) as filter_cnt_14, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (15, 19) then O_ORDERSTATUS else null end) as filter_cnt_15, " + + "count(distinct case when O_SHIPPRIORITY > 1 and O_ORDERKEY IN (13, 21) then O_ORDERSTATUS else null end) as filter_cnt_16, " + + "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (14, 22) then O_ORDERSTATUS else null end) as filter_cnt_17 " + + "from lineitem " + + "left join " + + "orders " + + "on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "where orders.O_ORDERDATE < '2023-12-30' and orders.O_ORDERDATE > '2023-12-01' " + + "group by " + + "lineitem.L_ORDERKEY, " + + "orders.O_SHIPPRIORITY, " + + "orders.O_COMMENT " + order_qt_query2_0_before "${query2_0}" + check_rewrite(mv2_0, query2_0, "mv2_0") + order_qt_query2_0_after "${query2_0}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_0""" +} diff --git a/regression-test/suites/nereids_rules_p0/mv/inner_join.groovy b/regression-test/suites/nereids_rules_p0/mv/inner_join.groovy index 9d0bc62f05c22e..70bf2f9bcb6ecc 100644 --- a/regression-test/suites/nereids_rules_p0/mv/inner_join.groovy +++ b/regression-test/suites/nereids_rules_p0/mv/inner_join.groovy @@ -22,6 +22,8 @@ suite("inner_join") { sql "SET enable_fallback_to_original_planner=false" sql "SET enable_materialized_view_rewrite=true" sql "SET enable_nereids_timeout = false" + // tmp disable to rewrite, will be removed in the future + sql "SET disable_nereids_rules = 'INFER_PREDICATES, ELIMINATE_OUTER_JOIN'" sql """ drop table if exists orders @@ -95,6 +97,19 @@ suite("inner_join") { ) """ + sql """ insert into lineitem values (1, 2, 3, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-08', '2023-12-09', '2023-12-10', 'a', 'b', 'yyyyyyyyy'), + (2, 2, 3, 6, 7.5, 8.5, 9.5, 10.5, 'k', 'o', '2023-12-11', '2023-12-12', '2023-12-13', 'c', 'd', 'xxxxxxxxx');""" + + sql """ + insert into orders values (1, 1, 'ok', 99.5, '2023-12-08', 'a', 'b', 1, 'yy'), + (2, 2, 'ok', 109.2, '2023-12-09', 'c','d',2, 'mm'); + """ + + sql """ + insert into partsupp values (2, 3, 9, 10.01, 'supply1'), + (2, 3, 10, 11.01, 'supply2'); + """ + def check_rewrite = { mv_sql, query_sql, mv_name -> sql """DROP MATERIALIZED VIEW IF EXISTS ${mv_name}""" @@ -114,6 +129,25 @@ suite("inner_join") { } } + def check_not_match = { mv_sql, query_sql, mv_name -> + + sql """DROP MATERIALIZED VIEW IF EXISTS ${mv_name}""" + sql""" + CREATE MATERIALIZED VIEW ${mv_name} + BUILD IMMEDIATE REFRESH COMPLETE ON MANUAL + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS ${mv_sql} + """ + + def job_name = getJobName(db, mv_name); + waitingMTMVTaskFinished(job_name) + explain { + sql("${query_sql}") + notContains "(${mv_name})" + } + } + // select + from + inner join def mv1_0 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY " + "from lineitem " + @@ -121,8 +155,9 @@ suite("inner_join") { def query1_0 = "select lineitem.L_LINENUMBER " + "from lineitem " + "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + order_qt_query1_0_before "${query1_0}" check_rewrite(mv1_0, query1_0, "mv1_0") - order_qt_query1_0 "${query1_0}" + order_qt_query1_0_after "${query1_0}" sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_0""" @@ -136,8 +171,9 @@ suite("inner_join") { "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + "inner join partsupp on lineitem.L_PARTKEY = partsupp.PS_PARTKEY " + "and lineitem.L_SUPPKEY = partsupp.PS_SUPPKEY" + order_qt_query1_1_before "${query1_1}" check_rewrite(mv1_1, query1_1, "mv1_1") - order_qt_query1_1 "${query1_1}" + order_qt_query1_1_after "${query1_1}" sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_1""" @@ -147,8 +183,9 @@ suite("inner_join") { def query1_2 = "select lineitem.L_LINENUMBER " + "from lineitem " + "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + order_qt_query1_2_before "${query1_2}" check_rewrite(mv1_2, query1_2, "mv1_2") - order_qt_query1_2 "${query1_2}" + order_qt_query1_2_after "${query1_2}" sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_2""" // select + from + inner join + filter @@ -159,7 +196,36 @@ suite("inner_join") { "from lineitem " + "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + "where lineitem.L_LINENUMBER > 10" + order_qt_query1_3_before "${query1_3}" check_rewrite(mv1_3, query1_3, "mv1_3") - order_qt_query1_3 "${query1_3}" + // tmp annotation, will fix later + order_qt_query1_3_after "${query1_3}" sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_3""" + + // select with complex expression + from + inner join + def mv1_4 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY " + + "from orders " + + "inner join lineitem on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + def query1_4 = "select IFNULL(orders.O_CUSTKEY, 0) as custkey_not_null " + + "from orders " + + "inner join lineitem on orders.O_ORDERKEY = lineitem.L_ORDERKEY" + order_qt_query1_4_before "${query1_4}" + check_rewrite(mv1_4, query1_4, "mv1_4") + order_qt_query1_4_after "${query1_4}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_4""" + + + // check not match, because use a filed orders.O_SHIPPRIORITY which not in mv + def mv10_0 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY " + + "from lineitem " + + "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY" + def query10_0 = "select orders.O_CUSTKEY " + + "from orders " + + "inner join lineitem on orders.O_ORDERKEY = lineitem.L_ORDERKEY " + + "WHERE lineitem.L_LINENUMBER > 10 AND orders.O_CUSTKEY = 5 AND " + + "orders.O_SHIPPRIORITY = 1" + order_qt_query10_0_before "${query10_0}" + check_not_match(mv10_0, query10_0, "mv10_0") + order_qt_query10_0_after "${query10_0}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv10_0""" } diff --git a/regression-test/suites/nereids_rules_p0/mv/outer_join.groovy b/regression-test/suites/nereids_rules_p0/mv/outer_join.groovy new file mode 100644 index 00000000000000..08046c20d445f2 --- /dev/null +++ b/regression-test/suites/nereids_rules_p0/mv/outer_join.groovy @@ -0,0 +1,288 @@ + +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you 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. + +suite("outer_join") { + String db = context.config.getDbNameByFile(context.file) + sql "use ${db}" + sql "SET enable_nereids_planner=true" + sql "SET enable_fallback_to_original_planner=false" + sql "SET enable_materialized_view_rewrite=true" + sql "SET enable_nereids_timeout = false" + // tmp disable to rewrite, will be removed in the future + sql "SET disable_nereids_rules = 'INFER_PREDICATES, ELIMINATE_OUTER_JOIN'" + + sql """ + drop table if exists orders + """ + + sql """ + CREATE TABLE IF NOT EXISTS orders ( + O_ORDERKEY INTEGER NOT NULL, + O_CUSTKEY INTEGER NOT NULL, + O_ORDERSTATUS CHAR(1) NOT NULL, + O_TOTALPRICE DECIMALV3(15,2) NOT NULL, + O_ORDERDATE DATE NOT NULL, + O_ORDERPRIORITY CHAR(15) NOT NULL, + O_CLERK CHAR(15) NOT NULL, + O_SHIPPRIORITY INTEGER NOT NULL, + O_COMMENT VARCHAR(79) NOT NULL + ) + DUPLICATE KEY(O_ORDERKEY, O_CUSTKEY) + DISTRIBUTED BY HASH(O_ORDERKEY) BUCKETS 3 + PROPERTIES ( + "replication_num" = "1" + ) + """ + + sql """ + drop table if exists lineitem + """ + + sql""" + CREATE TABLE IF NOT EXISTS lineitem ( + L_ORDERKEY INTEGER NOT NULL, + L_PARTKEY INTEGER NOT NULL, + L_SUPPKEY INTEGER NOT NULL, + L_LINENUMBER INTEGER NOT NULL, + L_QUANTITY DECIMALV3(15,2) NOT NULL, + L_EXTENDEDPRICE DECIMALV3(15,2) NOT NULL, + L_DISCOUNT DECIMALV3(15,2) NOT NULL, + L_TAX DECIMALV3(15,2) NOT NULL, + L_RETURNFLAG CHAR(1) NOT NULL, + L_LINESTATUS CHAR(1) NOT NULL, + L_SHIPDATE DATE NOT NULL, + L_COMMITDATE DATE NOT NULL, + L_RECEIPTDATE DATE NOT NULL, + L_SHIPINSTRUCT CHAR(25) NOT NULL, + L_SHIPMODE CHAR(10) NOT NULL, + L_COMMENT VARCHAR(44) NOT NULL + ) + DUPLICATE KEY(L_ORDERKEY, L_PARTKEY, L_SUPPKEY, L_LINENUMBER) + DISTRIBUTED BY HASH(L_ORDERKEY) BUCKETS 3 + PROPERTIES ( + "replication_num" = "1" + ) + """ + + sql """ + drop table if exists partsupp + """ + + sql """ + CREATE TABLE IF NOT EXISTS partsupp ( + PS_PARTKEY INTEGER NOT NULL, + PS_SUPPKEY INTEGER NOT NULL, + PS_AVAILQTY INTEGER NOT NULL, + PS_SUPPLYCOST DECIMALV3(15,2) NOT NULL, + PS_COMMENT VARCHAR(199) NOT NULL + ) + DUPLICATE KEY(PS_PARTKEY, PS_SUPPKEY) + DISTRIBUTED BY HASH(PS_PARTKEY) BUCKETS 3 + PROPERTIES ( + "replication_num" = "1" + ) + """ + + sql """ insert into lineitem values (1, 2, 3, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-08', '2023-12-09', '2023-12-10', 'a', 'b', 'yyyyyyyyy'), + (2, 2, 3, 6, 7.5, 8.5, 9.5, 10.5, 'k', 'o', '2023-12-11', '2023-12-12', '2023-12-13', 'c', 'd', 'xxxxxxxxx');""" + + sql """ + insert into orders values (1, 1, 'ok', 99.5, '2023-12-08', 'a', 'b', 1, 'yy'), + (2, 2, 'ok', 109.2, '2023-12-09', 'c','d',2, 'mm'); + """ + + sql """ + insert into partsupp values (2, 3, 9, 10.01, 'supply1'), + (2, 3, 10, 11.01, 'supply2'); + """ + + def check_rewrite = { mv_sql, query_sql, mv_name -> + + sql """DROP MATERIALIZED VIEW IF EXISTS ${mv_name}""" + sql""" + CREATE MATERIALIZED VIEW ${mv_name} + BUILD IMMEDIATE REFRESH COMPLETE ON MANUAL + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS ${mv_sql} + """ + + def job_name = getJobName(db, mv_name); + waitingMTMVTaskFinished(job_name) + explain { + sql("${query_sql}") + contains "(${mv_name})" + } + } + + def check_not_match = { mv_sql, query_sql, mv_name -> + + sql """DROP MATERIALIZED VIEW IF EXISTS ${mv_name}""" + sql""" + CREATE MATERIALIZED VIEW ${mv_name} + BUILD IMMEDIATE REFRESH COMPLETE ON MANUAL + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS ${mv_sql} + """ + + def job_name = getJobName(db, mv_name); + waitingMTMVTaskFinished(job_name) + explain { + sql("${query_sql}") + notContains "(${mv_name})" + } + } + + // select + from + left outer join + def mv1_0 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + def query1_0 = "select lineitem.L_LINENUMBER " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + order_qt_query1_0_before "${query1_0}" + check_rewrite(mv1_0, query1_0, "mv1_0") + order_qt_query1_0_after "${query1_0}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_0""" + + + def mv1_1 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY, partsupp.PS_AVAILQTY " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "left join partsupp on lineitem.L_PARTKEY = partsupp.PS_PARTKEY " + + "and lineitem.L_SUPPKEY = partsupp.PS_SUPPKEY" + def query1_1 = "select lineitem.L_LINENUMBER " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "left join partsupp on lineitem.L_PARTKEY = partsupp.PS_PARTKEY " + + "and lineitem.L_SUPPKEY = partsupp.PS_SUPPKEY" + order_qt_query1_1_before "${query1_1}" + check_rewrite(mv1_1, query1_1, "mv1_1") + order_qt_query1_1_after "${query1_1}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_1""" + + + def mv1_2 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY " + + "from orders " + + "left join lineitem on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + def query1_2 = "select lineitem.L_LINENUMBER " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + order_qt_query1_2_before "${query1_2}" + check_not_match(mv1_2, query1_2, "mv1_2") + order_qt_query1_2_after "${query1_2}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_2""" + + // select with complex expression + from + inner join + def mv1_3 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY " + + "from orders " + + "left join lineitem on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + def query1_3 = "select IFNULL(orders.O_CUSTKEY, 0) as custkey_not_null " + + "from orders " + + "left join lineitem on orders.O_ORDERKEY = lineitem.L_ORDERKEY" + order_qt_query1_3_before "${query1_3}" + check_rewrite(mv1_3, query1_3, "mv1_3") + order_qt_query1_3_after "${query1_3}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_3""" + + + // select + from + left outer join + filter + def mv2_0 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY " + + "from orders " + + "left join lineitem on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + def query2_0 = "select lineitem.L_LINENUMBER " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "where lineitem.L_LINENUMBER > 10" + order_qt_query2_0_before "${query2_0}" + // should not match, because the join direction is not same + check_not_match(mv2_0, query2_0, "mv2_0") + order_qt_query2_0_after "${query2_0}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_0""" + + + def mv2_1 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + def query2_1 = "select lineitem.L_LINENUMBER " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "where orders.O_ORDERSTATUS = 'ok'" + order_qt_query2_1_before "${query2_1}" + // use a filed not from mv, should not success + check_not_match(mv2_1, query2_1, "mv2_1") + order_qt_query2_1_after "${query2_1}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_1""" + + + def mv2_2 = "select t1.L_LINENUMBER, orders.O_CUSTKEY " + + "from (select * from lineitem where L_LINENUMBER > 1) t1 " + + "left join orders on t1.L_ORDERKEY = orders.O_ORDERKEY " + def query2_2 = "select lineitem.L_LINENUMBER " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "where lineitem.L_LINENUMBER > 1" + order_qt_query2_2_before "${query2_2}" + check_rewrite(mv2_2, query2_2, "mv2_2") + order_qt_query2_2_after "${query2_2}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_2""" + + + def mv2_3 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY, orders.O_ORDERSTATUS " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + def query2_3 = "select lineitem.L_LINENUMBER " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "where orders.O_ORDERSTATUS = 'ok'" + order_qt_query2_3_before "${query2_3}" + // left outer -> inner jon because orders.O_ORDERSTATUS = 'ok', but mv is left join, will fix in the future +// check_rewrite(mv2_3, query2_3, "mv2_3") + order_qt_query2_3_after "${query2_3}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_3""" + + + def mv2_4 = "select lineitem.L_LINENUMBER, t2.O_CUSTKEY, t2.O_ORDERSTATUS " + + "from lineitem " + + "left join " + + "(select * from orders where O_ORDERSTATUS = 'ok') t2 " + + "on lineitem.L_ORDERKEY = t2.O_ORDERKEY " + def query2_4 = "select lineitem.L_LINENUMBER " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "where orders.O_ORDERSTATUS = 'ok'" + order_qt_query2_4_before "${query2_4}" + // should not success, as mv filter is under left outer input + check_not_match(mv2_4, query2_4, "mv2_4") + order_qt_query2_4_after "${query2_4}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_4""" + + + def mv2_5 = "select lineitem.L_LINENUMBER, orders.O_CUSTKEY " + + "from lineitem " + + "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " + + "where lineitem.L_LINENUMBER > 1" + def query2_5 = "select t1.L_LINENUMBER " + + "from (select * from lineitem where L_LINENUMBER > 1) t1 " + + "left join orders on t1.L_ORDERKEY = orders.O_ORDERKEY " + order_qt_query2_5_before "${query2_5}" + check_rewrite(mv2_5, query2_5, "mv2_5") + order_qt_query2_5_after "${query2_5}" + sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_5""" +}