This is an automated email from the ASF dual-hosted git repository. morrysnow pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/doris.git
The following commit(s) were added to refs/heads/master by this push: new 6291cf2b208 [fix](Nereids) column pruning should prune map in cte consumer (#34079) 6291cf2b208 is described below commit 6291cf2b2084cf3ef6386c21881b7016c353bffe Author: morrySnow <101034200+morrys...@users.noreply.github.com> AuthorDate: Fri Apr 26 12:37:08 2024 +0800 [fix](Nereids) column pruning should prune map in cte consumer (#34079) we save bi-map in cte consumer to get the maping between producer and consumer. the consumer's output is decided by the map in it. so, cte consumer should be output prunable, and should remove useless entry from map when do column pruning --- .../org/apache/doris/nereids/CascadesContext.java | 26 +--- .../org/apache/doris/nereids/StatementContext.java | 41 ++---- .../glue/translator/PhysicalPlanTranslator.java | 6 +- .../doris/nereids/jobs/executor/Rewriter.java | 4 +- .../org/apache/doris/nereids/rules/RuleType.java | 3 +- .../rules/rewrite/CollectCteConsumerOutput.java | 41 ++++++ .../rules/rewrite/CollectProjectAboveConsumer.java | 81 ------------ .../doris/nereids/rules/rewrite/ColumnPruning.java | 31 +++-- .../nereids/rules/rewrite/RewriteCteChildren.java | 17 ++- .../trees/plans/logical/LogicalCTEConsumer.java | 22 +++- .../data/nereids_hint_tpcds_p0/shape/query1.out | 3 +- .../data/nereids_hint_tpcds_p0/shape/query24.out | 3 +- .../data/nereids_p0/hint/multi_leading.out | 6 +- .../nereids_tpcds_shape_sf1000_p0/shape/query1.out | 3 +- .../shape/query23.out | 12 +- .../shape/query24.out | 3 +- .../shape/query30.out | 3 +- .../shape/query31.out | 48 ++++--- .../shape/query39.out | 10 +- .../shape/query47.out | 5 +- .../shape/query57.out | 5 +- .../shape/query59.out | 19 ++- .../shape/query70.out | 1 + .../shape/query81.out | 3 +- .../shape/query95.out | 6 +- .../constraints/query23.out | 12 +- .../noStatsRfPrune/query1.out | 3 +- .../noStatsRfPrune/query23.out | 12 +- .../noStatsRfPrune/query24.out | 3 +- .../noStatsRfPrune/query30.out | 3 +- .../noStatsRfPrune/query31.out | 40 +++--- .../noStatsRfPrune/query39.out | 10 +- .../noStatsRfPrune/query47.out | 5 +- .../noStatsRfPrune/query57.out | 5 +- .../noStatsRfPrune/query59.out | 19 ++- .../noStatsRfPrune/query70.out | 1 + .../noStatsRfPrune/query81.out | 3 +- .../noStatsRfPrune/query95.out | 6 +- .../no_stats_shape/query1.out | 3 +- .../no_stats_shape/query23.out | 12 +- .../no_stats_shape/query24.out | 3 +- .../no_stats_shape/query30.out | 3 +- .../no_stats_shape/query31.out | 40 +++--- .../no_stats_shape/query39.out | 10 +- .../no_stats_shape/query47.out | 5 +- .../no_stats_shape/query57.out | 5 +- .../no_stats_shape/query59.out | 19 ++- .../no_stats_shape/query81.out | 3 +- .../no_stats_shape/query95.out | 6 +- .../rf_prune/query1.out | 3 +- .../rf_prune/query23.out | 12 +- .../rf_prune/query24.out | 3 +- .../rf_prune/query30.out | 3 +- .../rf_prune/query31.out | 48 ++++--- .../rf_prune/query39.out | 10 +- .../rf_prune/query47.out | 5 +- .../rf_prune/query57.out | 5 +- .../rf_prune/query59.out | 19 ++- .../rf_prune/query70.out | 1 + .../rf_prune/query81.out | 3 +- .../rf_prune/query95.out | 6 +- .../nereids_tpcds_shape_sf100_p0/shape/query1.out | 3 +- .../nereids_tpcds_shape_sf100_p0/shape/query23.out | 12 +- .../nereids_tpcds_shape_sf100_p0/shape/query24.out | 3 +- .../nereids_tpcds_shape_sf100_p0/shape/query30.out | 3 +- .../nereids_tpcds_shape_sf100_p0/shape/query31.out | 48 ++++--- .../nereids_tpcds_shape_sf100_p0/shape/query39.out | 10 +- .../nereids_tpcds_shape_sf100_p0/shape/query47.out | 5 +- .../nereids_tpcds_shape_sf100_p0/shape/query57.out | 5 +- .../nereids_tpcds_shape_sf100_p0/shape/query59.out | 19 ++- .../nereids_tpcds_shape_sf100_p0/shape/query81.out | 3 +- .../nereids_tpcds_shape_sf100_p0/shape/query95.out | 6 +- .../nereids_p0/cte/test_cte_column_pruning.groovy | 143 +++++++++++++++++++++ 73 files changed, 513 insertions(+), 498 deletions(-) diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/CascadesContext.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/CascadesContext.java index dd569ef8f75..3b9ba912383 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/CascadesContext.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/CascadesContext.java @@ -50,7 +50,6 @@ import org.apache.doris.nereids.rules.analysis.BindRelation.CustomTableResolver; import org.apache.doris.nereids.rules.exploration.mv.MaterializationContext; import org.apache.doris.nereids.trees.expressions.CTEId; import org.apache.doris.nereids.trees.expressions.Expression; -import org.apache.doris.nereids.trees.expressions.NamedExpression; import org.apache.doris.nereids.trees.expressions.Slot; import org.apache.doris.nereids.trees.expressions.SubqueryExpr; import org.apache.doris.nereids.trees.plans.Plan; @@ -102,7 +101,7 @@ public class CascadesContext implements ScheduleContext { private Optional<RootRewriteJobContext> currentRootRewriteJobContext; // in optimize stage, the plan will storage in the memo private Memo memo; - private StatementContext statementContext; + private final StatementContext statementContext; private final CTEContext cteContext; private final RuleSet ruleSet; @@ -616,16 +615,6 @@ public class CascadesContext implements ScheduleContext { consumers.add(cteConsumer); } - public void putCTEIdToProject(CTEId cteId, NamedExpression p) { - Set<NamedExpression> projects = this.statementContext.getCteIdToProjects() - .computeIfAbsent(cteId, k -> new HashSet<>()); - projects.add(p); - } - - public Set<NamedExpression> getProjectForProducer(CTEId cteId) { - return this.statementContext.getCteIdToProjects().get(cteId); - } - public Map<CTEId, Set<LogicalCTEConsumer>> getCteIdToConsumers() { return this.statementContext.getCteIdToConsumers(); } @@ -639,17 +628,6 @@ public class CascadesContext implements ScheduleContext { return this.statementContext.getConsumerIdToFilters(); } - public void markConsumerUnderProject(LogicalCTEConsumer cteConsumer) { - Set<RelationId> consumerIds = this.statementContext.getCteIdToConsumerUnderProjects() - .computeIfAbsent(cteConsumer.getCteId(), k -> new HashSet<>()); - consumerIds.add(cteConsumer.getRelationId()); - } - - public boolean couldPruneColumnOnProducer(CTEId cteId) { - Set<RelationId> consumerIds = this.statementContext.getCteIdToConsumerUnderProjects().get(cteId); - return consumerIds.size() == this.statementContext.getCteIdToConsumers().get(cteId).size(); - } - public void addCTEConsumerGroup(CTEId cteId, Group g, Map<Slot, Slot> producerSlotToConsumerSlot) { List<Pair<Map<Slot, Slot>, Group>> consumerGroups = this.statementContext.getCteIdToConsumerGroup().computeIfAbsent(cteId, k -> new ArrayList<>()); @@ -746,7 +724,7 @@ public class CascadesContext implements ScheduleContext { public static void printPlanProcess(List<PlanProcess> planProcesses) { for (PlanProcess row : planProcesses) { - LOG.info("RULE: " + row.ruleName + "\nBEFORE:\n" + row.beforeShape + "\nafter:\n" + row.afterShape); + LOG.info("RULE: {}\nBEFORE:\n{}\nafter:\n{}", row.ruleName, row.beforeShape, row.afterShape); } } 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 819ff032cd6..3274233c16b 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 @@ -27,7 +27,6 @@ import org.apache.doris.nereids.rules.analysis.ColumnAliasGenerator; import org.apache.doris.nereids.trees.expressions.CTEId; import org.apache.doris.nereids.trees.expressions.ExprId; import org.apache.doris.nereids.trees.expressions.Expression; -import org.apache.doris.nereids.trees.expressions.NamedExpression; import org.apache.doris.nereids.trees.expressions.Slot; import org.apache.doris.nereids.trees.expressions.SlotReference; import org.apache.doris.nereids.trees.plans.ObjectId; @@ -54,7 +53,6 @@ import java.io.Closeable; import java.util.ArrayList; import java.util.BitSet; import java.util.Collection; -import java.util.Comparator; import java.util.HashMap; import java.util.Iterator; import java.util.List; @@ -103,9 +101,8 @@ public class StatementContext implements Closeable { private final IdGenerator<CTEId> cteIdGenerator = CTEId.createGenerator(); private final Map<CTEId, Set<LogicalCTEConsumer>> cteIdToConsumers = new HashMap<>(); - private final Map<CTEId, Set<NamedExpression>> cteIdToProjects = new HashMap<>(); + private final Map<CTEId, Set<Slot>> cteIdToOutputIds = new HashMap<>(); private final Map<RelationId, Set<Expression>> consumerIdToFilters = new HashMap<>(); - private final Map<CTEId, Set<RelationId>> cteIdToConsumerUnderProjects = new HashMap<>(); // Used to update consumer's stats private final Map<CTEId, List<Pair<Map<Slot, Slot>, Group>>> cteIdToConsumerGroup = new HashMap<>(); private final Map<CTEId, LogicalPlan> rewrittenCteProducer = new HashMap<>(); @@ -134,12 +131,13 @@ public class StatementContext implements Closeable { private BitSet disableRules; // table locks - private Stack<CloseableResource> plannerResources = new Stack<>(); + private final Stack<CloseableResource> plannerResources = new Stack<>(); // for create view support in nereids // key is the start and end position of the sql substring that needs to be replaced, // and value is the new string used for replacement. - private TreeMap<Pair<Integer, Integer>, String> indexInSqlToString = new TreeMap<>(new Pair.PairComparator<>()); + private final TreeMap<Pair<Integer, Integer>, String> indexInSqlToString + = new TreeMap<>(new Pair.PairComparator<>()); public StatementContext() { this(ConnectContext.get(), null, 0); @@ -216,10 +214,6 @@ public class StatementContext implements Closeable { return Optional.ofNullable(sqlCacheContext); } - public int getMaxContinuousJoin() { - return joinCount; - } - public Set<SlotReference> getAllPathsSlots() { Set<SlotReference> allSlotReferences = Sets.newHashSet(); for (Map<List<String>, SlotReference> slotReferenceMap : subColumnSlotRefMap.values()) { @@ -240,19 +234,16 @@ public class StatementContext implements Closeable { * Add a slot ref attached with paths in context to avoid duplicated slot */ public void addPathSlotRef(Slot root, List<String> paths, SlotReference slotRef, Expression originalExpr) { - subColumnSlotRefMap.computeIfAbsent(root, k -> Maps.newTreeMap(new Comparator<List<String>>() { - @Override - public int compare(List<String> lst1, List<String> lst2) { - Iterator<String> it1 = lst1.iterator(); - Iterator<String> it2 = lst2.iterator(); - while (it1.hasNext() && it2.hasNext()) { - int result = it1.next().compareTo(it2.next()); - if (result != 0) { - return result; - } + subColumnSlotRefMap.computeIfAbsent(root, k -> Maps.newTreeMap((lst1, lst2) -> { + Iterator<String> it1 = lst1.iterator(); + Iterator<String> it2 = lst2.iterator(); + while (it1.hasNext() && it2.hasNext()) { + int result = it1.next().compareTo(it2.next()); + if (result != 0) { + return result; } - return Integer.compare(lst1.size(), lst2.size()); } + return Integer.compare(lst1.size(), lst2.size()); })); subColumnSlotRefMap.get(root).put(paths, slotRef); subColumnOriginalExprMap.put(slotRef, originalExpr); @@ -349,18 +340,14 @@ public class StatementContext implements Closeable { return cteIdToConsumers; } - public Map<CTEId, Set<NamedExpression>> getCteIdToProjects() { - return cteIdToProjects; + public Map<CTEId, Set<Slot>> getCteIdToOutputIds() { + return cteIdToOutputIds; } public Map<RelationId, Set<Expression>> getConsumerIdToFilters() { return consumerIdToFilters; } - public Map<CTEId, Set<RelationId>> getCteIdToConsumerUnderProjects() { - return cteIdToConsumerUnderProjects; - } - public Map<CTEId, List<Pair<Map<Slot, Slot>, Group>>> getCteIdToConsumerGroup() { return cteIdToConsumerGroup; } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PhysicalPlanTranslator.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PhysicalPlanTranslator.java index 98bc581fe8c..3184937e2e0 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PhysicalPlanTranslator.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PhysicalPlanTranslator.java @@ -1123,9 +1123,13 @@ public class PhysicalPlanTranslator extends DefaultPlanVisitor<PlanFragment, Pla // update expr to slot mapping TupleDescriptor tupleDescriptor = null; for (Slot producerSlot : cteProducer.getOutput()) { - Slot consumerSlot = cteConsumer.getProducerToConsumerSlotMap().get(producerSlot); SlotRef slotRef = context.findSlotRef(producerSlot.getExprId()); tupleDescriptor = slotRef.getDesc().getParent(); + Slot consumerSlot = cteConsumer.getProducerToConsumerSlotMap().get(producerSlot); + // consumerSlot could be null if we prune partial consumers' columns + if (consumerSlot == null) { + continue; + } context.addExprIdSlotRefPair(consumerSlot.getExprId(), slotRef); } CTEScanNode cteScanNode = new CTEScanNode(tupleDescriptor); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Rewriter.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Rewriter.java index 335de322c61..cf0d8a638d1 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Rewriter.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Rewriter.java @@ -44,8 +44,8 @@ import org.apache.doris.nereids.rules.rewrite.CheckDataTypes; import org.apache.doris.nereids.rules.rewrite.CheckMatchExpression; import org.apache.doris.nereids.rules.rewrite.CheckMultiDistinct; import org.apache.doris.nereids.rules.rewrite.CheckPrivileges; +import org.apache.doris.nereids.rules.rewrite.CollectCteConsumerOutput; import org.apache.doris.nereids.rules.rewrite.CollectFilterAboveConsumer; -import org.apache.doris.nereids.rules.rewrite.CollectProjectAboveConsumer; import org.apache.doris.nereids.rules.rewrite.ColumnPruning; import org.apache.doris.nereids.rules.rewrite.ConvertInnerOrCrossJoin; import org.apache.doris.nereids.rules.rewrite.CountDistinctRewrite; @@ -418,7 +418,7 @@ public class Rewriter extends AbstractBatchJobExecutor { topic("Push project and filter on cte consumer to cte producer", topDown( new CollectFilterAboveConsumer(), - new CollectProjectAboveConsumer() + new CollectCteConsumerOutput() ) ), topic("Collect used column", custom(RuleType.COLLECT_COLUMNS, QueryColumnCollector::new)) diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java index f1a797f4e2b..52e24827c31 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java @@ -306,8 +306,7 @@ public enum RuleType { COLLECT_FILTER(RuleTypeClass.REWRITE), COLLECT_JOIN_CONSTRAINT(RuleTypeClass.REWRITE), - COLLECT_PROJECT_ABOVE_CTE_CONSUMER(RuleTypeClass.REWRITE), - COLLECT_PROJECT_ABOVE_FILTER_CTE_CONSUMER(RuleTypeClass.REWRITE), + COLLECT_CTE_CONSUMER_OUTPUT(RuleTypeClass.REWRITE), LEADING_JOIN(RuleTypeClass.REWRITE), REWRITE_SENTINEL(RuleTypeClass.REWRITE), diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/CollectCteConsumerOutput.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/CollectCteConsumerOutput.java new file mode 100644 index 00000000000..20b6dc40230 --- /dev/null +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/CollectCteConsumerOutput.java @@ -0,0 +1,41 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you under the Apache License, Version 2.0 (the +// "License"); you may not use this file except in compliance +// with the License. You may obtain a copy of the License at +// +// http://www.apache.org/licenses/LICENSE-2.0 +// +// Unless required by applicable law or agreed to in writing, +// software distributed under the License is distributed on an +// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY +// KIND, either express or implied. See the License for the +// specific language governing permissions and limitations +// under the License. + +package org.apache.doris.nereids.rules.rewrite; + +import org.apache.doris.nereids.rules.Rule; +import org.apache.doris.nereids.rules.RuleType; +import org.apache.doris.nereids.trees.expressions.Slot; + +import java.util.HashSet; +import java.util.Set; + +/** + * Collect outputs of CTE Consumer. + */ +public class CollectCteConsumerOutput extends OneRewriteRuleFactory { + + @Override + public Rule build() { + return logicalCTEConsumer().thenApply(ctx -> { + Set<Slot> producerOutputs = ctx.statementContext + .getCteIdToOutputIds().computeIfAbsent(ctx.root.getCteId(), k -> new HashSet<>()); + producerOutputs.addAll(ctx.root.getProducerToConsumerOutputMap().keySet()); + return null; + }).toRule(RuleType.COLLECT_CTE_CONSUMER_OUTPUT); + } +} diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/CollectProjectAboveConsumer.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/CollectProjectAboveConsumer.java deleted file mode 100644 index 0ecace726db..00000000000 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/CollectProjectAboveConsumer.java +++ /dev/null @@ -1,81 +0,0 @@ -// Licensed to the Apache Software Foundation (ASF) under one -// or more contributor license agreements. See the NOTICE file -// distributed with this work for additional information -// regarding copyright ownership. The ASF licenses this file -// to you under the Apache License, Version 2.0 (the -// "License"); you may not use this file except in compliance -// with the License. You may obtain a copy of the License at -// -// http://www.apache.org/licenses/LICENSE-2.0 -// -// Unless required by applicable law or agreed to in writing, -// software distributed under the License is distributed on an -// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY -// KIND, either express or implied. See the License for the -// specific language governing permissions and limitations -// under the License. - -package org.apache.doris.nereids.rules.rewrite; - -import org.apache.doris.nereids.CascadesContext; -import org.apache.doris.nereids.rules.Rule; -import org.apache.doris.nereids.rules.RuleType; -import org.apache.doris.nereids.trees.expressions.Expression; -import org.apache.doris.nereids.trees.expressions.NamedExpression; -import org.apache.doris.nereids.trees.expressions.Slot; -import org.apache.doris.nereids.trees.plans.logical.LogicalCTEConsumer; -import org.apache.doris.nereids.trees.plans.logical.LogicalFilter; -import org.apache.doris.nereids.trees.plans.logical.LogicalProject; - -import com.google.common.collect.ImmutableList; - -import java.util.ArrayList; -import java.util.List; -import java.util.Set; - -/** - * Collect Projects Above CTE Consumer. - */ -public class CollectProjectAboveConsumer implements RewriteRuleFactory { - - @Override - public List<Rule> buildRules() { - return ImmutableList.of(RuleType.COLLECT_PROJECT_ABOVE_CTE_CONSUMER - .build(logicalProject(logicalCTEConsumer()).thenApply(ctx -> { - LogicalProject<LogicalCTEConsumer> project = ctx.root; - List<NamedExpression> namedExpressions = project.getProjects(); - LogicalCTEConsumer cteConsumer = project.child(); - collectProject(ctx.cascadesContext, namedExpressions, cteConsumer); - return ctx.root; - })), - RuleType.COLLECT_PROJECT_ABOVE_FILTER_CTE_CONSUMER - .build(logicalProject(logicalFilter(logicalCTEConsumer())).thenApply(ctx -> { - LogicalProject<LogicalFilter<LogicalCTEConsumer>> project = ctx.root; - LogicalFilter<LogicalCTEConsumer> filter = project.child(); - Set<Slot> filterSlots = filter.getInputSlots(); - List<NamedExpression> namedExpressions = new ArrayList<>(project.getProjects()); - for (Slot slot : filterSlots) { - if (!project.getOutput().contains(slot)) { - namedExpressions.add(slot); - } - } - collectProject(ctx.cascadesContext, namedExpressions, filter.child()); - return ctx.root; - })) - ); - } - - private static void collectProject(CascadesContext ctx, - List<NamedExpression> namedExpressions, LogicalCTEConsumer cteConsumer) { - for (Expression expr : namedExpressions) { - expr.foreach(node -> { - if (!(node instanceof Slot)) { - return; - } - Slot slot = cteConsumer.getProducerSlot((Slot) node); - ctx.putCTEIdToProject(cteConsumer.getCteId(), slot); - ctx.markConsumerUnderProject(cteConsumer); - }); - } - } -} diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ColumnPruning.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ColumnPruning.java index e36c0f5172a..4cb18e8a380 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ColumnPruning.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ColumnPruning.java @@ -29,6 +29,7 @@ import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.trees.plans.algebra.Aggregate; import org.apache.doris.nereids.trees.plans.algebra.SetOperation.Qualifier; import org.apache.doris.nereids.trees.plans.logical.LogicalAggregate; +import org.apache.doris.nereids.trees.plans.logical.LogicalCTEConsumer; import org.apache.doris.nereids.trees.plans.logical.LogicalCTEProducer; import org.apache.doris.nereids.trees.plans.logical.LogicalExcept; import org.apache.doris.nereids.trees.plans.logical.LogicalIntersect; @@ -200,13 +201,21 @@ public class ColumnPruning extends DefaultPlanRewriter<PruneContext> implements return pruneAggregate(repeat, context); } - private Plan pruneAggregate(Aggregate agg, PruneContext context) { - // first try to prune group by and aggregate functions - Aggregate prunedOutputAgg = pruneOutput(agg, agg.getOutputs(), agg::pruneOutputs, context); + @Override + public Plan visitLogicalCTEProducer(LogicalCTEProducer<? extends Plan> cteProducer, PruneContext context) { + return skipPruneThisAndFirstLevelChildren(cteProducer); + } - Aggregate fillUpAggr = fillUpGroupByAndOutput(prunedOutputAgg); + @Override + public Plan visitLogicalCTEConsumer(LogicalCTEConsumer cteConsumer, PruneContext context) { + return super.visitLogicalCTEConsumer(cteConsumer, context); + } - return pruneChildren(fillUpAggr); + private Plan pruneAggregate(Aggregate<?> agg, PruneContext context) { + // first try to prune group by and aggregate functions + Aggregate<? extends Plan> prunedOutputAgg = pruneOutput(agg, agg.getOutputs(), agg::pruneOutputs, context); + Aggregate<?> fillUpAggregate = fillUpGroupByAndOutput(prunedOutputAgg); + return pruneChildren(fillUpAggregate); } private Plan skipPruneThisAndFirstLevelChildren(Plan plan) { @@ -217,7 +226,7 @@ public class ColumnPruning extends DefaultPlanRewriter<PruneContext> implements return pruneChildren(plan, requireAllOutputOfChildren.build()); } - private static Aggregate<Plan> fillUpGroupByAndOutput(Aggregate<Plan> prunedOutputAgg) { + private static Aggregate<? extends Plan> fillUpGroupByAndOutput(Aggregate<? extends Plan> prunedOutputAgg) { List<Expression> groupBy = prunedOutputAgg.getGroupByExpressions(); List<NamedExpression> output = prunedOutputAgg.getOutputExpressions(); @@ -239,12 +248,11 @@ public class ColumnPruning extends DefaultPlanRewriter<PruneContext> implements ImmutableList.Builder<Expression> newGroupByExprList = ImmutableList.builderWithExpectedSize(newOutputList.size()); for (NamedExpression e : newOutputList) { - if (!(aggregateFunctions.contains(e) - || (e instanceof Alias && aggregateFunctions.contains(e.child(0))))) { + if (!(e instanceof Alias && aggregateFunctions.contains(e.child(0)))) { newGroupByExprList.add(e); } } - return ((LogicalAggregate<Plan>) prunedOutputAgg).withGroupByAndOutput( + return ((LogicalAggregate<? extends Plan>) prunedOutputAgg).withGroupByAndOutput( newGroupByExprList.build(), newOutputList); } @@ -371,11 +379,6 @@ public class ColumnPruning extends DefaultPlanRewriter<PruneContext> implements return prunedChild; } - @Override - public Plan visitLogicalCTEProducer(LogicalCTEProducer<? extends Plan> cteProducer, PruneContext context) { - return skipPruneThisAndFirstLevelChildren(cteProducer); - } - /** PruneContext */ public static class PruneContext { public Set<Slot> requiredSlots; diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/RewriteCteChildren.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/RewriteCteChildren.java index 72a4603fadc..3a2da623b4c 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/RewriteCteChildren.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/RewriteCteChildren.java @@ -26,6 +26,7 @@ import org.apache.doris.nereids.properties.PhysicalProperties; import org.apache.doris.nereids.trees.expressions.CTEId; import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.expressions.NamedExpression; +import org.apache.doris.nereids.trees.expressions.Slot; import org.apache.doris.nereids.trees.expressions.SlotReference; import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.trees.plans.RelationId; @@ -41,7 +42,6 @@ import org.apache.doris.nereids.util.ExpressionUtils; import com.google.common.collect.ImmutableList; import com.google.common.collect.ImmutableSet; -import org.apache.commons.collections.CollectionUtils; import java.util.HashSet; import java.util.List; @@ -109,10 +109,17 @@ public class RewriteCteChildren extends DefaultPlanRewriter<CascadesContext> imp } else { child = (LogicalPlan) cteProducer.child(); child = tryToConstructFilter(cascadesContext, cteProducer.getCteId(), child); - Set<NamedExpression> projects = cascadesContext.getProjectForProducer(cteProducer.getCteId()); - if (CollectionUtils.isNotEmpty(projects) - && cascadesContext.couldPruneColumnOnProducer(cteProducer.getCteId())) { - child = new LogicalProject<>(ImmutableList.copyOf(projects), child); + Set<Slot> producerOutputs = cascadesContext.getStatementContext() + .getCteIdToOutputIds().get(cteProducer.getCteId()); + if (producerOutputs.size() < child.getOutput().size()) { + ImmutableList.Builder<NamedExpression> projectsBuilder + = ImmutableList.builderWithExpectedSize(producerOutputs.size()); + for (Slot slot : child.getOutput()) { + if (producerOutputs.contains(slot)) { + projectsBuilder.add(slot); + } + } + child = new LogicalProject<>(projectsBuilder.build(), child); child = pushPlanUnderAnchor(child); } CascadesContext rewrittenCtx = CascadesContext.newSubtreeContext( diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalCTEConsumer.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalCTEConsumer.java index 71b1c43f791..5fd088a2bba 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalCTEConsumer.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalCTEConsumer.java @@ -20,6 +20,7 @@ package org.apache.doris.nereids.trees.plans.logical; import org.apache.doris.nereids.memo.GroupExpression; import org.apache.doris.nereids.properties.LogicalProperties; import org.apache.doris.nereids.trees.expressions.CTEId; +import org.apache.doris.nereids.trees.expressions.NamedExpression; import org.apache.doris.nereids.trees.expressions.Slot; import org.apache.doris.nereids.trees.expressions.SlotReference; import org.apache.doris.nereids.trees.expressions.StatementScopeIdGenerator; @@ -36,6 +37,7 @@ import com.google.common.collect.ImmutableList; import java.util.LinkedHashMap; import java.util.List; import java.util.Map; +import java.util.Map.Entry; import java.util.Objects; import java.util.Optional; @@ -43,7 +45,7 @@ import java.util.Optional; * LogicalCTEConsumer */ //TODO: find cte producer and propagate its functional dependencies -public class LogicalCTEConsumer extends LogicalRelation implements BlockFuncDepsPropagation { +public class LogicalCTEConsumer extends LogicalRelation implements BlockFuncDepsPropagation, OutputPrunable { private final String name; private final CTEId cteId; @@ -145,6 +147,24 @@ public class LogicalCTEConsumer extends LogicalRelation implements BlockFuncDeps return ImmutableList.copyOf(producerToConsumerOutputMap.values()); } + @Override + public Plan pruneOutputs(List<NamedExpression> prunedOutputs) { + Map<Slot, Slot> consumerToProducerOutputMap = new LinkedHashMap<>(this.consumerToProducerOutputMap.size()); + Map<Slot, Slot> producerToConsumerOutputMap = new LinkedHashMap<>(this.consumerToProducerOutputMap.size()); + for (Entry<Slot, Slot> consumerToProducerSlot : this.consumerToProducerOutputMap.entrySet()) { + if (prunedOutputs.contains(consumerToProducerSlot.getKey())) { + consumerToProducerOutputMap.put(consumerToProducerSlot.getKey(), consumerToProducerSlot.getValue()); + producerToConsumerOutputMap.put(consumerToProducerSlot.getValue(), consumerToProducerSlot.getKey()); + } + } + return withTwoMaps(consumerToProducerOutputMap, producerToConsumerOutputMap); + } + + @Override + public List<NamedExpression> getOutputs() { + return (List) this.getOutput(); + } + public CTEId getCteId() { return cteId; } diff --git a/regression-test/data/nereids_hint_tpcds_p0/shape/query1.out b/regression-test/data/nereids_hint_tpcds_p0/shape/query1.out index e0104b54a42..996ccd7623c 100644 --- a/regression-test/data/nereids_hint_tpcds_p0/shape/query1.out +++ b/regression-test/data/nereids_hint_tpcds_p0/shape/query1.out @@ -24,8 +24,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------PhysicalDistribute[DistributionSpecHash] ------------------hashAgg[LOCAL] --------------------PhysicalDistribute[DistributionSpecExecutionAny] -----------------------PhysicalProject -------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------PhysicalProject ----------------hashJoin[INNER_JOIN] hashCondition=((store.s_store_sk = ctr1.ctr_store_sk)) otherCondition=() build RFs:RF1 s_store_sk->[ctr_store_sk] ------------------PhysicalDistribute[DistributionSpecHash] diff --git a/regression-test/data/nereids_hint_tpcds_p0/shape/query24.out b/regression-test/data/nereids_hint_tpcds_p0/shape/query24.out index f37faa9bff4..2e2658db162 100644 --- a/regression-test/data/nereids_hint_tpcds_p0/shape/query24.out +++ b/regression-test/data/nereids_hint_tpcds_p0/shape/query24.out @@ -47,8 +47,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------PhysicalDistribute[DistributionSpecGather] ------------------hashAgg[LOCAL] --------------------PhysicalDistribute[DistributionSpecExecutionAny] -----------------------PhysicalProject -------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------PhysicalDistribute[DistributionSpecReplicated] --------------PhysicalProject ----------------hashAgg[GLOBAL] diff --git a/regression-test/data/nereids_p0/hint/multi_leading.out b/regression-test/data/nereids_p0/hint/multi_leading.out index 51ecab29494..71db5aec524 100644 --- a/regression-test/data/nereids_p0/hint/multi_leading.out +++ b/regression-test/data/nereids_p0/hint/multi_leading.out @@ -613,8 +613,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = cte.c11)) otherCondition=() ------------------------PhysicalOlapScan[t1] ------------------------PhysicalDistribute[DistributionSpecHash] ---------------------------PhysicalProject -----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) -- !sql5_2 -- PhysicalCteAnchor ( cteId=CTEId#0 ) @@ -639,8 +638,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalProject ----------------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = cte.c11)) otherCondition=() ------------------------PhysicalDistribute[DistributionSpecExecutionAny] ---------------------------PhysicalProject -----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------------------PhysicalDistribute[DistributionSpecReplicated] --------------------------PhysicalOlapScan[t1] diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query1.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query1.out index b2b5a87ac2b..2317f9435be 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query1.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query1.out @@ -39,6 +39,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------------PhysicalDistribute[DistributionSpecHash] --------------------------hashAgg[LOCAL] ----------------------------PhysicalDistribute[DistributionSpecExecutionAny] -------------------------------PhysicalProject ---------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query23.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query23.out index b937f23400b..0475cae9f95 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query23.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query23.out @@ -59,8 +59,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------PhysicalProject --------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((catalog_sales.cs_item_sk = frequent_ss_items.item_sk)) otherCondition=() build RFs:RF4 cs_item_sk->[item_sk] ----------------------PhysicalDistribute[DistributionSpecHash] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF4 +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF4 ----------------------PhysicalDistribute[DistributionSpecHash] ------------------------PhysicalProject --------------------------hashJoin[LEFT_SEMI_JOIN] hashCondition=((catalog_sales.cs_bill_customer_sk = best_ss_customer.c_customer_sk)) otherCondition=() @@ -73,13 +72,11 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------------------------filter((date_dim.d_moy = 7) and (date_dim.d_year = 2000)) --------------------------------------PhysicalOlapScan[date_dim] ----------------------------PhysicalDistribute[DistributionSpecHash] -------------------------------PhysicalProject ---------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) ------------------PhysicalProject --------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((web_sales.ws_item_sk = frequent_ss_items.item_sk)) otherCondition=() build RFs:RF6 ws_item_sk->[item_sk] ----------------------PhysicalDistribute[DistributionSpecHash] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF6 +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF6 ----------------------PhysicalDistribute[DistributionSpecHash] ------------------------PhysicalProject --------------------------hashJoin[LEFT_SEMI_JOIN] hashCondition=((web_sales.ws_bill_customer_sk = best_ss_customer.c_customer_sk)) otherCondition=() @@ -92,6 +89,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------------------------filter((date_dim.d_moy = 7) and (date_dim.d_year = 2000)) --------------------------------------PhysicalOlapScan[date_dim] ----------------------------PhysicalDistribute[DistributionSpecHash] -------------------------------PhysicalProject ---------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query24.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query24.out index 83f4e91b4a2..ebdb27b0c5d 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query24.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query24.out @@ -55,6 +55,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecGather] ----------------------hashAgg[LOCAL] ------------------------PhysicalDistribute[DistributionSpecExecutionAny] ---------------------------PhysicalProject -----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query30.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query30.out index 7dcac891ad1..7272f6c9e26 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query30.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query30.out @@ -43,6 +43,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] --------------------hashAgg[LOCAL] ----------------------PhysicalDistribute[DistributionSpecExecutionAny] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query31.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query31.out index 11e812ae39b..f22860f874d 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query31.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query31.out @@ -2,44 +2,42 @@ -- !ds_shape_31 -- PhysicalCteAnchor ( cteId=CTEId#0 ) --PhysicalCteProducer ( cteId=CTEId#0 ) -----PhysicalProject +----hashAgg[GLOBAL] +------PhysicalDistribute[DistributionSpecHash] +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF1 ca_address_sk->[ss_addr_sk] +--------------PhysicalDistribute[DistributionSpecHash] +----------------PhysicalProject +------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +--------------------PhysicalProject +----------------------PhysicalOlapScan[store_sales] apply RFs: RF0 RF1 +--------------------PhysicalDistribute[DistributionSpecReplicated] +----------------------PhysicalProject +------------------------filter((ss.d_year = 1999) and d_qoy IN (1, 2, 3)) +--------------------------PhysicalOlapScan[date_dim] +--------------PhysicalDistribute[DistributionSpecHash] +----------------PhysicalProject +------------------PhysicalOlapScan[customer_address] +--PhysicalCteAnchor ( cteId=CTEId#1 ) +----PhysicalCteProducer ( cteId=CTEId#1 ) ------hashAgg[GLOBAL] --------PhysicalDistribute[DistributionSpecHash] ----------hashAgg[LOCAL] ------------PhysicalProject ---------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF1 ca_address_sk->[ss_addr_sk] +--------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_bill_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF3 ca_address_sk->[ws_bill_addr_sk] ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject ---------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +--------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ws_sold_date_sk] ----------------------PhysicalProject -------------------------PhysicalOlapScan[store_sales] apply RFs: RF0 RF1 +------------------------PhysicalOlapScan[web_sales] apply RFs: RF2 RF3 ----------------------PhysicalDistribute[DistributionSpecReplicated] ------------------------PhysicalProject ---------------------------filter((ss.d_year = 1999) and d_qoy IN (1, 2, 3)) +--------------------------filter((ws.d_year = 1999) and d_qoy IN (1, 2, 3)) ----------------------------PhysicalOlapScan[date_dim] ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject --------------------PhysicalOlapScan[customer_address] ---PhysicalCteAnchor ( cteId=CTEId#1 ) -----PhysicalCteProducer ( cteId=CTEId#1 ) -------PhysicalProject ---------hashAgg[GLOBAL] -----------PhysicalDistribute[DistributionSpecHash] -------------hashAgg[LOCAL] ---------------PhysicalProject -----------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_bill_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF3 ca_address_sk->[ws_bill_addr_sk] -------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ws_sold_date_sk] -------------------------PhysicalProject ---------------------------PhysicalOlapScan[web_sales] apply RFs: RF2 RF3 -------------------------PhysicalDistribute[DistributionSpecReplicated] ---------------------------PhysicalProject -----------------------------filter((ws.d_year = 1999) and d_qoy IN (1, 2, 3)) -------------------------------PhysicalOlapScan[date_dim] -------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------PhysicalOlapScan[customer_address] ----PhysicalResultSink ------PhysicalQuickSort[MERGE_SORT] --------PhysicalDistribute[DistributionSpecGather] diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query39.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query39.out index 2fb33f5848e..9d2e1eb162d 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query39.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query39.out @@ -29,11 +29,9 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------PhysicalQuickSort[LOCAL_SORT] ----------hashJoin[INNER_JOIN] hashCondition=((inv1.i_item_sk = inv2.i_item_sk) and (inv1.w_warehouse_sk = inv2.w_warehouse_sk)) otherCondition=() ------------PhysicalDistribute[DistributionSpecHash] ---------------PhysicalProject -----------------filter((inv1.d_moy = 1)) -------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------filter((inv1.d_moy = 1)) +----------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------PhysicalDistribute[DistributionSpecHash] ---------------PhysicalProject -----------------filter((inv2.d_moy = 2)) -------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------filter((inv2.d_moy = 2)) +----------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query47.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query47.out index 1ff72561a11..e57e35ba769 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query47.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query47.out @@ -41,9 +41,8 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalProject ----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2000)) -------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2000)) +----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject --------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query57.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query57.out index 628a6aa98a3..79c67bde5cb 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query57.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query57.out @@ -41,9 +41,8 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalProject ----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2001)) -------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2001)) +----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject --------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query59.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query59.out index ca30f76b1d7..84bd2d432ac 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query59.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query59.out @@ -2,17 +2,16 @@ -- !ds_shape_59 -- PhysicalCteAnchor ( cteId=CTEId#0 ) --PhysicalCteProducer ( cteId=CTEId#0 ) -----PhysicalProject -------hashAgg[GLOBAL] ---------PhysicalDistribute[DistributionSpecHash] -----------hashAgg[LOCAL] -------------PhysicalProject ---------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_date_sk = store_sales.ss_sold_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +----hashAgg[GLOBAL] +------PhysicalDistribute[DistributionSpecHash] +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_date_sk = store_sales.ss_sold_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +--------------PhysicalProject +----------------PhysicalOlapScan[store_sales] apply RFs: RF0 +--------------PhysicalDistribute[DistributionSpecReplicated] ----------------PhysicalProject -------------------PhysicalOlapScan[store_sales] apply RFs: RF0 -----------------PhysicalDistribute[DistributionSpecReplicated] -------------------PhysicalProject ---------------------PhysicalOlapScan[date_dim] +------------------PhysicalOlapScan[date_dim] --PhysicalResultSink ----PhysicalTopN[MERGE_SORT] ------PhysicalDistribute[DistributionSpecGather] diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query70.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query70.out index fdbfa12a4b7..5eb9fba4824 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query70.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query70.out @@ -45,3 +45,4 @@ PhysicalResultSink --------------------------------------------------PhysicalDistribute[DistributionSpecReplicated] ----------------------------------------------------PhysicalProject ------------------------------------------------------PhysicalOlapScan[store] + diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query81.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query81.out index 20a169a0faa..61f4343fd1f 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query81.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query81.out @@ -44,6 +44,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] --------------------hashAgg[LOCAL] ----------------------PhysicalDistribute[DistributionSpecExecutionAny] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query95.out b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query95.out index b0a0655caff..c6f2d22db15 100644 --- a/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query95.out +++ b/regression-test/data/nereids_tpcds_shape_sf1000_p0/shape/query95.out @@ -23,16 +23,14 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------------PhysicalProject ------------------------hashJoin[INNER_JOIN] hashCondition=((web_returns.wr_order_number = ws_wh.ws_order_number)) otherCondition=() build RFs:RF5 wr_order_number->[ws_order_number] --------------------------PhysicalDistribute[DistributionSpecHash] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF5 RF6 +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF5 RF6 --------------------------PhysicalDistribute[DistributionSpecHash] ----------------------------PhysicalProject ------------------------------PhysicalOlapScan[web_returns] apply RFs: RF6 ----------------------PhysicalProject ------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((ws1.ws_order_number = ws_wh.ws_order_number)) otherCondition=() build RFs:RF7 ws_order_number->[ws_order_number,ws_order_number] --------------------------PhysicalDistribute[DistributionSpecHash] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------------------PhysicalDistribute[DistributionSpecHash] ----------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_web_site_sk = web_site.web_site_sk)) otherCondition=() build RFs:RF3 web_site_sk->[ws_web_site_sk] ------------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ws_ship_date_sk] diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/constraints/query23.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/constraints/query23.out index ddff36aebc7..8668943e20f 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/constraints/query23.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/constraints/query23.out @@ -69,11 +69,9 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------------------------filter((date_dim.d_moy = 5) and (date_dim.d_year = 2000)) --------------------------------------PhysicalOlapScan[date_dim] ----------------------------PhysicalDistribute[DistributionSpecHash] -------------------------------PhysicalProject ---------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) ----------------------PhysicalDistribute[DistributionSpecHash] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------------PhysicalProject --------------------hashJoin[LEFT_SEMI_JOIN] hashCondition=((web_sales.ws_item_sk = frequent_ss_items.item_sk)) otherCondition=() ----------------------PhysicalDistribute[DistributionSpecHash] @@ -88,9 +86,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------------------------filter((date_dim.d_moy = 5) and (date_dim.d_year = 2000)) --------------------------------------PhysicalOlapScan[date_dim] ----------------------------PhysicalDistribute[DistributionSpecHash] -------------------------------PhysicalProject ---------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) ----------------------PhysicalDistribute[DistributionSpecHash] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query1.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query1.out index 65dca9e89f3..8528bf7b49b 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query1.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query1.out @@ -37,6 +37,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------PhysicalDistribute[DistributionSpecHash] ------------------hashAgg[LOCAL] --------------------PhysicalDistribute[DistributionSpecExecutionAny] -----------------------PhysicalProject -------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query23.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query23.out index 431330e9039..22f483bb5a7 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query23.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query23.out @@ -65,11 +65,9 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------------------PhysicalProject ----------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF3 ------------------------------PhysicalDistribute[DistributionSpecHash] ---------------------------------PhysicalProject -----------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +--------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) --------------------------PhysicalDistribute[DistributionSpecHash] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ----------------------PhysicalDistribute[DistributionSpecReplicated] ------------------------PhysicalProject --------------------------filter((date_dim.d_moy = 5) and (date_dim.d_year = 2000)) @@ -84,11 +82,9 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------------------PhysicalProject ----------------------------------PhysicalOlapScan[web_sales] apply RFs: RF4 ------------------------------PhysicalDistribute[DistributionSpecHash] ---------------------------------PhysicalProject -----------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +--------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) --------------------------PhysicalDistribute[DistributionSpecHash] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ----------------------PhysicalDistribute[DistributionSpecReplicated] ------------------------PhysicalProject --------------------------filter((date_dim.d_moy = 5) and (date_dim.d_year = 2000)) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query24.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query24.out index b9a89aae5e9..9699ebf9255 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query24.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query24.out @@ -52,6 +52,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecGather] ----------------------hashAgg[LOCAL] ------------------------PhysicalDistribute[DistributionSpecExecutionAny] ---------------------------PhysicalProject -----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query30.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query30.out index 985c714b0ab..61524dd0138 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query30.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query30.out @@ -39,8 +39,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecHash] ----------------------hashAgg[LOCAL] ------------------------PhysicalDistribute[DistributionSpecExecutionAny] ---------------------------PhysicalProject -----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------PhysicalDistribute[DistributionSpecReplicated] ----------------PhysicalProject ------------------filter((customer_address.ca_state = 'IN')) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query31.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query31.out index 3cacf0ae184..0852d3abe2e 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query31.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query31.out @@ -2,30 +2,30 @@ -- !ds_shape_31 -- PhysicalCteAnchor ( cteId=CTEId#0 ) --PhysicalCteProducer ( cteId=CTEId#0 ) -----PhysicalProject -------hashAgg[GLOBAL] ---------PhysicalDistribute[DistributionSpecHash] -----------hashAgg[LOCAL] -------------PhysicalProject ---------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[ss_sold_date_sk] +----hashAgg[GLOBAL] +------PhysicalDistribute[DistributionSpecHash] +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[ss_sold_date_sk] +--------------PhysicalProject ----------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_addr_sk = customer_address.ca_address_sk)) otherCondition=() ------------------PhysicalProject --------------------PhysicalOlapScan[store_sales] apply RFs: RF1 ------------------PhysicalDistribute[DistributionSpecReplicated] --------------------PhysicalProject ----------------------PhysicalOlapScan[customer_address] -----------------PhysicalDistribute[DistributionSpecReplicated] -------------------PhysicalProject ---------------------filter((ss.d_year = 2000) and d_qoy IN (1, 2, 3)) -----------------------PhysicalOlapScan[date_dim] +--------------PhysicalDistribute[DistributionSpecReplicated] +----------------PhysicalProject +------------------filter((ss.d_year = 2000) and d_qoy IN (1, 2, 3)) +--------------------PhysicalOlapScan[date_dim] --PhysicalCteAnchor ( cteId=CTEId#1 ) ----PhysicalCteProducer ( cteId=CTEId#1 ) -------PhysicalProject ---------hashAgg[GLOBAL] -----------PhysicalDistribute[DistributionSpecHash] -------------hashAgg[LOCAL] ---------------PhysicalProject -----------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF3 d_date_sk->[ws_sold_date_sk] +------hashAgg[GLOBAL] +--------PhysicalDistribute[DistributionSpecHash] +----------hashAgg[LOCAL] +------------PhysicalProject +--------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF3 d_date_sk->[ws_sold_date_sk] +----------------PhysicalProject ------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_bill_addr_sk = customer_address.ca_address_sk)) otherCondition=() --------------------PhysicalDistribute[DistributionSpecHash] ----------------------PhysicalProject @@ -33,10 +33,10 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecHash] ----------------------PhysicalProject ------------------------PhysicalOlapScan[customer_address] -------------------PhysicalDistribute[DistributionSpecReplicated] ---------------------PhysicalProject -----------------------filter((ws.d_year = 2000) and d_qoy IN (1, 2, 3)) -------------------------PhysicalOlapScan[date_dim] +----------------PhysicalDistribute[DistributionSpecReplicated] +------------------PhysicalProject +--------------------filter((ws.d_year = 2000) and d_qoy IN (1, 2, 3)) +----------------------PhysicalOlapScan[date_dim] ----PhysicalResultSink ------PhysicalQuickSort[MERGE_SORT] --------PhysicalDistribute[DistributionSpecGather] diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query39.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query39.out index 9ffc6dc5e07..421fa8a7499 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query39.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query39.out @@ -28,11 +28,9 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------PhysicalQuickSort[LOCAL_SORT] ----------hashJoin[INNER_JOIN] hashCondition=((inv1.i_item_sk = inv2.i_item_sk) and (inv1.w_warehouse_sk = inv2.w_warehouse_sk)) otherCondition=() ------------PhysicalDistribute[DistributionSpecHash] ---------------PhysicalProject -----------------filter((inv1.d_moy = 1)) -------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------filter((inv1.d_moy = 1)) +----------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------PhysicalDistribute[DistributionSpecHash] ---------------PhysicalProject -----------------filter((inv2.d_moy = 2)) -------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------filter((inv2.d_moy = 2)) +----------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query47.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query47.out index fb20900a0b6..430c3c40673 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query47.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query47.out @@ -46,7 +46,6 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------------PhysicalProject ------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecHash] -----------------------PhysicalProject -------------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2001)) ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2001)) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query57.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query57.out index 18d9a45e7d6..ed1d2952975 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query57.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query57.out @@ -46,7 +46,6 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------------PhysicalProject ------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecHash] -----------------------PhysicalProject -------------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 1999)) ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 1999)) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query59.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query59.out index e871b1a1a44..3347fd02ab4 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query59.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query59.out @@ -2,17 +2,16 @@ -- !ds_shape_59 -- PhysicalCteAnchor ( cteId=CTEId#0 ) --PhysicalCteProducer ( cteId=CTEId#0 ) -----PhysicalProject -------hashAgg[GLOBAL] ---------PhysicalDistribute[DistributionSpecHash] -----------hashAgg[LOCAL] -------------PhysicalProject ---------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_date_sk = store_sales.ss_sold_date_sk)) otherCondition=() +----hashAgg[GLOBAL] +------PhysicalDistribute[DistributionSpecHash] +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_date_sk = store_sales.ss_sold_date_sk)) otherCondition=() +--------------PhysicalProject +----------------PhysicalOlapScan[store_sales] +--------------PhysicalDistribute[DistributionSpecReplicated] ----------------PhysicalProject -------------------PhysicalOlapScan[store_sales] -----------------PhysicalDistribute[DistributionSpecReplicated] -------------------PhysicalProject ---------------------PhysicalOlapScan[date_dim] +------------------PhysicalOlapScan[date_dim] --PhysicalResultSink ----PhysicalTopN[MERGE_SORT] ------PhysicalDistribute[DistributionSpecGather] diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query70.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query70.out index abdcd1b0149..d3e7d441cec 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query70.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query70.out @@ -45,3 +45,4 @@ PhysicalResultSink ------------------------------------PhysicalDistribute[DistributionSpecHash] --------------------------------------PhysicalProject ----------------------------------------PhysicalOlapScan[store] + diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query81.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query81.out index ac9cf29ee12..e9fea1c43c2 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query81.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query81.out @@ -40,8 +40,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------------PhysicalDistribute[DistributionSpecHash] ------------------------hashAgg[LOCAL] --------------------------PhysicalDistribute[DistributionSpecExecutionAny] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------PhysicalDistribute[DistributionSpecHash] ----------------PhysicalProject ------------------filter((customer_address.ca_state = 'CA')) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query95.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query95.out index b042e353174..3cc3f5843b2 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query95.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/noStatsRfPrune/query95.out @@ -21,8 +21,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------PhysicalProject --------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((ws1.ws_order_number = ws_wh.ws_order_number)) otherCondition=() build RFs:RF6 ws_order_number->[ws_order_number] ----------------------PhysicalDistribute[DistributionSpecHash] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF6 +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF6 ----------------------PhysicalProject ------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_web_site_sk = web_site.web_site_sk)) otherCondition=() build RFs:RF5 web_site_sk->[ws_web_site_sk] --------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF4 ca_address_sk->[ws_ship_addr_sk] @@ -31,8 +30,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------------------PhysicalProject ----------------------------------hashJoin[INNER_JOIN] hashCondition=((web_returns.wr_order_number = ws_wh.ws_order_number)) otherCondition=() ------------------------------------PhysicalDistribute[DistributionSpecHash] ---------------------------------------PhysicalProject -----------------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------------------------------PhysicalDistribute[DistributionSpecHash] --------------------------------------PhysicalProject ----------------------------------------PhysicalOlapScan[web_returns] apply RFs: RF2 diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query1.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query1.out index e1300825fc2..422dff53364 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query1.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query1.out @@ -37,6 +37,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------PhysicalDistribute[DistributionSpecHash] ------------------hashAgg[LOCAL] --------------------PhysicalDistribute[DistributionSpecExecutionAny] -----------------------PhysicalProject -------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query23.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query23.out index 049a85a8860..662fc3aa2dd 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query23.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query23.out @@ -65,11 +65,9 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------------------PhysicalProject ----------------------------------PhysicalOlapScan[catalog_sales] apply RFs: RF3 ------------------------------PhysicalDistribute[DistributionSpecHash] ---------------------------------PhysicalProject -----------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +--------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) --------------------------PhysicalDistribute[DistributionSpecHash] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ----------------------PhysicalDistribute[DistributionSpecReplicated] ------------------------PhysicalProject --------------------------filter((date_dim.d_moy = 5) and (date_dim.d_year = 2000)) @@ -84,11 +82,9 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------------------PhysicalProject ----------------------------------PhysicalOlapScan[web_sales] apply RFs: RF4 ------------------------------PhysicalDistribute[DistributionSpecHash] ---------------------------------PhysicalProject -----------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +--------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) --------------------------PhysicalDistribute[DistributionSpecHash] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ----------------------PhysicalDistribute[DistributionSpecReplicated] ------------------------PhysicalProject --------------------------filter((date_dim.d_moy = 5) and (date_dim.d_year = 2000)) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query24.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query24.out index 9f6191dac84..15252933e01 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query24.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query24.out @@ -52,6 +52,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecGather] ----------------------hashAgg[LOCAL] ------------------------PhysicalDistribute[DistributionSpecExecutionAny] ---------------------------PhysicalProject -----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query30.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query30.out index bfa6bc4fe65..ccd62e114b4 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query30.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query30.out @@ -39,8 +39,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecHash] ----------------------hashAgg[LOCAL] ------------------------PhysicalDistribute[DistributionSpecExecutionAny] ---------------------------PhysicalProject -----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------PhysicalDistribute[DistributionSpecReplicated] ----------------PhysicalProject ------------------filter((customer_address.ca_state = 'IN')) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query31.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query31.out index 5a0036ae4d5..65a1a9afe0c 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query31.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query31.out @@ -2,30 +2,30 @@ -- !ds_shape_31 -- PhysicalCteAnchor ( cteId=CTEId#0 ) --PhysicalCteProducer ( cteId=CTEId#0 ) -----PhysicalProject -------hashAgg[GLOBAL] ---------PhysicalDistribute[DistributionSpecHash] -----------hashAgg[LOCAL] -------------PhysicalProject ---------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[ss_sold_date_sk] +----hashAgg[GLOBAL] +------PhysicalDistribute[DistributionSpecHash] +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF1 d_date_sk->[ss_sold_date_sk] +--------------PhysicalProject ----------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF0 ca_address_sk->[ss_addr_sk] ------------------PhysicalProject --------------------PhysicalOlapScan[store_sales] apply RFs: RF0 RF1 ------------------PhysicalDistribute[DistributionSpecReplicated] --------------------PhysicalProject ----------------------PhysicalOlapScan[customer_address] -----------------PhysicalDistribute[DistributionSpecReplicated] -------------------PhysicalProject ---------------------filter((ss.d_year = 2000) and d_qoy IN (1, 2, 3)) -----------------------PhysicalOlapScan[date_dim] +--------------PhysicalDistribute[DistributionSpecReplicated] +----------------PhysicalProject +------------------filter((ss.d_year = 2000) and d_qoy IN (1, 2, 3)) +--------------------PhysicalOlapScan[date_dim] --PhysicalCteAnchor ( cteId=CTEId#1 ) ----PhysicalCteProducer ( cteId=CTEId#1 ) -------PhysicalProject ---------hashAgg[GLOBAL] -----------PhysicalDistribute[DistributionSpecHash] -------------hashAgg[LOCAL] ---------------PhysicalProject -----------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF3 d_date_sk->[ws_sold_date_sk] +------hashAgg[GLOBAL] +--------PhysicalDistribute[DistributionSpecHash] +----------hashAgg[LOCAL] +------------PhysicalProject +--------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF3 d_date_sk->[ws_sold_date_sk] +----------------PhysicalProject ------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_bill_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF2 ca_address_sk->[ws_bill_addr_sk] --------------------PhysicalDistribute[DistributionSpecHash] ----------------------PhysicalProject @@ -33,10 +33,10 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecHash] ----------------------PhysicalProject ------------------------PhysicalOlapScan[customer_address] -------------------PhysicalDistribute[DistributionSpecReplicated] ---------------------PhysicalProject -----------------------filter((ws.d_year = 2000) and d_qoy IN (1, 2, 3)) -------------------------PhysicalOlapScan[date_dim] +----------------PhysicalDistribute[DistributionSpecReplicated] +------------------PhysicalProject +--------------------filter((ws.d_year = 2000) and d_qoy IN (1, 2, 3)) +----------------------PhysicalOlapScan[date_dim] ----PhysicalResultSink ------PhysicalQuickSort[MERGE_SORT] --------PhysicalDistribute[DistributionSpecGather] diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query39.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query39.out index 40f877acac5..d1d9fb39429 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query39.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query39.out @@ -28,11 +28,9 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------PhysicalQuickSort[LOCAL_SORT] ----------hashJoin[INNER_JOIN] hashCondition=((inv1.i_item_sk = inv2.i_item_sk) and (inv1.w_warehouse_sk = inv2.w_warehouse_sk)) otherCondition=() ------------PhysicalDistribute[DistributionSpecHash] ---------------PhysicalProject -----------------filter((inv1.d_moy = 1)) -------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------filter((inv1.d_moy = 1)) +----------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------PhysicalDistribute[DistributionSpecHash] ---------------PhysicalProject -----------------filter((inv2.d_moy = 2)) -------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------filter((inv2.d_moy = 2)) +----------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query47.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query47.out index 59d526865c4..788b686c26f 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query47.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query47.out @@ -46,7 +46,6 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------------PhysicalProject ------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecHash] -----------------------PhysicalProject -------------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2001)) ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2001)) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query57.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query57.out index 52cd80d56cc..e2c13ea729c 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query57.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query57.out @@ -46,7 +46,6 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------------PhysicalProject ------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecHash] -----------------------PhysicalProject -------------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 1999)) ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 1999)) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query59.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query59.out index 6027c756904..fd888cc3382 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query59.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query59.out @@ -2,17 +2,16 @@ -- !ds_shape_59 -- PhysicalCteAnchor ( cteId=CTEId#0 ) --PhysicalCteProducer ( cteId=CTEId#0 ) -----PhysicalProject -------hashAgg[GLOBAL] ---------PhysicalDistribute[DistributionSpecHash] -----------hashAgg[LOCAL] -------------PhysicalProject ---------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_date_sk = store_sales.ss_sold_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +----hashAgg[GLOBAL] +------PhysicalDistribute[DistributionSpecHash] +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_date_sk = store_sales.ss_sold_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +--------------PhysicalProject +----------------PhysicalOlapScan[store_sales] apply RFs: RF0 +--------------PhysicalDistribute[DistributionSpecReplicated] ----------------PhysicalProject -------------------PhysicalOlapScan[store_sales] apply RFs: RF0 -----------------PhysicalDistribute[DistributionSpecReplicated] -------------------PhysicalProject ---------------------PhysicalOlapScan[date_dim] +------------------PhysicalOlapScan[date_dim] --PhysicalResultSink ----PhysicalTopN[MERGE_SORT] ------PhysicalDistribute[DistributionSpecGather] diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query81.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query81.out index 6463028a8fe..fb68f6ce1a3 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query81.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query81.out @@ -40,8 +40,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------------PhysicalDistribute[DistributionSpecHash] ------------------------hashAgg[LOCAL] --------------------------PhysicalDistribute[DistributionSpecExecutionAny] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------PhysicalDistribute[DistributionSpecHash] ----------------PhysicalProject ------------------filter((customer_address.ca_state = 'CA')) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query95.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query95.out index 9e96715c5e4..4fd762ec994 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query95.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/no_stats_shape/query95.out @@ -21,8 +21,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------PhysicalProject --------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((ws1.ws_order_number = ws_wh.ws_order_number)) otherCondition=() build RFs:RF12 ws_order_number->[ws_order_number];RF13 ws_order_number->[ws_order_number] ----------------------PhysicalDistribute[DistributionSpecHash] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF12 RF13 +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF12 RF13 ----------------------PhysicalProject ------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_web_site_sk = web_site.web_site_sk)) otherCondition=() build RFs:RF10 web_site_sk->[ws_web_site_sk];RF11 web_site_sk->[ws_web_site_sk] --------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF8 ca_address_sk->[ws_ship_addr_sk];RF9 ca_address_sk->[ws_ship_addr_sk] @@ -31,8 +30,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------------------PhysicalProject ----------------------------------hashJoin[INNER_JOIN] hashCondition=((web_returns.wr_order_number = ws_wh.ws_order_number)) otherCondition=() build RFs:RF2 wr_order_number->[ws_order_number];RF3 wr_order_number->[ws_order_number] ------------------------------------PhysicalDistribute[DistributionSpecHash] ---------------------------------------PhysicalProject -----------------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF2 RF3 +--------------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF2 RF3 ------------------------------------PhysicalDistribute[DistributionSpecHash] --------------------------------------PhysicalProject ----------------------------------------PhysicalOlapScan[web_returns] apply RFs: RF4 RF5 diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query1.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query1.out index eca7d46a8f3..8996d789efa 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query1.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query1.out @@ -39,6 +39,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------------PhysicalDistribute[DistributionSpecHash] --------------------------hashAgg[LOCAL] ----------------------------PhysicalDistribute[DistributionSpecExecutionAny] -------------------------------PhysicalProject ---------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query23.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query23.out index 8132fd343d5..4d4463732f0 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query23.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query23.out @@ -59,8 +59,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------PhysicalProject --------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((catalog_sales.cs_item_sk = frequent_ss_items.item_sk)) otherCondition=() build RFs:RF4 cs_item_sk->[item_sk] ----------------------PhysicalDistribute[DistributionSpecHash] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF4 +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF4 ----------------------PhysicalDistribute[DistributionSpecHash] ------------------------PhysicalProject --------------------------hashJoin[LEFT_SEMI_JOIN] hashCondition=((catalog_sales.cs_bill_customer_sk = best_ss_customer.c_customer_sk)) otherCondition=() @@ -73,13 +72,11 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------------------------filter((date_dim.d_moy = 5) and (date_dim.d_year = 2000)) --------------------------------------PhysicalOlapScan[date_dim] ----------------------------PhysicalDistribute[DistributionSpecHash] -------------------------------PhysicalProject ---------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) ------------------PhysicalProject --------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((web_sales.ws_item_sk = frequent_ss_items.item_sk)) otherCondition=() build RFs:RF6 ws_item_sk->[item_sk] ----------------------PhysicalDistribute[DistributionSpecHash] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF6 +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF6 ----------------------PhysicalDistribute[DistributionSpecHash] ------------------------PhysicalProject --------------------------hashJoin[LEFT_SEMI_JOIN] hashCondition=((web_sales.ws_bill_customer_sk = best_ss_customer.c_customer_sk)) otherCondition=() @@ -92,6 +89,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------------------------filter((date_dim.d_moy = 5) and (date_dim.d_year = 2000)) --------------------------------------PhysicalOlapScan[date_dim] ----------------------------PhysicalDistribute[DistributionSpecHash] -------------------------------PhysicalProject ---------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query24.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query24.out index 67183c68f45..ddf2dfd6c05 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query24.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query24.out @@ -55,6 +55,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecGather] ----------------------hashAgg[LOCAL] ------------------------PhysicalDistribute[DistributionSpecExecutionAny] ---------------------------PhysicalProject -----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query30.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query30.out index 9f72a79a16e..40293a67cb9 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query30.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query30.out @@ -43,6 +43,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] --------------------hashAgg[LOCAL] ----------------------PhysicalDistribute[DistributionSpecExecutionAny] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query31.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query31.out index 835238e3d70..21bd99fb9f3 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query31.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query31.out @@ -2,44 +2,42 @@ -- !ds_shape_31 -- PhysicalCteAnchor ( cteId=CTEId#0 ) --PhysicalCteProducer ( cteId=CTEId#0 ) -----PhysicalProject +----hashAgg[GLOBAL] +------PhysicalDistribute[DistributionSpecHash] +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_addr_sk = customer_address.ca_address_sk)) otherCondition=() +--------------PhysicalDistribute[DistributionSpecHash] +----------------PhysicalProject +------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +--------------------PhysicalProject +----------------------PhysicalOlapScan[store_sales] apply RFs: RF0 +--------------------PhysicalDistribute[DistributionSpecReplicated] +----------------------PhysicalProject +------------------------filter((ss.d_year = 2000) and d_qoy IN (1, 2, 3)) +--------------------------PhysicalOlapScan[date_dim] +--------------PhysicalDistribute[DistributionSpecHash] +----------------PhysicalProject +------------------PhysicalOlapScan[customer_address] +--PhysicalCteAnchor ( cteId=CTEId#1 ) +----PhysicalCteProducer ( cteId=CTEId#1 ) ------hashAgg[GLOBAL] --------PhysicalDistribute[DistributionSpecHash] ----------hashAgg[LOCAL] ------------PhysicalProject ---------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_addr_sk = customer_address.ca_address_sk)) otherCondition=() +--------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_bill_addr_sk = customer_address.ca_address_sk)) otherCondition=() ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject ---------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +--------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ws_sold_date_sk] ----------------------PhysicalProject -------------------------PhysicalOlapScan[store_sales] apply RFs: RF0 +------------------------PhysicalOlapScan[web_sales] apply RFs: RF2 ----------------------PhysicalDistribute[DistributionSpecReplicated] ------------------------PhysicalProject ---------------------------filter((ss.d_year = 2000) and d_qoy IN (1, 2, 3)) +--------------------------filter((ws.d_year = 2000) and d_qoy IN (1, 2, 3)) ----------------------------PhysicalOlapScan[date_dim] ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject --------------------PhysicalOlapScan[customer_address] ---PhysicalCteAnchor ( cteId=CTEId#1 ) -----PhysicalCteProducer ( cteId=CTEId#1 ) -------PhysicalProject ---------hashAgg[GLOBAL] -----------PhysicalDistribute[DistributionSpecHash] -------------hashAgg[LOCAL] ---------------PhysicalProject -----------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_bill_addr_sk = customer_address.ca_address_sk)) otherCondition=() -------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ws_sold_date_sk] -------------------------PhysicalProject ---------------------------PhysicalOlapScan[web_sales] apply RFs: RF2 -------------------------PhysicalDistribute[DistributionSpecReplicated] ---------------------------PhysicalProject -----------------------------filter((ws.d_year = 2000) and d_qoy IN (1, 2, 3)) -------------------------------PhysicalOlapScan[date_dim] -------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------PhysicalOlapScan[customer_address] ----PhysicalResultSink ------PhysicalQuickSort[MERGE_SORT] --------PhysicalDistribute[DistributionSpecGather] diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query39.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query39.out index 7a0a69965bd..11ec8af267c 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query39.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query39.out @@ -28,11 +28,9 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------PhysicalQuickSort[LOCAL_SORT] ----------hashJoin[INNER_JOIN] hashCondition=((inv1.i_item_sk = inv2.i_item_sk) and (inv1.w_warehouse_sk = inv2.w_warehouse_sk)) otherCondition=() ------------PhysicalDistribute[DistributionSpecHash] ---------------PhysicalProject -----------------filter((inv1.d_moy = 1)) -------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------filter((inv1.d_moy = 1)) +----------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------PhysicalDistribute[DistributionSpecHash] ---------------PhysicalProject -----------------filter((inv2.d_moy = 2)) -------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------filter((inv2.d_moy = 2)) +----------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query47.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query47.out index 174fd05d61d..03cb37e0f81 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query47.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query47.out @@ -41,9 +41,8 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalProject ----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2001)) -------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2001)) +----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject --------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query57.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query57.out index d09ed9ca40f..555d7716af6 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query57.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query57.out @@ -42,9 +42,8 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalProject ----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 1999)) -------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 1999)) +----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject --------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query59.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query59.out index 644b59d4db3..b9e7a9b40de 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query59.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query59.out @@ -2,17 +2,16 @@ -- !ds_shape_59 -- PhysicalCteAnchor ( cteId=CTEId#0 ) --PhysicalCteProducer ( cteId=CTEId#0 ) -----PhysicalProject -------hashAgg[GLOBAL] ---------PhysicalDistribute[DistributionSpecHash] -----------hashAgg[LOCAL] -------------PhysicalProject ---------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_date_sk = store_sales.ss_sold_date_sk)) otherCondition=() +----hashAgg[GLOBAL] +------PhysicalDistribute[DistributionSpecHash] +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_date_sk = store_sales.ss_sold_date_sk)) otherCondition=() +--------------PhysicalProject +----------------PhysicalOlapScan[store_sales] +--------------PhysicalDistribute[DistributionSpecReplicated] ----------------PhysicalProject -------------------PhysicalOlapScan[store_sales] -----------------PhysicalDistribute[DistributionSpecReplicated] -------------------PhysicalProject ---------------------PhysicalOlapScan[date_dim] +------------------PhysicalOlapScan[date_dim] --PhysicalResultSink ----PhysicalTopN[MERGE_SORT] ------PhysicalDistribute[DistributionSpecGather] diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query70.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query70.out index f3e524aabcf..6fc8a52f839 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query70.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query70.out @@ -45,3 +45,4 @@ PhysicalResultSink --------------------------------------------------PhysicalDistribute[DistributionSpecReplicated] ----------------------------------------------------PhysicalProject ------------------------------------------------------PhysicalOlapScan[store] + diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query81.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query81.out index 9b9a03af8fe..22ab8efaf19 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query81.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query81.out @@ -44,6 +44,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] --------------------hashAgg[LOCAL] ----------------------PhysicalDistribute[DistributionSpecExecutionAny] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query95.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query95.out index a835868fd8c..2a0ae9b4138 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query95.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/rf_prune/query95.out @@ -23,16 +23,14 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------------PhysicalProject ------------------------hashJoin[INNER_JOIN] hashCondition=((web_returns.wr_order_number = ws_wh.ws_order_number)) otherCondition=() --------------------------PhysicalDistribute[DistributionSpecHash] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF6 +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF6 --------------------------PhysicalDistribute[DistributionSpecHash] ----------------------------PhysicalProject ------------------------------PhysicalOlapScan[web_returns] apply RFs: RF6 ----------------------PhysicalProject ------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((ws1.ws_order_number = ws_wh.ws_order_number)) otherCondition=() build RFs:RF7 ws_order_number->[ws_order_number,ws_order_number] --------------------------PhysicalDistribute[DistributionSpecHash] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------------------PhysicalDistribute[DistributionSpecHash] ----------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_web_site_sk = web_site.web_site_sk)) otherCondition=() build RFs:RF3 web_site_sk->[ws_web_site_sk] ------------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ws_ship_date_sk] diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query1.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query1.out index eca7d46a8f3..8996d789efa 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query1.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query1.out @@ -39,6 +39,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------------PhysicalDistribute[DistributionSpecHash] --------------------------hashAgg[LOCAL] ----------------------------PhysicalDistribute[DistributionSpecExecutionAny] -------------------------------PhysicalProject ---------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query23.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query23.out index 520c3910152..edd849ea421 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query23.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query23.out @@ -59,8 +59,7 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------PhysicalProject --------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((catalog_sales.cs_item_sk = frequent_ss_items.item_sk)) otherCondition=() build RFs:RF4 cs_item_sk->[item_sk] ----------------------PhysicalDistribute[DistributionSpecHash] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF4 +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF4 ----------------------PhysicalDistribute[DistributionSpecHash] ------------------------PhysicalProject --------------------------hashJoin[LEFT_SEMI_JOIN] hashCondition=((catalog_sales.cs_bill_customer_sk = best_ss_customer.c_customer_sk)) otherCondition=() @@ -73,13 +72,11 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------------------------filter((date_dim.d_moy = 5) and (date_dim.d_year = 2000)) --------------------------------------PhysicalOlapScan[date_dim] ----------------------------PhysicalDistribute[DistributionSpecHash] -------------------------------PhysicalProject ---------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) ------------------PhysicalProject --------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((web_sales.ws_item_sk = frequent_ss_items.item_sk)) otherCondition=() build RFs:RF6 ws_item_sk->[item_sk] ----------------------PhysicalDistribute[DistributionSpecHash] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF6 +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF6 ----------------------PhysicalDistribute[DistributionSpecHash] ------------------------PhysicalProject --------------------------hashJoin[LEFT_SEMI_JOIN] hashCondition=((web_sales.ws_bill_customer_sk = best_ss_customer.c_customer_sk)) otherCondition=() @@ -92,6 +89,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------------------------filter((date_dim.d_moy = 5) and (date_dim.d_year = 2000)) --------------------------------------PhysicalOlapScan[date_dim] ----------------------------PhysicalDistribute[DistributionSpecHash] -------------------------------PhysicalProject ---------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) +------------------------------PhysicalCteConsumer ( cteId=CTEId#2 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query24.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query24.out index cf64374e507..c0d202025b6 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query24.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query24.out @@ -55,6 +55,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalDistribute[DistributionSpecGather] ----------------------hashAgg[LOCAL] ------------------------PhysicalDistribute[DistributionSpecExecutionAny] ---------------------------PhysicalProject -----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query30.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query30.out index 0160329ec26..2880145be20 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query30.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query30.out @@ -43,6 +43,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] --------------------hashAgg[LOCAL] ----------------------PhysicalDistribute[DistributionSpecExecutionAny] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query31.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query31.out index f12c5e5cb28..f759ca84798 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query31.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query31.out @@ -2,44 +2,42 @@ -- !ds_shape_31 -- PhysicalCteAnchor ( cteId=CTEId#0 ) --PhysicalCteProducer ( cteId=CTEId#0 ) -----PhysicalProject +----hashAgg[GLOBAL] +------PhysicalDistribute[DistributionSpecHash] +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF1 ca_address_sk->[ss_addr_sk] +--------------PhysicalDistribute[DistributionSpecHash] +----------------PhysicalProject +------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +--------------------PhysicalProject +----------------------PhysicalOlapScan[store_sales] apply RFs: RF0 RF1 +--------------------PhysicalDistribute[DistributionSpecReplicated] +----------------------PhysicalProject +------------------------filter((ss.d_year = 2000) and d_qoy IN (1, 2, 3)) +--------------------------PhysicalOlapScan[date_dim] +--------------PhysicalDistribute[DistributionSpecHash] +----------------PhysicalProject +------------------PhysicalOlapScan[customer_address] +--PhysicalCteAnchor ( cteId=CTEId#1 ) +----PhysicalCteProducer ( cteId=CTEId#1 ) ------hashAgg[GLOBAL] --------PhysicalDistribute[DistributionSpecHash] ----------hashAgg[LOCAL] ------------PhysicalProject ---------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF1 ca_address_sk->[ss_addr_sk] +--------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_bill_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF3 ca_address_sk->[ws_bill_addr_sk] ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject ---------------------hashJoin[INNER_JOIN] hashCondition=((store_sales.ss_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +--------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ws_sold_date_sk] ----------------------PhysicalProject -------------------------PhysicalOlapScan[store_sales] apply RFs: RF0 RF1 +------------------------PhysicalOlapScan[web_sales] apply RFs: RF2 RF3 ----------------------PhysicalDistribute[DistributionSpecReplicated] ------------------------PhysicalProject ---------------------------filter((ss.d_year = 2000) and d_qoy IN (1, 2, 3)) +--------------------------filter((ws.d_year = 2000) and d_qoy IN (1, 2, 3)) ----------------------------PhysicalOlapScan[date_dim] ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject --------------------PhysicalOlapScan[customer_address] ---PhysicalCteAnchor ( cteId=CTEId#1 ) -----PhysicalCteProducer ( cteId=CTEId#1 ) -------PhysicalProject ---------hashAgg[GLOBAL] -----------PhysicalDistribute[DistributionSpecHash] -------------hashAgg[LOCAL] ---------------PhysicalProject -----------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_bill_addr_sk = customer_address.ca_address_sk)) otherCondition=() build RFs:RF3 ca_address_sk->[ws_bill_addr_sk] -------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------hashJoin[INNER_JOIN] hashCondition=((web_sales.ws_sold_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF2 d_date_sk->[ws_sold_date_sk] -------------------------PhysicalProject ---------------------------PhysicalOlapScan[web_sales] apply RFs: RF2 RF3 -------------------------PhysicalDistribute[DistributionSpecReplicated] ---------------------------PhysicalProject -----------------------------filter((ws.d_year = 2000) and d_qoy IN (1, 2, 3)) -------------------------------PhysicalOlapScan[date_dim] -------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------PhysicalOlapScan[customer_address] ----PhysicalResultSink ------PhysicalQuickSort[MERGE_SORT] --------PhysicalDistribute[DistributionSpecGather] diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query39.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query39.out index 4ccedd31402..88b2869175c 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query39.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query39.out @@ -28,11 +28,9 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------PhysicalQuickSort[LOCAL_SORT] ----------hashJoin[INNER_JOIN] hashCondition=((inv1.i_item_sk = inv2.i_item_sk) and (inv1.w_warehouse_sk = inv2.w_warehouse_sk)) otherCondition=() ------------PhysicalDistribute[DistributionSpecHash] ---------------PhysicalProject -----------------filter((inv1.d_moy = 1)) -------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------filter((inv1.d_moy = 1)) +----------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------PhysicalDistribute[DistributionSpecHash] ---------------PhysicalProject -----------------filter((inv2.d_moy = 2)) -------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------filter((inv2.d_moy = 2)) +----------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query47.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query47.out index 214cdaaee62..e8f28d6ea4e 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query47.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query47.out @@ -41,9 +41,8 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalProject ----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2001)) -------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 2001)) +----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject --------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query57.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query57.out index ea7531482a7..f479209035e 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query57.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query57.out @@ -42,9 +42,8 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) --------------------PhysicalProject ----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] ---------------------PhysicalProject -----------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 1999)) -------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +--------------------filter((if((avg_monthly_sales > 0.0000), (cast(abs((sum_sales - cast(avg_monthly_sales as DECIMALV3(38, 2)))) as DECIMALV3(38, 10)) / avg_monthly_sales), NULL) > 0.100000) and (v2.avg_monthly_sales > 0.0000) and (v2.d_year = 1999)) +----------------------PhysicalCteConsumer ( cteId=CTEId#0 ) ----------------PhysicalDistribute[DistributionSpecHash] ------------------PhysicalProject --------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query59.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query59.out index 08d5a9d3f1f..2db7d57863d 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query59.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query59.out @@ -2,17 +2,16 @@ -- !ds_shape_59 -- PhysicalCteAnchor ( cteId=CTEId#0 ) --PhysicalCteProducer ( cteId=CTEId#0 ) -----PhysicalProject -------hashAgg[GLOBAL] ---------PhysicalDistribute[DistributionSpecHash] -----------hashAgg[LOCAL] -------------PhysicalProject ---------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_date_sk = store_sales.ss_sold_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +----hashAgg[GLOBAL] +------PhysicalDistribute[DistributionSpecHash] +--------hashAgg[LOCAL] +----------PhysicalProject +------------hashJoin[INNER_JOIN] hashCondition=((date_dim.d_date_sk = store_sales.ss_sold_date_sk)) otherCondition=() build RFs:RF0 d_date_sk->[ss_sold_date_sk] +--------------PhysicalProject +----------------PhysicalOlapScan[store_sales] apply RFs: RF0 +--------------PhysicalDistribute[DistributionSpecReplicated] ----------------PhysicalProject -------------------PhysicalOlapScan[store_sales] apply RFs: RF0 -----------------PhysicalDistribute[DistributionSpecReplicated] -------------------PhysicalProject ---------------------PhysicalOlapScan[date_dim] +------------------PhysicalOlapScan[date_dim] --PhysicalResultSink ----PhysicalTopN[MERGE_SORT] ------PhysicalDistribute[DistributionSpecGather] diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query81.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query81.out index aa637bc6469..99cf6c48bb7 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query81.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query81.out @@ -44,6 +44,5 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ------------------PhysicalDistribute[DistributionSpecHash] --------------------hashAgg[LOCAL] ----------------------PhysicalDistribute[DistributionSpecExecutionAny] -------------------------PhysicalProject ---------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) diff --git a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query95.out b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query95.out index 4763e6aa34c..9a971d30aec 100644 --- a/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query95.out +++ b/regression-test/data/nereids_tpcds_shape_sf100_p0/shape/query95.out @@ -23,16 +23,14 @@ PhysicalCteAnchor ( cteId=CTEId#0 ) ----------------------PhysicalProject ------------------------hashJoin[INNER_JOIN] hashCondition=((web_returns.wr_order_number = ws_wh.ws_order_number)) otherCondition=() build RFs:RF10 wr_order_number->[ws_order_number];RF11 wr_order_number->[ws_order_number] --------------------------PhysicalDistribute[DistributionSpecHash] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF10 RF11 RF12 RF13 +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) apply RFs: RF10 RF11 RF12 RF13 --------------------------PhysicalDistribute[DistributionSpecHash] ----------------------------PhysicalProject ------------------------------PhysicalOlapScan[web_returns] apply RFs: RF12 RF13 ----------------------PhysicalProject ------------------------hashJoin[RIGHT_SEMI_JOIN] hashCondition=((ws1.ws_order_number = ws_wh.ws_order_number)) otherCondition=() build RFs:RF14 ws_order_number->[ws_order_number,ws_order_number];RF15 ws_order_number->[ws_order_number,ws_order_number] --------------------------PhysicalDistribute[DistributionSpecHash] -----------------------------PhysicalProject -------------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) +----------------------------PhysicalCteConsumer ( cteId=CTEId#0 ) --------------------------PhysicalDistribute[DistributionSpecHash] ----------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_web_site_sk = web_site.web_site_sk)) otherCondition=() build RFs:RF6 web_site_sk->[ws_web_site_sk];RF7 web_site_sk->[ws_web_site_sk] ------------------------------hashJoin[INNER_JOIN] hashCondition=((ws1.ws_ship_date_sk = date_dim.d_date_sk)) otherCondition=() build RFs:RF4 d_date_sk->[ws_ship_date_sk];RF5 d_date_sk->[ws_ship_date_sk] diff --git a/regression-test/suites/nereids_p0/cte/test_cte_column_pruning.groovy b/regression-test/suites/nereids_p0/cte/test_cte_column_pruning.groovy new file mode 100644 index 00000000000..7a465c21251 --- /dev/null +++ b/regression-test/suites/nereids_p0/cte/test_cte_column_pruning.groovy @@ -0,0 +1,143 @@ +// 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("test_cte_column_pruning") { + sql "SET enable_nereids_planner=true" + sql "SET enable_pipeline_engine=true" + sql "SET enable_fallback_to_original_planner=false" + + sql """drop table if exists t1""" + sql """drop table if exists t2""" + sql """drop table if exists t3""" + sql """drop table if exists t4""" + + sql """ + create table if not exists t1 ( + c2 int , + c1 int , + c3 int , + c4 int , + pk int + ) + distributed by hash(pk) buckets 10 + properties("replication_num" = "1"); + """ + + sql """ + create table if not exists t2 ( + c1 int , + c2 int , + c3 int , + c4 int , + pk int + ) + distributed by hash(pk) buckets 10 + properties("replication_num" = "1"); + """ + + sql """ + create table if not exists t3 ( + c2 int , + c1 int , + c3 int , + c4 int , + pk int + ) + distributed by hash(pk) buckets 10 + properties("replication_num" = "1"); + """ + + sql """ + create table if not exists t4 ( + c1 int , + c2 int , + c3 int , + c4 int , + pk int + ) + distributed by hash(pk) buckets 10 + properties("replication_num" = "1"); + """ + + sql """ + insert into t1(pk,c1,c2,c3,c4) values (0,7,2,3328056,7),(1,3,5,3,3045349),(2,2130015,0,7,-7116176),(3,4411710,1203314,1,2336164),(4,4,-8001461,0,8),(5,9,3,6,2),(6,-8088092,null,-7256698,-2025142),(7,8,2,5,1),(8,4,4953685,3,null),(9,-6662413,-3845449,4,2),(10,5315281,0,5,null),(11,9,3,7,7),(12,4341905,null,null,8),(13,3,6,5,1),(14,5,9,6541164,3),(15,1,-582319,1,9),(16,5533636,4,39841,0),(17,1,1,null,7),(18,742881,-1420303,6,1),(19,281430,6753011,3,2),(20,7,1,4,-31350),(21,-5663089 [...] + """ + + sql """ + insert into t2(pk,c1,c2,c3,c4) values (0,5,4,189864,-7663457),(1,7,null,6,1),(2,null,8,-3362640,9),(3,3,2,5,-2197130),(4,2,3,7160615,1),(5,null,-57834,420441,3),(6,0,null,2,2),(7,1,-3681539,3,4),(8,548866,3,0,5),(9,8,-2824887,0,3246956),(10,5,3,7,2),(11,8,8,6,8),(12,0,2,7,9),(13,8,6,null,null),(14,-4103729,4,5,8),(15,-3659292,2,7,5),(16,8,7,1,null),(17,2526018,4,8069607,5),(18,6,6,5,2802235),(19,9,0,6379201,null),(20,3,null,4,3),(21,0,8,-5506402,2),(22,6,4,3,1),(23,4,5225086,3,1) [...] + """ + + sql """ + insert into t3(pk,c1,c2,c3,c4) values (0,3,2,6,-3164679),(1,-6216443,3437690,-288827,6),(2,4,-5352286,-1005469,4118240),(3,9,6795167,5,1616205),(4,8,-4659990,-4816829,6),(5,0,9,4,8),(6,-4454766,2,2510766,3),(7,7860071,-3434966,8,3),(8,null,0,2,1),(9,8031908,2,-6673194,-5981416),(10,5,6716310,8,2529959),(11,null,-3622116,1,-7891010),(12,null,3527222,7993802,null),(13,null,1,2,1),(14,2,8,7,7),(15,0,9,5,null),(16,7452083,null,-4620796,0),(17,9,9,null,6),(18,3,1,-1578776,5),(19,9,253 [...] + """ + + sql """ + insert into t4(pk,c1,c2,c3,c4) values (0,-4263513,null,null,6),(1,1,3,4,null),(2,2460936,6,5,6299003),(3,null,7,7107446,-2366754),(4,6247611,4785035,3,-8014875),(5,0,2,5249218,3),(6,null,253825,4,3),(7,null,2,9,-350785),(8,6,null,null,4),(9,1,3,1,3422691),(10,0,-6596165,1808018,3),(11,2,752342,null,1),(12,-5220927,2676278,9,7),(13,6025864,2,1,4),(14,7,4,4,9),(15,5,9,9,849881),(16,-4253076,null,-4404479,-6365351),(17,null,6,4240023,3),(18,7,1276495,7,6),(19,null,-4459040,178194,-6 [...] + """ + + sql """ + sync + """ + + sql """ + WITH tbl1 AS ( + SELECT + tbl2.c1 AS c1, + tbl3.c2 AS c2, + tbl5.c4 AS c3, + tbl3.c1 AS c4 + FROM + t1 AS tbl1 + JOIN t2 AS tbl2 ON tbl1.c2 = tbl1.c4 + RIGHT JOIN t1 AS tbl3 ON tbl3.c3 = tbl1.c3 + INNER JOIN t3 AS tbl4 ON tbl3.c2 = tbl4.c3 + INNER JOIN t4 AS tbl5 ON tbl5.c4 = tbl4.c2 + AND tbl3.c3 = tbl3.c4 + WHERE + ( + tbl2.c4 = (0 + 9) + AND ((4 + 1) IS NULL) + ) + ORDER BY + 1, + 2, + 3, + 4 DESC + LIMIT + 6666 OFFSET 500 + ) + SELECT + tbl3.c4 AS c1, + tbl2.c4 AS c2, + tbl3.c3 AS c3, + tbl2.c2 AS c4 + FROM + tbl1 AS tbl2 + JOIN tbl1 AS tbl3 ON tbl3.c2 = tbl2.c2 + WHERE + ( + tbl2.c3 != tbl2.c3 + AND ((2 + 0) IS NOT NULL) + ) + ORDER BY + 2, + 4, + 1, + 3 ASC + LIMIT + 6666 OFFSET 2; + """ +} --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org