This is an automated email from the ASF dual-hosted git repository. kxiao pushed a commit to branch branch-2.0 in repository https://gitbox.apache.org/repos/asf/doris.git
The following commit(s) were added to refs/heads/branch-2.0 by this push: new 73e0a1ccad3 [feature](nereids)optimize plan when column stats are unaviable (for branch 2.0) (#30314) 73e0a1ccad3 is described below commit 73e0a1ccad3522913794ba94ee16bf40cc5048fa Author: minghong <engle...@gmail.com> AuthorDate: Sat Jan 27 12:10:28 2024 +0800 [feature](nereids)optimize plan when column stats are unaviable (for branch 2.0) (#30314) --- .../org/apache/doris/nereids/StatementContext.java | 11 +++ .../org/apache/doris/nereids/cost/CostModelV1.java | 33 ++++++++ .../jobs/cascades/OptimizeGroupExpressionJob.java | 5 ++ .../java/org/apache/doris/nereids/memo/Group.java | 9 +++ .../org/apache/doris/nereids/rules/RuleSet.java | 12 +++ .../rules/exploration/join/InnerJoinLAsscom.java | 30 ++++++-- .../exploration/join/InnerJoinLAsscomProject.java | 11 ++- .../rules/exploration/join/JoinCommute.java | 14 +++- .../join/SemiJoinSemiJoinTransposeProject.java | 2 +- .../doris/nereids/rules/rewrite/ReorderJoin.java | 1 + .../apache/doris/nereids/stats/JoinEstimation.java | 28 ++++++- .../doris/nereids/stats/StatsCalculator.java | 27 ++++++- .../java/org/apache/doris/qe/SessionVariable.java | 12 +++ .../shape_no_stats/q1.out | 13 ++++ .../shape_no_stats/q10.out | 31 ++++++++ .../shape_no_stats/q11.out | 45 +++++++++++ .../shape_no_stats/q12.out | 17 +++++ .../shape_no_stats/q13.out | 21 ++++++ .../shape_no_stats/q14.out | 17 +++++ .../shape_no_stats/q15.out | 31 ++++++++ .../shape_no_stats/q16.out | 25 +++++++ .../shape_no_stats/q17.out | 21 ++++++ .../shape_no_stats/q18.out | 27 +++++++ .../shape_no_stats/q19.out | 17 +++++ .../shape_no_stats/q2.out | 35 +++++++++ .../shape_no_stats/q20-rewrite.out | 35 +++++++++ .../shape_no_stats/q20.out | 34 +++++++++ .../shape_no_stats/q21.out | 36 +++++++++ .../shape_no_stats/q22.out | 27 +++++++ .../shape_no_stats/q3.out | 25 +++++++ .../shape_no_stats/q4.out | 18 +++++ .../shape_no_stats/q5.out | 41 ++++++++++ .../shape_no_stats/q6.out | 10 +++ .../shape_no_stats/q7.out | 40 ++++++++++ .../shape_no_stats/q8.out | 51 +++++++++++++ .../shape_no_stats/q9.out | 38 ++++++++++ .../shape_no_stats/q1.groovy | 60 +++++++++++++++ .../shape_no_stats/q10.groovy | 71 ++++++++++++++++++ .../shape_no_stats/q11.groovy | 68 +++++++++++++++++ .../shape_no_stats/q12.groovy | 66 ++++++++++++++++ .../shape_no_stats/q13.groovy | 62 +++++++++++++++ .../shape_no_stats/q14.groovy | 52 +++++++++++++ .../shape_no_stats/q15.groovy | 62 +++++++++++++++ .../shape_no_stats/q16.groovy | 73 ++++++++++++++++++ .../shape_no_stats/q17.groovy | 60 +++++++++++++++ .../shape_no_stats/q18.groovy | 76 +++++++++++++++++++ .../shape_no_stats/q19.groovy | 80 ++++++++++++++++++++ .../shape_no_stats/q2.groovy | 87 ++++++++++++++++++++++ .../shape_no_stats/q20-rewrite.groovy | 71 ++++++++++++++++++ .../shape_no_stats/q20.groovy | 81 ++++++++++++++++++++ .../shape_no_stats/q21.groovy | 83 +++++++++++++++++++++ .../shape_no_stats/q22.groovy | 80 ++++++++++++++++++++ .../shape_no_stats/q3.groovy | 67 +++++++++++++++++ .../shape_no_stats/q4.groovy | 63 ++++++++++++++++ .../shape_no_stats/q5.groovy | 67 +++++++++++++++++ .../shape_no_stats/q6.groovy | 52 +++++++++++++ .../shape_no_stats/q7.groovy | 82 ++++++++++++++++++++ .../shape_no_stats/q8.groovy | 80 ++++++++++++++++++++ .../shape_no_stats/q9.groovy | 75 +++++++++++++++++++ 59 files changed, 2452 insertions(+), 16 deletions(-) diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java index 887f237d347..9db05187cbb 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java @@ -43,6 +43,7 @@ import com.google.common.collect.Maps; import com.google.common.collect.Sets; import java.util.ArrayList; +import java.util.Collection; import java.util.HashMap; import java.util.List; import java.util.Map; @@ -88,6 +89,8 @@ public class StatementContext { private final Map<CTEId, LogicalPlan> rewrittenCteConsumer = new HashMap<>(); private final Set<String> viewDdlSqlSet = Sets.newHashSet(); + // collect all hash join conditions to compute node connectivity in join graph + private final List<Expression> joinFilters = new ArrayList<>(); private final Map<String, Hint> hintMap = Maps.newLinkedHashMap(); private final List<Hint> hints = new ArrayList<>(); @@ -244,4 +247,12 @@ public class StatementContext { public List<Hint> getHints() { return ImmutableList.copyOf(hints); } + + public List<Expression> getJoinFilters() { + return joinFilters; + } + + public void addJoinFilters(Collection<Expression> newJoinFilters) { + this.joinFilters.addAll(newJoinFilters); + } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/cost/CostModelV1.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/cost/CostModelV1.java index 9d43b3c3195..9f504dad475 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/cost/CostModelV1.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/cost/CostModelV1.java @@ -18,10 +18,12 @@ package org.apache.doris.nereids.cost; import org.apache.doris.nereids.PlanContext; +import org.apache.doris.nereids.StatementContext; import org.apache.doris.nereids.properties.DistributionSpec; import org.apache.doris.nereids.properties.DistributionSpecGather; import org.apache.doris.nereids.properties.DistributionSpecHash; import org.apache.doris.nereids.properties.DistributionSpecReplicated; +import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.trees.plans.physical.PhysicalAssertNumRows; import org.apache.doris.nereids.trees.plans.physical.PhysicalDeferMaterializeOlapScan; @@ -48,6 +50,8 @@ import org.apache.doris.statistics.Statistics; import com.google.common.base.Preconditions; +import java.util.Collections; + class CostModelV1 extends PlanVisitor<Cost, PlanContext> { /** * The intuition behind `HEAVY_OPERATOR_PUNISH_FACTOR` is we need to avoid this form of join patterns: @@ -274,6 +278,25 @@ class CostModelV1 extends PlanVisitor<Cost, PlanContext> { return broadcastJoinPenalty; } + /* + in a join cluster graph, if a node has higher connectivity, it is more likely to be reduced + by runtime filters, and it is also more likely to produce effective runtime filters. + Thus, we prefer to put the node with higher connectivity on the join right side. + */ + private int computeConnectivity(Plan plan) { + int connectCount = 0; + if (ConnectContext.get() != null) { + StatementContext stmtContext = ConnectContext.get().getStatementContext(); + // in UT, stmtContext is null + if (stmtContext != null) { + for (Expression expr : stmtContext.getJoinFilters()) { + connectCount += Collections.disjoint(expr.getInputSlots(), plan.getOutputSet()) ? 0 : 1; + } + } + } + return connectCount; + } + @Override public Cost visitPhysicalHashJoin( PhysicalHashJoin<? extends Plan, ? extends Plan> physicalHashJoin, PlanContext context) { @@ -286,6 +309,16 @@ class CostModelV1 extends PlanVisitor<Cost, PlanContext> { double leftRowCount = probeStats.getRowCount(); double rightRowCount = buildStats.getRowCount(); + if (leftRowCount == rightRowCount + && physicalHashJoin.getGroupExpression().isPresent() + && physicalHashJoin.getGroupExpression().get().getOwnerGroup() != null + && !physicalHashJoin.getGroupExpression().get().getOwnerGroup().isStatsReliable()) { + int leftConnectivity = computeConnectivity(physicalHashJoin.left()); + int rightConnectivity = computeConnectivity(physicalHashJoin.right()); + if (rightConnectivity < leftConnectivity) { + leftRowCount += 1; + } + } /* pattern1: L join1 (Agg1() join2 Agg2()) result number of join2 may much less than Agg1. diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/cascades/OptimizeGroupExpressionJob.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/cascades/OptimizeGroupExpressionJob.java index 0bc7393dc0b..c53ba3c9ad8 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/cascades/OptimizeGroupExpressionJob.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/cascades/OptimizeGroupExpressionJob.java @@ -61,6 +61,9 @@ public class OptimizeGroupExpressionJob extends Job { boolean isOtherJoinReorder = context.getCascadesContext().getStatementContext().isOtherJoinReorder(); boolean isEnableBushyTree = context.getCascadesContext().getConnectContext().getSessionVariable() .isEnableBushyTree(); + boolean isLeftZigZagTree = context.getCascadesContext().getConnectContext() + .getSessionVariable().isEnableLeftZigZag() + || (groupExpression.getOwnerGroup() != null && !groupExpression.getOwnerGroup().isStatsReliable()); if (isDisableJoinReorder) { return Collections.emptyList(); } else if (isDpHyp) { @@ -69,6 +72,8 @@ public class OptimizeGroupExpressionJob extends Job { } else { return Collections.emptyList(); } + } else if (isLeftZigZagTree) { + return getRuleSet().getLeftZigZagTreeJoinReorder(); } else if (isEnableBushyTree) { return getRuleSet().getBushyTreeJoinReorder(); } else if (context.getCascadesContext().getStatementContext().getMaxNAryInnerJoin() <= 5) { diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Group.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Group.java index c816151731d..e6b2256b410 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Group.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Group.java @@ -60,6 +60,7 @@ public class Group { private final List<GroupExpression> physicalExpressions = Lists.newArrayList(); private final List<GroupExpression> enforcers = Lists.newArrayList(); + private boolean isStatsReliable = true; private LogicalProperties logicalProperties; // Map of cost lower bounds @@ -115,6 +116,14 @@ public class Group { return groupExpression; } + public void setStatsReliable(boolean statsReliable) { + this.isStatsReliable = statsReliable; + } + + public boolean isStatsReliable() { + return isStatsReliable; + } + public void addLogicalExpression(GroupExpression groupExpression) { groupExpression.setOwnerGroup(this); logicalExpressions.add(groupExpression); 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 05846fbd123..81ae5530192 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 @@ -180,6 +180,14 @@ public class RuleSet { .add(new LogicalDeferMaterializeResultSinkToPhysicalDeferMaterializeResultSink()) .build(); + // left-zig-zag tree is used when column stats are not available. + public static final List<Rule> LEFT_ZIG_ZAG_TREE_JOIN_REORDER = planRuleFactories() + .add(JoinCommute.LEFT_ZIG_ZAG) + .add(InnerJoinLAsscom.LEFT_ZIG_ZAG) + .add(InnerJoinLAsscomProject.LEFT_ZIG_ZAG) + .addAll(OTHER_REORDER_RULES) + .build(); + public static final List<Rule> ZIG_ZAG_TREE_JOIN_REORDER = planRuleFactories() .add(JoinCommute.ZIG_ZAG) .add(InnerJoinLAsscom.INSTANCE) @@ -216,6 +224,10 @@ public class RuleSet { return DPHYP_REORDER_RULES; } + public List<Rule> getLeftZigZagTreeJoinReorder() { + return LEFT_ZIG_ZAG_TREE_JOIN_REORDER; + } + public List<Rule> getZigZagTreeJoinReorder() { return ZIG_ZAG_TREE_JOIN_REORDER_RULES; } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/InnerJoinLAsscom.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/InnerJoinLAsscom.java index fa90adf6303..d7d81e3461c 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/InnerJoinLAsscom.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/InnerJoinLAsscom.java @@ -36,7 +36,13 @@ import java.util.stream.Collectors; * Rule for change inner join LAsscom (associative and commutive). */ public class InnerJoinLAsscom extends OneExplorationRuleFactory { - public static final InnerJoinLAsscom INSTANCE = new InnerJoinLAsscom(); + public static final InnerJoinLAsscom INSTANCE = new InnerJoinLAsscom(false); + public static final InnerJoinLAsscom LEFT_ZIG_ZAG = new InnerJoinLAsscom(true); + private boolean leftZigZag = false; + + public InnerJoinLAsscom(boolean leftZigZag) { + this.leftZigZag = leftZigZag; + } /* * topJoin newTopJoin @@ -48,7 +54,7 @@ public class InnerJoinLAsscom extends OneExplorationRuleFactory { @Override public Rule build() { return innerLogicalJoin(innerLogicalJoin(), group()) - .when(topJoin -> checkReorder(topJoin, topJoin.left())) + .when(topJoin -> checkReorder(topJoin, topJoin.left(), leftZigZag)) .whenNot(join -> join.hasJoinHint() || join.left().hasJoinHint()) .whenNot(join -> join.isMarkJoin() || join.left().isMarkJoin()) .then(topJoin -> { @@ -88,11 +94,23 @@ public class InnerJoinLAsscom extends OneExplorationRuleFactory { }).toRule(RuleType.LOGICAL_INNER_JOIN_LASSCOM); } + /** + * + * check reorder + */ public static boolean checkReorder(LogicalJoin<? extends Plan, GroupPlan> topJoin, - LogicalJoin<GroupPlan, GroupPlan> bottomJoin) { - return !bottomJoin.getJoinReorderContext().hasCommuteZigZag() - && !topJoin.getJoinReorderContext().hasLAsscom() - && (!bottomJoin.isMarkJoin() && !topJoin.isMarkJoin()); + LogicalJoin<GroupPlan, GroupPlan> bottomJoin, boolean leftZigZag) { + if (leftZigZag) { + double bRows = bottomJoin.right().getGroup().getStatistics().getRowCount(); + double cRows = topJoin.right().getGroup().getStatistics().getRowCount(); + return bRows < cRows && !bottomJoin.getJoinReorderContext().hasCommuteZigZag() + && !topJoin.getJoinReorderContext().hasLAsscom() + && (!bottomJoin.isMarkJoin() && !topJoin.isMarkJoin()); + } else { + return !bottomJoin.getJoinReorderContext().hasCommuteZigZag() + && !topJoin.getJoinReorderContext().hasLAsscom() + && (!bottomJoin.isMarkJoin() && !topJoin.isMarkJoin()); + } } /** diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/InnerJoinLAsscomProject.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/InnerJoinLAsscomProject.java index 1c73a41a945..3587354fd57 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/InnerJoinLAsscomProject.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/InnerJoinLAsscomProject.java @@ -40,7 +40,14 @@ import java.util.stream.Collectors; * Rule for change inner join LAsscom (associative and commutive). */ public class InnerJoinLAsscomProject extends OneExplorationRuleFactory { - public static final InnerJoinLAsscomProject INSTANCE = new InnerJoinLAsscomProject(); + public static final InnerJoinLAsscomProject INSTANCE = new InnerJoinLAsscomProject(false); + public static final InnerJoinLAsscomProject LEFT_ZIG_ZAG = new InnerJoinLAsscomProject(true); + private final boolean enableLeftZigZag; + + public InnerJoinLAsscomProject(boolean enableLeftZigZag) { + this.enableLeftZigZag = enableLeftZigZag; + } + /* * topJoin newTopJoin @@ -54,7 +61,7 @@ public class InnerJoinLAsscomProject extends OneExplorationRuleFactory { @Override public Rule build() { return innerLogicalJoin(logicalProject(innerLogicalJoin()), group()) - .when(topJoin -> InnerJoinLAsscom.checkReorder(topJoin, topJoin.left().child())) + .when(topJoin -> InnerJoinLAsscom.checkReorder(topJoin, topJoin.left().child(), enableLeftZigZag)) .whenNot(join -> join.hasJoinHint() || join.left().child().hasJoinHint()) .whenNot(join -> join.isMarkJoin() || join.left().child().isMarkJoin()) .when(join -> join.left().isAllSlots()) diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/JoinCommute.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/JoinCommute.java index c231640a798..8dd3e6abe71 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/JoinCommute.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/JoinCommute.java @@ -38,6 +38,7 @@ import java.util.List; public class JoinCommute extends OneExplorationRuleFactory { public static final JoinCommute LEFT_DEEP = new JoinCommute(SwapType.LEFT_DEEP, false); + public static final JoinCommute LEFT_ZIG_ZAG = new JoinCommute(SwapType.LEFT_ZIG_ZAG, false); public static final JoinCommute ZIG_ZAG = new JoinCommute(SwapType.ZIG_ZAG, false); public static final JoinCommute BUSHY = new JoinCommute(SwapType.BUSHY, false); public static final JoinCommute NON_INNER = new JoinCommute(SwapType.BUSHY, true); @@ -72,7 +73,8 @@ public class JoinCommute extends OneExplorationRuleFactory { } enum SwapType { - LEFT_DEEP, ZIG_ZAG, BUSHY + LEFT_DEEP, ZIG_ZAG, BUSHY, + LEFT_ZIG_ZAG } /** @@ -87,6 +89,12 @@ public class JoinCommute extends OneExplorationRuleFactory { return false; } + if (swapType == SwapType.LEFT_ZIG_ZAG) { + double leftRows = join.left().getGroup().getStatistics().getRowCount(); + double rightRows = join.right().getGroup().getStatistics().getRowCount(); + return leftRows <= rightRows && isZigZagJoin(join); + } + return !join.getJoinReorderContext().hasCommute() && !join.getJoinReorderContext().hasExchange(); } @@ -95,6 +103,10 @@ public class JoinCommute extends OneExplorationRuleFactory { return containJoin(join.left()) || containJoin(join.right()); } + public static boolean isZigZagJoin(LogicalJoin<GroupPlan, GroupPlan> join) { + return !containJoin(join.left()) || !containJoin(join.right()); + } + private static boolean containJoin(GroupPlan groupPlan) { // TODO: tmp way to judge containJoin List<Slot> output = groupPlan.getOutput(); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinSemiJoinTransposeProject.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinSemiJoinTransposeProject.java index 0db1c800871..13e9c7cd461 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinSemiJoinTransposeProject.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinSemiJoinTransposeProject.java @@ -53,7 +53,7 @@ public class SemiJoinSemiJoinTransposeProject extends OneExplorationRuleFactory public Rule build() { return logicalJoin(logicalProject(logicalJoin()), group()) .when(this::typeChecker) - .when(topSemi -> InnerJoinLAsscom.checkReorder(topSemi, topSemi.left().child())) + .when(topSemi -> InnerJoinLAsscom.checkReorder(topSemi, topSemi.left().child(), false)) .whenNot(join -> join.hasJoinHint() || join.left().child().hasJoinHint()) .whenNot(join -> join.isMarkJoin() || join.left().child().isMarkJoin()) .when(join -> join.left().isAllSlots()) diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ReorderJoin.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ReorderJoin.java index 9a5c7d833c6..03c4e7e8ed8 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ReorderJoin.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ReorderJoin.java @@ -87,6 +87,7 @@ public class ReorderJoin extends OneRewriteRuleFactory { Plan plan = joinToMultiJoin(filter, planToHintType); Preconditions.checkState(plan instanceof MultiJoin); MultiJoin multiJoin = (MultiJoin) plan; + ctx.statementContext.addJoinFilters(multiJoin.getJoinFilter()); ctx.statementContext.setMaxNAryInnerJoin(multiJoin.children().size()); Plan after = multiJoinToJoin(multiJoin, planToHintType); return after; diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/JoinEstimation.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/JoinEstimation.java index f9d25cab171..1776bb33cd1 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/JoinEstimation.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/JoinEstimation.java @@ -20,6 +20,7 @@ package org.apache.doris.nereids.stats; import org.apache.doris.nereids.exceptions.AnalysisException; import org.apache.doris.nereids.trees.expressions.Cast; import org.apache.doris.nereids.trees.expressions.EqualPredicate; +import org.apache.doris.nereids.trees.expressions.EqualTo; import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.expressions.Slot; import org.apache.doris.nereids.trees.plans.JoinType; @@ -44,6 +45,7 @@ import java.util.stream.Collectors; */ public class JoinEstimation { private static double DEFAULT_ANTI_JOIN_SELECTIVITY_COEFFICIENT = 0.3; + private static double UNKNOWN_COL_STATS_FILTER_SEL_LOWER_BOUND = 0.5; private static EqualPredicate normalizeHashJoinCondition(EqualPredicate equal, Statistics leftStats, Statistics rightStats) { @@ -154,6 +156,27 @@ public class JoinEstimation { .build(); } + private static double computeSelectivityForBuildSideWhenColStatsUnknown(Statistics buildStats, Join join) { + double sel = 1.0; + for (Expression cond : join.getHashJoinConjuncts()) { + if (cond instanceof EqualTo) { + EqualTo equal = (EqualTo) cond; + if (equal.left() instanceof Slot && equal.right() instanceof Slot) { + ColumnStatistic buildColStats = buildStats.findColumnStatistics(equal.left()); + if (buildColStats == null) { + buildColStats = buildStats.findColumnStatistics(equal.right()); + } + if (buildColStats != null) { + double buildSel = Math.min(buildStats.getRowCount() / buildColStats.count, 1.0); + buildSel = Math.max(buildSel, UNKNOWN_COL_STATS_FILTER_SEL_LOWER_BOUND); + sel = Math.min(sel, buildSel); + } + } + } + } + return sel; + } + private static Statistics estimateInnerJoin(Statistics leftStats, Statistics rightStats, Join join) { if (hashJoinConditionContainsUnknownColumnStats(leftStats, rightStats, join)) { double rowCount = Math.max(leftStats.getRowCount(), rightStats.getRowCount()); @@ -230,14 +253,15 @@ public class JoinEstimation { private static Statistics estimateSemiOrAnti(Statistics leftStats, Statistics rightStats, Join join) { if (hashJoinConditionContainsUnknownColumnStats(leftStats, rightStats, join)) { + double sel = computeSelectivityForBuildSideWhenColStatsUnknown(rightStats, join); if (join.getJoinType().isLeftSemiOrAntiJoin()) { - return new StatisticsBuilder().setRowCount(leftStats.getRowCount()) + return new StatisticsBuilder().setRowCount(leftStats.getRowCount() * sel) .putColumnStatistics(leftStats.columnStatistics()) .putColumnStatistics(rightStats.columnStatistics()) .build(); } else { //right semi or anti - return new StatisticsBuilder().setRowCount(rightStats.getRowCount()) + return new StatisticsBuilder().setRowCount(rightStats.getRowCount() * sel) .putColumnStatistics(leftStats.columnStatistics()) .putColumnStatistics(rightStats.columnStatistics()) .build(); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/StatsCalculator.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/StatsCalculator.java index 64f85c47a80..eabb8b14fb9 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/StatsCalculator.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/stats/StatsCalculator.java @@ -227,6 +227,9 @@ public class StatsCalculator extends DefaultPlanVisitor<Statistics, Void> { we record the lowest expression cost as group cost to avoid missing this group. */ if (originStats == null || originStats.getRowCount() > stats.getRowCount()) { + boolean isReliable = groupExpression.getPlan().getExpressions().stream() + .noneMatch(e -> stats.isInputSlotsUnknown(e.getInputSlots())); + groupExpression.getOwnerGroup().setStatsReliable(isReliable); groupExpression.getOwnerGroup().setStatistics(stats); } else { // the reason why we update col stats here. @@ -629,8 +632,7 @@ public class StatsCalculator extends DefaultPlanVisitor<Statistics, Void> { throw new RuntimeException(String.format("Invalid slot: %s", slotReference.getExprId())); } ColumnStatistic cache; - if (ConnectContext.get() == null || !ConnectContext.get().getSessionVariable().enableStats - || !FeConstants.enableInternalSchemaDb + if (!FeConstants.enableInternalSchemaDb || shouldIgnoreThisCol) { cache = ColumnStatistic.UNKNOWN; } else { @@ -644,9 +646,26 @@ public class StatsCalculator extends DefaultPlanVisitor<Statistics, Void> { if (!cache.isUnKnown) { rowCount = Math.max(rowCount, cache.count); } - columnStatisticMap.put(slotReference, cache); + if (ConnectContext.get() != null && ConnectContext.get().getSessionVariable().enableStats) { + columnStatisticMap.put(slotReference, cache); + } else { + columnStatisticMap.put(slotReference, ColumnStatistic.UNKNOWN); + } + } + Statistics stats = new Statistics(rowCount, columnStatisticMap); + stats = normalizeCatalogRelationColumnStatsRowCount(stats); + return stats; + } + + private Statistics normalizeCatalogRelationColumnStatsRowCount(Statistics stats) { + for (Expression slot : stats.columnStatistics().keySet()) { + ColumnStatistic colStats = stats.findColumnStatistics(slot); + Preconditions.checkArgument(colStats != null, + "can not find col stats for %s in table", slot.toSql()); + stats.addColumnStats(slot, + new ColumnStatisticBuilder(colStats).setCount(stats.getRowCount()).build()); } - return new Statistics(rowCount, columnStatisticMap); + return stats; } private Statistics computeTopN(TopN topN) { diff --git a/fe/fe-core/src/main/java/org/apache/doris/qe/SessionVariable.java b/fe/fe-core/src/main/java/org/apache/doris/qe/SessionVariable.java index 227b5b98f88..036d8da6836 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/qe/SessionVariable.java +++ b/fe/fe-core/src/main/java/org/apache/doris/qe/SessionVariable.java @@ -240,6 +240,7 @@ public class SessionVariable implements Serializable, Writable { public static final String NTH_OPTIMIZED_PLAN = "nth_optimized_plan"; public static final String ENABLE_NEREIDS_PLANNER = "enable_nereids_planner"; + public static final String ENABLE_LEFT_ZIG_ZAG = "enable_left_zig_zag"; public static final String DISABLE_NEREIDS_RULES = "disable_nereids_rules"; public static final String ENABLE_NEW_COST_MODEL = "enable_new_cost_model"; public static final String ENABLE_FALLBACK_TO_ORIGINAL_PLANNER = "enable_fallback_to_original_planner"; @@ -1023,6 +1024,17 @@ public class SessionVariable implements Serializable, Writable { checker = "checkExternalAggBytesThreshold", fuzzy = true) public long externalAggBytesThreshold = 0; + public boolean isEnableLeftZigZag() { + return enableLeftZigZag; + } + + public void setEnableLeftZigZag(boolean enableLeftZigZag) { + this.enableLeftZigZag = enableLeftZigZag; + } + + @VariableMgr.VarAttr(name = ENABLE_LEFT_ZIG_ZAG) + private boolean enableLeftZigZag = false; + public static final int MIN_EXTERNAL_AGG_PARTITION_BITS = 4; public static final int MAX_EXTERNAL_AGG_PARTITION_BITS = 8; @VariableMgr.VarAttr(name = EXTERNAL_AGG_PARTITION_BITS, diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q1.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q1.out new file mode 100644 index 00000000000..6d8f63d340c --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q1.out @@ -0,0 +1,13 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------hashAgg[GLOBAL] +----------PhysicalDistribute +------------hashAgg[LOCAL] +--------------PhysicalProject +----------------filter((lineitem.l_shipdate <= '1998-09-02')) +------------------PhysicalOlapScan[lineitem] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q10.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q10.out new file mode 100644 index 00000000000..7c65ed9171f --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q10.out @@ -0,0 +1,31 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalTopN +----PhysicalDistribute +------PhysicalTopN +--------PhysicalProject +----------hashAgg[GLOBAL] +------------PhysicalDistribute +--------------hashAgg[LOCAL] +----------------PhysicalProject +------------------hashJoin[INNER_JOIN](customer.c_nationkey = nation.n_nationkey) +--------------------PhysicalProject +----------------------hashJoin[INNER_JOIN](lineitem.l_orderkey = orders.o_orderkey) +------------------------PhysicalProject +--------------------------filter((lineitem.l_returnflag = 'R')) +----------------------------PhysicalOlapScan[lineitem] +------------------------PhysicalDistribute +--------------------------PhysicalProject +----------------------------hashJoin[INNER_JOIN](customer.c_custkey = orders.o_custkey) +------------------------------PhysicalDistribute +--------------------------------PhysicalProject +----------------------------------filter((orders.o_orderdate < '1994-01-01')(orders.o_orderdate >= '1993-10-01')) +------------------------------------PhysicalOlapScan[orders] +------------------------------PhysicalDistribute +--------------------------------PhysicalProject +----------------------------------PhysicalOlapScan[customer] +--------------------PhysicalDistribute +----------------------PhysicalProject +------------------------PhysicalOlapScan[nation] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q11.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q11.out new file mode 100644 index 00000000000..585fbbff52c --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q11.out @@ -0,0 +1,45 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------PhysicalProject +----------NestedLoopJoin[INNER_JOIN](cast(value as DOUBLE) > cast((sum((ps_supplycost * cast(ps_availqty as DECIMALV3(10, 0)))) * 0.000002) as DOUBLE)) +------------hashAgg[GLOBAL] +--------------PhysicalDistribute +----------------hashAgg[LOCAL] +------------------PhysicalProject +--------------------hashJoin[INNER_JOIN](supplier.s_nationkey = nation.n_nationkey) +----------------------PhysicalProject +------------------------hashJoin[INNER_JOIN](partsupp.ps_suppkey = supplier.s_suppkey) +--------------------------PhysicalDistribute +----------------------------PhysicalProject +------------------------------PhysicalOlapScan[partsupp] +--------------------------PhysicalDistribute +----------------------------PhysicalProject +------------------------------PhysicalOlapScan[supplier] +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------filter((nation.n_name = 'GERMANY')) +----------------------------PhysicalOlapScan[nation] +------------PhysicalDistribute +--------------PhysicalProject +----------------hashAgg[GLOBAL] +------------------PhysicalDistribute +--------------------hashAgg[LOCAL] +----------------------PhysicalProject +------------------------hashJoin[INNER_JOIN](supplier.s_nationkey = nation.n_nationkey) +--------------------------PhysicalProject +----------------------------hashJoin[INNER_JOIN](partsupp.ps_suppkey = supplier.s_suppkey) +------------------------------PhysicalDistribute +--------------------------------PhysicalProject +----------------------------------PhysicalOlapScan[partsupp] +------------------------------PhysicalDistribute +--------------------------------PhysicalProject +----------------------------------PhysicalOlapScan[supplier] +--------------------------PhysicalDistribute +----------------------------PhysicalProject +------------------------------filter((nation.n_name = 'GERMANY')) +--------------------------------PhysicalOlapScan[nation] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q12.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q12.out new file mode 100644 index 00000000000..045e260d0d0 --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q12.out @@ -0,0 +1,17 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------hashAgg[GLOBAL] +----------PhysicalDistribute +------------hashAgg[LOCAL] +--------------PhysicalProject +----------------hashJoin[INNER_JOIN](orders.o_orderkey = lineitem.l_orderkey) +------------------PhysicalProject +--------------------PhysicalOlapScan[orders] +------------------PhysicalProject +--------------------filter(l_shipmode IN ('MAIL', 'SHIP')(lineitem.l_shipdate < lineitem.l_commitdate)(lineitem.l_receiptdate < '1995-01-01')(lineitem.l_receiptdate >= '1994-01-01')(lineitem.l_commitdate < lineitem.l_receiptdate)) +----------------------PhysicalOlapScan[lineitem] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q13.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q13.out new file mode 100644 index 00000000000..ab3e4ce172b --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q13.out @@ -0,0 +1,21 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------hashAgg[GLOBAL] +----------PhysicalDistribute +------------hashAgg[LOCAL] +--------------PhysicalProject +----------------hashAgg[LOCAL] +------------------PhysicalProject +--------------------hashJoin[RIGHT_OUTER_JOIN](customer.c_custkey = orders.o_custkey) +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------filter(( not (o_comment like '%special%requests%'))) +----------------------------PhysicalOlapScan[orders] +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------PhysicalOlapScan[customer] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q14.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q14.out new file mode 100644 index 00000000000..d0da3ce9aa2 --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q14.out @@ -0,0 +1,17 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalProject +----hashAgg[GLOBAL] +------PhysicalDistribute +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN](lineitem.l_partkey = part.p_partkey) +--------------PhysicalDistribute +----------------PhysicalProject +------------------filter((lineitem.l_shipdate < '1995-10-01')(lineitem.l_shipdate >= '1995-09-01')) +--------------------PhysicalOlapScan[lineitem] +--------------PhysicalDistribute +----------------PhysicalProject +------------------PhysicalOlapScan[part] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q15.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q15.out new file mode 100644 index 00000000000..6d60f8b8052 --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q15.out @@ -0,0 +1,31 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------PhysicalProject +----------hashJoin[INNER_JOIN](supplier.s_suppkey = revenue0.supplier_no) +------------hashJoin[INNER_JOIN](revenue0.total_revenue = max(total_revenue)) +--------------PhysicalProject +----------------hashAgg[GLOBAL] +------------------PhysicalDistribute +--------------------hashAgg[LOCAL] +----------------------PhysicalProject +------------------------filter((lineitem.l_shipdate >= '1996-01-01')(lineitem.l_shipdate < '1996-04-01')) +--------------------------PhysicalOlapScan[lineitem] +--------------PhysicalDistribute +----------------hashAgg[GLOBAL] +------------------PhysicalDistribute +--------------------hashAgg[LOCAL] +----------------------PhysicalProject +------------------------hashAgg[GLOBAL] +--------------------------PhysicalDistribute +----------------------------hashAgg[LOCAL] +------------------------------PhysicalProject +--------------------------------filter((lineitem.l_shipdate >= '1996-01-01')(lineitem.l_shipdate < '1996-04-01')) +----------------------------------PhysicalOlapScan[lineitem] +------------PhysicalDistribute +--------------PhysicalProject +----------------PhysicalOlapScan[supplier] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q16.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q16.out new file mode 100644 index 00000000000..6eabacb321d --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q16.out @@ -0,0 +1,25 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------hashAgg[GLOBAL] +----------PhysicalDistribute +------------hashAgg[LOCAL] +--------------PhysicalProject +----------------hashJoin[INNER_JOIN](part.p_partkey = partsupp.ps_partkey) +------------------PhysicalDistribute +--------------------hashJoin[LEFT_ANTI_JOIN](partsupp.ps_suppkey = supplier.s_suppkey) +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------PhysicalOlapScan[partsupp] +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------filter((s_comment like '%Customer%Complaints%')) +----------------------------PhysicalOlapScan[supplier] +------------------PhysicalDistribute +--------------------PhysicalProject +----------------------filter(( not (p_type like 'MEDIUM POLISHED%'))( not (p_brand = 'Brand#45'))p_size IN (3, 9, 14, 19, 23, 36, 45, 49)) +------------------------PhysicalOlapScan[part] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q17.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q17.out new file mode 100644 index 00000000000..1649f86cbd4 --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q17.out @@ -0,0 +1,21 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalProject +----hashAgg[GLOBAL] +------PhysicalDistribute +--------hashAgg[LOCAL] +----------PhysicalProject +------------filter((cast(l_quantity as DECIMALV3(38, 5)) < (0.2 * avg(cast(l_quantity as DECIMALV3(17, 4))) OVER(PARTITION BY p_partkey)))) +--------------PhysicalWindow +----------------PhysicalQuickSort +------------------PhysicalProject +--------------------hashJoin[INNER_JOIN](part.p_partkey = lineitem.l_partkey) +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------PhysicalOlapScan[lineitem] +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------filter((part.p_container = 'MED BOX')(part.p_brand = 'Brand#23')) +----------------------------PhysicalOlapScan[part] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q18.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q18.out new file mode 100644 index 00000000000..93e539d9bbc --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q18.out @@ -0,0 +1,27 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalTopN +----PhysicalDistribute +------PhysicalTopN +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN](orders.o_orderkey = lineitem.l_orderkey) +--------------PhysicalProject +----------------PhysicalOlapScan[lineitem] +--------------PhysicalDistribute +----------------PhysicalProject +------------------hashJoin[INNER_JOIN](customer.c_custkey = orders.o_custkey) +--------------------PhysicalDistribute +----------------------hashJoin[LEFT_SEMI_JOIN](orders.o_orderkey = lineitem.l_orderkey) +------------------------PhysicalProject +--------------------------PhysicalOlapScan[orders] +------------------------PhysicalProject +--------------------------filter((sum(l_quantity) > 300.00)) +----------------------------hashAgg[LOCAL] +------------------------------PhysicalProject +--------------------------------PhysicalOlapScan[lineitem] +--------------------PhysicalDistribute +----------------------PhysicalProject +------------------------PhysicalOlapScan[customer] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q19.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q19.out new file mode 100644 index 00000000000..47874ac1c16 --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q19.out @@ -0,0 +1,17 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--hashAgg[GLOBAL] +----PhysicalDistribute +------hashAgg[LOCAL] +--------PhysicalProject +----------hashJoin[INNER_JOIN](part.p_partkey = lineitem.l_partkey)((((((part.p_brand = 'Brand#12') AND p_container IN ('SM CASE', 'SM BOX', 'SM PACK', 'SM PKG')) AND ((lineitem.l_quantity >= 1.00) AND (lineitem.l_quantity <= 11.00))) AND (part.p_size <= 5)) OR ((((part.p_brand = 'Brand#23') AND p_container IN ('MED BAG', 'MED BOX', 'MED PKG', 'MED PACK')) AND ((lineitem.l_quantity >= 10.00) AND (lineitem.l_quantity <= 20.00))) AND (part.p_size <= 10))) OR ((((part.p_brand = 'Brand#34') [...] +------------PhysicalDistribute +--------------PhysicalProject +----------------filter(l_shipmode IN ('AIR', 'AIR REG')(lineitem.l_shipinstruct = 'DELIVER IN PERSON')((((lineitem.l_quantity >= 1.00) AND (lineitem.l_quantity <= 11.00)) OR ((lineitem.l_quantity >= 10.00) AND (lineitem.l_quantity <= 20.00))) OR ((lineitem.l_quantity >= 20.00) AND (lineitem.l_quantity <= 30.00)))) +------------------PhysicalOlapScan[lineitem] +------------PhysicalDistribute +--------------PhysicalProject +----------------filter((part.p_size >= 1)(((((part.p_brand = 'Brand#12') AND p_container IN ('SM CASE', 'SM BOX', 'SM PACK', 'SM PKG')) AND (part.p_size <= 5)) OR (((part.p_brand = 'Brand#23') AND p_container IN ('MED BAG', 'MED BOX', 'MED PKG', 'MED PACK')) AND (part.p_size <= 10))) OR (((part.p_brand = 'Brand#34') AND p_container IN ('LG CASE', 'LG BOX', 'LG PACK', 'LG PKG')) AND (part.p_size <= 15)))) +------------------PhysicalOlapScan[part] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q2.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q2.out new file mode 100644 index 00000000000..5501599e35a --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q2.out @@ -0,0 +1,35 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalTopN +----PhysicalDistribute +------PhysicalTopN +--------PhysicalProject +----------filter((partsupp.ps_supplycost = min(ps_supplycost) OVER(PARTITION BY p_partkey))) +------------PhysicalWindow +--------------PhysicalQuickSort +----------------PhysicalDistribute +------------------PhysicalProject +--------------------hashJoin[INNER_JOIN](nation.n_regionkey = region.r_regionkey) +----------------------PhysicalProject +------------------------hashJoin[INNER_JOIN](supplier.s_nationkey = nation.n_nationkey) +--------------------------PhysicalProject +----------------------------hashJoin[INNER_JOIN](supplier.s_suppkey = partsupp.ps_suppkey) +------------------------------PhysicalDistribute +--------------------------------PhysicalProject +----------------------------------hashJoin[INNER_JOIN](part.p_partkey = partsupp.ps_partkey) +------------------------------------PhysicalProject +--------------------------------------PhysicalOlapScan[partsupp] +------------------------------------PhysicalProject +--------------------------------------filter((part.p_size = 15)(p_type like '%BRASS')) +----------------------------------------PhysicalOlapScan[part] +------------------------------PhysicalDistribute +--------------------------------PhysicalOlapScan[supplier] +--------------------------PhysicalDistribute +----------------------------PhysicalProject +------------------------------PhysicalOlapScan[nation] +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------filter((region.r_name = 'EUROPE')) +----------------------------PhysicalOlapScan[region] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20-rewrite.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20-rewrite.out new file mode 100644 index 00000000000..38af22ce935 --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20-rewrite.out @@ -0,0 +1,35 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------PhysicalProject +----------hashJoin[INNER_JOIN](supplier.s_nationkey = nation.n_nationkey) +------------PhysicalProject +--------------hashJoin[RIGHT_SEMI_JOIN](supplier.s_suppkey = t3.ps_suppkey) +----------------PhysicalDistribute +------------------PhysicalProject +--------------------hashJoin[INNER_JOIN](t2.l_partkey = t1.ps_partkey)(t2.l_suppkey = t1.ps_suppkey)(cast(ps_availqty as DECIMALV3(38, 3)) > t2.l_q) +----------------------hashJoin[LEFT_SEMI_JOIN](partsupp.ps_partkey = part.p_partkey) +------------------------PhysicalProject +--------------------------PhysicalOlapScan[partsupp] +------------------------PhysicalProject +--------------------------filter((p_name like 'forest%')) +----------------------------PhysicalOlapScan[part] +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------hashAgg[GLOBAL] +----------------------------PhysicalDistribute +------------------------------hashAgg[LOCAL] +--------------------------------PhysicalProject +----------------------------------filter((lineitem.l_shipdate < '1995-01-01')(lineitem.l_shipdate >= '1994-01-01')) +------------------------------------PhysicalOlapScan[lineitem] +----------------PhysicalDistribute +------------------PhysicalProject +--------------------PhysicalOlapScan[supplier] +------------PhysicalDistribute +--------------PhysicalProject +----------------filter((nation.n_name = 'CANADA')) +------------------PhysicalOlapScan[nation] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20.out new file mode 100644 index 00000000000..147cc2ec5f0 --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20.out @@ -0,0 +1,34 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------PhysicalProject +----------hashJoin[INNER_JOIN](supplier.s_nationkey = nation.n_nationkey) +------------PhysicalProject +--------------hashJoin[RIGHT_SEMI_JOIN](supplier.s_suppkey = partsupp.ps_suppkey) +----------------PhysicalDistribute +------------------PhysicalProject +--------------------hashJoin[INNER_JOIN](lineitem.l_partkey = partsupp.ps_partkey)(lineitem.l_suppkey = partsupp.ps_suppkey)(cast(ps_availqty as DECIMALV3(38, 3)) > (0.5 * sum(l_quantity))) +----------------------hashJoin[LEFT_SEMI_JOIN](partsupp.ps_partkey = part.p_partkey) +------------------------PhysicalProject +--------------------------PhysicalOlapScan[partsupp] +------------------------PhysicalProject +--------------------------filter((p_name like 'forest%')) +----------------------------PhysicalOlapScan[part] +----------------------PhysicalDistribute +------------------------hashAgg[GLOBAL] +--------------------------PhysicalDistribute +----------------------------hashAgg[LOCAL] +------------------------------PhysicalProject +--------------------------------filter((lineitem.l_shipdate < '1995-01-01')(lineitem.l_shipdate >= '1994-01-01')) +----------------------------------PhysicalOlapScan[lineitem] +----------------PhysicalDistribute +------------------PhysicalProject +--------------------PhysicalOlapScan[supplier] +------------PhysicalDistribute +--------------PhysicalProject +----------------filter((nation.n_name = 'CANADA')) +------------------PhysicalOlapScan[nation] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q21.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q21.out new file mode 100644 index 00000000000..3017266befd --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q21.out @@ -0,0 +1,36 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalTopN +----PhysicalDistribute +------PhysicalTopN +--------hashAgg[GLOBAL] +----------PhysicalDistribute +------------hashAgg[LOCAL] +--------------PhysicalProject +----------------hashJoin[INNER_JOIN](supplier.s_nationkey = nation.n_nationkey) +------------------PhysicalProject +--------------------hashJoin[INNER_JOIN](supplier.s_suppkey = l1.l_suppkey) +----------------------PhysicalDistribute +------------------------hashJoin[INNER_JOIN](orders.o_orderkey = l1.l_orderkey) +--------------------------PhysicalProject +----------------------------filter((orders.o_orderstatus = 'F')) +------------------------------PhysicalOlapScan[orders] +--------------------------hashJoin[RIGHT_SEMI_JOIN](l2.l_orderkey = l1.l_orderkey)( not (l_suppkey = l_suppkey)) +----------------------------PhysicalProject +------------------------------PhysicalOlapScan[lineitem] +----------------------------hashJoin[RIGHT_ANTI_JOIN](l3.l_orderkey = l1.l_orderkey)( not (l_suppkey = l_suppkey)) +------------------------------PhysicalProject +--------------------------------filter((l3.l_receiptdate > l3.l_commitdate)) +----------------------------------PhysicalOlapScan[lineitem] +------------------------------PhysicalProject +--------------------------------filter((l1.l_receiptdate > l1.l_commitdate)) +----------------------------------PhysicalOlapScan[lineitem] +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------PhysicalOlapScan[supplier] +------------------PhysicalDistribute +--------------------PhysicalProject +----------------------filter((nation.n_name = 'SAUDI ARABIA')) +------------------------PhysicalOlapScan[nation] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q22.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q22.out new file mode 100644 index 00000000000..7845eba2baf --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q22.out @@ -0,0 +1,27 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------hashAgg[GLOBAL] +----------PhysicalDistribute +------------hashAgg[LOCAL] +--------------PhysicalProject +----------------hashJoin[RIGHT_ANTI_JOIN](orders.o_custkey = customer.c_custkey) +------------------PhysicalDistribute +--------------------PhysicalProject +----------------------PhysicalOlapScan[orders] +------------------PhysicalDistribute +--------------------NestedLoopJoin[INNER_JOIN](cast(c_acctbal as DECIMALV3(38, 4)) > avg(cast(c_acctbal as DECIMALV3(17, 4)))) +----------------------PhysicalProject +------------------------filter(substring(c_phone, 1, 2) IN ('13', '31', '23', '29', '30', '18', '17')) +--------------------------PhysicalOlapScan[customer] +----------------------PhysicalDistribute +------------------------hashAgg[GLOBAL] +--------------------------PhysicalDistribute +----------------------------hashAgg[LOCAL] +------------------------------PhysicalProject +--------------------------------filter((customer.c_acctbal > 0.00)substring(c_phone, 1, 2) IN ('13', '31', '23', '29', '30', '18', '17')) +----------------------------------PhysicalOlapScan[customer] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q3.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q3.out new file mode 100644 index 00000000000..3ca7b3d419b --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q3.out @@ -0,0 +1,25 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalTopN +----PhysicalDistribute +------PhysicalTopN +--------PhysicalProject +----------hashAgg[LOCAL] +------------PhysicalProject +--------------hashJoin[INNER_JOIN](lineitem.l_orderkey = orders.o_orderkey) +----------------PhysicalProject +------------------filter((lineitem.l_shipdate > '1995-03-15')) +--------------------PhysicalOlapScan[lineitem] +----------------PhysicalDistribute +------------------PhysicalProject +--------------------hashJoin[INNER_JOIN](customer.c_custkey = orders.o_custkey) +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------filter((orders.o_orderdate < '1995-03-15')) +----------------------------PhysicalOlapScan[orders] +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------filter((customer.c_mktsegment = 'BUILDING')) +----------------------------PhysicalOlapScan[customer] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q4.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q4.out new file mode 100644 index 00000000000..00c53a53674 --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q4.out @@ -0,0 +1,18 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------hashAgg[GLOBAL] +----------PhysicalDistribute +------------hashAgg[LOCAL] +--------------PhysicalProject +----------------hashJoin[RIGHT_SEMI_JOIN](lineitem.l_orderkey = orders.o_orderkey) +------------------PhysicalProject +--------------------filter((lineitem.l_commitdate < lineitem.l_receiptdate)) +----------------------PhysicalOlapScan[lineitem] +------------------PhysicalProject +--------------------filter((orders.o_orderdate >= '1993-07-01')(orders.o_orderdate < '1993-10-01')) +----------------------PhysicalOlapScan[orders] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q5.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q5.out new file mode 100644 index 00000000000..9fc242ea918 --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q5.out @@ -0,0 +1,41 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------hashAgg[GLOBAL] +----------PhysicalDistribute +------------hashAgg[LOCAL] +--------------PhysicalProject +----------------hashJoin[INNER_JOIN](nation.n_regionkey = region.r_regionkey) +------------------PhysicalProject +--------------------hashJoin[INNER_JOIN](supplier.s_nationkey = nation.n_nationkey) +----------------------PhysicalProject +------------------------hashJoin[INNER_JOIN](customer.c_nationkey = supplier.s_nationkey)(lineitem.l_suppkey = supplier.s_suppkey) +--------------------------PhysicalDistribute +----------------------------PhysicalProject +------------------------------hashJoin[INNER_JOIN](lineitem.l_orderkey = orders.o_orderkey) +--------------------------------PhysicalProject +----------------------------------PhysicalOlapScan[lineitem] +--------------------------------PhysicalDistribute +----------------------------------PhysicalProject +------------------------------------hashJoin[INNER_JOIN](customer.c_custkey = orders.o_custkey) +--------------------------------------PhysicalDistribute +----------------------------------------PhysicalProject +------------------------------------------filter((orders.o_orderdate < '1995-01-01')(orders.o_orderdate >= '1994-01-01')) +--------------------------------------------PhysicalOlapScan[orders] +--------------------------------------PhysicalDistribute +----------------------------------------PhysicalProject +------------------------------------------PhysicalOlapScan[customer] +--------------------------PhysicalDistribute +----------------------------PhysicalProject +------------------------------PhysicalOlapScan[supplier] +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------PhysicalOlapScan[nation] +------------------PhysicalDistribute +--------------------PhysicalProject +----------------------filter((region.r_name = 'ASIA')) +------------------------PhysicalOlapScan[region] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q6.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q6.out new file mode 100644 index 00000000000..c9714f3608d --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q6.out @@ -0,0 +1,10 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--hashAgg[GLOBAL] +----PhysicalDistribute +------hashAgg[LOCAL] +--------PhysicalProject +----------filter((lineitem.l_shipdate >= '1994-01-01')(lineitem.l_discount <= 0.07)(lineitem.l_discount >= 0.05)(lineitem.l_quantity < 24.00)(lineitem.l_shipdate < '1995-01-01')) +------------PhysicalOlapScan[lineitem] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q7.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q7.out new file mode 100644 index 00000000000..2356083c4e8 --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q7.out @@ -0,0 +1,40 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------hashAgg[GLOBAL] +----------PhysicalDistribute +------------hashAgg[LOCAL] +--------------PhysicalProject +----------------hashJoin[INNER_JOIN](customer.c_nationkey = n2.n_nationkey)(((n1.n_name = 'FRANCE') AND (n2.n_name = 'GERMANY')) OR ((n1.n_name = 'GERMANY') AND (n2.n_name = 'FRANCE'))) +------------------PhysicalProject +--------------------hashJoin[INNER_JOIN](supplier.s_nationkey = n1.n_nationkey) +----------------------PhysicalProject +------------------------hashJoin[INNER_JOIN](customer.c_custkey = orders.o_custkey) +--------------------------PhysicalDistribute +----------------------------PhysicalProject +------------------------------hashJoin[INNER_JOIN](supplier.s_suppkey = lineitem.l_suppkey) +--------------------------------PhysicalDistribute +----------------------------------hashJoin[INNER_JOIN](orders.o_orderkey = lineitem.l_orderkey) +------------------------------------PhysicalProject +--------------------------------------PhysicalOlapScan[orders] +------------------------------------PhysicalProject +--------------------------------------filter((lineitem.l_shipdate <= '1996-12-31')(lineitem.l_shipdate >= '1995-01-01')) +----------------------------------------PhysicalOlapScan[lineitem] +--------------------------------PhysicalDistribute +----------------------------------PhysicalProject +------------------------------------PhysicalOlapScan[supplier] +--------------------------PhysicalDistribute +----------------------------PhysicalProject +------------------------------PhysicalOlapScan[customer] +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------filter(((n1.n_name = 'FRANCE') OR (n1.n_name = 'GERMANY'))) +----------------------------PhysicalOlapScan[nation] +------------------PhysicalDistribute +--------------------PhysicalProject +----------------------filter(((n2.n_name = 'GERMANY') OR (n2.n_name = 'FRANCE'))) +------------------------PhysicalOlapScan[nation] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q8.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q8.out new file mode 100644 index 00000000000..4ce12cd85a3 --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q8.out @@ -0,0 +1,51 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------PhysicalProject +----------hashAgg[GLOBAL] +------------PhysicalDistribute +--------------hashAgg[LOCAL] +----------------PhysicalProject +------------------hashJoin[INNER_JOIN](n1.n_regionkey = region.r_regionkey) +--------------------PhysicalProject +----------------------hashJoin[INNER_JOIN](supplier.s_nationkey = n2.n_nationkey) +------------------------PhysicalProject +--------------------------hashJoin[INNER_JOIN](customer.c_nationkey = n1.n_nationkey) +----------------------------PhysicalProject +------------------------------hashJoin[INNER_JOIN](supplier.s_suppkey = lineitem.l_suppkey) +--------------------------------PhysicalDistribute +----------------------------------PhysicalProject +------------------------------------hashJoin[INNER_JOIN](part.p_partkey = lineitem.l_partkey) +--------------------------------------PhysicalDistribute +----------------------------------------hashJoin[INNER_JOIN](orders.o_custkey = customer.c_custkey) +------------------------------------------PhysicalDistribute +--------------------------------------------hashJoin[INNER_JOIN](lineitem.l_orderkey = orders.o_orderkey) +----------------------------------------------PhysicalProject +------------------------------------------------PhysicalOlapScan[lineitem] +----------------------------------------------PhysicalProject +------------------------------------------------filter((orders.o_orderdate <= '1996-12-31')(orders.o_orderdate >= '1995-01-01')) +--------------------------------------------------PhysicalOlapScan[orders] +------------------------------------------PhysicalDistribute +--------------------------------------------PhysicalProject +----------------------------------------------PhysicalOlapScan[customer] +--------------------------------------PhysicalDistribute +----------------------------------------PhysicalProject +------------------------------------------filter((part.p_type = 'ECONOMY ANODIZED STEEL')) +--------------------------------------------PhysicalOlapScan[part] +--------------------------------PhysicalDistribute +----------------------------------PhysicalProject +------------------------------------PhysicalOlapScan[supplier] +----------------------------PhysicalDistribute +------------------------------PhysicalProject +--------------------------------PhysicalOlapScan[nation] +------------------------PhysicalDistribute +--------------------------PhysicalProject +----------------------------PhysicalOlapScan[nation] +--------------------PhysicalDistribute +----------------------PhysicalProject +------------------------filter((region.r_name = 'AMERICA')) +--------------------------PhysicalOlapScan[region] + diff --git a/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q9.out b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q9.out new file mode 100644 index 00000000000..e173453102b --- /dev/null +++ b/regression-test/data/nereids_tpch_shape_sf1000_p0/shape_no_stats/q9.out @@ -0,0 +1,38 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +PhysicalResultSink +--PhysicalQuickSort +----PhysicalDistribute +------PhysicalQuickSort +--------hashAgg[GLOBAL] +----------PhysicalDistribute +------------hashAgg[LOCAL] +--------------PhysicalProject +----------------hashJoin[INNER_JOIN](supplier.s_nationkey = nation.n_nationkey) +------------------PhysicalProject +--------------------hashJoin[INNER_JOIN](supplier.s_suppkey = lineitem.l_suppkey) +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------hashJoin[INNER_JOIN](partsupp.ps_partkey = lineitem.l_partkey)(partsupp.ps_suppkey = lineitem.l_suppkey) +----------------------------PhysicalProject +------------------------------hashJoin[INNER_JOIN](part.p_partkey = lineitem.l_partkey) +--------------------------------PhysicalDistribute +----------------------------------hashJoin[INNER_JOIN](orders.o_orderkey = lineitem.l_orderkey) +------------------------------------PhysicalProject +--------------------------------------PhysicalOlapScan[lineitem] +------------------------------------PhysicalProject +--------------------------------------PhysicalOlapScan[orders] +--------------------------------PhysicalDistribute +----------------------------------PhysicalProject +------------------------------------filter((p_name like '%green%')) +--------------------------------------PhysicalOlapScan[part] +----------------------------PhysicalDistribute +------------------------------PhysicalProject +--------------------------------PhysicalOlapScan[partsupp] +----------------------PhysicalDistribute +------------------------PhysicalProject +--------------------------PhysicalOlapScan[supplier] +------------------PhysicalDistribute +--------------------PhysicalProject +----------------------PhysicalOlapScan[nation] + diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q1.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q1.groovy new file mode 100644 index 00000000000..59f149db8ee --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q1.groovy @@ -0,0 +1,60 @@ +/* + * 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("q1") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + + qt_select """ + explain shape plan + select + l_returnflag, + l_linestatus, + sum(l_quantity) as sum_qty, + sum(l_extendedprice) as sum_base_price, + sum(l_extendedprice * (1 - l_discount)) as sum_disc_price, + sum(l_extendedprice * (1 - l_discount) * (1 + l_tax)) as sum_charge, + avg(l_quantity) as avg_qty, + avg(l_extendedprice) as avg_price, + avg(l_discount) as avg_disc, + count(*) as count_order + from + lineitem + where + l_shipdate <= date '1998-12-01' - interval '90' day + group by + l_returnflag, + l_linestatus + order by + l_returnflag, + l_linestatus; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q10.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q10.groovy new file mode 100644 index 00000000000..9f696db6592 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q10.groovy @@ -0,0 +1,71 @@ +/* + * 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("q10") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + qt_select """ + explain shape plan + select + c_custkey, + c_name, + sum(l_extendedprice * (1 - l_discount)) as revenue, + c_acctbal, + n_name, + c_address, + c_phone, + c_comment + from + customer, + orders, + lineitem, + nation + where + c_custkey = o_custkey + and l_orderkey = o_orderkey + and o_orderdate >= date '1993-10-01' + and o_orderdate < date '1993-10-01' + interval '3' month + and l_returnflag = 'R' + and c_nationkey = n_nationkey + group by + c_custkey, + c_name, + c_acctbal, + c_phone, + n_name, + c_address, + c_comment + order by + revenue desc + limit 20; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q11.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q11.groovy new file mode 100644 index 00000000000..b72e0b392f7 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q11.groovy @@ -0,0 +1,68 @@ +/* + * 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("q11") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + sql 'set parallel_pipeline_task_num=8' + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + + + qt_select """ + explain shape plan + select + ps_partkey, + sum(ps_supplycost * ps_availqty) as value + from + partsupp, + supplier, + nation + where + ps_suppkey = s_suppkey + and s_nationkey = n_nationkey + and n_name = 'GERMANY' + group by + ps_partkey having + sum(ps_supplycost * ps_availqty) > ( + select + sum(ps_supplycost * ps_availqty) * 0.000002 + from + partsupp, + supplier, + nation + where + ps_suppkey = s_suppkey + and s_nationkey = n_nationkey + and n_name = 'GERMANY' + ) + order by + value desc; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q12.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q12.groovy new file mode 100644 index 00000000000..e4c0f670d56 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q12.groovy @@ -0,0 +1,66 @@ +/* + * 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("q12") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + sql 'set parallel_pipeline_task_num=8' + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + qt_select """ + explain shape plan + select + l_shipmode, + sum(case + when o_orderpriority = '1-URGENT' + or o_orderpriority = '2-HIGH' + then 1 + else 0 + end) as high_line_count, + sum(case + when o_orderpriority <> '1-URGENT' + and o_orderpriority <> '2-HIGH' + then 1 + else 0 + end) as low_line_count + from + orders, + lineitem + where + o_orderkey = l_orderkey + and l_shipmode in ('MAIL', 'SHIP') + and l_commitdate < l_receiptdate + and l_shipdate < l_commitdate + and l_receiptdate >= date '1994-01-01' + and l_receiptdate < date '1994-01-01' + interval '1' year + group by + l_shipmode + order by + l_shipmode; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q13.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q13.groovy new file mode 100644 index 00000000000..991a00d739e --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q13.groovy @@ -0,0 +1,62 @@ +/* + * 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("q13") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + qt_select """ + explain shape plan + select + c_count, + count(*) as custdist + from + ( + select + c_custkey, + count(o_orderkey) as c_count + from + customer left outer join orders on + c_custkey = o_custkey + and o_comment not like '%special%requests%' + group by + c_custkey + ) as c_orders + group by + c_count + order by + custdist desc, + c_count desc; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q14.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q14.groovy new file mode 100644 index 00000000000..2b109a4996b --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q14.groovy @@ -0,0 +1,52 @@ +/* + * 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("q14") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + qt_select """ + explain shape plan + select + 100.00 * sum(case + when p_type like 'PROMO%' + then l_extendedprice * (1 - l_discount) + else 0 + end) / sum(l_extendedprice * (1 - l_discount)) as promo_revenue + from + lineitem, + part + where + l_partkey = p_partkey + and l_shipdate >= date '1995-09-01' + and l_shipdate < date '1995-09-01' + interval '1' month; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q15.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q15.groovy new file mode 100644 index 00000000000..eb627a0a068 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q15.groovy @@ -0,0 +1,62 @@ +/* + * 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("q15") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + s_suppkey, + s_name, + s_address, + s_phone, + total_revenue + from + supplier, + revenue0 + where + s_suppkey = supplier_no + and total_revenue = ( + select + max(total_revenue) + from + revenue0 + ) + order by + s_suppkey; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q16.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q16.groovy new file mode 100644 index 00000000000..5e7a35611c0 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q16.groovy @@ -0,0 +1,73 @@ +/* + * 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("q16") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + p_brand, + p_type, + p_size, + count(distinct ps_suppkey) as supplier_cnt + from + partsupp, + part + where + p_partkey = ps_partkey + and p_brand <> 'Brand#45' + and p_type not like 'MEDIUM POLISHED%' + and p_size in (49, 14, 23, 45, 19, 3, 36, 9) + and ps_suppkey not in ( + select + s_suppkey + from + supplier + where + s_comment like '%Customer%Complaints%' + ) + group by + p_brand, + p_type, + p_size + order by + supplier_cnt desc, + p_brand, + p_type, + p_size; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q17.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q17.groovy new file mode 100644 index 00000000000..f365a230ac6 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q17.groovy @@ -0,0 +1,60 @@ +/* + * 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("q17") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + sum(l_extendedprice) / 7.0 as avg_yearly + from + lineitem, + part + where + p_partkey = l_partkey + and p_brand = 'Brand#23' + and p_container = 'MED BOX' + and l_quantity < ( + select + 0.2 * avg(l_quantity) + from + lineitem + where + l_partkey = p_partkey + ); + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q18.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q18.groovy new file mode 100644 index 00000000000..ffdf6163269 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q18.groovy @@ -0,0 +1,76 @@ +/* + * 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("q18") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + c_name, + c_custkey, + o_orderkey, + o_orderdate, + o_totalprice, + sum(l_quantity) + from + customer, + orders, + lineitem + where + o_orderkey in ( + select + l_orderkey + from + lineitem + group by + l_orderkey having + sum(l_quantity) > 300 + ) + and c_custkey = o_custkey + and o_orderkey = l_orderkey + group by + c_name, + c_custkey, + o_orderkey, + o_orderdate, + o_totalprice + order by + o_totalprice desc, + o_orderdate + limit 100; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q19.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q19.groovy new file mode 100644 index 00000000000..ec91b1f745b --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q19.groovy @@ -0,0 +1,80 @@ +/* + * 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("q19") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + sum(l_extendedprice* (1 - l_discount)) as revenue + from + lineitem, + part + where + ( + p_partkey = l_partkey + and p_brand = 'Brand#12' + and p_container in ('SM CASE', 'SM BOX', 'SM PACK', 'SM PKG') + and l_quantity >= 1 and l_quantity <= 1 + 10 + and p_size between 1 and 5 + and l_shipmode in ('AIR', 'AIR REG') + and l_shipinstruct = 'DELIVER IN PERSON' + ) + or + ( + p_partkey = l_partkey + and p_brand = 'Brand#23' + and p_container in ('MED BAG', 'MED BOX', 'MED PKG', 'MED PACK') + and l_quantity >= 10 and l_quantity <= 10 + 10 + and p_size between 1 and 10 + and l_shipmode in ('AIR', 'AIR REG') + and l_shipinstruct = 'DELIVER IN PERSON' + ) + or + ( + p_partkey = l_partkey + and p_brand = 'Brand#34' + and p_container in ('LG CASE', 'LG BOX', 'LG PACK', 'LG PKG') + and l_quantity >= 20 and l_quantity <= 20 + 10 + and p_size between 1 and 15 + and l_shipmode in ('AIR', 'AIR REG') + and l_shipinstruct = 'DELIVER IN PERSON' + ); + + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q2.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q2.groovy new file mode 100644 index 00000000000..8f0d99c4644 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q2.groovy @@ -0,0 +1,87 @@ +/* + * 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("q2") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + s_acctbal, + s_name, + n_name, + p_partkey, + p_mfgr, + s_address, + s_phone, + s_comment + from + part, + supplier, + partsupp, + nation, + region + where + p_partkey = ps_partkey + and s_suppkey = ps_suppkey + and p_size = 15 + and p_type like '%BRASS' + and s_nationkey = n_nationkey + and n_regionkey = r_regionkey + and r_name = 'EUROPE' + and ps_supplycost = ( + select + min(ps_supplycost) + from + partsupp, + supplier, + nation, + region + where + p_partkey = ps_partkey + and s_suppkey = ps_suppkey + and s_nationkey = n_nationkey + and n_regionkey = r_regionkey + and r_name = 'EUROPE' + ) + order by + s_acctbal desc, + n_name, + s_name, + p_partkey + limit 100; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20-rewrite.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20-rewrite.groovy new file mode 100644 index 00000000000..77cc8368822 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20-rewrite.groovy @@ -0,0 +1,71 @@ +/* + * 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("q20-rewrite") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + sql 'set parallel_pipeline_task_num=8' + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan +select +s_name, s_address +from +supplier left semi join +( + select * from + ( + select l_partkey,l_suppkey, 0.5 * sum(l_quantity) as l_q + from lineitem + where l_shipdate >= date '1994-01-01' + and l_shipdate < date '1994-01-01' + interval '1' year + group by l_partkey,l_suppkey + ) t2 join + ( + select ps_partkey, ps_suppkey, ps_availqty + from partsupp left semi join part + on ps_partkey = p_partkey and p_name like 'forest%' + ) t1 + on t2.l_partkey = t1.ps_partkey and t2.l_suppkey = t1.ps_suppkey + and t1.ps_availqty > t2.l_q +) t3 +on s_suppkey = t3.ps_suppkey +join nation +where s_nationkey = n_nationkey + and n_name = 'CANADA' +order by s_name +; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20.groovy new file mode 100644 index 00000000000..e59a0a70b08 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q20.groovy @@ -0,0 +1,81 @@ +/* + * 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("q20") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + s_name, + s_address + from + supplier, + nation + where + s_suppkey in ( + select + ps_suppkey + from + partsupp + where + ps_partkey in ( + select + p_partkey + from + part + where + p_name like 'forest%' + ) + and ps_availqty > ( + select + 0.5 * sum(l_quantity) + from + lineitem + where + l_partkey = ps_partkey + and l_suppkey = ps_suppkey + and l_shipdate >= date '1994-01-01' + and l_shipdate < date '1994-01-01' + interval '1' year + ) + ) + and s_nationkey = n_nationkey + and n_name = 'CANADA' + order by + s_name; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q21.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q21.groovy new file mode 100644 index 00000000000..74fd3aa486a --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q21.groovy @@ -0,0 +1,83 @@ +/* + * 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("q21") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + s_name, + count(*) as numwait + from + supplier, + lineitem l1, + orders, + nation + where + s_suppkey = l1.l_suppkey + and o_orderkey = l1.l_orderkey + and o_orderstatus = 'F' + and l1.l_receiptdate > l1.l_commitdate + and exists ( + select + * + from + lineitem l2 + where + l2.l_orderkey = l1.l_orderkey + and l2.l_suppkey <> l1.l_suppkey + ) + and not exists ( + select + * + from + lineitem l3 + where + l3.l_orderkey = l1.l_orderkey + and l3.l_suppkey <> l1.l_suppkey + and l3.l_receiptdate > l3.l_commitdate + ) + and s_nationkey = n_nationkey + and n_name = 'SAUDI ARABIA' + group by + s_name + order by + numwait desc, + s_name + limit 100; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q22.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q22.groovy new file mode 100644 index 00000000000..1ec5264ef30 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q22.groovy @@ -0,0 +1,80 @@ +/* + * 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("q22") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + cntrycode, + count(*) as numcust, + sum(c_acctbal) as totacctbal + from + ( + select + substring(c_phone, 1, 2) as cntrycode, + c_acctbal + from + customer + where + substring(c_phone, 1, 2) in + ('13', '31', '23', '29', '30', '18', '17') + and c_acctbal > ( + select + avg(c_acctbal) + from + customer + where + c_acctbal > 0.00 + and substring(c_phone, 1, 2) in + ('13', '31', '23', '29', '30', '18', '17') + ) + and not exists ( + select + * + from + orders + where + o_custkey = c_custkey + ) + ) as custsale + group by + cntrycode + order by + cntrycode; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q3.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q3.groovy new file mode 100644 index 00000000000..7d563162383 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q3.groovy @@ -0,0 +1,67 @@ +/* + * 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("q3") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + // db = "tpch" + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + l_orderkey, + sum(l_extendedprice * (1 - l_discount)) as revenue, + o_orderdate, + o_shippriority + from + customer, + orders, + lineitem + where + c_mktsegment = 'BUILDING' + and c_custkey = o_custkey + and l_orderkey = o_orderkey + and o_orderdate < date '1995-03-15' + and l_shipdate > date '1995-03-15' + group by + l_orderkey, + o_orderdate, + o_shippriority + order by + revenue desc, + o_orderdate + limit 10; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q4.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q4.groovy new file mode 100644 index 00000000000..25da27e3b5e --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q4.groovy @@ -0,0 +1,63 @@ +/* + * 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("q4") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + sql 'set parallel_pipeline_task_num=8' + + + + + qt_select """ + explain shape plan + select + o_orderpriority, + count(*) as order_count + from + orders + where + o_orderdate >= date '1993-07-01' + and o_orderdate < date '1993-07-01' + interval '3' month + and exists ( + select + * + from + lineitem + where + l_orderkey = o_orderkey + and l_commitdate < l_receiptdate + ) + group by + o_orderpriority + order by + o_orderpriority; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q5.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q5.groovy new file mode 100644 index 00000000000..d10c624ff5c --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q5.groovy @@ -0,0 +1,67 @@ +/* + * 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("q5") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + n_name, + sum(l_extendedprice * (1 - l_discount)) as revenue + from + customer, + orders, + lineitem, + supplier, + nation, + region + where + c_custkey = o_custkey + and l_orderkey = o_orderkey + and l_suppkey = s_suppkey + and c_nationkey = s_nationkey + and s_nationkey = n_nationkey + and n_regionkey = r_regionkey + and r_name = 'ASIA' + and o_orderdate >= date '1994-01-01' + and o_orderdate < date '1994-01-01' + interval '1' year + group by + n_name + order by + revenue desc; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q6.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q6.groovy new file mode 100644 index 00000000000..12abff48e66 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q6.groovy @@ -0,0 +1,52 @@ +/* + * 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("q6") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + sum(l_extendedprice * l_discount) as revenue + from + lineitem + where + l_shipdate >= date '1994-01-01' + and l_shipdate < date '1994-01-01' + interval '1' year + and l_discount between .06 - 0.01 and .06 + 0.01 + and l_quantity < 24; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q7.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q7.groovy new file mode 100644 index 00000000000..544f09cbdf7 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q7.groovy @@ -0,0 +1,82 @@ +/* + * 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("q7") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + supp_nation, + cust_nation, + l_year, + sum(volume) as revenue + from + ( + select + n1.n_name as supp_nation, + n2.n_name as cust_nation, + extract(year from l_shipdate) as l_year, + l_extendedprice * (1 - l_discount) as volume + from + supplier, + lineitem, + orders, + customer, + nation n1, + nation n2 + where + s_suppkey = l_suppkey + and o_orderkey = l_orderkey + and c_custkey = o_custkey + and s_nationkey = n1.n_nationkey + and c_nationkey = n2.n_nationkey + and ( + (n1.n_name = 'FRANCE' and n2.n_name = 'GERMANY') + or (n1.n_name = 'GERMANY' and n2.n_name = 'FRANCE') + ) + and l_shipdate between date '1995-01-01' and date '1996-12-31' + ) as shipping + group by + supp_nation, + cust_nation, + l_year + order by + supp_nation, + cust_nation, + l_year; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q8.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q8.groovy new file mode 100644 index 00000000000..178ac21e789 --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q8.groovy @@ -0,0 +1,80 @@ +/* + * 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("q8") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + o_year, + sum(case + when nation = 'BRAZIL' then volume + else 0 + end) / sum(volume) as mkt_share + from + ( + select + extract(year from o_orderdate) as o_year, + l_extendedprice * (1 - l_discount) as volume, + n2.n_name as nation + from + part, + supplier, + lineitem, + orders, + customer, + nation n1, + nation n2, + region + where + p_partkey = l_partkey + and s_suppkey = l_suppkey + and l_orderkey = o_orderkey + and o_custkey = c_custkey + and c_nationkey = n1.n_nationkey + and n1.n_regionkey = r_regionkey + and r_name = 'AMERICA' + and s_nationkey = n2.n_nationkey + and o_orderdate between date '1995-01-01' and date '1996-12-31' + and p_type = 'ECONOMY ANODIZED STEEL' + ) as all_nations + group by + o_year + order by + o_year; + """ +} diff --git a/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q9.groovy b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q9.groovy new file mode 100644 index 00000000000..a724369a9ce --- /dev/null +++ b/regression-test/suites/nereids_tpch_shape_sf1000_p0/shape_no_stats/q9.groovy @@ -0,0 +1,75 @@ +/* + * 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("q9") { + String db = context.config.getDbNameByFile(new File(context.file.parent)) + sql "use ${db}" + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql "set runtime_filter_mode='GLOBAL'" + + sql 'set exec_mem_limit=21G' + sql 'SET enable_pipeline_engine = true' + sql 'set parallel_pipeline_task_num=8' + + + + sql 'set be_number_for_test=3' + sql "set runtime_filter_type=8" +sql 'set enable_runtime_filter_prune=false' +sql 'set forbid_unknown_col_stats=false;' +sql 'set enable_runtime_filter_prune=false' +sql 'set enable_stats=false' + + qt_select """ + explain shape plan + select + nation, + o_year, + sum(amount) as sum_profit + from + ( + select + n_name as nation, + extract(year from o_orderdate) as o_year, + l_extendedprice * (1 - l_discount) - ps_supplycost * l_quantity as amount + from + part, + supplier, + lineitem, + partsupp, + orders, + nation + where + s_suppkey = l_suppkey + and ps_suppkey = l_suppkey + and ps_partkey = l_partkey + and p_partkey = l_partkey + and o_orderkey = l_orderkey + and s_nationkey = n_nationkey + and p_name like '%green%' + ) as profit + group by + nation, + o_year + order by + nation, + o_year desc; + """ +} --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org