From 84c602a71347849b60aa1be7390de57883726de2 Mon Sep 17 00:00:00 2001 From: morrySnow <101034200+morrySnow@users.noreply.github.com> Date: Fri, 21 Jun 2024 11:31:50 +0800 Subject: [PATCH] [refactor](variant) refactor sub path push down on variant type (#36478) pick from master #36478 intro a new rule VARIANT_SUB_PATH_PRUNING to prune variant sub path. for example, variant slot v in table t has two sub path: 'c1' and 'c2', after this rule, select v['c1'] from t will only scan one sub path 'c1' of v to reduce scan time. This rule accomplishes all the work using two components. The Collector traverses from the top down, collecting all the element_at functions on the variant types, and recording the required path from the original variant slot to the current element_at. The Replacer traverses from the bottom up, generating the slots for the required sub path on scan, union, and cte consumer. Then, it replaces the element_at with the corresponding slot. --- .../doris/nereids/StatementContext.java | 59 -- .../glue/translator/ExpressionTranslator.java | 16 - .../translator/PhysicalPlanTranslator.java | 39 +- .../translator/PlanTranslatorContext.java | 6 +- .../executor/AbstractBatchJobExecutor.java | 6 +- .../doris/nereids/jobs/executor/Analyzer.java | 2 - .../doris/nereids/jobs/executor/Rewriter.java | 82 +- .../post/CommonSubExpressionOpt.java | 11 +- .../post/PushDownFilterThroughProject.java | 7 +- .../nereids/processor/post/Validator.java | 2 +- .../apache/doris/nereids/rules/RuleType.java | 8 +- .../rules/analysis/BindSlotWithPaths.java | 81 -- .../rules/analysis/CheckAfterRewrite.java | 5 +- .../rules/analysis/ExpressionAnalyzer.java | 66 +- .../nereids/rules/analysis/SlotBinder.java | 6 - .../expression/rules/FunctionBinder.java | 43 - .../rules/rewrite/ClearContextStatus.java | 40 + .../rewrite/DeferMaterializeTopNResult.java | 4 +- .../rules/rewrite/PruneOlapScanPartition.java | 117 ++- .../rules/rewrite/PruneOlapScanTablet.java | 69 +- .../rewrite/PushDownFilterThroughProject.java | 21 +- .../rules/rewrite/VariantSubPathPruning.java | 775 ++++++++++++++++++ .../nereids/trees/expressions/Alias.java | 4 +- .../trees/expressions/ArrayItemReference.java | 3 +- .../trees/expressions/SlotReference.java | 69 +- .../functions/scalar/ElementAt.java | 2 +- .../scalar/PushDownToProjectionFunction.java | 88 -- .../nereids/trees/plans/algebra/Project.java | 30 - .../plans/logical/LogicalCTEConsumer.java | 23 +- .../plans/logical/LogicalCatalogRelation.java | 2 +- .../trees/plans/logical/LogicalOlapScan.java | 145 ++-- .../plans/logical/LogicalTVFRelation.java | 2 +- .../physical/PhysicalCatalogRelation.java | 2 +- .../plans/physical/PhysicalOlapScan.java | 4 +- .../plans/physical/PhysicalTVFRelation.java | 2 +- .../expression/PredicatesSplitterTest.java | 2 +- .../data/variant_p0/test_sub_path_pruning.out | 324 ++++++++ .../variant_p0/test_sub_path_pruning.groovy | 217 +++++ 38 files changed, 1660 insertions(+), 724 deletions(-) delete mode 100644 fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSlotWithPaths.java create mode 100644 fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ClearContextStatus.java create mode 100644 fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/VariantSubPathPruning.java delete mode 100644 fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/scalar/PushDownToProjectionFunction.java create mode 100644 regression-test/data/variant_p0/test_sub_path_pruning.out create mode 100644 regression-test/suites/variant_p0/test_sub_path_pruning.groovy 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 4d026cf9ce35a2..528a002bd2517e 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 @@ -31,7 +31,6 @@ import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.expressions.Placeholder; import org.apache.doris.nereids.trees.expressions.Slot; -import org.apache.doris.nereids.trees.expressions.SlotReference; import org.apache.doris.nereids.trees.expressions.StatementScopeIdGenerator; import org.apache.doris.nereids.trees.plans.ObjectId; import org.apache.doris.nereids.trees.plans.PlaceholderId; @@ -63,7 +62,6 @@ import java.util.BitSet; import java.util.Collection; import java.util.HashMap; -import java.util.Iterator; import java.util.LinkedHashMap; import java.util.List; import java.util.Map; @@ -130,15 +128,6 @@ public class StatementContext implements Closeable { private final List joinFilters = new ArrayList<>(); private final List hints = new ArrayList<>(); - // Root Slot -> Paths -> Sub-column Slots - private final Map, SlotReference>> subColumnSlotRefMap - = Maps.newHashMap(); - - // Map from rewritten slot to original expr - private final Map subColumnOriginalExprMap = Maps.newHashMap(); - - // Map from original expr to rewritten slot - private final Map originalExprToRewrittenSubColumn = Maps.newHashMap(); // Map slot to its relation, currently used in SlotReference to find its original // Relation for example LogicalOlapScan @@ -262,58 +251,10 @@ public Optional getSqlCacheContext() { return Optional.ofNullable(sqlCacheContext); } - public Set getAllPathsSlots() { - Set allSlotReferences = Sets.newHashSet(); - for (Map, SlotReference> slotReferenceMap : subColumnSlotRefMap.values()) { - allSlotReferences.addAll(slotReferenceMap.values()); - } - return allSlotReferences; - } - - public Expression getOriginalExpr(SlotReference rewriteSlot) { - return subColumnOriginalExprMap.getOrDefault(rewriteSlot, null); - } - - public Slot getRewrittenSlotRefByOriginalExpr(Expression originalExpr) { - return originalExprToRewrittenSubColumn.getOrDefault(originalExpr, null); - } - - /** - * Add a slot ref attached with paths in context to avoid duplicated slot - */ - public void addPathSlotRef(Slot root, List paths, SlotReference slotRef, Expression originalExpr) { - subColumnSlotRefMap.computeIfAbsent(root, k -> Maps.newTreeMap((lst1, lst2) -> { - Iterator it1 = lst1.iterator(); - Iterator 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()); - })); - subColumnSlotRefMap.get(root).put(paths, slotRef); - subColumnOriginalExprMap.put(slotRef, originalExpr); - originalExprToRewrittenSubColumn.put(originalExpr, slotRef); - } - - public SlotReference getPathSlot(Slot root, List paths) { - Map, SlotReference> pathsSlotsMap = subColumnSlotRefMap.getOrDefault(root, null); - if (pathsSlotsMap == null) { - return null; - } - return pathsSlotsMap.getOrDefault(paths, null); - } - public void addSlotToRelation(Slot slot, Relation relation) { slotToRelation.put(slot, relation); } - public Relation getRelationBySlot(Slot slot) { - return slotToRelation.getOrDefault(slot, null); - } - public boolean isDpHyp() { return isDpHyp; } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/ExpressionTranslator.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/ExpressionTranslator.java index 2482bb50e32938..b5dabeefb94b5b 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/ExpressionTranslator.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/ExpressionTranslator.java @@ -92,7 +92,6 @@ import org.apache.doris.nereids.trees.expressions.functions.scalar.ElementAt; import org.apache.doris.nereids.trees.expressions.functions.scalar.HighOrderFunction; import org.apache.doris.nereids.trees.expressions.functions.scalar.Lambda; -import org.apache.doris.nereids.trees.expressions.functions.scalar.PushDownToProjectionFunction; import org.apache.doris.nereids.trees.expressions.functions.scalar.ScalarFunction; import org.apache.doris.nereids.trees.expressions.functions.udf.JavaUdaf; import org.apache.doris.nereids.trees.expressions.functions.udf.JavaUdf; @@ -101,7 +100,6 @@ import org.apache.doris.nereids.trees.expressions.literal.NullLiteral; import org.apache.doris.nereids.trees.expressions.visitor.DefaultExpressionVisitor; import org.apache.doris.nereids.types.DataType; -import org.apache.doris.qe.ConnectContext; import org.apache.doris.thrift.TFunctionBinaryType; import com.google.common.base.Preconditions; @@ -210,20 +208,6 @@ private OlapTable getOlapTableDirectly(SlotRef left) { @Override public Expr visitElementAt(ElementAt elementAt, PlanTranslatorContext context) { - if (PushDownToProjectionFunction.validToPushDown(elementAt)) { - if (ConnectContext.get() != null - && ConnectContext.get().getSessionVariable() != null - && !ConnectContext.get().getSessionVariable().isEnableRewriteElementAtToSlot()) { - throw new AnalysisException( - "set enable_rewrite_element_at_to_slot=true when using element_at function for variant type"); - } - SlotReference rewrittenSlot = (SlotReference) context.getConnectContext() - .getStatementContext().getRewrittenSlotRefByOriginalExpr(elementAt); - // rewrittenSlot == null means variant is not from table. so keep element_at function - if (rewrittenSlot != null) { - return context.findSlotRef(rewrittenSlot.getExprId()); - } - } return visitScalarFunction(elementAt, context); } 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 d81b0ee7a22def..6dc72634b191be 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 @@ -94,7 +94,6 @@ import org.apache.doris.nereids.trees.expressions.WindowFrame; import org.apache.doris.nereids.trees.expressions.functions.agg.AggregateFunction; import org.apache.doris.nereids.trees.expressions.functions.agg.AggregateParam; -import org.apache.doris.nereids.trees.expressions.functions.scalar.PushDownToProjectionFunction; import org.apache.doris.nereids.trees.plans.AbstractPlan; import org.apache.doris.nereids.trees.plans.AggMode; import org.apache.doris.nereids.trees.plans.AggPhase; @@ -1244,8 +1243,7 @@ public PlanFragment visitPhysicalFilter(PhysicalFilter filter, P } if (planNode instanceof ExchangeNode || planNode instanceof SortNode || planNode instanceof UnionNode // this means we have filter->limit->project, need a SelectNode - || (child instanceof PhysicalProject - && !((PhysicalProject) child).hasPushedDownToProjectionFunctions())) { + || child instanceof PhysicalProject) { // the three nodes don't support conjuncts, need create a SelectNode to filter data SelectNode selectNode = new SelectNode(context.nextPlanNodeId(), planNode); selectNode.setNereidsId(filter.getId()); @@ -1827,35 +1825,6 @@ && findOlapScanNodesByPassExchangeAndJoinNode(inputFragment.getPlanRoot())) { return inputFragment; } - // collect all valid PushDownToProjectionFunction from expression - private List getPushDownToProjectionFunctionForRewritten(NamedExpression expression) { - List targetExprList = expression.collectToList(PushDownToProjectionFunction.class::isInstance); - return targetExprList.stream() - .filter(PushDownToProjectionFunction::validToPushDown) - .collect(Collectors.toList()); - } - - // register rewritten slots from original PushDownToProjectionFunction - private void registerRewrittenSlot(PhysicalProject project, OlapScanNode olapScanNode) { - // register slots that are rewritten from element_at/etc.. - List allPushDownProjectionFunctions = project.getProjects().stream() - .map(this::getPushDownToProjectionFunctionForRewritten) - .flatMap(List::stream) - .collect(Collectors.toList()); - for (Expression expr : allPushDownProjectionFunctions) { - PushDownToProjectionFunction function = (PushDownToProjectionFunction) expr; - if (context != null - && context.getConnectContext() != null - && context.getConnectContext().getStatementContext() != null) { - Slot argumentSlot = function.getInputSlots().stream().findFirst().get(); - Expression rewrittenSlot = PushDownToProjectionFunction.rewriteToSlot( - function, (SlotReference) argumentSlot); - TupleDescriptor tupleDescriptor = context.getTupleDesc(olapScanNode.getTupleId()); - context.createSlotDesc(tupleDescriptor, (SlotReference) rewrittenSlot); - } - } - } - // TODO: generate expression mapping when be project could do in ExecNode. @Override public PlanFragment visitPhysicalProject(PhysicalProject project, PlanTranslatorContext context) { @@ -1870,12 +1839,6 @@ public PlanFragment visitPhysicalProject(PhysicalProject project PlanFragment inputFragment = project.child(0).accept(this, context); - if (inputFragment.getPlanRoot() instanceof OlapScanNode) { - // function already pushed down in projection - // e.g. select count(distinct cast(element_at(v, 'a') as int)) from tbl; - registerRewrittenSlot(project, (OlapScanNode) inputFragment.getPlanRoot()); - } - PlanNode inputPlanNode = inputFragment.getPlanRoot(); List projectionExprs = null; List allProjectionExprs = Lists.newArrayList(); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PlanTranslatorContext.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PlanTranslatorContext.java index 31e54a92ab95a3..869dc71120e794 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PlanTranslatorContext.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PlanTranslatorContext.java @@ -287,12 +287,12 @@ public SlotDescriptor createSlotDesc(TupleDescriptor tupleDesc, SlotReference sl slotDescriptor.setLabel(slotReference.getName()); } else { slotRef = new SlotRef(slotDescriptor); - if (slotReference.hasSubColPath()) { - slotDescriptor.setSubColLables(slotReference.getSubColPath()); + if (slotReference.hasSubColPath() && slotReference.getColumn().isPresent()) { + slotDescriptor.setSubColLables(slotReference.getSubPath()); // use lower case name for variant's root, since backend treat parent column as lower case // see issue: https://github.com/apache/doris/pull/32999/commits slotDescriptor.setMaterializedColumnName(slotRef.getColumnName().toLowerCase() - + "." + String.join(".", slotReference.getSubColPath())); + + "." + String.join(".", slotReference.getSubPath())); } } slotRef.setTable(table); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/AbstractBatchJobExecutor.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/AbstractBatchJobExecutor.java index 52b248076203e9..bec86debc9ecac 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/AbstractBatchJobExecutor.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/AbstractBatchJobExecutor.java @@ -53,10 +53,14 @@ public AbstractBatchJobExecutor(CascadesContext cascadesContext) { this.cascadesContext = Objects.requireNonNull(cascadesContext, "cascadesContext can not null"); } + /** + * flat map jobs in TopicRewriteJob to could really run jobs, and filter null. + */ public static List jobs(RewriteJob... jobs) { return Arrays.stream(jobs) + .filter(Objects::nonNull) .flatMap(job -> job instanceof TopicRewriteJob - ? ((TopicRewriteJob) job).jobs.stream() + ? ((TopicRewriteJob) job).jobs.stream().filter(Objects::nonNull) : Stream.of(job) ).collect(ImmutableList.toImmutableList()); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Analyzer.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Analyzer.java index 77d23464e65965..b72240cb8e503e 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Analyzer.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/executor/Analyzer.java @@ -25,7 +25,6 @@ import org.apache.doris.nereids.rules.analysis.BindRelation; import org.apache.doris.nereids.rules.analysis.BindRelation.CustomTableResolver; import org.apache.doris.nereids.rules.analysis.BindSink; -import org.apache.doris.nereids.rules.analysis.BindSlotWithPaths; import org.apache.doris.nereids.rules.analysis.BuildAggForRandomDistributedTable; import org.apache.doris.nereids.rules.analysis.CheckAfterBind; import org.apache.doris.nereids.rules.analysis.CheckAnalysis; @@ -136,7 +135,6 @@ private static List buildAnalyzeJobs(Optional c new CheckPolicy() ), bottomUp(new BindExpression()), - bottomUp(new BindSlotWithPaths()), topDown(new BindSink()), bottomUp(new CheckAfterBind()), bottomUp( 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 a240bebd90578b..2c81a60e0ed697 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,6 +44,7 @@ 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.ClearContextStatus; import org.apache.doris.nereids.rules.rewrite.CollectCteConsumerOutput; import org.apache.doris.nereids.rules.rewrite.CollectFilterAboveConsumer; import org.apache.doris.nereids.rules.rewrite.ColumnPruning; @@ -132,12 +133,16 @@ import org.apache.doris.nereids.rules.rewrite.TransposeSemiJoinAggProject; import org.apache.doris.nereids.rules.rewrite.TransposeSemiJoinLogicalJoin; import org.apache.doris.nereids.rules.rewrite.TransposeSemiJoinLogicalJoinProject; +import org.apache.doris.nereids.rules.rewrite.VariantSubPathPruning; import org.apache.doris.nereids.rules.rewrite.batch.ApplyToJoin; import org.apache.doris.nereids.rules.rewrite.batch.CorrelateApplyToUnCorrelateApply; import org.apache.doris.nereids.rules.rewrite.batch.EliminateUselessPlanUnderApply; import org.apache.doris.nereids.rules.rewrite.mv.SelectMaterializedIndexWithAggregate; import org.apache.doris.nereids.rules.rewrite.mv.SelectMaterializedIndexWithoutAggregate; +import com.google.common.collect.ImmutableList; +import com.google.common.collect.Lists; + import java.util.List; import java.util.stream.Collectors; @@ -146,7 +151,7 @@ */ public class Rewriter extends AbstractBatchJobExecutor { - private static final List CTE_CHILDREN_REWRITE_JOBS = jobs( + private static final List CTE_CHILDREN_REWRITE_JOBS_BEFORE_SUB_PATH_PUSH_DOWN = jobs( topic("Plan Normalization", topDown( new EliminateOrderByConstant(), @@ -396,9 +401,6 @@ public class Rewriter extends AbstractBatchJobExecutor { topic("adjust preagg status", topDown(new AdjustPreAggStatus()) ), - topic("topn optimize", - topDown(new DeferMaterializeTopNResult()) - ), topic("Point query short circuit", topDown(new LogicalResultSinkToShortCircuitPointQuery())), topic("eliminate", @@ -411,6 +413,25 @@ public class Rewriter extends AbstractBatchJobExecutor { topDown(new SumLiteralRewrite(), new MergePercentileToArray()) ), + topic("Push project and filter on cte consumer to cte producer", + topDown( + new CollectFilterAboveConsumer(), + new CollectCteConsumerOutput() + ) + ) + ); + + private static final List CTE_CHILDREN_REWRITE_JOBS_AFTER_SUB_PATH_PUSH_DOWN = jobs( + // after variant sub path pruning, we need do column pruning again + custom(RuleType.COLUMN_PRUNING, ColumnPruning::new), + bottomUp(ImmutableList.of( + new PushDownFilterThroughProject(), + new MergeProjects() + )), + custom(RuleType.ELIMINATE_UNNECESSARY_PROJECT, EliminateUnnecessaryProject::new), + topic("topn optimize", + topDown(new DeferMaterializeTopNResult()) + ), // this rule batch must keep at the end of rewrite to do some plan check topic("Final rewrite and check", custom(RuleType.CHECK_DATA_TYPES, CheckDataTypes::new), @@ -423,12 +444,7 @@ public class Rewriter extends AbstractBatchJobExecutor { new CheckAfterRewrite() ) ), - topic("Push project and filter on cte consumer to cte producer", - topDown( - new CollectFilterAboveConsumer(), - new CollectCteConsumerOutput() - ) - ) + topDown(new CollectCteConsumerOutput()) ); private static final List WHOLE_TREE_REWRITE_JOBS @@ -456,19 +472,30 @@ public static Rewriter getCteChildrenRewriter(CascadesContext cascadesContext, L return new Rewriter(cascadesContext, jobs); } + /** + * only + */ public static Rewriter getWholeTreeRewriterWithCustomJobs(CascadesContext cascadesContext, List jobs) { - return new Rewriter(cascadesContext, getWholeTreeRewriteJobs(jobs)); + return new Rewriter(cascadesContext, getWholeTreeRewriteJobs(false, false, jobs, ImmutableList.of())); } private static List getWholeTreeRewriteJobs(boolean withCostBased) { - List withoutCostBased = Rewriter.CTE_CHILDREN_REWRITE_JOBS.stream() + List withoutCostBased = Rewriter.CTE_CHILDREN_REWRITE_JOBS_BEFORE_SUB_PATH_PUSH_DOWN.stream() .filter(j -> !(j instanceof CostBasedRewriteJob)) .collect(Collectors.toList()); - return getWholeTreeRewriteJobs(withCostBased ? CTE_CHILDREN_REWRITE_JOBS : withoutCostBased); + return getWholeTreeRewriteJobs(true, true, + withCostBased ? CTE_CHILDREN_REWRITE_JOBS_BEFORE_SUB_PATH_PUSH_DOWN : withoutCostBased, + CTE_CHILDREN_REWRITE_JOBS_AFTER_SUB_PATH_PUSH_DOWN); } - private static List getWholeTreeRewriteJobs(List jobs) { - return jobs( + private static List getWholeTreeRewriteJobs( + boolean needSubPathPushDown, + boolean needOrExpansion, + List beforePushDownJobs, + List afterPushDownJobs) { + + List rewriteJobs = Lists.newArrayListWithExpectedSize(300); + rewriteJobs.addAll(jobs( topic("cte inline and pull up all cte anchor", custom(RuleType.PULL_UP_CTE_ANCHOR, PullUpCteAnchor::new), custom(RuleType.CTE_INLINE, CTEInline::new) @@ -476,15 +503,30 @@ private static List getWholeTreeRewriteJobs(List jobs) { topic("process limit session variables", custom(RuleType.ADD_DEFAULT_LIMIT, AddDefaultLimit::new) ), - topic("rewrite cte sub-tree", - custom(RuleType.REWRITE_CTE_CHILDREN, () -> new RewriteCteChildren(jobs)) + topic("rewrite cte sub-tree before sub path push down", + custom(RuleType.REWRITE_CTE_CHILDREN, () -> new RewriteCteChildren(beforePushDownJobs)) + ))); + if (needOrExpansion) { + rewriteJobs.addAll(jobs(topic("or expansion", + custom(RuleType.OR_EXPANSION, () -> OrExpansion.INSTANCE)))); + } + if (needSubPathPushDown) { + rewriteJobs.addAll(jobs( + topic("variant element_at push down", + custom(RuleType.VARIANT_SUB_PATH_PRUNING, VariantSubPathPruning::new) + ) + )); + } + rewriteJobs.addAll(jobs( + topic("rewrite cte sub-tree after sub path push down", + custom(RuleType.CLEAR_CONTEXT_STATUS, ClearContextStatus::new), + custom(RuleType.REWRITE_CTE_CHILDREN, () -> new RewriteCteChildren(afterPushDownJobs)) ), - topic("or expansion", - custom(RuleType.OR_EXPANSION, () -> OrExpansion.INSTANCE)), topic("whole plan check", custom(RuleType.ADJUST_NULLABLE, AdjustNullable::new) ) - ); + )); + return rewriteJobs; } @Override diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/CommonSubExpressionOpt.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/CommonSubExpressionOpt.java index 8f558aaba0239b..fca84167994d08 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/CommonSubExpressionOpt.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/CommonSubExpressionOpt.java @@ -50,13 +50,12 @@ */ public class CommonSubExpressionOpt extends PlanPostProcessor { @Override - public PhysicalProject visitPhysicalProject(PhysicalProject project, CascadesContext ctx) { + public PhysicalProject visitPhysicalProject( + PhysicalProject project, CascadesContext ctx) { project.child().accept(this, ctx); - if (!project.hasPushedDownToProjectionFunctions()) { - List> multiLayers = computeMultiLayerProjections( - project.getInputSlots(), project.getProjects()); - project.setMultiLayerProjects(multiLayers); - } + List> multiLayers = computeMultiLayerProjections( + project.getInputSlots(), project.getProjects()); + project.setMultiLayerProjects(multiLayers); return project; } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/PushDownFilterThroughProject.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/PushDownFilterThroughProject.java index e0ee6bbd6fd4ff..864e817dc1f4de 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/PushDownFilterThroughProject.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/PushDownFilterThroughProject.java @@ -19,6 +19,7 @@ import org.apache.doris.nereids.CascadesContext; import org.apache.doris.nereids.trees.plans.Plan; +import org.apache.doris.nereids.trees.plans.physical.AbstractPhysicalPlan; import org.apache.doris.nereids.trees.plans.physical.PhysicalFilter; import org.apache.doris.nereids.trees.plans.physical.PhysicalProject; import org.apache.doris.nereids.util.ExpressionUtils; @@ -36,14 +37,10 @@ public Plan visitPhysicalFilter(PhysicalFilter filter, CascadesC } PhysicalProject project = (PhysicalProject) child; - if (project.hasPushedDownToProjectionFunctions()) { - // ignore project which is pulled up from LogicalOlapScan - return filter; - } PhysicalFilter newFilter = filter.withConjunctsAndChild( ExpressionUtils.replace(filter.getConjuncts(), project.getAliasToProducer()), project.child()); - return ((PhysicalProject) project.withChildren(newFilter.accept(this, context))) + return ((AbstractPhysicalPlan) project.withChildren(newFilter.accept(this, context))) .copyStatsAndGroupIdFrom(project); } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/Validator.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/Validator.java index 561e09ed404ad2..8ff3a43bf79e21 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/Validator.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/processor/post/Validator.java @@ -61,7 +61,7 @@ public Plan visitPhysicalFilter(PhysicalFilter filter, CascadesC Plan child = filter.child(); // Forbidden filter-project, we must make filter-project -> project-filter. - if (child instanceof PhysicalProject && !((PhysicalProject) child).hasPushedDownToProjectionFunctions()) { + if (child instanceof PhysicalProject) { throw new AnalysisException( "Nereids generate a filter-project plan, but backend not support:\n" + filter.treeString()); } 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 74ad4bdb53d3fd..d8fd26c7949362 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 @@ -52,7 +52,6 @@ public enum RuleType { BINDING_SET_OPERATION_SLOT(RuleTypeClass.REWRITE), BINDING_INLINE_TABLE_SLOT(RuleTypeClass.REWRITE), - BINDING_SLOT_WITH_PATHS_SCAN(RuleTypeClass.REWRITE), COUNT_LITERAL_REWRITE(RuleTypeClass.REWRITE), SUM_LITERAL_REWRITE(RuleTypeClass.REWRITE), REPLACE_SORT_EXPRESSION_BY_CHILD_OUTPUT(RuleTypeClass.REWRITE), @@ -178,6 +177,9 @@ public enum RuleType { PUSH_DOWN_DISTINCT_THROUGH_JOIN(RuleTypeClass.REWRITE), + VARIANT_SUB_PATH_PRUNING(RuleTypeClass.REWRITE), + CLEAR_CONTEXT_STATUS(RuleTypeClass.REWRITE), + COLUMN_PRUNING(RuleTypeClass.REWRITE), ELIMINATE_SORT(RuleTypeClass.REWRITE), @@ -268,13 +270,11 @@ public enum RuleType { OLAP_SCAN_PARTITION_PRUNE(RuleTypeClass.REWRITE), - OLAP_SCAN_WITH_PROJECT_PARTITION_PRUNE(RuleTypeClass.REWRITE), FILE_SCAN_PARTITION_PRUNE(RuleTypeClass.REWRITE), PUSH_CONJUNCTS_INTO_JDBC_SCAN(RuleTypeClass.REWRITE), PUSH_CONJUNCTS_INTO_ODBC_SCAN(RuleTypeClass.REWRITE), PUSH_CONJUNCTS_INTO_ES_SCAN(RuleTypeClass.REWRITE), OLAP_SCAN_TABLET_PRUNE(RuleTypeClass.REWRITE), - OLAP_SCAN_WITH_PROJECT_TABLET_PRUNE(RuleTypeClass.REWRITE), PUSH_AGGREGATE_TO_OLAP_SCAN(RuleTypeClass.REWRITE), EXTRACT_SINGLE_TABLE_EXPRESSION_FROM_DISJUNCTION(RuleTypeClass.REWRITE), HIDE_ONE_ROW_RELATION_UNDER_UNION(RuleTypeClass.REWRITE), @@ -317,8 +317,6 @@ public enum RuleType { // adjust nullable ADJUST_NULLABLE(RuleTypeClass.REWRITE), ADJUST_CONJUNCTS_RETURN_TYPE(RuleTypeClass.REWRITE), - // ensure having project on the top join - ENSURE_PROJECT_ON_TOP_JOIN(RuleTypeClass.REWRITE), PULL_UP_CTE_ANCHOR(RuleTypeClass.REWRITE), CTE_INLINE(RuleTypeClass.REWRITE), diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSlotWithPaths.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSlotWithPaths.java deleted file mode 100644 index 714e6e48794ba0..00000000000000 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSlotWithPaths.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.analysis; - -import org.apache.doris.nereids.StatementContext; -import org.apache.doris.nereids.rules.Rule; -import org.apache.doris.nereids.rules.RuleType; -import org.apache.doris.nereids.trees.expressions.Alias; -import org.apache.doris.nereids.trees.expressions.NamedExpression; -import org.apache.doris.nereids.trees.expressions.SlotReference; -import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan; -import org.apache.doris.nereids.trees.plans.logical.LogicalProject; -import org.apache.doris.qe.ConnectContext; - -import com.google.common.base.Preconditions; -import com.google.common.collect.ImmutableList; - -import java.util.ArrayList; -import java.util.List; -import java.util.Set; - -/** - * Rule to bind slot with path in query plan. - * Slots with paths do not exist in OlapTable so in order to materialize them, - * generate a LogicalProject on LogicalOlapScan which merges both slots from LogicalOlapScan - * and alias functions from original expressions before rewritten. - */ -public class BindSlotWithPaths implements AnalysisRuleFactory { - - @Override - public List buildRules() { - return ImmutableList.of( - // only scan - RuleType.BINDING_SLOT_WITH_PATHS_SCAN.build( - logicalOlapScan().whenNot(LogicalOlapScan::isProjectPulledUp).thenApply(ctx -> { - if (ConnectContext.get() != null - && ConnectContext.get().getSessionVariable() != null - && !ConnectContext.get().getSessionVariable().isEnableRewriteElementAtToSlot()) { - return ctx.root; - } - LogicalOlapScan logicalOlapScan = ctx.root; - List newProjectsExpr = new ArrayList<>(logicalOlapScan.getOutput()); - Set pathsSlots = ctx.statementContext.getAllPathsSlots(); - // With new logical properties that contains new slots with paths - StatementContext stmtCtx = ConnectContext.get().getStatementContext(); - ImmutableList.Builder newExprsBuilder - = ImmutableList.builderWithExpectedSize(pathsSlots.size()); - for (SlotReference slot : pathsSlots) { - Preconditions.checkNotNull(stmtCtx.getRelationBySlot(slot), - "[Not implemented] Slot not found in relation map, slot ", slot); - if (stmtCtx.getRelationBySlot(slot).getRelationId() - == logicalOlapScan.getRelationId()) { - newExprsBuilder.add(new Alias(slot.getExprId(), - stmtCtx.getOriginalExpr(slot), slot.getName())); - } - } - ImmutableList newExprs = newExprsBuilder.build(); - if (newExprs.isEmpty()) { - return ctx.root; - } - newProjectsExpr.addAll(newExprs); - return new LogicalProject<>(newProjectsExpr, logicalOlapScan.withProjectPulledUp()); - })) - ); - } -} diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckAfterRewrite.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckAfterRewrite.java index 3e079cd19af9d0..df8ec64fc2e1ff 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckAfterRewrite.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckAfterRewrite.java @@ -42,7 +42,6 @@ import org.apache.doris.nereids.trees.plans.logical.LogicalDeferMaterializeOlapScan; import org.apache.doris.nereids.trees.plans.logical.LogicalFilter; import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan; -import org.apache.doris.nereids.trees.plans.logical.LogicalProject; import org.apache.doris.nereids.trees.plans.logical.LogicalSort; import org.apache.doris.nereids.trees.plans.logical.LogicalTopN; import org.apache.doris.nereids.trees.plans.logical.LogicalWindow; @@ -187,9 +186,7 @@ private void checkMatchIsUsedCorrectly(Plan plan) { for (Expression expression : plan.getExpressions()) { if (expression instanceof Match) { if (plan instanceof LogicalFilter && (plan.child(0) instanceof LogicalOlapScan - || plan.child(0) instanceof LogicalDeferMaterializeOlapScan - || plan.child(0) instanceof LogicalProject - && ((LogicalProject) plan.child(0)).hasPushedDownToProjectionFunctions())) { + || plan.child(0) instanceof LogicalDeferMaterializeOlapScan)) { return; } else { throw new AnalysisException(String.format( diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/ExpressionAnalyzer.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/ExpressionAnalyzer.java index 0f4641cc425f6b..f6ccb87c8baa8a 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/ExpressionAnalyzer.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/ExpressionAnalyzer.java @@ -69,7 +69,6 @@ import org.apache.doris.nereids.trees.expressions.functions.scalar.ElementAt; import org.apache.doris.nereids.trees.expressions.functions.scalar.Lambda; import org.apache.doris.nereids.trees.expressions.functions.scalar.Nvl; -import org.apache.doris.nereids.trees.expressions.functions.scalar.PushDownToProjectionFunction; import org.apache.doris.nereids.trees.expressions.functions.udf.AliasUdfBuilder; import org.apache.doris.nereids.trees.expressions.functions.udf.JavaUdaf; import org.apache.doris.nereids.trees.expressions.functions.udf.JavaUdf; @@ -102,7 +101,6 @@ import java.util.List; import java.util.Locale; import java.util.Optional; -import java.util.Set; import java.util.stream.Collectors; /** ExpressionAnalyzer */ @@ -123,12 +121,6 @@ public class ExpressionAnalyzer extends SubExprAnalyzer !(slot instanceof SlotReference) || (((SlotReference) slot).isVisible()) || showHidden) - .filter(slot -> !(((SlotReference) slot).hasSubColPath())) .collect(Collectors.toList()); switch (qualifier.size()) { case 0: // select * @@ -336,9 +327,6 @@ public Expression visitUnboundStar(UnboundStar unboundStar, ExpressionRewriteCon * ******************************************************************************************** */ @Override public Expression visitUnboundFunction(UnboundFunction unboundFunction, ExpressionRewriteContext context) { - if (unboundFunction.getName().equalsIgnoreCase("element_at")) { - ++currentElementAtLevel; - } if (unboundFunction.isHighOrder()) { unboundFunction = bindHighOrderFunction(unboundFunction, context); } else { @@ -399,17 +387,6 @@ public Expression visitUnboundFunction(UnboundFunction unboundFunction, Expressi // so wrap COUNT with Nvl to ensure it's result is 0 instead of null to get the correct result castFunction = new Nvl(castFunction, new BigIntLiteral(0)); } - - if (currentElementAtLevel == 1 - && PushDownToProjectionFunction.validToPushDown(castFunction)) { - // Only rewrite the top level of PushDownToProjectionFunction, otherwise invalid slot will be generated - // currentElementAtLevel == 1 means at the top of element_at function, other levels will be ignored. - currentElementAtLevel = 0; - return visitElementAt((ElementAt) castFunction, context); - } - if (castFunction instanceof ElementAt) { - --currentElementAtLevel; - } return castFunction; } } @@ -420,39 +397,6 @@ public Expression visitBoundFunction(BoundFunction boundFunction, ExpressionRewr return TypeCoercionUtils.processBoundFunction(boundFunction); } - @Override - public Expression visitElementAt(ElementAt elementAt, ExpressionRewriteContext context) { - ElementAt boundFunction = (ElementAt) visitBoundFunction(elementAt, context); - if (PushDownToProjectionFunction.validToPushDown(boundFunction)) { - if (ConnectContext.get() != null - && ConnectContext.get().getSessionVariable() != null - && !ConnectContext.get().getSessionVariable().isEnableRewriteElementAtToSlot()) { - return boundFunction; - } - // TODO: push down logic here is very tricky, we will refactor it later - Set inputSlots = boundFunction.getInputSlots(); - if (inputSlots.isEmpty()) { - return boundFunction; - } - Slot slot = inputSlots.iterator().next(); - if (slot.hasUnbound()) { - slot = (Slot) slot.accept(this, context); - } - StatementContext statementContext = context.cascadesContext.getStatementContext(); - Expression originBoundFunction = boundFunction.rewriteUp(expr -> { - if (expr instanceof SlotReference) { - Expression originalExpr = statementContext.getOriginalExpr((SlotReference) expr); - return originalExpr == null ? expr : originalExpr; - } - return expr; - }); - // rewrite to slot and bound this slot - return PushDownToProjectionFunction.rewriteToSlot( - (PushDownToProjectionFunction) originBoundFunction, (SlotReference) slot); - } - return boundFunction; - } - /** * gets the method for calculating the time. * e.g. YEARS_ADD、YEARS_SUB、DAYS_ADD 、DAYS_SUB @@ -813,15 +757,7 @@ private UnboundFunction bindHighOrderFunction(UnboundFunction unboundFunction, E } private boolean shouldBindSlotBy(int namePartSize, Slot boundSlot) { - if (boundSlot instanceof SlotReference - && ((SlotReference) boundSlot).hasSubColPath()) { - // already bounded - return false; - } - if (namePartSize > boundSlot.getQualifier().size() + 1) { - return false; - } - return true; + return namePartSize <= boundSlot.getQualifier().size() + 1; } private List bindSingleSlotByName(String name, Scope scope) { diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/SlotBinder.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/SlotBinder.java index 8b6d82087d97ea..3fa272568c3772 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/SlotBinder.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/SlotBinder.java @@ -184,7 +184,6 @@ public Expression visitUnboundStar(UnboundStar unboundStar, CascadesContext cont .stream() .filter(slot -> !(slot instanceof SlotReference) || (((SlotReference) slot).isVisible()) || showHidden) - .filter(slot -> !(((SlotReference) slot).hasSubColPath())) .collect(Collectors.toList()); switch (qualifier.size()) { case 0: // select * @@ -268,11 +267,6 @@ && compareDbName(qualifierStar.get(1), boundSlotQualifier.get(1)) private List bindSlot(UnboundSlot unboundSlot, List boundSlots) { return boundSlots.stream().distinct().filter(boundSlot -> { - if (boundSlot instanceof SlotReference - && ((SlotReference) boundSlot).hasSubColPath()) { - // already bounded - return false; - } List nameParts = unboundSlot.getNameParts(); int qualifierSize = boundSlot.getQualifier().size(); int namePartsSize = nameParts.size(); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/expression/rules/FunctionBinder.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/expression/rules/FunctionBinder.java index 8e79e60abad1bd..1f170479e0763e 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/expression/rules/FunctionBinder.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/expression/rules/FunctionBinder.java @@ -45,16 +45,13 @@ import org.apache.doris.nereids.trees.expressions.Match; import org.apache.doris.nereids.trees.expressions.Not; import org.apache.doris.nereids.trees.expressions.Slot; -import org.apache.doris.nereids.trees.expressions.SlotReference; import org.apache.doris.nereids.trees.expressions.TimestampArithmetic; import org.apache.doris.nereids.trees.expressions.WhenClause; import org.apache.doris.nereids.trees.expressions.functions.BoundFunction; import org.apache.doris.nereids.trees.expressions.functions.FunctionBuilder; import org.apache.doris.nereids.trees.expressions.functions.agg.Count; -import org.apache.doris.nereids.trees.expressions.functions.scalar.ElementAt; import org.apache.doris.nereids.trees.expressions.functions.scalar.Lambda; import org.apache.doris.nereids.trees.expressions.functions.scalar.Nvl; -import org.apache.doris.nereids.trees.expressions.functions.scalar.PushDownToProjectionFunction; import org.apache.doris.nereids.trees.expressions.functions.udf.AliasUdfBuilder; import org.apache.doris.nereids.trees.expressions.literal.BigIntLiteral; import org.apache.doris.nereids.trees.expressions.typecoercion.ImplicitCastInputTypes; @@ -63,7 +60,6 @@ import org.apache.doris.nereids.types.BooleanType; import org.apache.doris.nereids.types.DataType; import org.apache.doris.nereids.util.TypeCoercionUtils; -import org.apache.doris.qe.ConnectContext; import com.google.common.collect.ImmutableList; import org.apache.commons.lang3.StringUtils; @@ -80,12 +76,6 @@ public class FunctionBinder extends AbstractExpressionRewriteRule { public static final FunctionBinder INSTANCE = new FunctionBinder(); - // Keep track of which element_at function's level - // e.g. element_at(element_at(v, 'repo'), 'name') level 1 - // element_at(v, 'repo') level 2 - // Only works with function ElementAt which satisfy condition PushDownToProjectionFunction.validToPushDown - private int currentElementAtLevel = 0; - @Override public Expression visit(Expression expr, ExpressionRewriteContext context) { expr = super.visit(expr, context); @@ -147,9 +137,6 @@ private UnboundFunction bindHighOrderFunction(UnboundFunction unboundFunction, E @Override public Expression visitUnboundFunction(UnboundFunction unboundFunction, ExpressionRewriteContext context) { - if (unboundFunction.getName().equalsIgnoreCase("element_at")) { - ++currentElementAtLevel; - } if (unboundFunction.isHighOrder()) { unboundFunction = bindHighOrderFunction(unboundFunction, context); } else { @@ -197,16 +184,6 @@ public Expression visitUnboundFunction(UnboundFunction unboundFunction, Expressi // so wrap COUNT with Nvl to ensure it's result is 0 instead of null to get the correct result boundFunction = new Nvl(boundFunction, new BigIntLiteral(0)); } - if (currentElementAtLevel == 1 - && PushDownToProjectionFunction.validToPushDown(boundFunction)) { - // Only rewrite the top level of PushDownToProjectionFunction, otherwise invalid slot will be generated - // currentElementAtLevel == 1 means at the top of element_at function, other levels will be ignored. - currentElementAtLevel = 0; - return visitElementAt((ElementAt) boundFunction, context); - } - if (boundFunction instanceof ElementAt) { - --currentElementAtLevel; - } return boundFunction; } } @@ -217,26 +194,6 @@ public Expression visitBoundFunction(BoundFunction boundFunction, ExpressionRewr return TypeCoercionUtils.processBoundFunction(boundFunction); } - @Override - public Expression visitElementAt(ElementAt elementAt, ExpressionRewriteContext context) { - Expression boundFunction = visitBoundFunction(elementAt, context); - - if (PushDownToProjectionFunction.validToPushDown(boundFunction)) { - if (ConnectContext.get() != null - && ConnectContext.get().getSessionVariable() != null - && !ConnectContext.get().getSessionVariable().isEnableRewriteElementAtToSlot()) { - return boundFunction; - } - Slot slot = elementAt.getInputSlots().stream().findFirst().get(); - if (slot.hasUnbound()) { - slot = (Slot) super.visit(slot, context); - } - // rewrite to slot and bound this slot - return PushDownToProjectionFunction.rewriteToSlot(elementAt, (SlotReference) slot); - } - return boundFunction; - } - /** * gets the method for calculating the time. * e.g. YEARS_ADD、YEARS_SUB、DAYS_ADD 、DAYS_SUB diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ClearContextStatus.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ClearContextStatus.java new file mode 100644 index 00000000000000..cb5dcf5526a4d5 --- /dev/null +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/ClearContextStatus.java @@ -0,0 +1,40 @@ +// 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.jobs.JobContext; +import org.apache.doris.nereids.trees.plans.Plan; +import org.apache.doris.nereids.trees.plans.visitor.CustomRewriter; + +/** + * pull up LogicalCteAnchor to the top of plan to avoid CteAnchor break other rewrite rules pattern + * The front producer may depend on the back producer in {@code List>} + * After this rule, we normalize all CteAnchor in plan, all CteAnchor under CteProducer should pull out + * and put all of them to the top of plan depends on dependency tree of them. + */ +public class ClearContextStatus implements CustomRewriter { + + @Override + public Plan rewriteRoot(Plan plan, JobContext jobContext) { + jobContext.getCascadesContext().getStatementContext().getRewrittenCteConsumer().clear(); + jobContext.getCascadesContext().getStatementContext().getRewrittenCteProducer().clear(); + jobContext.getCascadesContext().getStatementContext().getCteIdToOutputIds().clear(); + jobContext.getCascadesContext().getStatementContext().getConsumerIdToFilters().clear(); + return plan; + } +} diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/DeferMaterializeTopNResult.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/DeferMaterializeTopNResult.java index 398034638411fc..6d90d81349d08f 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/DeferMaterializeTopNResult.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/DeferMaterializeTopNResult.java @@ -83,8 +83,8 @@ private Plan deferMaterialize(LogicalResultSink logicalResultSin LogicalTopN logicalTopN, Optional> logicalFilter, LogicalOlapScan logicalOlapScan) { Column rowId = new Column(Column.ROWID_COL, Type.STRING, false, null, false, "", "rowid column"); - SlotReference columnId = SlotReference.fromColumn(logicalOlapScan.getTable(), rowId, - logicalOlapScan.getQualifier(), logicalOlapScan); + SlotReference columnId = SlotReference.fromColumn( + logicalOlapScan.getTable(), rowId, logicalOlapScan.getQualifier()); Set deferredMaterializedExprIds = Sets.newHashSet(logicalOlapScan.getOutputExprIdSet()); logicalFilter.ifPresent(filter -> filter.getConjuncts() .forEach(e -> deferredMaterializedExprIds.removeAll(e.getInputSlotExprIds()))); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PruneOlapScanPartition.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PruneOlapScanPartition.java index 0aacde1cc1984c..0d5054086117d9 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PruneOlapScanPartition.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PruneOlapScanPartition.java @@ -21,17 +21,14 @@ import org.apache.doris.catalog.OlapTable; import org.apache.doris.catalog.PartitionInfo; import org.apache.doris.catalog.PartitionItem; -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.rules.expression.rules.PartitionPruner; import org.apache.doris.nereids.rules.expression.rules.PartitionPruner.PartitionTableType; import org.apache.doris.nereids.trees.expressions.Slot; -import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.trees.plans.logical.LogicalEmptyRelation; import org.apache.doris.nereids.trees.plans.logical.LogicalFilter; import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan; -import org.apache.doris.nereids.trees.plans.logical.LogicalProject; import org.apache.doris.nereids.util.Utils; import org.apache.doris.qe.ConnectContext; @@ -48,79 +45,57 @@ * Used to prune partition of olap scan, should execute after SwapProjectAndFilter, MergeConsecutiveFilters, * MergeConsecutiveProjects and all predicate push down related rules. */ -public class PruneOlapScanPartition implements RewriteRuleFactory { +public class PruneOlapScanPartition extends OneRewriteRuleFactory { @Override - public List buildRules() { - return ImmutableList.of( - logicalFilter(logicalOlapScan()) - .when(p -> !p.child().isPartitionPruned()) - .thenApply(ctx -> prunePartitions(ctx.cascadesContext, ctx.root.child(), ctx.root)) - .toRule(RuleType.OLAP_SCAN_PARTITION_PRUNE), - - logicalFilter(logicalProject(logicalOlapScan())) - .when(p -> !p.child().child().isPartitionPruned()) - .when(p -> p.child().hasPushedDownToProjectionFunctions()) - .thenApply(ctx -> prunePartitions(ctx.cascadesContext, ctx.root.child().child(), ctx.root)) - .toRule(RuleType.OLAP_SCAN_WITH_PROJECT_PARTITION_PRUNE) - ); - } - - private Plan prunePartitions(CascadesContext ctx, - LogicalOlapScan scan, LogicalFilter originalFilter) { - OlapTable table = scan.getTable(); - Set partitionColumnNameSet = Utils.execWithReturnVal(table::getPartitionColumnNames); - if (partitionColumnNameSet.isEmpty()) { - return originalFilter; - } - - List output = scan.getOutput(); - PartitionInfo partitionInfo = table.getPartitionInfo(); - List partitionColumns = partitionInfo.getPartitionColumns(); - List partitionSlots = new ArrayList<>(partitionColumns.size()); - for (Column column : partitionColumns) { - Slot partitionSlot = null; - // loop search is faster than build a map - for (Slot slot : output) { - if (slot.getName().equalsIgnoreCase(column.getName())) { - partitionSlot = slot; - break; + public Rule build() { + return logicalFilter(logicalOlapScan()).when(p -> !p.child().isPartitionPruned()).thenApply(ctx -> { + LogicalFilter filter = ctx.root; + LogicalOlapScan scan = filter.child(); + OlapTable table = scan.getTable(); + Set partitionColumnNameSet = Utils.execWithReturnVal(table::getPartitionColumnNames); + if (partitionColumnNameSet.isEmpty()) { + return null; + } + List output = scan.getOutput(); + PartitionInfo partitionInfo = table.getPartitionInfo(); + List partitionColumns = partitionInfo.getPartitionColumns(); + List partitionSlots = new ArrayList<>(partitionColumns.size()); + for (Column column : partitionColumns) { + Slot partitionSlot = null; + // loop search is faster than build a map + for (Slot slot : output) { + if (slot.getName().equalsIgnoreCase(column.getName())) { + partitionSlot = slot; + break; + } + } + if (partitionSlot == null) { + return null; + } else { + partitionSlots.add(partitionSlot); } } - if (partitionSlot == null) { - return originalFilter; + List manuallySpecifiedPartitions = scan.getManuallySpecifiedPartitions(); + Map idToPartitions; + if (manuallySpecifiedPartitions.isEmpty()) { + idToPartitions = partitionInfo.getIdToItem(false); } else { - partitionSlots.add(partitionSlot); + Map allPartitions = partitionInfo.getAllPartitions(); + idToPartitions = allPartitions.keySet().stream() + .filter(manuallySpecifiedPartitions::contains) + .collect(Collectors.toMap(Function.identity(), allPartitions::get)); } - } - - List manuallySpecifiedPartitions = scan.getManuallySpecifiedPartitions(); - - Map idToPartitions; - if (manuallySpecifiedPartitions.isEmpty()) { - idToPartitions = partitionInfo.getIdToItem(false); - } else { - Map allPartitions = partitionInfo.getAllPartitions(); - idToPartitions = allPartitions.keySet().stream() - .filter(id -> manuallySpecifiedPartitions.contains(id)) - .collect(Collectors.toMap(Function.identity(), id -> allPartitions.get(id))); - } - List prunedPartitions = PartitionPruner.prune( - partitionSlots, originalFilter.getPredicate(), idToPartitions, ctx, - PartitionTableType.OLAP); - if (prunedPartitions.isEmpty()) { - return new LogicalEmptyRelation( - ConnectContext.get().getStatementContext().getNextRelationId(), - originalFilter.getOutput()); - } - - LogicalOlapScan rewrittenScan = scan.withSelectedPartitionIds(prunedPartitions); - if (originalFilter.child() instanceof LogicalProject) { - LogicalProject rewrittenProject - = (LogicalProject) originalFilter.child() - .withChildren(ImmutableList.of(rewrittenScan)); - return new LogicalFilter<>(originalFilter.getConjuncts(), rewrittenProject); - } - return originalFilter.withChildren(ImmutableList.of(rewrittenScan)); + List prunedPartitions = PartitionPruner.prune( + partitionSlots, filter.getPredicate(), idToPartitions, ctx.cascadesContext, + PartitionTableType.OLAP); + if (prunedPartitions.isEmpty()) { + return new LogicalEmptyRelation( + ConnectContext.get().getStatementContext().getNextRelationId(), + filter.getOutput()); + } + LogicalOlapScan rewrittenScan = scan.withSelectedPartitionIds(prunedPartitions); + return filter.withChildren(ImmutableList.of(rewrittenScan)); + }).toRule(RuleType.OLAP_SCAN_PARTITION_PRUNE); } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PruneOlapScanTablet.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PruneOlapScanTablet.java index c4468ca81202fa..8cf4f91deecea1 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PruneOlapScanTablet.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PruneOlapScanTablet.java @@ -27,10 +27,7 @@ import org.apache.doris.nereids.rules.RuleType; import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.expressions.visitor.ExpressionColumnFilterConverter; -import org.apache.doris.nereids.trees.plans.Plan; -import org.apache.doris.nereids.trees.plans.logical.LogicalFilter; import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan; -import org.apache.doris.nereids.trees.plans.logical.LogicalProject; import org.apache.doris.nereids.util.ExpressionUtils; import org.apache.doris.planner.HashDistributionPruner; import org.apache.doris.planner.PartitionColumnFilter; @@ -49,50 +46,32 @@ /** * prune bucket */ -public class PruneOlapScanTablet implements RewriteRuleFactory { +public class PruneOlapScanTablet extends OneRewriteRuleFactory { @Override - public List buildRules() { - return ImmutableList.of( - logicalFilter(logicalOlapScan()) - .then(filter -> { - return pruneTablets(filter.child(), filter); - }).toRule(RuleType.OLAP_SCAN_TABLET_PRUNE), - - logicalFilter(logicalProject(logicalOlapScan())) - .when(p -> p.child().hasPushedDownToProjectionFunctions()).then(filter -> { - return pruneTablets(filter.child().child(), filter); - }).toRule(RuleType.OLAP_SCAN_WITH_PROJECT_TABLET_PRUNE) - ); - } - - private Plan pruneTablets(LogicalOlapScan olapScan, LogicalFilter originalFilter) { - OlapTable table = olapScan.getTable(); - Builder selectedTabletIdsBuilder = ImmutableList.builder(); - if (olapScan.getSelectedTabletIds().isEmpty()) { - for (Long id : olapScan.getSelectedPartitionIds()) { - Partition partition = table.getPartition(id); - MaterializedIndex index = partition.getIndex(olapScan.getSelectedIndexId()); - selectedTabletIdsBuilder - .addAll(getSelectedTabletIds(originalFilter.getConjuncts(), index, - olapScan.getSelectedIndexId() == olapScan.getTable() - .getBaseIndexId(), - partition.getDistributionInfo())); + public Rule build() { + return logicalFilter(logicalOlapScan()).then(filter -> { + LogicalOlapScan olapScan = filter.child(); + OlapTable table = olapScan.getTable(); + Builder selectedTabletIdsBuilder = ImmutableList.builder(); + if (olapScan.getSelectedTabletIds().isEmpty()) { + for (Long id : olapScan.getSelectedPartitionIds()) { + Partition partition = table.getPartition(id); + MaterializedIndex index = partition.getIndex(olapScan.getSelectedIndexId()); + selectedTabletIdsBuilder + .addAll(getSelectedTabletIds(filter.getConjuncts(), index, + olapScan.getSelectedIndexId() == olapScan.getTable() + .getBaseIndexId(), + partition.getDistributionInfo())); + } + } else { + selectedTabletIdsBuilder.addAll(olapScan.getSelectedTabletIds()); } - } else { - selectedTabletIdsBuilder.addAll(olapScan.getSelectedTabletIds()); - } - List selectedTabletIds = selectedTabletIdsBuilder.build(); - if (new HashSet(selectedTabletIds).equals(new HashSet(olapScan.getSelectedTabletIds()))) { - return null; - } - LogicalOlapScan rewrittenScan = olapScan.withSelectedTabletIds(selectedTabletIds); - if (originalFilter.child() instanceof LogicalProject) { - LogicalProject rewrittenProject - = (LogicalProject) originalFilter.child() - .withChildren(ImmutableList.of(rewrittenScan)); - return new LogicalFilter<>(originalFilter.getConjuncts(), rewrittenProject); - } - return originalFilter.withChildren(rewrittenScan); + List selectedTabletIds = selectedTabletIdsBuilder.build(); + if (new HashSet<>(selectedTabletIds).equals(new HashSet<>(olapScan.getSelectedTabletIds()))) { + return null; + } + return filter.withChildren(olapScan.withSelectedTabletIds(selectedTabletIds)); + }).toRule(RuleType.OLAP_SCAN_TABLET_PRUNE); } private Collection getSelectedTabletIds(Set expressions, diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PushDownFilterThroughProject.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PushDownFilterThroughProject.java index 71834a66b19a2f..5842beaf3d6328 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PushDownFilterThroughProject.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/PushDownFilterThroughProject.java @@ -49,28 +49,27 @@ public List buildRules() { return ImmutableList.of( logicalFilter(logicalProject()) .whenNot(filter -> ExpressionUtils.containsWindowExpression(filter.child().getProjects())) - .whenNot(filter -> filter.child().hasPushedDownToProjectionFunctions()) - .then(PushDownFilterThroughProject::pushdownFilterThroughProject) + .then(PushDownFilterThroughProject::pushDownFilterThroughProject) .toRule(RuleType.PUSH_DOWN_FILTER_THROUGH_PROJECT), // filter(project(limit)) will change to filter(limit(project)) by PushdownProjectThroughLimit, // then we should change filter(limit(project)) to project(filter(limit)) + // TODO maybe we could remove this rule, because translator already support filter(limit(project)) logicalFilter(logicalLimit(logicalProject())) .whenNot(filter -> ExpressionUtils.containsWindowExpression(filter.child().child().getProjects()) ) - .whenNot(filter -> filter.child().child().hasPushedDownToProjectionFunctions()) - .then(PushDownFilterThroughProject::pushdownFilterThroughLimitProject) + .then(PushDownFilterThroughProject::pushDownFilterThroughLimitProject) .toRule(RuleType.PUSH_DOWN_FILTER_THROUGH_PROJECT_UNDER_LIMIT) ); } - /** pushdown Filter through project */ - private static Plan pushdownFilterThroughProject(LogicalFilter> filter) { - LogicalProject project = filter.child(); + /** push down Filter through project */ + private static Plan pushDownFilterThroughProject(LogicalFilter> filter) { + LogicalProject project = filter.child(); Set childOutputs = project.getOutputSet(); - // we need run this rule before subquey unnesting + // we need run this rule before subquery unnesting // therefore the conjuncts may contain slots from outer query - // we should only push down conjuncts without any outer query's slot + // we should only push down conjuncts without any outer query's slot, // so we split the conjuncts into two parts: // splitConjuncts.first -> conjuncts having outer query slots which should NOT be pushed down // splitConjuncts.second -> conjuncts without any outer query slots which should be pushed down @@ -81,13 +80,13 @@ private static Plan pushdownFilterThroughProject(LogicalFilter) project.withChildren(new LogicalFilter<>( + project = (LogicalProject) project.withChildren(new LogicalFilter<>( ExpressionUtils.replace(splitConjuncts.second, project.getAliasToProducer()), project.child())); return PlanUtils.filterOrSelf(splitConjuncts.first, project); } - private static Plan pushdownFilterThroughLimitProject( + private static Plan pushDownFilterThroughLimitProject( LogicalFilter>> filter) { LogicalLimit> limit = filter.child(); LogicalProject project = limit.child(); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/VariantSubPathPruning.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/VariantSubPathPruning.java new file mode 100644 index 00000000000000..111493837a0cdd --- /dev/null +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/VariantSubPathPruning.java @@ -0,0 +1,775 @@ +// 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.common.Pair; +import org.apache.doris.nereids.exceptions.AnalysisException; +import org.apache.doris.nereids.jobs.JobContext; +import org.apache.doris.nereids.properties.OrderKey; +import org.apache.doris.nereids.rules.rewrite.ColumnPruning.PruneContext; +import org.apache.doris.nereids.trees.expressions.Alias; +import org.apache.doris.nereids.trees.expressions.Expression; +import org.apache.doris.nereids.trees.expressions.NamedExpression; +import org.apache.doris.nereids.trees.expressions.OrderExpression; +import org.apache.doris.nereids.trees.expressions.Slot; +import org.apache.doris.nereids.trees.expressions.SlotReference; +import org.apache.doris.nereids.trees.expressions.StatementScopeIdGenerator; +import org.apache.doris.nereids.trees.expressions.functions.ExpressionTrait; +import org.apache.doris.nereids.trees.expressions.functions.Function; +import org.apache.doris.nereids.trees.expressions.functions.scalar.ElementAt; +import org.apache.doris.nereids.trees.expressions.literal.StringLikeLiteral; +import org.apache.doris.nereids.trees.expressions.literal.VarcharLiteral; +import org.apache.doris.nereids.trees.expressions.visitor.DefaultExpressionVisitor; +import org.apache.doris.nereids.trees.plans.Plan; +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.LogicalCTEAnchor; +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.LogicalGenerate; +import org.apache.doris.nereids.trees.plans.logical.LogicalJoin; +import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan; +import org.apache.doris.nereids.trees.plans.logical.LogicalOneRowRelation; +import org.apache.doris.nereids.trees.plans.logical.LogicalPartitionTopN; +import org.apache.doris.nereids.trees.plans.logical.LogicalProject; +import org.apache.doris.nereids.trees.plans.logical.LogicalSort; +import org.apache.doris.nereids.trees.plans.logical.LogicalTopN; +import org.apache.doris.nereids.trees.plans.logical.LogicalUnion; +import org.apache.doris.nereids.trees.plans.logical.LogicalWindow; +import org.apache.doris.nereids.trees.plans.visitor.CustomRewriter; +import org.apache.doris.nereids.trees.plans.visitor.DefaultPlanRewriter; +import org.apache.doris.nereids.trees.plans.visitor.PlanVisitor; +import org.apache.doris.nereids.types.VariantType; +import org.apache.doris.nereids.util.ExpressionUtils; + +import com.google.common.collect.ImmutableList; +import com.google.common.collect.ImmutableSet; +import com.google.common.collect.Lists; +import com.google.common.collect.Maps; +import com.google.common.collect.Sets; + +import java.util.Collections; +import java.util.List; +import java.util.Map; +import java.util.Map.Entry; +import java.util.Optional; +import java.util.Set; + +/** + * prune sub path of variant type slot. + * for example, variant slot v in table t has two sub path: 'c1' and 'c2' + * after this rule, select v['c1'] from t will only scan one sub path 'c1' of v to reduce scan time + * + * This rule accomplishes all the work using two components. The Collector traverses from the top down, + * collecting all the element_at functions on the variant types, and recording the required path from + * the original variant slot to the current element_at. The Replacer traverses from the bottom up, + * generating the slots for the required sub path on scan, union, and cte consumer. + * Then, it replaces the element_at with the corresponding slot. + */ +public class VariantSubPathPruning extends DefaultPlanRewriter implements CustomRewriter { + + @Override + public Plan rewriteRoot(Plan plan, JobContext jobContext) { + Context context = new Context(); + plan.accept(VariantSubPathCollector.INSTANCE, context); + if (context.elementAtToSubPathMap.isEmpty()) { + return plan; + } else { + return plan.accept(VariantSubPathReplacer.INSTANCE, context); + } + } + + private static class Context { + + // user for collector + // record slot to its original expr. for example, Alias(c1, a1) will put a1 -> c1 to this map + private final Map slotToOriginalExprMap = Maps.newHashMap(); + // record element_at to root slot with sub path. for example, element_at(c1, 'a') as c2 + element_at(c2, 'b') + // will put element(c2, 'b') -> {c1, ['a', 'b']} and element_at(c1, 'a') -> {c1, ['a']} to this map + private final Map>> elementAtToSubPathMap = Maps.newHashMap(); + // record sub path need from slot. for example, element_at(c1, 'a') as c2 + element_at(c2, 'b') + // will put c1 -> [['a'], ['a', 'b']] to this map + private final Map>> slotToSubPathsMap = Maps.newHashMap(); + + // we need to record elementAt to consumer slot, and generate right slot when do consumer slot replace + private final Map elementAtToCteConsumer = Maps.newHashMap(); + + // use for replacer + // record element_at should be replaced with which slot + private final Map elementAtToSlotMap = Maps.newHashMap(); + // record which slots of prefix-matched sub paths need to be replaced + // in addition to the slots of the exactly matched sub path. + // for example, we have element_at(c1, 'a') as c2 + element_at(c2, 'b') + // if element_at(c1, 'a') -> s1, element_at(c2, 'b') -> s2, then + // in this map we have element_at(c1, 'a') -> {['a', 'b'] -> s2} + // this is used in replace element_at in project. since upper node may need its sub path, so we must put all + // slot could be generated from it into project list. + private final Map, SlotReference>> elementAtToSlotsMap = Maps.newHashMap(); + // same as elementAtToSlotsMap, record variant slot should be replaced by which slots. + private final Map, SlotReference>> slotToSlotsMap = Maps.newHashMap(); + + public void putSlotToOriginal(Slot slot, Expression expression) { + this.slotToOriginalExprMap.put(slot, expression); + // update existed entry + // element_at(3, c) -> 3, ['c'] + // + + // slot3 -> element_at(1, b) -> 1, ['b'] + // ==> + // element_at(3, c) -> 1, ['b', 'c'] + for (Map.Entry>> entry : elementAtToSubPathMap.entrySet()) { + ElementAt elementAt = entry.getKey(); + Pair> oldSlotSubPathPair = entry.getValue(); + if (slot.equals(oldSlotSubPathPair.first)) { + if (expression instanceof ElementAt) { + Pair> newSlotSubPathPair = elementAtToSubPathMap.get(expression); + List newPath = Lists.newArrayList(newSlotSubPathPair.second); + newPath.addAll(oldSlotSubPathPair.second); + elementAtToSubPathMap.put(elementAt, Pair.of(newSlotSubPathPair.first, newPath)); + slotToSubPathsMap.computeIfAbsent(newSlotSubPathPair.first, + k -> Sets.newHashSet()).add(newPath); + } else if (expression instanceof Slot) { + Pair> newSlotSubPathPair + = Pair.of((SlotReference) expression, oldSlotSubPathPair.second); + elementAtToSubPathMap.put(elementAt, newSlotSubPathPair); + } + } + } + if (expression instanceof SlotReference && slotToSubPathsMap.containsKey((SlotReference) slot)) { + Set> subPaths = slotToSubPathsMap + .computeIfAbsent((SlotReference) expression, k -> Sets.newHashSet()); + subPaths.addAll(slotToSubPathsMap.get(slot)); + } + } + + public void putElementAtToSubPath(ElementAt elementAt, + Pair> pair, Slot parent) { + this.elementAtToSubPathMap.put(elementAt, pair); + Set> subPaths = slotToSubPathsMap.computeIfAbsent(pair.first, k -> Sets.newHashSet()); + subPaths.add(pair.second); + if (parent != null) { + for (List parentSubPath : slotToSubPathsMap.computeIfAbsent( + (SlotReference) parent, k -> Sets.newHashSet())) { + List subPathWithParents = Lists.newArrayList(pair.second); + subPathWithParents.addAll(parentSubPath); + subPaths.add(subPathWithParents); + } + } + } + + public void putAllElementAtToSubPath(Map>> elementAtToSubPathMap) { + for (Map.Entry>> entry : elementAtToSubPathMap.entrySet()) { + putElementAtToSubPath(entry.getKey(), entry.getValue(), null); + } + } + } + + private static class VariantSubPathReplacer extends DefaultPlanRewriter { + + public static VariantSubPathReplacer INSTANCE = new VariantSubPathReplacer(); + + @Override + public Plan visitLogicalOlapScan(LogicalOlapScan olapScan, Context context) { + List outputs = olapScan.getOutput(); + Map>> colToSubPaths = Maps.newHashMap(); + for (Slot slot : outputs) { + if (slot.getDataType() instanceof VariantType + && context.slotToSubPathsMap.containsKey((SlotReference) slot)) { + Set> subPaths = context.slotToSubPathsMap.get(slot); + if (((SlotReference) slot).getColumn().isPresent()) { + colToSubPaths.put(((SlotReference) slot).getColumn().get().getName(), subPaths); + } + } + } + LogicalOlapScan newScan = olapScan.withColToSubPathsMap(colToSubPaths); + Map, SlotReference>> oriSlotToSubPathToSlot = newScan.getSubPathToSlotMap(); + generateElementAtMaps(context, oriSlotToSubPathToSlot); + return newScan; + } + + @Override + public Plan visitLogicalUnion(LogicalUnion union, Context context) { + union = (LogicalUnion) this.visit(union, context); + if (union.getQualifier() == Qualifier.DISTINCT) { + return super.visitLogicalUnion(union, context); + } + List> regularChildrenOutputs + = Lists.newArrayListWithExpectedSize(union.getRegularChildrenOutputs().size()); + List> constExprs + = Lists.newArrayListWithExpectedSize(union.getConstantExprsList().size()); + for (int i = 0; i < union.getRegularChildrenOutputs().size(); i++) { + regularChildrenOutputs.add(Lists.newArrayListWithExpectedSize(union.getOutput().size() * 2)); + } + for (int i = 0; i < union.getConstantExprsList().size(); i++) { + constExprs.add(Lists.newArrayListWithExpectedSize(union.getOutput().size() * 2)); + } + List outputs = Lists.newArrayListWithExpectedSize(union.getOutput().size() * 2); + + Map, SlotReference>> oriSlotToSubPathToSlot = Maps.newHashMap(); + for (int i = 0; i < union.getOutput().size(); i++) { + // put back original slot + for (int j = 0; j < regularChildrenOutputs.size(); j++) { + regularChildrenOutputs.get(j).add(union.getRegularChildOutput(j).get(i)); + } + for (int j = 0; j < constExprs.size(); j++) { + constExprs.get(j).add(union.getConstantExprsList().get(j).get(i)); + } + outputs.add(union.getOutputs().get(i)); + // if not variant, no need to process + if (!union.getOutput().get(i).getDataType().isVariantType()) { + continue; + } + // put new slots generated by sub path push down + Map, List> subPathSlots = Maps.newHashMap(); + for (int j = 0; j < regularChildrenOutputs.size(); j++) { + List regularChildOutput = union.getRegularChildOutput(j); + Expression output = regularChildOutput.get(i); + if (!context.slotToSlotsMap.containsKey(output) + || !context.slotToSubPathsMap.containsKey(outputs.get(i))) { + // no sub path request for this column + continue; + } + // find sub path generated by union children + Expression key = output; + while (context.slotToOriginalExprMap.containsKey(key)) { + key = context.slotToOriginalExprMap.get(key); + } + List subPathByChildren = Collections.emptyList(); + if (key instanceof ElementAt) { + // this means need to find common sub path of its slots. + subPathByChildren = context.elementAtToSubPathMap.get(key).second; + } + + for (Map.Entry, SlotReference> entry : context.slotToSlotsMap.get(output).entrySet()) { + List slotsForSubPath; + // remove subPath generated by children, + // because context only record sub path generated by parent + List parentPaths = entry.getKey() + .subList(subPathByChildren.size(), entry.getKey().size()); + if (!context.slotToSubPathsMap.get(outputs.get(i)).contains(parentPaths)) { + continue; + } + if (j == 0) { + // first child, need to put entry to subPathToSlots + slotsForSubPath = subPathSlots.computeIfAbsent(parentPaths, k -> Lists.newArrayList()); + } else { + // other children, should find try from map. otherwise bug comes + if (!subPathSlots.containsKey(parentPaths)) { + throw new AnalysisException("push down variant sub path failed." + + " cannot find sub path for child " + j + "." + + " Sub path set is " + subPathSlots.keySet()); + } + slotsForSubPath = subPathSlots.get(parentPaths); + } + slotsForSubPath.add(entry.getValue()); + } + } + if (regularChildrenOutputs.isEmpty()) { + // use output sub paths exprs to generate subPathSlots + for (List subPath : context.slotToSubPathsMap.get(outputs.get(i))) { + subPathSlots.put(subPath, ImmutableList.of((SlotReference) outputs.get(i).toSlot())); + } + + } + for (Map.Entry, List> entry : subPathSlots.entrySet()) { + for (int j = 0; j < regularChildrenOutputs.size(); j++) { + regularChildrenOutputs.get(j).add(entry.getValue().get(j)); + } + for (int j = 0; j < constExprs.size(); j++) { + NamedExpression constExpr = union.getConstantExprsList().get(j).get(i); + Expression pushDownExpr; + if (constExpr instanceof Alias) { + pushDownExpr = ((Alias) constExpr).child(); + } else { + pushDownExpr = constExpr; + } + for (int sp = entry.getKey().size() - 1; sp >= 0; sp--) { + VarcharLiteral path = new VarcharLiteral(entry.getKey().get(sp)); + pushDownExpr = new ElementAt(pushDownExpr, path); + } + constExprs.get(j).add(new Alias(pushDownExpr)); + + } + SlotReference outputSlot = new SlotReference(StatementScopeIdGenerator.newExprId(), + entry.getValue().get(0).getName(), VariantType.INSTANCE, + true, ImmutableList.of(), + null, + null, + Optional.empty()); + outputs.add(outputSlot); + // update element to slot map + Map, SlotReference> s = oriSlotToSubPathToSlot.computeIfAbsent( + (Slot) outputs.get(i), k -> Maps.newHashMap()); + s.put(entry.getKey(), outputSlot); + } + } + generateElementAtMaps(context, oriSlotToSubPathToSlot); + + return union.withNewOutputsChildrenAndConstExprsList(outputs, union.children(), + regularChildrenOutputs, constExprs); + } + + @Override + public Plan visitLogicalOneRowRelation(LogicalOneRowRelation oneRowRelation, Context context) { + ImmutableList.Builder newProjections + = ImmutableList.builderWithExpectedSize(oneRowRelation.getProjects().size()); + for (NamedExpression projection : oneRowRelation.getProjects()) { + newProjections.add(projection); + newProjections.addAll(pushDownToProject(context, projection)); + } + return oneRowRelation.withProjects(newProjections.build()); + } + + @Override + public Plan visitLogicalProject(LogicalProject project, Context context) { + project = (LogicalProject) this.visit(project, context); + ImmutableList.Builder newProjections + = ImmutableList.builderWithExpectedSize(project.getProjects().size()); + for (NamedExpression projection : project.getProjects()) { + boolean addOthers = projection.getDataType().isVariantType(); + if (projection instanceof SlotReference) { + newProjections.add(projection); + } else { + Expression child = ((Alias) projection).child(); + NamedExpression newProjection; + if (child instanceof SlotReference) { + newProjection = projection; + } else if (child instanceof ElementAt) { + if (context.elementAtToSlotMap.containsKey(child)) { + newProjection = (NamedExpression) projection + .withChildren(context.elementAtToSlotMap.get(child)); + } else { + addOthers = false; + newProjection = projection; + + // try push element_at on this slot + if (extractSlotToSubPathPair((ElementAt) child) == null) { + newProjections.addAll(pushDownToProject(context, projection)); + } + } + } else { + addOthers = false; + newProjection = (NamedExpression) ExpressionUtils.replace( + projection, context.elementAtToSlotMap); + // try push element_at on this slot + newProjections.addAll(pushDownToProject(context, projection)); + } + newProjections.add(newProjection); + } + if (addOthers) { + Expression key = projection.toSlot(); + while (key instanceof Slot && context.slotToOriginalExprMap.containsKey(key)) { + key = context.slotToOriginalExprMap.get(key); + } + if (key instanceof ElementAt && context.elementAtToSlotsMap.containsKey(key)) { + newProjections.addAll(context.elementAtToSlotsMap.get(key).values()); + context.slotToSlotsMap.put(projection.toSlot(), context.elementAtToSlotsMap.get(key)); + } else if (key instanceof Slot && context.slotToSlotsMap.containsKey(key)) { + newProjections.addAll(context.slotToSlotsMap.get(key).values()); + context.slotToSlotsMap.put(projection.toSlot(), context.slotToSlotsMap.get(key)); + } + } + } + return project.withProjects(newProjections.build()); + } + + @Override + public Plan visitLogicalCTEConsumer(LogicalCTEConsumer cteConsumer, Context context) { + if (cteConsumer.getProducerToConsumerOutputMap().keySet().stream() + .map(ExpressionTrait::getDataType).noneMatch(VariantType.class::isInstance)) { + return cteConsumer; + } + Map consumerToProducerOutputMap = Maps.newHashMap(); + Map producerToConsumerOutputMap = Maps.newHashMap(); + Map, SlotReference>> oriSlotToSubPathToSlot = Maps.newHashMap(); + for (Map.Entry consumerToProducer : cteConsumer.getConsumerToProducerOutputMap().entrySet()) { + Slot consumer = consumerToProducer.getKey(); + Slot producer = consumerToProducer.getValue(); + consumerToProducerOutputMap.put(consumer, producer); + producerToConsumerOutputMap.put(producer, consumer); + if (!(consumer.getDataType() instanceof VariantType)) { + continue; + } + + if (context.slotToSlotsMap.containsKey(producer)) { + Map, SlotReference> consumerSlots = Maps.newHashMap(); + for (Map.Entry, SlotReference> producerSlot + : context.slotToSlotsMap.get(producer).entrySet()) { + SlotReference consumerSlot = LogicalCTEConsumer.generateConsumerSlot( + cteConsumer.getName(), producerSlot.getValue()); + consumerToProducerOutputMap.put(consumerSlot, producerSlot.getValue()); + producerToConsumerOutputMap.put(producerSlot.getValue(), consumerSlot); + consumerSlots.put(producerSlot.getKey(), consumerSlot); + } + context.slotToSlotsMap.put(consumer, consumerSlots); + oriSlotToSubPathToSlot.put(consumer, consumerSlots); + } + } + + for (Entry>> elementAtToSubPath + : context.elementAtToSubPathMap.entrySet()) { + ElementAt elementAt = elementAtToSubPath.getKey(); + Pair> slotWithSubPath = elementAtToSubPath.getValue(); + SlotReference key = slotWithSubPath.first; + if (context.elementAtToCteConsumer.containsKey(elementAt)) { + key = context.elementAtToCteConsumer.get(elementAt); + } + // find exactly sub-path slot + if (oriSlotToSubPathToSlot.containsKey(key)) { + context.elementAtToSlotMap.put(elementAtToSubPath.getKey(), + oriSlotToSubPathToSlot.get(key).get(slotWithSubPath.second)); + } + // find prefix sub-path slots + if (oriSlotToSubPathToSlot.containsKey(key)) { + Map, SlotReference> subPathToSlotMap = oriSlotToSubPathToSlot.get(key); + for (Map.Entry, SlotReference> subPathWithSlot : subPathToSlotMap.entrySet()) { + if (subPathWithSlot.getKey().size() > slotWithSubPath.second.size() + && subPathWithSlot.getKey().subList(0, slotWithSubPath.second.size()) + .equals(slotWithSubPath.second)) { + Map, SlotReference> slots = context.elementAtToSlotsMap + .computeIfAbsent(elementAt, e -> Maps.newHashMap()); + slots.put(subPathWithSlot.getKey(), subPathWithSlot.getValue()); + + } + } + } + } + return cteConsumer.withTwoMaps(consumerToProducerOutputMap, producerToConsumerOutputMap); + } + + @Override + public Plan visitLogicalFilter(LogicalFilter filter, Context context) { + filter = (LogicalFilter) this.visit(filter, context); + ImmutableSet.Builder newConjuncts + = ImmutableSet.builderWithExpectedSize(filter.getConjuncts().size()); + for (Expression conjunct : filter.getConjuncts()) { + newConjuncts.add(ExpressionUtils.replace(conjunct, context.elementAtToSlotMap)); + } + return filter.withConjuncts(newConjuncts.build()); + } + + @Override + public Plan visitLogicalJoin(LogicalJoin join, Context context) { + join = (LogicalJoin) this.visit(join, context); + ImmutableList.Builder hashConditions + = ImmutableList.builderWithExpectedSize(join.getHashJoinConjuncts().size()); + ImmutableList.Builder otherConditions + = ImmutableList.builderWithExpectedSize(join.getOtherJoinConjuncts().size()); + ImmutableList.Builder markConditions + = ImmutableList.builderWithExpectedSize(join.getMarkJoinConjuncts().size()); + for (Expression conjunct : join.getHashJoinConjuncts()) { + hashConditions.add(ExpressionUtils.replace(conjunct, context.elementAtToSlotMap)); + } + for (Expression conjunct : join.getOtherJoinConjuncts()) { + otherConditions.add(ExpressionUtils.replace(conjunct, context.elementAtToSlotMap)); + } + for (Expression conjunct : join.getMarkJoinConjuncts()) { + markConditions.add(ExpressionUtils.replace(conjunct, context.elementAtToSlotMap)); + } + return join.withJoinConjuncts(hashConditions.build(), otherConditions.build(), + markConditions.build(), join.getJoinReorderContext()); + } + + @Override + public Plan visitLogicalSort(LogicalSort sort, Context context) { + sort = (LogicalSort) this.visit(sort, context); + ImmutableList.Builder orderKeyBuilder + = ImmutableList.builderWithExpectedSize(sort.getOrderKeys().size()); + for (OrderKey orderKey : sort.getOrderKeys()) { + orderKeyBuilder.add(orderKey.withExpression( + ExpressionUtils.replace(orderKey.getExpr(), context.elementAtToSlotMap))); + } + return sort.withOrderKeys(orderKeyBuilder.build()); + } + + @Override + public Plan visitLogicalTopN(LogicalTopN topN, Context context) { + topN = (LogicalTopN) this.visit(topN, context); + ImmutableList.Builder orderKeyBuilder + = ImmutableList.builderWithExpectedSize(topN.getOrderKeys().size()); + for (OrderKey orderKey : topN.getOrderKeys()) { + orderKeyBuilder.add(orderKey.withExpression( + ExpressionUtils.replace(orderKey.getExpr(), context.elementAtToSlotMap))); + } + return topN.withOrderKeys(orderKeyBuilder.build()); + } + + @Override + public Plan visitLogicalPartitionTopN(LogicalPartitionTopN partitionTopN, Context context) { + partitionTopN = (LogicalPartitionTopN) this.visit(partitionTopN, context); + ImmutableList.Builder orderKeyBuilder + = ImmutableList.builderWithExpectedSize(partitionTopN.getOrderKeys().size()); + for (OrderExpression orderExpression : partitionTopN.getOrderKeys()) { + orderKeyBuilder.add(new OrderExpression(orderExpression.getOrderKey().withExpression( + ExpressionUtils.replace(orderExpression.getOrderKey().getExpr(), context.elementAtToSlotMap)) + )); + } + ImmutableList.Builder partitionKeysBuilder + = ImmutableList.builderWithExpectedSize(partitionTopN.getPartitionKeys().size()); + for (Expression partitionKey : partitionTopN.getPartitionKeys()) { + partitionKeysBuilder.add(ExpressionUtils.replace(partitionKey, context.elementAtToSlotMap)); + } + return partitionTopN.withPartitionKeysAndOrderKeys(partitionKeysBuilder.build(), orderKeyBuilder.build()); + } + + @Override + public Plan visitLogicalGenerate(LogicalGenerate generate, Context context) { + generate = (LogicalGenerate) this.visit(generate, context); + ImmutableList.Builder generatorBuilder + = ImmutableList.builderWithExpectedSize(generate.getGenerators().size()); + for (Function generator : generate.getGenerators()) { + generatorBuilder.add((Function) ExpressionUtils.replace(generator, context.elementAtToSlotMap)); + } + return generate.withGenerators(generatorBuilder.build()); + } + + @Override + public Plan visitLogicalWindow(LogicalWindow window, Context context) { + window = (LogicalWindow) this.visit(window, context); + ImmutableList.Builder windowBuilder + = ImmutableList.builderWithExpectedSize(window.getWindowExpressions().size()); + for (NamedExpression windowFunction : window.getWindowExpressions()) { + windowBuilder.add((NamedExpression) ExpressionUtils.replace( + windowFunction, context.elementAtToSlotMap)); + } + return window.withExpressionsAndChild(windowBuilder.build(), window.child()); + } + + @Override + public Plan visitLogicalAggregate(LogicalAggregate aggregate, Context context) { + aggregate = (LogicalAggregate) this.visit(aggregate, context); + ImmutableList.Builder outputsBuilder + = ImmutableList.builderWithExpectedSize(aggregate.getOutputExpressions().size()); + for (NamedExpression output : aggregate.getOutputExpressions()) { + outputsBuilder.add((NamedExpression) ExpressionUtils.replace( + output, context.elementAtToSlotMap)); + } + ImmutableList.Builder groupByKeysBuilder + = ImmutableList.builderWithExpectedSize(aggregate.getGroupByExpressions().size()); + for (Expression groupByKey : aggregate.getGroupByExpressions()) { + groupByKeysBuilder.add(ExpressionUtils.replace(groupByKey, context.elementAtToSlotMap)); + } + return aggregate.withGroupByAndOutput(groupByKeysBuilder.build(), outputsBuilder.build()); + } + + private List pushDownToProject(Context context, NamedExpression projection) { + if (!projection.getDataType().isVariantType() + || !context.slotToSubPathsMap.containsKey((SlotReference) projection.toSlot())) { + return Collections.emptyList(); + } + List newProjections = Lists.newArrayList(); + Expression child = projection.child(0); + Map, SlotReference> subPathToSlot = Maps.newHashMap(); + Set> subPaths = context.slotToSubPathsMap + .get((SlotReference) projection.toSlot()); + for (List subPath : subPaths) { + Expression pushDownExpr = child; + for (int i = subPath.size() - 1; i >= 0; i--) { + VarcharLiteral path = new VarcharLiteral(subPath.get(i)); + pushDownExpr = new ElementAt(pushDownExpr, path); + } + Alias alias = new Alias(pushDownExpr); + newProjections.add(alias); + subPathToSlot.put(subPath, (SlotReference) alias.toSlot()); + } + Map, SlotReference>> oriSlotToSubPathToSlot = Maps.newHashMap(); + oriSlotToSubPathToSlot.put(projection.toSlot(), subPathToSlot); + generateElementAtMaps(context, oriSlotToSubPathToSlot); + return newProjections; + } + + private void generateElementAtMaps(Context context, Map, SlotReference>> oriSlotToSubPathToSlot) { + context.slotToSlotsMap.putAll(oriSlotToSubPathToSlot); + for (Entry>> elementAtToSubPath + : context.elementAtToSubPathMap.entrySet()) { + ElementAt elementAt = elementAtToSubPath.getKey(); + Pair> slotWithSubPath = elementAtToSubPath.getValue(); + // find exactly sub-path slot + if (oriSlotToSubPathToSlot.containsKey(slotWithSubPath.first)) { + context.elementAtToSlotMap.put(elementAtToSubPath.getKey(), oriSlotToSubPathToSlot.get( + slotWithSubPath.first).get(slotWithSubPath.second)); + } + // find prefix sub-path slots + if (oriSlotToSubPathToSlot.containsKey(slotWithSubPath.first)) { + Map, SlotReference> subPathToSlotMap = oriSlotToSubPathToSlot.get( + slotWithSubPath.first); + for (Map.Entry, SlotReference> subPathWithSlot : subPathToSlotMap.entrySet()) { + if (subPathWithSlot.getKey().size() > slotWithSubPath.second.size() + && subPathWithSlot.getKey().subList(0, slotWithSubPath.second.size()) + .equals(slotWithSubPath.second)) { + Map, SlotReference> slots = context.elementAtToSlotsMap + .computeIfAbsent(elementAt, e -> Maps.newHashMap()); + slots.put(subPathWithSlot.getKey(), subPathWithSlot.getValue()); + } + } + } + } + } + } + + private static class VariantSubPathCollector extends PlanVisitor { + + public static VariantSubPathCollector INSTANCE = new VariantSubPathCollector(); + + /** + * Extract sequential element_at from expression tree. + * if extract success, put it into context map and stop traverse + * other-wise, traverse its children + */ + private static class ExtractSlotToSubPathPairFromTree + extends DefaultExpressionVisitor>>> { + + public static ExtractSlotToSubPathPairFromTree INSTANCE = new ExtractSlotToSubPathPairFromTree(); + + @Override + public Void visitElementAt(ElementAt elementAt, Map>> context) { + Pair> pair = extractSlotToSubPathPair(elementAt); + if (pair == null) { + visit(elementAt, context); + } else { + context.put(elementAt, pair); + } + return null; + } + } + + @Override + public Void visitLogicalCTEAnchor(LogicalCTEAnchor cteAnchor, + Context context) { + cteAnchor.right().accept(this, context); + return cteAnchor.left().accept(this, context); + } + + @Override + public Void visitLogicalCTEConsumer(LogicalCTEConsumer cteConsumer, Context context) { + for (Map.Entry consumerToProducer : cteConsumer.getConsumerToProducerOutputMap().entrySet()) { + Slot consumer = consumerToProducer.getKey(); + if (!(consumer.getDataType() instanceof VariantType)) { + continue; + } + Slot producer = consumerToProducer.getValue(); + if (context.slotToSubPathsMap.containsKey((SlotReference) consumer)) { + Set> subPaths = context.slotToSubPathsMap + .computeIfAbsent((SlotReference) producer, k -> Sets.newHashSet()); + subPaths.addAll(context.slotToSubPathsMap.get(consumer)); + } + for (Entry>> elementAtToSubPath + : context.elementAtToSubPathMap.entrySet()) { + ElementAt elementAt = elementAtToSubPath.getKey(); + Pair> slotWithSubPath = elementAtToSubPath.getValue(); + if (slotWithSubPath.first.equals(consumer)) { + context.elementAtToCteConsumer.putIfAbsent(elementAt, (SlotReference) consumer); + context.elementAtToSubPathMap.put(elementAt, + Pair.of((SlotReference) producer, slotWithSubPath.second)); + } + } + } + return null; + } + + @Override + public Void visitLogicalUnion(LogicalUnion union, Context context) { + if (union.getQualifier() == Qualifier.DISTINCT) { + return super.visitLogicalUnion(union, context); + } + for (List childOutputs : union.getRegularChildrenOutputs()) { + for (int i = 0; i < union.getOutputs().size(); i++) { + Slot unionOutput = union.getOutput().get(i); + SlotReference childOutput = childOutputs.get(i); + if (context.slotToSubPathsMap.containsKey((SlotReference) unionOutput)) { + Set> subPaths = context.slotToSubPathsMap + .computeIfAbsent(childOutput, k -> Sets.newHashSet()); + subPaths.addAll(context.slotToSubPathsMap.get(unionOutput)); + } + } + } + + this.visit(union, context); + return null; + } + + @Override + public Void visitLogicalProject(LogicalProject project, Context context) { + for (NamedExpression projection : project.getProjects()) { + if (!(projection instanceof Alias)) { + continue; + } + Alias alias = (Alias) projection; + Expression child = alias.child(); + if (child instanceof SlotReference && child.getDataType() instanceof VariantType) { + context.putSlotToOriginal(alias.toSlot(), child); + } + // process expression like v['a']['b']['c'] just in root + // The reason for handling this situation separately is that + // it will have an impact on the upper level. So, we need to record the mapping of slots to it. + if (child instanceof ElementAt) { + Pair> pair = extractSlotToSubPathPair((ElementAt) child); + if (pair != null) { + context.putElementAtToSubPath((ElementAt) child, pair, alias.toSlot()); + context.putSlotToOriginal(alias.toSlot(), child); + continue; + } + } + // process other situation of expression like v['a']['b']['c'] + Map>> elementAtToSubPathMap = Maps.newHashMap(); + child.accept(ExtractSlotToSubPathPairFromTree.INSTANCE, elementAtToSubPathMap); + context.putAllElementAtToSubPath(elementAtToSubPathMap); + } + this.visit(project, context); + return null; + } + + @Override + public Void visit(Plan plan, Context context) { + Map>> elementAtToSubPathMap = Maps.newHashMap(); + for (Expression expression : plan.getExpressions()) { + expression.accept(ExtractSlotToSubPathPairFromTree.INSTANCE, elementAtToSubPathMap); + } + context.putAllElementAtToSubPath(elementAtToSubPathMap); + for (Plan child : plan.children()) { + child.accept(this, context); + } + return null; + } + } + + protected static Pair> extractSlotToSubPathPair(ElementAt elementAt) { + List subPath = Lists.newArrayList(); + while (true) { + if (!(elementAt.left().getDataType() instanceof VariantType)) { + return null; + } + if (!(elementAt.left() instanceof ElementAt || elementAt.left() instanceof SlotReference)) { + return null; + } + if (!(elementAt.right() instanceof StringLikeLiteral)) { + return null; + } + subPath.add(((StringLikeLiteral) elementAt.right()).getStringValue()); + if (elementAt.left() instanceof SlotReference) { + // ElementAt's left child is SlotReference + // reverse subPath because we put them by reverse order + Collections.reverse(subPath); + return Pair.of((SlotReference) elementAt.left(), subPath); + } else { + // ElementAt's left child is ElementAt + elementAt = (ElementAt) elementAt.left(); + } + } + } +} diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/Alias.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/Alias.java index a15cec47228b27..af9347a169b832 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/Alias.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/Alias.java @@ -94,8 +94,8 @@ public Slot toSlot() throws UnboundException { : null, internalName, slotReference != null - ? slotReference.getSubColPath() - : null + ? slotReference.getSubPath() + : ImmutableList.of() ); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/ArrayItemReference.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/ArrayItemReference.java index b43e4811192319..62137a4d30c047 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/ArrayItemReference.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/ArrayItemReference.java @@ -142,7 +142,8 @@ public static class ArrayItemSlot extends SlotReference implements SlotNotFromCh * @param nullable true if nullable */ public ArrayItemSlot(ExprId exprId, String name, DataType dataType, boolean nullable) { - super(exprId, name, dataType, nullable, ImmutableList.of(), null, null, Optional.empty(), null); + super(exprId, name, dataType, nullable, ImmutableList.of(), + null, null, Optional.empty(), ImmutableList.of()); } @Override diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/SlotReference.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/SlotReference.java index 3d11a7d011da15..b5fb57abf3c704 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/SlotReference.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/SlotReference.java @@ -21,10 +21,8 @@ import org.apache.doris.catalog.TableIf; import org.apache.doris.nereids.exceptions.UnboundException; import org.apache.doris.nereids.trees.expressions.visitor.ExpressionVisitor; -import org.apache.doris.nereids.trees.plans.algebra.Relation; import org.apache.doris.nereids.types.DataType; import org.apache.doris.nereids.util.Utils; -import org.apache.doris.qe.ConnectContext; import com.google.common.base.Preconditions; import com.google.common.base.Suppliers; @@ -48,7 +46,7 @@ public class SlotReference extends Slot { // The sub column path to access type like struct or variant // e.g. For accessing variant["a"]["b"], the parsed paths is ["a", "b"] - protected final List subColPath; + protected final List subPath; // the unique string representation of a SlotReference // different SlotReference will have different internalName @@ -60,31 +58,31 @@ public class SlotReference extends Slot { public SlotReference(String name, DataType dataType) { this(StatementScopeIdGenerator.newExprId(), name, dataType, true, ImmutableList.of(), - null, null, Optional.empty(), null); + null, null, Optional.empty(), ImmutableList.of()); } public SlotReference(String name, DataType dataType, boolean nullable) { this(StatementScopeIdGenerator.newExprId(), name, dataType, nullable, ImmutableList.of(), - null, null, Optional.empty(), null); + null, null, Optional.empty(), ImmutableList.of()); } public SlotReference(String name, DataType dataType, boolean nullable, List qualifier) { this(StatementScopeIdGenerator.newExprId(), name, dataType, nullable, - qualifier, null, null, Optional.empty(), null); + qualifier, null, null, Optional.empty(), ImmutableList.of()); } public SlotReference(ExprId exprId, String name, DataType dataType, boolean nullable, List qualifier) { - this(exprId, name, dataType, nullable, qualifier, null, null, Optional.empty(), null); + this(exprId, name, dataType, nullable, qualifier, null, null, Optional.empty(), ImmutableList.of()); } public SlotReference(ExprId exprId, String name, DataType dataType, boolean nullable, List qualifier, @Nullable TableIf table, @Nullable Column column) { - this(exprId, name, dataType, nullable, qualifier, table, column, Optional.empty(), null); + this(exprId, name, dataType, nullable, qualifier, table, column, Optional.empty(), ImmutableList.of()); } public SlotReference(ExprId exprId, String name, DataType dataType, boolean nullable, List qualifier, @Nullable TableIf table, @Nullable Column column, Optional internalName) { - this(exprId, name, dataType, nullable, qualifier, table, column, internalName, null); + this(exprId, name, dataType, nullable, qualifier, table, column, internalName, ImmutableList.of()); } public SlotReference(ExprId exprId, String name, DataType dataType, boolean nullable, @@ -104,11 +102,11 @@ public SlotReference(ExprId exprId, String name, DataType dataType, boolean null * @param qualifier slot reference qualifier * @param column the column which this slot come from * @param internalName the internalName of this slot - * @param subColLabels subColumn access labels + * @param subPath subColumn access labels */ public SlotReference(ExprId exprId, Supplier name, DataType dataType, boolean nullable, List qualifier, @Nullable TableIf table, @Nullable Column column, - Supplier> internalName, List subColLabels) { + Supplier> internalName, List subPath) { this.exprId = exprId; this.name = name; this.dataType = dataType; @@ -117,7 +115,7 @@ public SlotReference(ExprId exprId, Supplier name, DataType dataType, bo this.nullable = nullable; this.table = table; this.column = column; - this.subColPath = subColLabels; + this.subPath = Objects.requireNonNull(subPath, "subPath can not be null"); this.internalName = internalName; } @@ -129,23 +127,18 @@ public static SlotReference of(String name, DataType type) { * get SlotReference from a column * @param column the column which contains type info * @param qualifier the qualifier of SlotReference - * @param relation the relation which column is from */ - public static SlotReference fromColumn(TableIf table, Column column, List qualifier, Relation relation) { + public static SlotReference fromColumn(TableIf table, Column column, List qualifier) { DataType dataType = DataType.fromCatalogType(column.getType()); - SlotReference slot = new SlotReference(StatementScopeIdGenerator.newExprId(), () -> column.getName(), dataType, - column.isAllowNull(), qualifier, table, column, () -> Optional.of(column.getName()), null); - if (relation != null && ConnectContext.get() != null - && ConnectContext.get().getStatementContext() != null) { - ConnectContext.get().getStatementContext().addSlotToRelation(slot, relation); - } - return slot; + return new SlotReference(StatementScopeIdGenerator.newExprId(), column::getName, dataType, + column.isAllowNull(), qualifier, table, column, + () -> Optional.of(column.getName()), ImmutableList.of()); } public static SlotReference fromColumn(TableIf table, Column column, String name, List qualifier) { DataType dataType = DataType.fromCatalogType(column.getType()); return new SlotReference(StatementScopeIdGenerator.newExprId(), name, dataType, - column.isAllowNull(), qualifier, table, column, Optional.empty(), null); + column.isAllowNull(), qualifier, table, column, Optional.empty(), ImmutableList.of()); } @Override @@ -188,13 +181,20 @@ public Optional getTable() { @Override public String toSql() { - return name.get(); + if (subPath.isEmpty()) { + return name.get(); + } else { + return name.get() + "['" + String.join("']['", subPath) + "']"; + } } @Override public String toString() { - // Just return name and exprId, add another method to show fully qualified name when it's necessary. - return name.get() + "#" + exprId; + if (subPath.isEmpty()) { + // Just return name and exprId, add another method to show fully qualified name when it's necessary. + return name.get() + "#" + exprId; + } + return name.get() + "['" + String.join("']['", subPath) + "']" + "#" + exprId; } @Override @@ -251,12 +251,12 @@ public SlotReference withNullable(boolean newNullable) { return this; } return new SlotReference(exprId, name, dataType, newNullable, - qualifier, table, column, internalName, subColPath); + qualifier, table, column, internalName, subPath); } @Override public SlotReference withQualifier(List qualifier) { - return new SlotReference(exprId, name, dataType, nullable, qualifier, table, column, internalName, subColPath); + return new SlotReference(exprId, name, dataType, nullable, qualifier, table, column, internalName, subPath); } @Override @@ -265,24 +265,29 @@ public SlotReference withName(String name) { return this; } return new SlotReference( - exprId, () -> name, dataType, nullable, qualifier, table, column, internalName, subColPath); + exprId, () -> name, dataType, nullable, qualifier, table, column, internalName, subPath); } @Override public SlotReference withExprId(ExprId exprId) { - return new SlotReference(exprId, name, dataType, nullable, qualifier, table, column, internalName, subColPath); + return new SlotReference(exprId, name, dataType, nullable, qualifier, table, column, internalName, subPath); + } + + public SlotReference withSubPath(List subPath) { + return new SlotReference(exprId, name, dataType, !subPath.isEmpty() || nullable, + qualifier, table, column, internalName, subPath); } public boolean isVisible() { return column == null || column.isVisible(); } - public List getSubColPath() { - return subColPath; + public List getSubPath() { + return subPath; } public boolean hasSubColPath() { - return subColPath != null && !subColPath.isEmpty(); + return !subPath.isEmpty(); } private static Supplier> buildInternalName( diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/scalar/ElementAt.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/scalar/ElementAt.java index d4fe6d50438673..c8d54189d379b4 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/scalar/ElementAt.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/scalar/ElementAt.java @@ -39,7 +39,7 @@ /** * ScalarFunction 'element_at'. This class is generated by GenerateFunction. */ -public class ElementAt extends PushDownToProjectionFunction +public class ElementAt extends ScalarFunction implements BinaryExpression, ExplicitlyCastableSignature, AlwaysNullable { public static final List SIGNATURES = ImmutableList.of( diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/scalar/PushDownToProjectionFunction.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/scalar/PushDownToProjectionFunction.java deleted file mode 100644 index 8662dffcadc8a9..00000000000000 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/scalar/PushDownToProjectionFunction.java +++ /dev/null @@ -1,88 +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.trees.expressions.functions.scalar; - -import org.apache.doris.nereids.StatementContext; -import org.apache.doris.nereids.trees.expressions.Expression; -import org.apache.doris.nereids.trees.expressions.SlotReference; -import org.apache.doris.nereids.trees.expressions.StatementScopeIdGenerator; -import org.apache.doris.nereids.trees.expressions.literal.VarcharLiteral; -import org.apache.doris.qe.ConnectContext; - -import java.util.List; -import java.util.Optional; -import java.util.stream.Collectors; - -/** - * Function that could be rewritten and pushed down to projection - */ -public abstract class PushDownToProjectionFunction extends ScalarFunction { - public PushDownToProjectionFunction(String name, Expression... arguments) { - super(name, arguments); - } - - /** - * check if specified function could be pushed down to project - * @param pushDownExpr expr to check - * @return if it is valid to push down input expr - */ - public static boolean validToPushDown(Expression pushDownExpr) { - // Currently only element at for variant type could be pushed down - return pushDownExpr != null && pushDownExpr.anyMatch(expr -> - expr instanceof PushDownToProjectionFunction && ((Expression) expr).getDataType().isVariantType() - ); - } - - /** - * Rewrites an {@link PushDownToProjectionFunction} instance to a {@link SlotReference}. - * This method is used to transform an PushDownToProjectionFunction expr into a SlotReference, - * based on the provided topColumnSlot and the context of the statement. - * - * @param pushedFunction The {@link PushDownToProjectionFunction} instance to be rewritten. - * @param topColumnSlot The {@link SlotReference} that represents the top column slot. - * @return A {@link SlotReference} that represents the rewritten element. - * If a target column slot is found in the context, it is returned to avoid duplicates. - * Otherwise, a new SlotReference is created and added to the context. - */ - public static Expression rewriteToSlot(PushDownToProjectionFunction pushedFunction, SlotReference topColumnSlot) { - // push down could not work well with variant that not belong to table, so skip it. - if (!topColumnSlot.getColumn().isPresent() || !topColumnSlot.getTable().isPresent()) { - return pushedFunction; - } - // rewrite to slotRef - StatementContext ctx = ConnectContext.get().getStatementContext(); - List fullPaths = pushedFunction.collectToList(node -> node instanceof VarcharLiteral).stream() - .map(node -> ((VarcharLiteral) node).getValue()) - .collect(Collectors.toList()); - SlotReference targetColumnSlot = ctx.getPathSlot(topColumnSlot, fullPaths); - if (targetColumnSlot != null) { - // avoid duplicated slots - return targetColumnSlot; - } - boolean nullable = true; // always nullable at present - SlotReference slotRef = new SlotReference(StatementScopeIdGenerator.newExprId(), - topColumnSlot.getName(), topColumnSlot.getDataType(), - nullable, topColumnSlot.getQualifier(), topColumnSlot.getTable().get(), - topColumnSlot.getColumn().get(), Optional.of(topColumnSlot.getInternalName()), - fullPaths); - ctx.addPathSlotRef(topColumnSlot, fullPaths, slotRef, pushedFunction); - ctx.addSlotToRelation(slotRef, ctx.getRelationBySlot(topColumnSlot)); - - return slotRef; - } -} diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/algebra/Project.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/algebra/Project.java index 15b095bee3bd28..73d4cb36448eb4 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/algebra/Project.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/algebra/Project.java @@ -18,16 +18,12 @@ package org.apache.doris.nereids.trees.plans.algebra; import org.apache.doris.nereids.exceptions.AnalysisException; -import org.apache.doris.nereids.trees.expressions.Alias; 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.expressions.functions.scalar.PushDownToProjectionFunction; import org.apache.doris.nereids.util.ExpressionUtils; import org.apache.doris.nereids.util.PlanUtils; -import org.apache.doris.qe.ConnectContext; import com.google.common.collect.ImmutableMap; @@ -68,32 +64,6 @@ default List mergeProjections(Project childProject) { return PlanUtils.mergeProjections(childProject.getProjects(), getProjects()); } - /** - * Check if it is a project that is pull up from scan in analyze rule - * e.g. BindSlotWithPaths - * And check if contains PushDownToProjectionFunction that can pushed down to project - */ - default boolean hasPushedDownToProjectionFunctions() { - if ((ConnectContext.get() == null - || ConnectContext.get().getSessionVariable() == null - || !ConnectContext.get().getSessionVariable().isEnableRewriteElementAtToSlot())) { - return false; - } - - boolean hasValidAlias = false; - for (NamedExpression namedExpr : getProjects()) { - if (namedExpr instanceof Alias) { - if (!PushDownToProjectionFunction.validToPushDown(((Alias) namedExpr).child())) { - return false; - } - hasValidAlias = true; - } else if (!(namedExpr instanceof SlotReference)) { - return false; - } - } - return hasValidAlias; - } - /** * find projects, if not found the slot, then throw AnalysisException */ 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 aed808d811d50b..cd73c96d02e192 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 @@ -93,17 +93,24 @@ public LogicalCTEConsumer(RelationId relationId, CTEId cteId, String name, "producerToConsumerOutputMap should not null"); } + /** + * generate a consumer slot mapping from producer slot. + */ + public static SlotReference generateConsumerSlot(String cteName, Slot producerOutputSlot) { + SlotReference slotRef = + producerOutputSlot instanceof SlotReference ? (SlotReference) producerOutputSlot : null; + return new SlotReference(StatementScopeIdGenerator.newExprId(), + producerOutputSlot.getName(), producerOutputSlot.getDataType(), + producerOutputSlot.nullable(), ImmutableList.of(cteName), + slotRef != null ? (slotRef.getTable().isPresent() ? slotRef.getTable().get() : null) : null, + slotRef != null ? (slotRef.getColumn().isPresent() ? slotRef.getColumn().get() : null) : null, + slotRef != null ? Optional.of(slotRef.getInternalName()) : Optional.empty()); + } + private void initOutputMaps(LogicalPlan childPlan) { List producerOutput = childPlan.getOutput(); for (Slot producerOutputSlot : producerOutput) { - SlotReference slotRef = - producerOutputSlot instanceof SlotReference ? (SlotReference) producerOutputSlot : null; - Slot consumerSlot = new SlotReference(StatementScopeIdGenerator.newExprId(), - producerOutputSlot.getName(), producerOutputSlot.getDataType(), - producerOutputSlot.nullable(), ImmutableList.of(name), - slotRef != null ? (slotRef.getTable().isPresent() ? slotRef.getTable().get() : null) : null, - slotRef != null ? (slotRef.getColumn().isPresent() ? slotRef.getColumn().get() : null) : null, - slotRef != null ? Optional.of(slotRef.getInternalName()) : Optional.empty()); + Slot consumerSlot = generateConsumerSlot(this.name, producerOutputSlot); producerToConsumerOutputMap.put(producerOutputSlot, consumerSlot); consumerToProducerOutputMap.put(consumerSlot, producerOutputSlot); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalCatalogRelation.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalCatalogRelation.java index 4d8fba447b329e..30d861dd87023b 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalCatalogRelation.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalCatalogRelation.java @@ -104,7 +104,7 @@ public DatabaseIf getDatabase() throws AnalysisException { public List computeOutput() { return table.getBaseSchema() .stream() - .map(col -> SlotReference.fromColumn(table, col, qualified(), this)) + .map(col -> SlotReference.fromColumn(table, col, qualified())) .collect(ImmutableList.toImmutableList()); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalOlapScan.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalOlapScan.java index 714f540524f1a8..76d360ed531226 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalOlapScan.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalOlapScan.java @@ -38,9 +38,9 @@ import com.google.common.base.Preconditions; import com.google.common.collect.ImmutableList; import com.google.common.collect.ImmutableList.Builder; +import com.google.common.collect.ImmutableMap; import com.google.common.collect.Lists; import com.google.common.collect.Maps; -import com.google.common.collect.Sets; import org.apache.commons.lang3.tuple.Pair; import org.json.JSONObject; @@ -70,11 +70,6 @@ public class LogicalOlapScan extends LogicalCatalogRelation implements OlapScan */ private final boolean indexSelected; - /* - * Status to indicate a new project pulled up from this logicalOlapScan - */ - private final boolean projectPulledUp; - /** * Status to indicate using pre-aggregation or not. */ @@ -118,6 +113,9 @@ public class LogicalOlapScan extends LogicalCatalogRelation implements OlapScan private final boolean directMvScan; + private final Map>> colToSubPathsMap; + private final Map, SlotReference>> subPathToSlotMap; + public LogicalOlapScan(RelationId id, OlapTable table) { this(id, table, ImmutableList.of()); } @@ -127,7 +125,7 @@ public LogicalOlapScan(RelationId id, OlapTable table, List qualifier) { table.getPartitionIds(), false, ImmutableList.of(), -1, false, PreAggStatus.unset(), ImmutableList.of(), ImmutableList.of(), - Maps.newHashMap(), Optional.empty(), false, false); + Maps.newHashMap(), Optional.empty(), false, ImmutableMap.of()); } public LogicalOlapScan(RelationId id, OlapTable table, List qualifier, List tabletIds, @@ -135,7 +133,7 @@ public LogicalOlapScan(RelationId id, OlapTable table, List qualifier, L this(id, table, qualifier, Optional.empty(), Optional.empty(), table.getPartitionIds(), false, tabletIds, -1, false, PreAggStatus.unset(), ImmutableList.of(), hints, Maps.newHashMap(), - tableSample, false, false); + tableSample, false, ImmutableMap.of()); } public LogicalOlapScan(RelationId id, OlapTable table, List qualifier, List specifiedPartitions, @@ -144,7 +142,7 @@ public LogicalOlapScan(RelationId id, OlapTable table, List qualifier, L // must use specifiedPartitions here for prune partition by sql like 'select * from t partition p1' specifiedPartitions, false, tabletIds, -1, false, PreAggStatus.unset(), specifiedPartitions, hints, Maps.newHashMap(), - tableSample, false, false); + tableSample, false, ImmutableMap.of()); } public LogicalOlapScan(RelationId id, OlapTable table, List qualifier, List tabletIds, @@ -153,7 +151,7 @@ public LogicalOlapScan(RelationId id, OlapTable table, List qualifier, L this(id, table, qualifier, Optional.empty(), Optional.empty(), table.getPartitionIds(), false, tabletIds, selectedIndexId, true, preAggStatus, - ImmutableList.of(), hints, Maps.newHashMap(), tableSample, true, false); + ImmutableList.of(), hints, Maps.newHashMap(), tableSample, true, ImmutableMap.of()); } /** @@ -165,7 +163,8 @@ public LogicalOlapScan(RelationId id, Table table, List qualifier, List selectedTabletIds, long selectedIndexId, boolean indexSelected, PreAggStatus preAggStatus, List specifiedPartitions, List hints, Map, Slot> cacheSlotWithSlotName, - Optional tableSample, boolean directMvScan, boolean projectPulledUp) { + Optional tableSample, boolean directMvScan, + Map>> colToSubPathsMap) { super(id, PlanType.LOGICAL_OLAP_SCAN, table, qualifier, groupExpression, logicalProperties); Preconditions.checkArgument(selectedPartitionIds != null, @@ -177,28 +176,25 @@ public LogicalOlapScan(RelationId id, Table table, List qualifier, this.preAggStatus = preAggStatus; this.manuallySpecifiedPartitions = ImmutableList.copyOf(specifiedPartitions); - switch (selectedPartitionIds.size()) { - case 0: { - this.selectedPartitionIds = ImmutableList.of(); - break; - } - default: { - ImmutableList.Builder existPartitions - = ImmutableList.builderWithExpectedSize(selectedPartitionIds.size()); - for (Long partitionId : selectedPartitionIds) { - if (((OlapTable) table).getPartition(partitionId) != null) { - existPartitions.add(partitionId); - } + if (selectedPartitionIds.isEmpty()) { + this.selectedPartitionIds = ImmutableList.of(); + } else { + Builder existPartitions + = ImmutableList.builderWithExpectedSize(selectedPartitionIds.size()); + for (Long partitionId : selectedPartitionIds) { + if (((OlapTable) table).getPartition(partitionId) != null) { + existPartitions.add(partitionId); } - this.selectedPartitionIds = existPartitions.build(); } + this.selectedPartitionIds = existPartitions.build(); } this.hints = Objects.requireNonNull(hints, "hints can not be null"); this.cacheSlotWithSlotName = Objects.requireNonNull(cacheSlotWithSlotName, "mvNameToSlot can not be null"); this.tableSample = tableSample; this.directMvScan = directMvScan; - this.projectPulledUp = projectPulledUp; + this.colToSubPathsMap = colToSubPathsMap; + this.subPathToSlotMap = Maps.newHashMap(); } public List getSelectedPartitionIds() { @@ -239,7 +235,7 @@ public boolean equals(Object o) { && Objects.equals(manuallySpecifiedPartitions, that.manuallySpecifiedPartitions) && Objects.equals(selectedPartitionIds, that.selectedPartitionIds) && Objects.equals(hints, that.hints) - && Objects.equals(tableSample, tableSample); + && Objects.equals(tableSample, that.tableSample); } @Override @@ -255,7 +251,7 @@ public LogicalOlapScan withGroupExpression(Optional groupExpres groupExpression, Optional.of(getLogicalProperties()), selectedPartitionIds, partitionPruned, selectedTabletIds, selectedIndexId, indexSelected, preAggStatus, manuallySpecifiedPartitions, - hints, cacheSlotWithSlotName, tableSample, directMvScan, projectPulledUp); + hints, cacheSlotWithSlotName, tableSample, directMvScan, colToSubPathsMap); } @Override @@ -264,7 +260,7 @@ public Plan withGroupExprLogicalPropChildren(Optional groupExpr return new LogicalOlapScan(relationId, (Table) table, qualifier, groupExpression, logicalProperties, selectedPartitionIds, partitionPruned, selectedTabletIds, selectedIndexId, indexSelected, preAggStatus, manuallySpecifiedPartitions, - hints, cacheSlotWithSlotName, tableSample, directMvScan, projectPulledUp); + hints, cacheSlotWithSlotName, tableSample, directMvScan, colToSubPathsMap); } public LogicalOlapScan withSelectedPartitionIds(List selectedPartitionIds) { @@ -272,7 +268,7 @@ public LogicalOlapScan withSelectedPartitionIds(List selectedPartitionIds) Optional.empty(), Optional.of(getLogicalProperties()), selectedPartitionIds, true, selectedTabletIds, selectedIndexId, indexSelected, preAggStatus, manuallySpecifiedPartitions, - hints, cacheSlotWithSlotName, tableSample, directMvScan, projectPulledUp); + hints, cacheSlotWithSlotName, tableSample, directMvScan, colToSubPathsMap); } public LogicalOlapScan withMaterializedIndexSelected(long indexId) { @@ -280,35 +276,31 @@ public LogicalOlapScan withMaterializedIndexSelected(long indexId) { Optional.empty(), Optional.of(getLogicalProperties()), selectedPartitionIds, partitionPruned, selectedTabletIds, indexId, true, PreAggStatus.unset(), manuallySpecifiedPartitions, hints, cacheSlotWithSlotName, - tableSample, directMvScan, projectPulledUp); - } - - public boolean isProjectPulledUp() { - return projectPulledUp; + tableSample, directMvScan, colToSubPathsMap); } - public LogicalOlapScan withProjectPulledUp() { + public LogicalOlapScan withSelectedTabletIds(List selectedTabletIds) { return new LogicalOlapScan(relationId, (Table) table, qualifier, Optional.empty(), Optional.of(getLogicalProperties()), selectedPartitionIds, partitionPruned, selectedTabletIds, - selectedIndexId, indexSelected, preAggStatus, manuallySpecifiedPartitions, hints, cacheSlotWithSlotName, - tableSample, directMvScan, true); + selectedIndexId, indexSelected, preAggStatus, manuallySpecifiedPartitions, + hints, cacheSlotWithSlotName, tableSample, directMvScan, colToSubPathsMap); } - public LogicalOlapScan withSelectedTabletIds(List selectedTabletIds) { + public LogicalOlapScan withPreAggStatus(PreAggStatus preAggStatus) { return new LogicalOlapScan(relationId, (Table) table, qualifier, Optional.empty(), Optional.of(getLogicalProperties()), selectedPartitionIds, partitionPruned, selectedTabletIds, selectedIndexId, indexSelected, preAggStatus, manuallySpecifiedPartitions, - hints, cacheSlotWithSlotName, tableSample, directMvScan, projectPulledUp); + hints, cacheSlotWithSlotName, tableSample, directMvScan, colToSubPathsMap); } - public LogicalOlapScan withPreAggStatus(PreAggStatus preAggStatus) { + public LogicalOlapScan withColToSubPathsMap(Map>> colToSubPathsMap) { return new LogicalOlapScan(relationId, (Table) table, qualifier, - Optional.empty(), Optional.of(getLogicalProperties()), + Optional.empty(), Optional.empty(), selectedPartitionIds, partitionPruned, selectedTabletIds, selectedIndexId, indexSelected, preAggStatus, manuallySpecifiedPartitions, - hints, cacheSlotWithSlotName, tableSample, directMvScan, projectPulledUp); + hints, cacheSlotWithSlotName, tableSample, directMvScan, colToSubPathsMap); } @Override @@ -318,7 +310,7 @@ public LogicalOlapScan withRelationId(RelationId relationId) { Optional.empty(), Optional.empty(), selectedPartitionIds, false, selectedTabletIds, selectedIndexId, indexSelected, preAggStatus, manuallySpecifiedPartitions, - hints, Maps.newHashMap(), tableSample, directMvScan, projectPulledUp); + hints, Maps.newHashMap(), tableSample, directMvScan, colToSubPathsMap); } @Override @@ -347,6 +339,11 @@ public PreAggStatus getPreAggStatus() { return preAggStatus; } + public Map, SlotReference>> getSubPathToSlotMap() { + this.getOutput(); + return subPathToSlotMap; + } + @VisibleForTesting public Optional getSelectedMaterializedIndexName() { return indexSelected ? Optional.ofNullable(((OlapTable) table).getIndexNameById(selectedIndexId)) @@ -363,27 +360,27 @@ public List computeOutput() { Builder slots = ImmutableList.builder(); for (int i = 0; i < baseSchema.size(); i++) { + final int index = i; Column col = baseSchema.get(i); Pair key = Pair.of(selectedIndexId, col.getName()); - Slot slot = cacheSlotWithSlotName.get(key); - if (slot != null) { - slots.add(slot); - } else { - slot = slotFromColumn.get(i); - cacheSlotWithSlotName.put(key, slot); - slots.add(slot); + Slot slot = cacheSlotWithSlotName.computeIfAbsent(key, k -> slotFromColumn.get(index)); + slots.add(slot); + if (colToSubPathsMap.containsKey(key.getValue())) { + for (List subPath : colToSubPathsMap.get(key.getValue())) { + if (!subPath.isEmpty()) { + SlotReference slotReference = SlotReference.fromColumn( + table, baseSchema.get(i), qualified()).withSubPath(subPath); + slots.add(slotReference); + subPathToSlotMap.computeIfAbsent(slot, k -> Maps.newHashMap()) + .put(subPath, slotReference); + } + + } } } return slots.build(); } - @Override - public Set getInputRelations() { - Set relationIdSet = Sets.newHashSet(); - relationIdSet.add(relationId); - return relationIdSet; - } - /** * Get the slot under the index, * and create a new slotReference for the slot that has not appeared in the materialized view. @@ -395,25 +392,33 @@ public List getOutputByIndex(long indexId) { List schema = olapTable.getIndexMetaByIndexId(indexId).getSchema(); List slots = Lists.newArrayListWithCapacity(schema.size()); for (Column c : schema) { - Slot slot = generateUniqueSlot( - olapTable, c, indexId == ((OlapTable) table).getBaseIndexId(), indexId); - slots.add(slot); + slots.addAll(generateUniqueSlot( + olapTable, c, indexId == ((OlapTable) table).getBaseIndexId(), indexId)); } return slots; } - private Slot generateUniqueSlot(OlapTable table, Column column, boolean isBaseIndex, long indexId) { + private List generateUniqueSlot(OlapTable table, Column column, boolean isBaseIndex, long indexId) { String name = isBaseIndex || directMvScan ? column.getName() : AbstractSelectMaterializedIndexRule.parseMvColumnToMvName(column.getName(), column.isAggregated() ? Optional.of(column.getAggregationType().toSql()) : Optional.empty()); Pair key = Pair.of(indexId, name); - Slot slot = cacheSlotWithSlotName.get(key); - if (slot != null) { - return slot; + Slot slot = cacheSlotWithSlotName.computeIfAbsent(key, k -> + SlotReference.fromColumn(table, column, name, qualified())); + List slots = Lists.newArrayList(slot); + if (colToSubPathsMap.containsKey(key.getValue())) { + for (List subPath : colToSubPathsMap.get(key.getValue())) { + if (!subPath.isEmpty()) { + SlotReference slotReference + = SlotReference.fromColumn(table, column, name, qualified()).withSubPath(subPath); + slots.add(slotReference); + subPathToSlotMap.computeIfAbsent(slot, k -> Maps.newHashMap()) + .put(subPath, slotReference); + } + + } } - slot = SlotReference.fromColumn(table, column, name, qualified()); - cacheSlotWithSlotName.put(key, slot); - return slot; + return slots; } public List getManuallySpecifiedPartitions() { @@ -438,11 +443,11 @@ public boolean isPreAggStatusUnSet() { private List createSlotsVectorized(List columns) { List qualified = qualified(); - Object[] slots = new Object[columns.size()]; + SlotReference[] slots = new SlotReference[columns.size()]; for (int i = 0; i < columns.size(); i++) { - slots[i] = SlotReference.fromColumn(table, columns.get(i), qualified, this); + slots[i] = SlotReference.fromColumn(table, columns.get(i), qualified); } - return (List) Arrays.asList(slots); + return Arrays.asList(slots); } @Override diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalTVFRelation.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalTVFRelation.java index 9b098a6ddbaa13..f6fdfabe452891 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalTVFRelation.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalTVFRelation.java @@ -104,7 +104,7 @@ public String toString() { public List computeOutput() { return function.getTable().getBaseSchema() .stream() - .map(col -> SlotReference.fromColumn(function.getTable(), col, qualifier, this)) + .map(col -> SlotReference.fromColumn(function.getTable(), col, qualifier)) .collect(ImmutableList.toImmutableList()); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalCatalogRelation.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalCatalogRelation.java index ddc26f1041c1fa..17977fe054a664 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalCatalogRelation.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalCatalogRelation.java @@ -109,7 +109,7 @@ public DatabaseIf getDatabase() throws AnalysisException { public List computeOutput() { return table.getBaseSchema() .stream() - .map(col -> SlotReference.fromColumn(table, col, qualified(), this)) + .map(col -> SlotReference.fromColumn(table, col, qualified())) .collect(ImmutableList.toImmutableList()); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalOlapScan.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalOlapScan.java index 14629e528180a3..76713a51e29209 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalOlapScan.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalOlapScan.java @@ -51,7 +51,6 @@ public class PhysicalOlapScan extends PhysicalCatalogRelation implements OlapSca private final ImmutableList selectedPartitionIds; private final PreAggStatus preAggStatus; private final List baseOutputs; - private final Optional tableSample; /** @@ -121,8 +120,7 @@ public List getBaseOutputs() { public String toString() { StringBuilder builder = new StringBuilder(); if (!getAppliedRuntimeFilters().isEmpty()) { - getAppliedRuntimeFilters() - .stream().forEach(rf -> builder.append(" RF").append(rf.getId().asInt())); + getAppliedRuntimeFilters().forEach(rf -> builder.append(" RF").append(rf.getId().asInt())); } return Utils.toSqlString("PhysicalOlapScan[" + table.getName() + "]" + getGroupIdWithPrefix(), "stats", statistics, "RFs", builder diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalTVFRelation.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalTVFRelation.java index 1f409921c6cf36..ce7d32f9cfe292 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalTVFRelation.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/PhysicalTVFRelation.java @@ -107,7 +107,7 @@ public String toString() { public List computeOutput() { return function.getTable().getBaseSchema() .stream() - .map(col -> SlotReference.fromColumn(function.getTable(), col, ImmutableList.of(), this)) + .map(col -> SlotReference.fromColumn(function.getTable(), col, ImmutableList.of())) .collect(ImmutableList.toImmutableList()); } diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/expression/PredicatesSplitterTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/expression/PredicatesSplitterTest.java index a83ac620164806..56c0a6b8341b70 100644 --- a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/expression/PredicatesSplitterTest.java +++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/expression/PredicatesSplitterTest.java @@ -102,7 +102,7 @@ public Expression replaceUnboundSlot(Expression expression, Map me String name = ((UnboundSlot) expression).getName(); mem.putIfAbsent(name, SlotReference.fromColumn(null, new Column(name, getType(name.charAt(0)).toCatalogDataType()), - Lists.newArrayList("table"), null)); + Lists.newArrayList("table"))); return mem.get(name); } return hasNewChildren ? expression.withChildren(children) : expression; diff --git a/regression-test/data/variant_p0/test_sub_path_pruning.out b/regression-test/data/variant_p0/test_sub_path_pruning.out new file mode 100644 index 00000000000000..a48bc550d001e4 --- /dev/null +++ b/regression-test/data/variant_p0/test_sub_path_pruning.out @@ -0,0 +1,324 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !sql -- +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} {"c":{"d":{"e":21}},"d":{"e":22},"e":23} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} {"c":{"d":{"e":21}},"d":{"e":22},"e":23} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} {"c":{"d":{"e":21}},"d":{"e":22},"e":23} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} {"c":{"d":{"e":21}},"d":{"e":22},"e":23} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} {"c":{"d":{"e":21}},"d":{"e":22},"e":23} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} {"c":{"d":{"e":21}},"d":{"e":22},"e":23} + +-- !sql -- +1 {"c":{"d":{"e":11}}} {"c":{"d":{"e":21}},"d":{"e":22},"e":23} + +-- !sql -- +1 {"c":{"d":{"e":11}}} {"c":{"d":{"e":21}},"d":{"e":22},"e":23} + +-- !sql -- +1 {"c":{"d":{"e":11}}} \N + +-- !sql -- +1 {"c":{"d":{"e":11}}} \N + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"d":{"e":11}} + +-- !sql -- +1 {"d":{"e":11}} + +-- !sql -- +1 5 + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} 5 + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} 5 + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} 5 + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"d":{"e":11}} + +-- !sql -- +1 {"d":{"e":11}} + +-- !sql -- +1 {"c":{"d":{"e":11}}} {"d":{"e":12}} + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"c":{"d":{"e":11}}} \N + +-- !sql -- +1 {"c":{"d":{"e":11}}} \N + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} 1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} 1 {"c":{"d":{"e":21}},"d":{"e":22},"e":23} + +-- !sql -- +1 {"c":{"d":{"e":11}}} 1 {"a":{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14},"b":{"c":{"d":{"e":21}},"d":{"e":22},"e":23},"c":{"d":{"e":31},"e":32},"d":{"e":4},"e":5} + +-- !sql -- +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} {"c":{"d":{"e":21}},"d":{"e":22},"e":23} 1 {"a":{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14},"b":{"c":{"d":{"e":21}},"d":{"e":22},"e":23},"c":{"d":{"e":31},"e":32},"d":{"e":4},"e":5} + +-- !sql -- +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} {"c":{"d":{"e":21}},"d":{"e":22},"e":23} + +-- !sql -- +{"c":{"d":{"e":11}}} {"c":{"d":{"e":11}}} + +-- !sql -- +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +1 {"d":{"e":11}} + +-- !sql -- +1 {"d":{"e":11}} + +-- !sql -- +1 {"d":{"e":11}} \N + +-- !sql -- +1 {"d":{"e":11}} {"c":{"d":{"e":11}}} + +-- !sql -- +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +{"c":{"d":{"e":11}}} +{"c":{"d":{"e":11}}} + +-- !sql -- +{"c":{"d":{"e":11}}} +{"c":{"d":{"e":21}},"d":{"e":22},"e":23} + +-- !sql -- +{"e":11} +{"e":31} + +-- !sql -- +{"e":11} +{"e":12} + +-- !sql -- +{"e":22} +{"e":31} + +-- !sql -- +{"e":22} +{"e":31} + +-- !sql -- +{"e":22} 23 +{"e":31} 32 + +-- !sql -- +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +{"c":{"d":{"e":11}}} +{"c":{"d":{"e":11}}} +{"c":{"d":{"e":11}}} + +-- !sql -- +{"e":13} +{"e":22} +{"e":31} + +-- !sql -- +{"e":13} +{"e":22} +{"e":22} + +-- !sql -- +{"e":11} +{"e":12} +{"e":31} + +-- !sql -- +1 1 +1 {"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- +"1" +{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14} + +-- !sql -- + +{"c":{"d":{"e":11}}} + +-- !sql -- +"" +{"c":{"d":{"e":11}}} + +-- !sql -- +1 1 +1 {"d":{"e":11}} + +-- !sql -- +"1" +{"d":{"e":11}} + +-- !sql -- +\N + + +-- !sql -- +"" +{"e":11} + +-- !sql -- +1 1 +2 1 + +-- !sql -- +1 +1 + +-- !sql -- +1 +1 + +-- !sql -- + + + +-- !sql -- + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"c":{"d":{"e":11}}} + +-- !sql -- +1 {"d":{"e":12}} + +-- !sql -- +1 {"e":31} 1 + +-- !sql -- +1 {"d":{"e":12}} 11}}} +1 {"d":{"e":12}} {"c" +1 {"d":{"e":12}} {"d" +1 {"d":{"e":12}} {"e" + +-- !sql -- +1 {"e":31} 32 + +-- !sql -- +1 31 32 + +-- !sql -- +1 + +-- !sql -- +1 2 + +-- !sql -- +1 + +-- !sql -- +1 2 + +-- !sql -- +1 + +-- !sql -- +1 2 + +-- !sql -- +1 + +-- !sql -- +1 2 + diff --git a/regression-test/suites/variant_p0/test_sub_path_pruning.groovy b/regression-test/suites/variant_p0/test_sub_path_pruning.groovy new file mode 100644 index 00000000000000..a6bce99d5e4ec7 --- /dev/null +++ b/regression-test/suites/variant_p0/test_sub_path_pruning.groovy @@ -0,0 +1,217 @@ +// 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("variant_sub_path_pruning", "variant_type"){ + + sql "DROP TABLE IF EXISTS pruning_test" + + sql """ + CREATE TABLE `pruning_test` ( + `id` INT NULL, + `dt` VARIANT NULL + ) + DUPLICATE KEY(id) + DISTRIBUTED BY HASH(id) + PROPERTIES("replication_num"="1") + """ + + sql """ + insert into pruning_test values(1, '{"a":{"b":{"c":{"d":{"e":11}}},"c":{"d":{"e":12}},"d":{"e":13},"e":14},"b":{"c":{"d":{"e":21}},"d":{"e":22},"e":23},"c":{"d":{"e":31},"e":32},"d":{"e":4},"e":5}') + """ + + sql "sync" + + // project + + // simple case + order_qt_sql """select dt['a'] from pruning_test order by id limit 100;""" + + // one level + order_qt_sql """select id, c1['a'] from (select id, dt as c1 from pruning_test order by id limit 100) tmp;""" + order_qt_sql """select id, c1 from (select id, dt['a'] as c1 from pruning_test order by id limit 100) tmp;""" + + // two level, one level sub path + order_qt_sql """select id, c2 as c3 from (select id, c1 as c2 from (select id, dt['a'] as c1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2 as c3 from (select id, c1['a'] as c2 from (select id, dt as c1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2['a'] as c3 from (select id, c1 as c2 from (select id, dt as c1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + + // two level, two level sub paths + order_qt_sql """select id, c2 as c3 from (select id, c1['b'] as c2 from (select id, dt['a'] as c1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2['b'] as c3 from (select id, c1 as c2 from (select id, dt['a'] as c1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2['b'] as c3 from (select id, c1['a'] as c2 from (select id, dt as c1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + + // two level, two sub path on diff slot + order_qt_sql """select id, c2 as c3, d2 as d3 from (select id, c1 as c2, d1 as d2 from (select id, dt['a'] as c1, dt['b'] as d1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2 as c3, d2 as d3 from (select id, c1 as c2, d1['b'] as d2 from (select id, dt['a'] as c1, dt as d1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2 as c3, d2['b'] as d3 from (select id, c1 as c2, d1 as d2 from (select id, dt['a'] as c1, dt as d1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + + order_qt_sql """select id, c2 as c3, d2 as d3 from (select id, c1['a'] as c2, d1['b'] as d2 from (select id, dt as c1, dt as d1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2['a'] as c3, d2 as d3 from (select id, c1 as c2, d1['b'] as d2 from (select id, dt as c1, dt as d1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + + order_qt_sql """select id, c2['a'] as c3, d2['b'] as d3 from (select id, c1 as c2, d1 as d2 from (select id, dt as c1, dt as d1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + + // two level, two level sub path on diff slot + order_qt_sql """select id, c2 as c3, d2 as d3 from (select id, c1['b'] as c2, d1 as d2 from (select id, dt['a'] as c1, dt['b'] as d1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2['b'] as c3, d2 as d3 from (select id, c1 as c2, d1 as d2 from (select id, dt['a'] as c1, dt['b'] as d1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + + order_qt_sql """select id, c2 as c3, d2 as d3 from (select id, c1['b'] as c2, d1['b'] as d2 from (select id, dt['a'] as c1, dt['b'] as d1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2 as c3, d2['b'] as d3 from (select id, c1['b'] as c2, d1 as d2 from (select id, dt['a'] as c1, dt['b'] as d1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + + order_qt_sql """select id, c2 as c3 from (select id, c1 as c2 from (select id, dt['a']['b'] as c1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2 as c3 from (select id, c1['a']['b'] as c2 from (select id, dt as c1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2['a']['b'] as c3 from (select id, c1 as c2 from (select id, dt as c1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + + // two level, three level sub paths + order_qt_sql """select id, c2 as c3 from (select id, c1['c'] as c2 from (select id, dt['a']['b'] as c1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c2['c'] as c3 from (select id, c1['a']['b'] as c2 from (select id, dt as c1 from pruning_test order by id limit 100) tmp order by id limit 100) tmp;""" + + + // filter + order_qt_sql """select id, dt['e'] as c1 from pruning_test where dt['e'] > 1;""" + order_qt_sql """select id, dt['a'] as c1 from pruning_test where dt['e'] > 1;""" + order_qt_sql """select id, dt['a'] as c1, dt['e'] as c2 from pruning_test where dt['e'] > 1;""" + order_qt_sql """select id, dt['a'] as c1, dt['e'] as c2 from pruning_test where dt['d']['e'] > 1;""" + order_qt_sql """select id, c1, c2 from (select id, dt['a'] as c1, dt['e'] as c2 from pruning_test)tmp where c2 > 1;""" + + + // cte + + sql """set inline_cte_referenced_threshold = 0;""" + + // one level, sub path on consumer + order_qt_sql """with cte1 as (select id, dt as c1 from pruning_test) select id, c1['a'] as c2 from cte1;""" + order_qt_sql """with cte1 as (select id, dt as c1 from pruning_test) select id, c1['a']['b'] as c2 from cte1;""" + + // one level sub path on producer + order_qt_sql """with cte1 as (select id, dt['a'] as c1 from pruning_test) select id, c1 as c2 from cte1;""" + order_qt_sql """with cte1 as (select id, dt['a']['b'] as c1 from pruning_test) select id, c1 as c2 from cte1;""" + + // two level, sub path on both producer and consumer + order_qt_sql """with cte1 as (select id, dt['a'] as c1 from pruning_test) select id, c1['b'] as c2 from cte1;""" + order_qt_sql """with cte1 as (select id, dt['a']['b'] as c1 from pruning_test) select id, c1['c'] as c2 from cte1;""" + order_qt_sql """with cte1 as (select id, dt['a'] as c1 from pruning_test) select id, c1['b']['c'] as c2 from cte1;""" + + // two level, sub path on both producer and consumer on diff slots + order_qt_sql """with cte1 as (select id, dt['a'] as c1 from pruning_test) select id, c1['b'] as c2, c1['c'] as c3 from cte1;""" + order_qt_sql """with cte1 as (select id, dt['a'] as c1, dt['b'] as c2 from pruning_test) select id, c1['b'] as c3 from cte1;""" + order_qt_sql """with cte1 as (select id, dt['a'] as c1, dt['b'] as c2 from pruning_test) select id, c1['b'] as c3, c2['a'] as c4 from cte1;""" + order_qt_sql """with cte1 as (select id, dt['a'] as c1, dt['b'] as c2 from pruning_test) select id, c1['b'] as c3, c1['a'] as c4 from cte1;""" + + // more consumers + order_qt_sql """with cte1 as (select id, dt as c1 from pruning_test) select * from (select id, c1['a'] as c2 from cte1) v1 cross join (select id, c1['a'] as c2 from cte1) v2;""" + order_qt_sql """with cte1 as (select id, dt as c1 from pruning_test) select * from (select id, c1['a'] as c2 from cte1) v1 cross join (select id, c1['b'] as c2 from cte1) v2;""" + order_qt_sql """with cte1 as (select id, dt as c1 from pruning_test) select * from (select id, c1['a']['b'] as c2 from cte1) v1 cross join (select id, c1 as c2 from cte1) v2;""" + order_qt_sql """with cte1 as (select id, dt as c1 from pruning_test) select * from (select id, c1['a'] as c2, c1['b'] as c3 from cte1) v1 cross join (select id, c1 as c2 from cte1) v2;""" + + // more producers + order_qt_sql """with cte1 as (select id, dt as c1 from pruning_test), cte2 as (select id, dt as c1 from pruning_test) select cte1.c1['a'], cte2.c1['b'] from cte1, cte2;""" + order_qt_sql """with cte1 as (select id, dt['a'] as c1 from pruning_test), cte2 as (select id, dt['a'] as c1 from pruning_test) select cte1.c1['b'], cte2.c1['b'] from cte1, cte2;""" + + // two level cte + order_qt_sql """with cte1 as (select id, dt as c1 from pruning_test), cte2 as (select id, c1 as c2 from cte1) select c2['a'] from cte2;""" + order_qt_sql """with cte1 as (select id, dt as c1 from pruning_test), cte2 as (select id, c1['a'] as c2 from cte1) select c2 from cte2;""" + order_qt_sql """with cte1 as (select id, dt['a'] as c1 from pruning_test), cte2 as (select id, c1 as c2 from cte1) select c2 from cte2;""" + + order_qt_sql """with cte1 as (select id, dt['a'] as c1 from pruning_test), cte2 as (select id, c1['b'] as c2, c1['a'] as c3 from cte1) select id, c2['c'] as c4 from cte2;""" + order_qt_sql """with cte1 as (select id, dt['a']['b'] as c1 from pruning_test), cte2 as (select id, c1 as c2, c1['a'] as c3 from cte1) select id, c2['c'] as c4 from cte2;""" + order_qt_sql """with cte1 as (select id, dt['a']['b'] as c1 from pruning_test), cte2 as (select id, c1 as c2, c1['a'] as c3 from cte1) select id, c2['c'] as c4, c3['b'] as c5 from cte2;""" + + // two level cte + one level cte + order_qt_sql """with cte1 as (select id, dt['a'] as c1 from pruning_test), cte2 as (select id, c1['b'] as c2, c1['a'] as c3 from cte1) select cte2.id, cte2.c2['c'], cte1.c1['b'] from cte1, cte2;""" + + + // set operation + // distinct could not push down, only push down union all + + // two children + order_qt_sql """select dt['a'] as c1 from pruning_test union all select dt['a'] as c1 from pruning_test;""" + order_qt_sql """select c1['a'] from (select dt as c1 from pruning_test union all select dt as c1 from pruning_test) v1;""" + order_qt_sql """select c1['b'] from (select dt['a'] as c1 from pruning_test union all select dt['a'] as c1 from pruning_test) v1;""" + order_qt_sql """select c1['b'] from (select dt['a'] as c1 from pruning_test union all select dt as c1 from pruning_test) v1;""" + order_qt_sql """select c1['c']['d'] from (select dt['a']['b'] as c1 from pruning_test union all select dt as c1 from pruning_test) v1;""" + order_qt_sql """select c1['c']['d'] from (select dt['a']['b'] as c1 from pruning_test union all select dt['a'] as c1 from pruning_test) v1;""" + + // two children with diff sub path in children + order_qt_sql """select c1['d'] from (select dt['c'] as c1 from pruning_test where cast(dt['a']['b']['c']['d']['e'] as int) = 11 union all select dt['b'] as c1 from pruning_test where cast(dt['a']['c']['d']['e'] as int) = 12) tmp;""" + order_qt_sql """select c1['d'] from (select dt['c'] as c1 from pruning_test where cast(dt['c']['d']['e'] as int) = 31 union all select dt['b'] as c1 from pruning_test where cast(dt['c']['e'] as int) = 32) tmp;""" + order_qt_sql """select c1['d'], c1['e'] from (select dt['c'] as c1 from pruning_test where cast(dt['c']['d']['e'] as int) = 31 union all select dt['b'] as c1 from pruning_test where cast(dt['c']['e'] as int) = 32) tmp;""" + + // three children + order_qt_sql """select dt['a'] as c1 from pruning_test union all select dt['a'] as c1 from pruning_test union all select dt['a'] as c1 from pruning_test;""" + order_qt_sql """select c1['a'] from (select dt as c1 from pruning_test union all select dt as c1 from pruning_test union all select dt as c1 from pruning_test) v1;""" + order_qt_sql """select c1['b'] from (select dt['a'] as c1 from pruning_test union all select dt['a'] as c1 from pruning_test union all select dt['a'] as c1 from pruning_test) v1;""" + order_qt_sql """select c1['d'] from (select dt['a'] as c1 from pruning_test union all select dt['b'] as c1 from pruning_test union all select dt['c'] as c1 from pruning_test) v1;""" + order_qt_sql """select c1['d'] from (select dt['a'] as c1 from pruning_test union all select dt['b'] as c1 from pruning_test union all select dt['b'] as c1 from pruning_test) v1;""" + order_qt_sql """select c1['c']['d'] from (select dt['a']['b'] as c1 from pruning_test union all select dt['a'] as c1 from pruning_test union all select dt as c1 from pruning_test) v1;""" + + // one table + one const list + order_qt_sql """select id, c1['a'] from (select cast('{"a":1}' as variant) as c1, 1 as id union all select dt as c1, id from pruning_test) tmp order by id limit 100;""" + order_qt_sql """select c1['a'] from (select id, c1 from (select cast('{"a":1}' as variant) as c1, 1 as id union all select dt as c1, id from pruning_test) tmp order by id limit 100) tmp;""" + order_qt_sql """select c2['b'] from (select id, c1['a'] as c2 from (select cast('{"a":{"b":1}}' as variant) as c1, 1 as id union all select dt as c1, id from pruning_test) tmp order by id limit 100) tmp;""" + order_qt_sql """select c2['a']['b'] from (select id, c1 as c2 from (select cast('{"a":{"b":1}}' as variant) as c1, 1 as id union all select dt as c1, id from pruning_test) tmp order by id limit 100) tmp;""" + order_qt_sql """select id, c1['c'] from (select cast('{"c":1}' as variant) as c1, 1 as id union all select dt['a']['b'] as c1, id from pruning_test) tmp order by id limit 100;""" + order_qt_sql """select c1['c'] from (select id, c1 from (select cast('{"c":1}' as variant) as c1, 1 as id union all select dt['a']['b'] as c1, id from pruning_test) tmp order by id limit 100) tmp;""" + order_qt_sql """select c2['d'] from (select id, c1['a'] as c2 from (select cast('{"c":{"d":1}}' as variant) as c1, 1 as id union all select dt['a']['b'] as c1, id from pruning_test) tmp order by id limit 100) tmp;""" + order_qt_sql """select c2['c']['d'] from (select id, c1 as c2 from (select cast('{"c":{"d":1}}' as variant) as c1, 1 as id union all select dt['a']['b'] as c1, id from pruning_test) tmp order by id limit 100) tmp;""" + + // two const list + order_qt_sql """select id, c1['a'] from (select cast('{"a":1}' as variant) as c1, 1 as id union all select cast('{"a":1}' as variant) as c1, 2 as id) tmp order by id limit 100;""" + order_qt_sql """select c1['a'] from (select id, c1 from (select cast('{"a":1}' as variant) as c1, 1 as id union all select cast('{"a":1}' as variant) as c1, 2 as id) tmp order by id limit 100) tmp;""" + order_qt_sql """select c2['b'] from (select id, c1['a'] as c2 from (select cast('{"a":{"b":1}}' as variant) as c1, 1 as id union all select cast('{"a":{"b":1}}' as variant) as c1, 2 as id) tmp order by id limit 100) tmp;""" + order_qt_sql """select c2['a']['b'] from (select id, c1 as c2 from (select cast('{"a":{"b":1}}' as variant) as c1, 1 as id union all select cast('{"a":{"b":1}}' as variant) as c1, 2 as id) tmp order by id limit 100) tmp;""" + + + // join + order_qt_sql """select v1.id, v2.dt['a']['b'] from pruning_test v1 join pruning_test v2 on cast(v1.dt['b']['c']['d']['e'] as int) = cast(v2.dt['a']['d']['e'] as int);""" + order_qt_sql """select v1.id, v2.dt['a']['b'] from pruning_test v1 join pruning_test v2 on case when cast(v1.dt['b']['c']['d']['e'] as int) > cast(v2.dt['a']['d']['e'] as int) then true when cast(v1.dt['b']['c']['d']['e'] as int) < cast(v2.dt['a']['d']['e'] as int) then false else true end;""" + order_qt_sql """select v1.id, v2.dt['a']['b'] from pruning_test v1 join pruning_test v2 on v1.id = v2.id and cast(v1.dt['b']['c']['d']['e'] as int) > cast(v2.dt['a']['d']['e'] as int);""" + + + // sort / topn + order_qt_sql """select id, dt['a']['c'] from pruning_test order by cast(dt['a']['e'] as int), cast(dt['e'] as int);""" + + + // partition topn + order_qt_sql """select * from (select id, dt['c']['d'], row_number() over(partition by id order by cast(dt['c']['e'] as int)) as c1 from pruning_test) tmp where c1 < 10;""" + + + // generate + order_qt_sql """select id, dt['a']['c'] as c1, c2 from pruning_test lateral view explode_split(cast(dt['a']['b'] as string), ':') v1 as c2 order by c2;""" + + + // window + order_qt_sql """select id, dt['c']['d'], sum(cast(dt['c']['e'] as int)) over(partition by id) as c1 from pruning_test;""" + + + // aggregate + order_qt_sql """select id, cast(dt['c']['d']['e'] as int) c1, sum(cast(dt['c']['e'] as int)) as c2 from pruning_test group by 1, 2;""" + + + // variant in project / one row relation + + // variant in project + order_qt_sql """select c1['a'] from (select id, cast('{"a":1}' as variant) as c1 from pruning_test order by id limit 100) tmp;""" + order_qt_sql """select c1['a'] as c2, c1['b'] as c3 from (select id, cast('{"a":1, "b":2}' as variant) as c1 from pruning_test order by id limit 100) tmp;""" + order_qt_sql """select c1['a'] from (select id, cast('{"b":{"a":1}}' as variant)["b"] as c1 from pruning_test order by id limit 100) tmp;""" + order_qt_sql """select c1['a'] as c2, c1['b'] as c3 from (select id, cast('{"b":{"a":1, "b":2}}' as variant)["b"] as c1 from pruning_test order by id limit 100) tmp;""" + + // varaint in one row relation + order_qt_sql """select c1['a'] from (select 1 as id, cast('{"a":1}' as variant) as c1 order by id limit 100) tmp;""" + order_qt_sql """select c1['a'] as c2, c1['b'] as c3 from (select 1 as id, cast('{"a":1, "b":2}' as variant) as c1 order by id limit 100) tmp;""" + order_qt_sql """select c1['a'] from (select 1 as id, cast('{"b":{"a":1}}' as variant)["b"] as c1 order by id limit 100) tmp;""" + order_qt_sql """select c1['a'] as c2, c1['b'] as c3 from (select 1 as id, cast('{"b":{"a":1, "b":2}}' as variant)["b"] as c1 order by id limit 100) tmp;""" +} \ No newline at end of file