This is an automated email from the ASF dual-hosted git repository. jakevin pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/doris.git
The following commit(s) were added to refs/heads/master by this push: new 01fa2e6a555 [feat](Nereids): Refactor Eliminate_Group_By_Key by functional dependencies (#34948) 01fa2e6a555 is described below commit 01fa2e6a55511f1173f958293b091ba7e97bf74e Author: 谢健 <jianx...@gmail.com> AuthorDate: Wed May 22 10:26:46 2024 +0800 [feat](Nereids): Refactor Eliminate_Group_By_Key by functional dependencies (#34948) --- .../apache/doris/nereids/properties/FuncDeps.java | 34 +++ .../nereids/properties/FunctionalDependencies.java | 29 ++- .../org/apache/doris/nereids/rules/RuleType.java | 1 + .../nereids/rules/rewrite/EliminateGroupByKey.java | 236 ++++++------------- .../plans/logical/LogicalCatalogRelation.java | 9 - .../nereids/trees/plans/logical/LogicalPlan.java | 13 +- .../trees/plans/logical/LogicalProject.java | 5 + .../properties/{FuncDepsTest.java => FdTest.java} | 6 +- .../doris/nereids/properties/FuncDepsTest.java | 260 ++++++--------------- .../properties/FunctionalDependenciesTest.java | 6 +- .../rules/rewrite/EliminateGroupByKeyTest.java | 99 ++++++++ .../eliminate_gby_key/eliminate_gby_key.groovy | 22 +- 12 files changed, 329 insertions(+), 391 deletions(-) diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/FuncDeps.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/FuncDeps.java index 8c84d7a1755..9ecfbea6504 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/FuncDeps.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/FuncDeps.java @@ -20,8 +20,11 @@ package org.apache.doris.nereids.properties; import org.apache.doris.nereids.trees.expressions.Slot; import com.google.common.collect.ImmutableSet; +import com.google.common.collect.Sets; +import java.util.ArrayList; import java.util.HashSet; +import java.util.List; import java.util.Objects; import java.util.Set; @@ -71,6 +74,37 @@ public class FuncDeps { return items.size(); } + public boolean isEmpty() { + return items.isEmpty(); + } + + /** + * Eliminate all deps in slots + */ + public Set<Set<Slot>> eliminateDeps(Set<Set<Slot>> slots) { + Set<Set<Slot>> minSlotSet = slots; + List<Set<Set<Slot>>> reduceSlotSets = new ArrayList<>(); + reduceSlotSets.add(slots); + while (!reduceSlotSets.isEmpty()) { + List<Set<Set<Slot>>> newReduceSlotSets = new ArrayList<>(); + for (Set<Set<Slot>> slotSet : reduceSlotSets) { + for (FuncDepsItem funcDepsItem : items) { + if (slotSet.contains(funcDepsItem.dependencies) + && slotSet.contains(funcDepsItem.determinants)) { + Set<Set<Slot>> newSet = Sets.newHashSet(slotSet); + newSet.remove(funcDepsItem.dependencies); + if (minSlotSet.size() > newSet.size()) { + minSlotSet = newSet; + } + newReduceSlotSets.add(newSet); + } + } + } + reduceSlotSets = newReduceSlotSets; + } + return minSlotSet; + } + public boolean isFuncDeps(Set<Slot> dominate, Set<Slot> dependency) { return items.contains(new FuncDepsItem(dominate, dependency)); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/FunctionalDependencies.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/FunctionalDependencies.java index 079657df9b6..cee2be1c918 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/FunctionalDependencies.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/FunctionalDependencies.java @@ -18,6 +18,7 @@ package org.apache.doris.nereids.properties; import org.apache.doris.nereids.trees.expressions.Slot; +import org.apache.doris.nereids.trees.expressions.functions.ExpressionTrait; import org.apache.doris.nereids.util.ImmutableEqualSet; import com.google.common.collect.ImmutableSet; @@ -196,6 +197,9 @@ public class FunctionalDependencies { } public void addDeps(Set<Slot> dominate, Set<Slot> dependency) { + if (dominate.containsAll(dependency)) { + return; + } fdDgBuilder.addDeps(dominate, dependency); } @@ -265,10 +269,17 @@ public class FunctionalDependencies { /** * get all unique slots */ - public List<Set<Slot>> getAllUnique() { - List<Set<Slot>> res = new ArrayList<>(uniqueSet.slotSets); - for (Slot s : uniqueSet.slots) { - res.add(ImmutableSet.of(s)); + public List<Set<Slot>> getAllUniqueAndNotNull() { + List<Set<Slot>> res = new ArrayList<>(); + for (Slot slot : uniqueSet.slots) { + if (!slot.nullable()) { + res.add(ImmutableSet.of(slot)); + } + } + for (Set<Slot> slotSet : uniqueSet.slotSets) { + if (slotSet.stream().noneMatch(ExpressionTrait::nullable)) { + res.add(slotSet); + } } return res; } @@ -276,8 +287,14 @@ public class FunctionalDependencies { /** * get all uniform slots */ - public Set<Slot> getAllUniform() { - return uniformSet.slots; + public List<Set<Slot>> getAllUniformAndNotNull() { + List<Set<Slot>> res = new ArrayList<>(); + for (Slot s : uniformSet.slots) { + if (!s.nullable()) { + res.add(ImmutableSet.of(s)); + } + } + return res; } public void addEqualPair(Slot l, Slot r) { 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 b1101b7592b..0115274ea2b 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 @@ -226,6 +226,7 @@ public enum RuleType { ELIMINATE_JOIN_BY_UK(RuleTypeClass.REWRITE), ELIMINATE_JOIN_BY_FK(RuleTypeClass.REWRITE), ELIMINATE_GROUP_BY_KEY(RuleTypeClass.REWRITE), + ELIMINATE_FILTER_GROUP_BY_KEY(RuleTypeClass.REWRITE), ELIMINATE_DEDUP_JOIN_CONDITION(RuleTypeClass.REWRITE), ELIMINATE_NULL_AWARE_LEFT_ANTI_JOIN(RuleTypeClass.REWRITE), ELIMINATE_ASSERT_NUM_ROWS(RuleTypeClass.REWRITE), diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/EliminateGroupByKey.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/EliminateGroupByKey.java index 9853b7e6ff9..965c5d13e32 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/EliminateGroupByKey.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/EliminateGroupByKey.java @@ -17,193 +17,101 @@ package org.apache.doris.nereids.rules.rewrite; -import org.apache.doris.nereids.properties.FdItem; +import org.apache.doris.nereids.annotation.DependsRules; +import org.apache.doris.nereids.properties.FuncDeps; import org.apache.doris.nereids.rules.Rule; import org.apache.doris.nereids.rules.RuleType; import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.expressions.NamedExpression; -import org.apache.doris.nereids.trees.expressions.SlotReference; +import org.apache.doris.nereids.trees.expressions.Slot; +import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.trees.plans.logical.LogicalAggregate; -import org.apache.doris.nereids.trees.plans.logical.LogicalPlan; -import com.google.common.collect.ImmutableSet; +import com.google.common.collect.ImmutableList; import java.util.ArrayList; +import java.util.HashMap; import java.util.HashSet; import java.util.List; +import java.util.Map; +import java.util.Map.Entry; import java.util.Set; -import java.util.stream.Collectors; + /** * Eliminate group by key based on fd item information. + * such as: + * for a -> b, we can get: + * group by a, b, c => group by a, c */ -public class EliminateGroupByKey extends OneRewriteRuleFactory { - @Override - public Rule build() { - return logicalAggregate(logicalProject()).then(agg -> { - LogicalPlan childPlan = agg.child(); - List<FdItem> uniqueFdItems = new ArrayList<>(); - List<FdItem> nonUniqueFdItems = new ArrayList<>(); - if (agg.getGroupByExpressions().isEmpty() - || !agg.getGroupByExpressions().stream().allMatch(e -> e instanceof SlotReference)) { - return null; - } - ImmutableSet<FdItem> fdItems = childPlan.getLogicalProperties().getFunctionalDependencies().getFdItems(); - if (fdItems.isEmpty()) { - return null; - } - List<SlotReference> candiExprs = agg.getGroupByExpressions().stream() - .map(SlotReference.class::cast).collect(Collectors.toList()); +@DependsRules({EliminateGroupBy.class, ColumnPruning.class}) +public class EliminateGroupByKey implements RewriteRuleFactory { - fdItems.stream().filter(e -> !e.isCandidate()).forEach(e -> { - if (e.isUnique()) { - uniqueFdItems.add(e); - } else { - nonUniqueFdItems.add(e); - } - } - ); + @Override + public List<Rule> buildRules() { + return ImmutableList.of( + RuleType.ELIMINATE_GROUP_BY_KEY.build( + logicalProject(logicalAggregate().when(agg -> !agg.getSourceRepeat().isPresent())) + .then(proj -> { + LogicalAggregate<? extends Plan> agg = proj.child(); + LogicalAggregate<Plan> newAgg = eliminateGroupByKey(agg, proj.getInputSlots()); + if (newAgg == null) { + return null; + } + return proj.withChildren(newAgg); + })), + RuleType.ELIMINATE_FILTER_GROUP_BY_KEY.build( + logicalProject(logicalFilter(logicalAggregate() + .when(agg -> !agg.getSourceRepeat().isPresent()))) + .then(proj -> { + LogicalAggregate<? extends Plan> agg = proj.child().child(); + Set<Slot> requireSlots = new HashSet<>(proj.getInputSlots()); + requireSlots.addAll(proj.child(0).getInputSlots()); + LogicalAggregate<Plan> newAgg = eliminateGroupByKey(agg, proj.getOutputSet()); + if (newAgg == null) { + return null; + } + return proj.withChildren(proj.child().withChildren(newAgg)); + }) + ) + ); + } - int minParentExprCnt = -1; - ImmutableSet<SlotReference> minParentExprs = ImmutableSet.of(); - // if unique fd items exists, try to find the one which has the - // smallest parent exprs - for (int i = 0; i < uniqueFdItems.size(); i++) { - FdItem fdItem = uniqueFdItems.get(i); - ImmutableSet<SlotReference> parentExprs = fdItem.getParentExprs(); - if (minParentExprCnt == -1 || parentExprs.size() < minParentExprCnt) { - boolean isContain = isExprsContainFdParent(candiExprs, fdItem); - if (isContain) { - minParentExprCnt = parentExprs.size(); - minParentExprs = ImmutableSet.copyOf(parentExprs); - } - } - } + LogicalAggregate<Plan> eliminateGroupByKey(LogicalAggregate<? extends Plan> agg, Set<Slot> requireOutput) { + Map<Expression, Set<Slot>> groupBySlots = new HashMap<>(); + Set<Slot> validSlots = new HashSet<>(); + for (Expression expression : agg.getGroupByExpressions()) { + groupBySlots.put(expression, expression.getInputSlots()); + validSlots.addAll(expression.getInputSlots()); + } - Set<Integer> rootExprsSet = new HashSet<>(); - List<SlotReference> rootExprs = new ArrayList<>(); - Set<Integer> eliminateSet = new HashSet<>(); - if (minParentExprs.size() > 0) { - // if any unique fd item found, find the expr which matching parentExprs - // from candiExprs directly - for (int i = 0; i < minParentExprs.size(); i++) { - int index = findEqualExpr(candiExprs, minParentExprs.asList().get(i)); - if (index != -1) { - rootExprsSet.add(new Integer(index)); - } else { - return null; - } - } - } else { - // no unique fd item found, try to find the smallest root exprs set - // from non-unique fd items. - for (int i = 0; i < nonUniqueFdItems.size() && eliminateSet.size() < candiExprs.size(); i++) { - FdItem fdItem = nonUniqueFdItems.get(i); - ImmutableSet<SlotReference> parentExprs = fdItem.getParentExprs(); - boolean isContains = isExprsContainFdParent(candiExprs, fdItem); - if (isContains) { - List<SlotReference> leftDomain = new ArrayList<>(); - List<SlotReference> rightDomain = new ArrayList<>(); - // generate new root exprs - for (int j = 0; j < rootExprs.size(); j++) { - leftDomain.add(rootExprs.get(j)); - boolean isInChild = fdItem.checkExprInChild(rootExprs.get(j), childPlan); - if (!isInChild) { - rightDomain.add(rootExprs.get(j)); - } - } - for (int j = 0; j < parentExprs.size(); j++) { - int index = findEqualExpr(candiExprs, parentExprs.asList().get(j)); - if (index != -1) { - rightDomain.add(candiExprs.get(index)); - if (!eliminateSet.contains(index)) { - leftDomain.add(candiExprs.get(index)); - } - } - } - // check fd can eliminate new candi expr - for (int j = 0; j < candiExprs.size(); j++) { - if (!eliminateSet.contains(j)) { - boolean isInChild = fdItem.checkExprInChild(candiExprs.get(j), childPlan); - if (isInChild) { - eliminateSet.add(j); - } - } - } - // if fd eliminate new candi exprs or new root exprs is smaller than the older, - // than use new root expr to replace old ones - List<SlotReference> newRootExprs = leftDomain.size() <= rightDomain.size() - ? leftDomain : rightDomain; - rootExprs.clear(); - rootExprs.addAll(newRootExprs); - } - } - } - // find the root expr, add into root exprs set, indicate the index in - // candiExprs list - for (int i = 0; i < rootExprs.size(); i++) { - int index = findEqualExpr(candiExprs, rootExprs.get(i)); - if (index != -1) { - rootExprsSet.add(new Integer(index)); - } else { - return null; - } - } - // other can't be determined expr, add into root exprs directly - if (eliminateSet.size() < candiExprs.size()) { - for (int i = 0; i < candiExprs.size(); i++) { - if (!eliminateSet.contains(i)) { - rootExprsSet.add(i); - } - } - } - rootExprs.clear(); - for (int i = 0; i < candiExprs.size(); i++) { - if (rootExprsSet.contains(i)) { - rootExprs.add(candiExprs.get(i)); - } - } + FuncDeps funcDeps = agg.child().getLogicalProperties() + .getFunctionalDependencies().getAllValidFuncDeps(validSlots); + if (funcDeps.isEmpty()) { + return null; + } - // use the new rootExprs as new group by keys - List<Expression> resultExprs = new ArrayList<>(); - for (int i = 0; i < rootExprs.size(); i++) { - resultExprs.add(rootExprs.get(i)); + Set<Set<Slot>> minGroupBySlots = funcDeps.eliminateDeps(new HashSet<>(groupBySlots.values())); + Set<Expression> removeExpression = new HashSet<>(); + for (Entry<Expression, Set<Slot>> entry : groupBySlots.entrySet()) { + if (!minGroupBySlots.contains(entry.getValue()) + && !requireOutput.containsAll(entry.getValue())) { + removeExpression.add(entry.getKey()); } + } - // eliminate outputs keys - // TODO: remove outputExprList computing - List<NamedExpression> outputExprList = new ArrayList<>(); - for (int i = 0; i < agg.getOutputExpressions().size(); i++) { - if (rootExprsSet.contains(i)) { - outputExprList.add(agg.getOutputExpressions().get(i)); - } - } - // find the remained outputExprs list - List<NamedExpression> remainedOutputExprList = new ArrayList<>(); - for (int i = 0; i < agg.getOutputExpressions().size(); i++) { - NamedExpression outputExpr = agg.getOutputExpressions().get(i); - if (!agg.getGroupByExpressions().contains(outputExpr)) { - remainedOutputExprList.add(outputExpr); - } + List<Expression> newGroupExpression = new ArrayList<>(); + for (Expression expression : agg.getGroupByExpressions()) { + if (!removeExpression.contains(expression)) { + newGroupExpression.add(expression); } - outputExprList.addAll(remainedOutputExprList); - return new LogicalAggregate<>(resultExprs, agg.getOutputExpressions(), agg.child()); - }).toRule(RuleType.ELIMINATE_GROUP_BY_KEY); - } - - /** - * find the equal expr index from expr list. - */ - public int findEqualExpr(List<SlotReference> exprList, SlotReference expr) { - for (int i = 0; i < exprList.size(); i++) { - if (exprList.get(i).equals(expr)) { - return i; + } + List<NamedExpression> newOutput = new ArrayList<>(); + for (NamedExpression expression : agg.getOutputExpressions()) { + if (!removeExpression.contains(expression)) { + newOutput.add(expression); } } - return -1; - } - - private boolean isExprsContainFdParent(List<SlotReference> candiExprs, FdItem fdItem) { - return fdItem.getParentExprs().stream().allMatch(e -> candiExprs.contains(e)); + return agg.withGroupByAndOutput(newGroupExpression, newOutput); } } 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 63119f00e6d..6995a6b5737 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 @@ -30,7 +30,6 @@ import org.apache.doris.nereids.memo.GroupExpression; import org.apache.doris.nereids.properties.FdFactory; import org.apache.doris.nereids.properties.FdItem; import org.apache.doris.nereids.properties.FunctionalDependencies; -import org.apache.doris.nereids.properties.FunctionalDependencies.Builder; import org.apache.doris.nereids.properties.LogicalProperties; import org.apache.doris.nereids.properties.TableFdItem; import org.apache.doris.nereids.trees.expressions.Slot; @@ -126,14 +125,6 @@ public abstract class LogicalCatalogRelation extends LogicalRelation implements return Utils.qualifiedName(qualifier, table.getName()); } - @Override - public FunctionalDependencies computeFuncDeps() { - Builder fdBuilder = new Builder(); - computeUnique(fdBuilder); - fdBuilder.addFdItems(computeFdItems(Utils.fastToImmutableSet(getOutputSet()))); - return fdBuilder.build(); - } - @Override public void computeUnique(FunctionalDependencies.Builder fdBuilder) { Set<Slot> outputSet = Utils.fastToImmutableSet(getOutputSet()); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalPlan.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalPlan.java index 3c7758963cf..ea64f21fc9f 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalPlan.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalPlan.java @@ -26,7 +26,6 @@ import com.google.common.collect.ImmutableList; import com.google.common.collect.ImmutableSet; import com.google.common.collect.Sets; -import java.util.List; import java.util.Optional; import java.util.Set; import java.util.function.BiFunction; @@ -72,17 +71,15 @@ public interface LogicalPlan extends Plan { ImmutableSet<FdItem> fdItems = computeFdItems(); fdBuilder.addFdItems(fdItems); - List<Set<Slot>> uniqueSlots = fdBuilder.getAllUnique(); - Set<Slot> uniformSlots = fdBuilder.getAllUniform(); for (Slot slot : getOutput()) { Set<Slot> o = ImmutableSet.of(slot); - // all slot dependents unique slot - for (Set<Slot> uniqueSlot : uniqueSlots) { + // all slots dependent unique slot + for (Set<Slot> uniqueSlot : fdBuilder.getAllUniqueAndNotNull()) { fdBuilder.addDeps(uniqueSlot, o); } - // uniform slot dependents all unique slot - for (Slot uniformSlot : uniformSlots) { - fdBuilder.addDeps(o, ImmutableSet.of(uniformSlot)); + // uniform slot dependents all slots + for (Set<Slot> uniformSlot : fdBuilder.getAllUniformAndNotNull()) { + fdBuilder.addDeps(o, uniformSlot); } } for (Set<Slot> equalSet : fdBuilder.calEqualSetList()) { diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalProject.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalProject.java index 9acaea20552..02b9d52859a 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalProject.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalProject.java @@ -276,5 +276,10 @@ public class LogicalProject<CHILD_TYPE extends Plan> extends LogicalUnary<CHILD_ @Override public void computeFd(FunctionalDependencies.Builder fdBuilder) { fdBuilder.addFuncDepsDG(child().getLogicalProperties().getFunctionalDependencies()); + for (NamedExpression expr : getProjects()) { + if (!expr.isSlot()) { + fdBuilder.addDeps(expr.getInputSlots(), ImmutableSet.of(expr.toSlot())); + } + } } } diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FuncDepsTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FdTest.java similarity index 98% copy from fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FuncDepsTest.java copy to fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FdTest.java index 47871ce82e3..6deed993c3b 100644 --- a/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FuncDepsTest.java +++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FdTest.java @@ -30,7 +30,7 @@ import org.junit.jupiter.api.Test; import java.util.Set; -class FuncDepsTest extends TestWithFeService { +class FdTest extends TestWithFeService { Slot slot1 = new SlotReference("1", IntegerType.INSTANCE, false); Slot slot2 = new SlotReference("2", IntegerType.INSTANCE, false); Slot slot3 = new SlotReference("1", IntegerType.INSTANCE, false); @@ -60,7 +60,7 @@ class FuncDepsTest extends TestWithFeService { @Test void testAgg() { Plan plan = PlanChecker.from(connectContext) - .analyze("select id, id2 from agg group by id, id2") + .analyze("select sum(id2), id2 from agg group by id2") .getPlan(); Set<Slot> output = ImmutableSet.copyOf(plan.getOutputSet()); System.out.println(plan.getLogicalProperties() @@ -68,7 +68,7 @@ class FuncDepsTest extends TestWithFeService { Assertions.assertTrue( plan.getLogicalProperties() .getFunctionalDependencies().getAllValidFuncDeps(output) - .isFuncDeps(output, ImmutableSet.of(plan.getOutput().get(0)))); + .isFuncDeps(ImmutableSet.of(plan.getOutput().get(1)), ImmutableSet.of(plan.getOutput().get(0)))); } @Test diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FuncDepsTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FuncDepsTest.java index 47871ce82e3..a9496392fc6 100644 --- a/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FuncDepsTest.java +++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FuncDepsTest.java @@ -19,216 +19,102 @@ package org.apache.doris.nereids.properties; import org.apache.doris.nereids.trees.expressions.Slot; import org.apache.doris.nereids.trees.expressions.SlotReference; -import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.types.IntegerType; -import org.apache.doris.nereids.util.PlanChecker; -import org.apache.doris.utframe.TestWithFeService; -import com.google.common.collect.ImmutableSet; +import com.google.common.collect.Sets; import org.junit.jupiter.api.Assertions; import org.junit.jupiter.api.Test; +import java.util.HashSet; import java.util.Set; -class FuncDepsTest extends TestWithFeService { - Slot slot1 = new SlotReference("1", IntegerType.INSTANCE, false); - Slot slot2 = new SlotReference("2", IntegerType.INSTANCE, false); - Slot slot3 = new SlotReference("1", IntegerType.INSTANCE, false); - Slot slot4 = new SlotReference("1", IntegerType.INSTANCE, false); - - @Override - protected void runBeforeAll() throws Exception { - createDatabase("test"); - createTable("create table test.agg (\n" - + "id int not null,\n" - + "id2 int replace not null,\n" - + "name varchar(128) replace not null )\n" - + "AGGREGATE KEY(id)\n" - + "distributed by hash(id) buckets 10\n" - + "properties('replication_num' = '1');"); - createTable("create table test.uni (\n" - + "id int not null,\n" - + "id2 int not null,\n" - + "name varchar(128) not null)\n" - + "UNIQUE KEY(id)\n" - + "distributed by hash(id) buckets 10\n" - + "properties('replication_num' = '1');"); - connectContext.setDatabase("test"); - connectContext.getSessionVariable().setDisableNereidsRules("PRUNE_EMPTY_PARTITION"); - } - - @Test - void testAgg() { - Plan plan = PlanChecker.from(connectContext) - .analyze("select id, id2 from agg group by id, id2") - .getPlan(); - Set<Slot> output = ImmutableSet.copyOf(plan.getOutputSet()); - System.out.println(plan.getLogicalProperties() - .getFunctionalDependencies().getAllValidFuncDeps(output)); - Assertions.assertTrue( - plan.getLogicalProperties() - .getFunctionalDependencies().getAllValidFuncDeps(output) - .isFuncDeps(output, ImmutableSet.of(plan.getOutput().get(0)))); - } - - @Test - void testTopNLimit() { - Plan plan = PlanChecker.from(connectContext) - .analyze("select id, id2 from agg group by id, id2 order by id limit 1") - .getPlan(); - Set<Slot> output = ImmutableSet.copyOf(plan.getOutputSet()); - System.out.println(plan.getLogicalProperties() - .getFunctionalDependencies().getAllValidFuncDeps(output)); - Assertions.assertTrue( - plan.getLogicalProperties() - .getFunctionalDependencies().getAllValidFuncDeps(output) - .isFuncDeps(output, ImmutableSet.of(plan.getOutput().get(0)))); - } - - @Test - void testSetOp() { - Plan plan = PlanChecker.from(connectContext) - .analyze("select id, id2 from agg where id2 = id intersect select id, id2 from agg") - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isNullSafeEqual(plan.getOutput().get(0), plan.getOutput().get(1))); - plan = PlanChecker.from(connectContext) - .analyze("select id, id2 from agg where id2 = id except select id, id2 from agg") - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isNullSafeEqual(plan.getOutput().get(0), plan.getOutput().get(1))); - plan = PlanChecker.from(connectContext) - .analyze("select id, id2 from agg where id2 = id union all select id, id2 from agg") - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isEmpty()); - plan = PlanChecker.from(connectContext) - .analyze("select id, id2 from agg union all select id, id2 from agg where id2 = id") - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isEmpty()); - } +class FuncDepsTest { + Slot s1 = new SlotReference("1", IntegerType.INSTANCE, false); + Slot s2 = new SlotReference("2", IntegerType.INSTANCE, false); + Slot s3 = new SlotReference("3", IntegerType.INSTANCE, false); + Slot s4 = new SlotReference("4", IntegerType.INSTANCE, false); + Set<Slot> set1 = Sets.newHashSet(s1); + Set<Slot> set2 = Sets.newHashSet(s2); + Set<Slot> set3 = Sets.newHashSet(s3); + Set<Slot> set4 = Sets.newHashSet(s4); @Test - void testFilterHaving() { - Plan plan = PlanChecker.from(connectContext) - .analyze("select id, id2 from agg where id = 1") - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(0)), ImmutableSet.of(plan.getOutput().get(1)))); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(1)), ImmutableSet.of(plan.getOutput().get(0)))); - plan = PlanChecker.from(connectContext) - .analyze("select id, id2 from agg group by id, id2 having id = 1") - .rewrite() - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(0)), ImmutableSet.of(plan.getOutput().get(1)))); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(1)), ImmutableSet.of(plan.getOutput().get(0)))); + void testOneEliminate() { + Set<Set<Slot>> slotSet = Sets.newHashSet(set1, set2, set3, set4); + FuncDeps funcDeps = new FuncDeps(); + funcDeps.addFuncItems(Sets.newHashSet(s1), Sets.newHashSet(s2)); + Set<Set<Slot>> slots = funcDeps.eliminateDeps(slotSet); + Set<Set<Slot>> expected = new HashSet<>(); + expected.add(set1); + expected.add(set3); + expected.add(set4); + Assertions.assertEquals(expected, slots); } @Test - void testGenerate() { - Plan plan = PlanChecker.from(connectContext) - .analyze("select id, id2 from agg lateral view explode([1,2,3]) tmp1 as e1") - .rewrite() - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(0)), ImmutableSet.of(plan.getOutput().get(1)))); + void testChainEliminate() { + Set<Set<Slot>> slotSet = Sets.newHashSet(set1, set2, set3, set4); + FuncDeps funcDeps = new FuncDeps(); + funcDeps.addFuncItems(Sets.newHashSet(s1), Sets.newHashSet(s2)); + funcDeps.addFuncItems(Sets.newHashSet(s2), Sets.newHashSet(s3)); + funcDeps.addFuncItems(Sets.newHashSet(s3), Sets.newHashSet(s4)); + Set<Set<Slot>> slots = funcDeps.eliminateDeps(slotSet); + Set<Set<Slot>> expected = new HashSet<>(); + expected.add(set1); + Assertions.assertEquals(expected, slots); } @Test - void testJoin() { - // inner join - Plan plan = PlanChecker.from(connectContext) - .analyze("select uni.id, agg.id, agg.id2 from agg join uni " - + "where agg.id = uni.id") - .rewrite() - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(0)), ImmutableSet.of(plan.getOutput().get(1)))); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(1)), ImmutableSet.of(plan.getOutput().get(0)))); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(1)), ImmutableSet.of(plan.getOutput().get(2)))); - - // foj - plan = PlanChecker.from(connectContext) - .analyze("select t1.id, t1.id2, t2.id, t2.id2 " - + "from uni as t1 full outer join uni as t2 on t1.id2 = t2.id2") - .rewrite() - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(0)), ImmutableSet.of(plan.getOutput().get(1)))); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(2)), ImmutableSet.of(plan.getOutput().get(3)))); - - // loj - plan = PlanChecker.from(connectContext) - .analyze("select t1.id, t1.id2, t2.id, t2.id2 " - + "from uni as t1 left outer join uni as t2 on t1.id2 = t2.id2") - .rewrite() - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(0)), ImmutableSet.of(plan.getOutput().get(1)))); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(2)), ImmutableSet.of(plan.getOutput().get(3)))); - - // roj - plan = PlanChecker.from(connectContext) - .analyze("select t1.id, t1.id2, t2.id, t2.id2 " - + "from uni as t1 right outer join uni as t2 on t1.id2 = t2.id2") - .rewrite() - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(0)), ImmutableSet.of(plan.getOutput().get(1)))); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(2)), ImmutableSet.of(plan.getOutput().get(3)))); + void testTreeEliminate() { + Set<Set<Slot>> slotSet = Sets.newHashSet(set1, set2, set3, set4); + FuncDeps funcDeps = new FuncDeps(); + funcDeps.addFuncItems(Sets.newHashSet(s1), Sets.newHashSet(s2)); + funcDeps.addFuncItems(Sets.newHashSet(s1), Sets.newHashSet(s3)); + funcDeps.addFuncItems(Sets.newHashSet(s1), Sets.newHashSet(s4)); + Set<Set<Slot>> slots = funcDeps.eliminateDeps(slotSet); + Set<Set<Slot>> expected = new HashSet<>(); + expected.add(set1); + Assertions.assertEquals(expected, slots); } @Test - void testOneRowRelation() { - Plan plan = PlanChecker.from(connectContext) - .analyze("select 1, 1") - .rewrite() - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(1)), ImmutableSet.of(plan.getOutput().get(0)))); + void testCircleEliminate1() { + Set<Set<Slot>> slotSet = Sets.newHashSet(set1, set2, set3, set4); + FuncDeps funcDeps = new FuncDeps(); + funcDeps.addFuncItems(Sets.newHashSet(s1), Sets.newHashSet(s2)); + funcDeps.addFuncItems(Sets.newHashSet(s2), Sets.newHashSet(s1)); + Set<Set<Slot>> slots = funcDeps.eliminateDeps(slotSet); + Set<Set<Slot>> expected = new HashSet<>(); + expected.add(set2); + expected.add(set3); + expected.add(set4); + Assertions.assertEquals(expected, slots); } @Test - void testProject() { - Plan plan = PlanChecker.from(connectContext) - .analyze("select id as o1, id as o2, id2 as o4, 1 as c1, 1 as c2 from uni where id = id2") - .rewrite() - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(1)), ImmutableSet.of(plan.getOutput().get(0)))); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(0)), ImmutableSet.of(plan.getOutput().get(1)))); + void testCircleEliminate2() { + Set<Set<Slot>> slotSet = Sets.newHashSet(set1, set2, set3, set4); + FuncDeps funcDeps = new FuncDeps(); + funcDeps.addFuncItems(Sets.newHashSet(s1), Sets.newHashSet(s2)); + funcDeps.addFuncItems(Sets.newHashSet(s2), Sets.newHashSet(s3)); + funcDeps.addFuncItems(Sets.newHashSet(s3), Sets.newHashSet(s4)); + funcDeps.addFuncItems(Sets.newHashSet(s4), Sets.newHashSet(s1)); + Set<Set<Slot>> slots = funcDeps.eliminateDeps(slotSet); + Set<Set<Slot>> expected = new HashSet<>(); + expected.add(set3); + Assertions.assertEquals(expected, slots); } @Test - void testSubQuery() { - Plan plan = PlanChecker.from(connectContext) - .analyze("select id, id2 from (select id, id2 from agg where id = id2) t") - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(0)), ImmutableSet.of(plan.getOutput().get(1)))); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(1)), ImmutableSet.of(plan.getOutput().get(0)))); + void testGraphEliminate1() { + Set<Set<Slot>> slotSet = Sets.newHashSet(set1, set2, set3, set4); + FuncDeps funcDeps = new FuncDeps(); + funcDeps.addFuncItems(Sets.newHashSet(s1), Sets.newHashSet(s2)); + funcDeps.addFuncItems(Sets.newHashSet(s1), Sets.newHashSet(s3)); + funcDeps.addFuncItems(Sets.newHashSet(s3), Sets.newHashSet(s4)); + Set<Set<Slot>> slots = funcDeps.eliminateDeps(slotSet); + Set<Set<Slot>> expected = new HashSet<>(); + expected.add(set1); + Assertions.assertEquals(expected, slots); } - - @Test - void testWindow() { - // partition by uniform - Plan plan = PlanChecker.from(connectContext) - .analyze("select id, id2, row_number() over(partition by id) from agg where id = id2") - .rewrite() - .getPlan(); - Assertions.assertTrue(plan.getLogicalProperties().getFunctionalDependencies() - .isDependent(ImmutableSet.of(plan.getOutput().get(1)), ImmutableSet.of(plan.getOutput().get(0)))); - } - } diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FunctionalDependenciesTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FunctionalDependenciesTest.java index 6d180168522..53e2542fb5f 100644 --- a/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FunctionalDependenciesTest.java +++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/FunctionalDependenciesTest.java @@ -281,7 +281,6 @@ class FunctionalDependenciesTest extends TestWithFeService { .getPlan(); LogicalPartitionTopN<?> ptopn = (LogicalPartitionTopN<?>) plan.child(0).child(0).child(0).child(0).child(0); System.out.println(ptopn.getLogicalProperties().getFunctionalDependencies()); - System.out.println(ptopn.getOutput()); Assertions.assertTrue(ptopn.getLogicalProperties() .getFunctionalDependencies().isUniformAndNotNull(ImmutableSet.copyOf(ptopn.getOutputSet()))); @@ -290,8 +289,9 @@ class FunctionalDependenciesTest extends TestWithFeService { .rewrite() .getPlan(); ptopn = (LogicalPartitionTopN<?>) plan.child(0).child(0).child(0).child(0).child(0); - Assertions.assertTrue(ptopn.getLogicalProperties() - .getFunctionalDependencies().isEmpty()); + + Assertions.assertFalse(ptopn.getLogicalProperties() + .getFunctionalDependencies().isUnique(plan.getOutputSet())); } @Test diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/EliminateGroupByKeyTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/EliminateGroupByKeyTest.java new file mode 100644 index 00000000000..ccf4f036cd3 --- /dev/null +++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/EliminateGroupByKeyTest.java @@ -0,0 +1,99 @@ +// 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.util.MemoPatternMatchSupported; +import org.apache.doris.nereids.util.PlanChecker; +import org.apache.doris.utframe.TestWithFeService; + +import org.junit.jupiter.api.Test; + +class EliminateGroupByKeyTest extends TestWithFeService implements MemoPatternMatchSupported { + @Override + protected void runBeforeAll() throws Exception { + createDatabase("test"); + createTable("create table test.t1 (\n" + + "id int not null,\n" + + "name varchar(128) not null)\n" + + "distributed by hash(id) buckets 10\n" + + "properties('replication_num' = '1');"); + createTable("create table test.uni (\n" + + "id int not null,\n" + + "name varchar(128) not null)\n" + + "UNIQUE KEY(id)\n" + + "distributed by hash(id) buckets 10\n" + + "properties('replication_num' = '1');"); + connectContext.setDatabase("test"); + connectContext.getSessionVariable().setDisableNereidsRules("PRUNE_EMPTY_PARTITION"); + } + + @Test + void testEliminateByUniform() { + PlanChecker.from(connectContext) + .analyze("select count(name) from t1 where id = 1 group by name, id") + .rewrite() + .printlnTree() + .matches(logicalAggregate().when(agg -> + agg.getGroupByExpressions().size() == 1 && agg.getGroupByExpressions().get(0).toSql().equals("name"))); + } + + @Test + void testEliminateByUnique() { + PlanChecker.from(connectContext) + .analyze("select count(t1.id) from uni as t1 cross join uni as t2 group by t1.name, t1.id") + .rewrite() + .printlnTree() + .matches(logicalAggregate().when(agg -> + agg.getGroupByExpressions().size() == 1 && agg.getGroupByExpressions().get(0).toSql().equals("id"))); + PlanChecker.from(connectContext) + .analyze("select count(t1.id) from uni as t1 cross join uni as t2 group by t1.name, t2.id") + .rewrite() + .printlnTree() + .matches(logicalAggregate().when(agg -> + agg.getGroupByExpressions().size() == 2)); + } + + @Test + void testEliminateByPk() throws Exception { + addConstraint("alter table t1 add constraint pk primary key (id)"); + PlanChecker.from(connectContext) + .analyze("select count(t1.id) from t1 as t1 cross join t1 as t2 group by t1.name, t1.id") + .rewrite() + .printlnTree() + .matches(logicalAggregate().when(agg -> + agg.getGroupByExpressions().size() == 1 && agg.getGroupByExpressions().get(0).toSql().equals("id"))); + PlanChecker.from(connectContext) + .analyze("select count(t1.id) from t1 as t1 cross join t1 as t2 group by t1.name, t2.id") + .rewrite() + .printlnTree() + .matches(logicalAggregate().when(agg -> + agg.getGroupByExpressions().size() == 2)); + dropConstraint("alter table t1 drop constraint pk"); + } + + @Test + void testEliminateByEqual() { + PlanChecker.from(connectContext) + .analyze("select count(t1.name) from t1 as t1 join t1 as t2 on t1.name = t2.name group by t1.name, t2.name") + .rewrite() + .printlnTree() + .matches(logicalAggregate().when(agg -> + agg.getGroupByExpressions().size() == 1 && agg.getGroupByExpressions().get(0).toSql().equals("name"))); + } + +} diff --git a/regression-test/suites/nereids_rules_p0/eliminate_gby_key/eliminate_gby_key.groovy b/regression-test/suites/nereids_rules_p0/eliminate_gby_key/eliminate_gby_key.groovy index 41a2f8f852c..0c79a5b6746 100644 --- a/regression-test/suites/nereids_rules_p0/eliminate_gby_key/eliminate_gby_key.groovy +++ b/regression-test/suites/nereids_rules_p0/eliminate_gby_key/eliminate_gby_key.groovy @@ -25,9 +25,9 @@ suite("eliminate_gby_key") { sql """ CREATE TABLE `t1` ( - `c1` int(20) DEFAULT NULL, - `c2` int(20) DEFAULT NULL, - `c3` int(20) DEFAULT NULL + `c1` int(20) NOT NULL, + `c2` int(20) NOT NULL, + `c3` int(20) NOT NULL ) DUPLICATE KEY (`c1`) DISTRIBUTED BY HASH(`c1`) BUCKETS 3 PROPERTIES("replication_num"="1"); @@ -35,9 +35,9 @@ suite("eliminate_gby_key") { sql """ CREATE TABLE `t2` ( - `c1` int(20) DEFAULT NULL, - `c2` varchar(20) DEFAULT NULL, - `c3` int(20) DEFAULT NULL + `c1` int(20) NOT NULL, + `c2` varchar(20) NOT NULL, + `c3` int(20) NOT NULL ) DUPLICATE KEY (`c1`) DISTRIBUTED BY HASH(`c1`) BUCKETS 3 PROPERTIES("replication_num"="1"); @@ -84,7 +84,7 @@ suite("eliminate_gby_key") { select t2_c2 from temp; """) - contains("groupByExpr=[c1#13, c3#18, t2_c2#19], outputExpr=[c1#13, c3#18, t2_c2#19]") + contains("groupByExpr=[t2_c2#19, c1#13, c3#18], outputExpr=[t2_c2#19, c1#13, c3#18]") } explain { @@ -144,7 +144,7 @@ suite("eliminate_gby_key") { select t2_c2, t2_c1 from temp; """) - contains("groupByExpr=[c1#13, c3#18, t2_c2#19], outputExpr=[c1#13, c3#18, t2_c2#19]") + contains("groupByExpr=[t2_c2#19, c1#13, c3#18], outputExpr=[t2_c2#19, c1#13, c3#18]") } explain { @@ -184,7 +184,7 @@ suite("eliminate_gby_key") { select c3, t2_c2 from temp; """) - contains("groupByExpr=[c1#13, c3#18, t2_c2#19], outputExpr=[c1#13, c3#18, t2_c2#19]") + contains("groupByExpr=[t2_c2#19, c1#13, c3#18], outputExpr=[t2_c2#19, c1#13, c3#18]") } explain { @@ -264,7 +264,7 @@ suite("eliminate_gby_key") { select t2_c2, c3, t2_c1 from temp; """) - contains("groupByExpr=[c1#13, c3#18, t2_c2#19], outputExpr=[c1#13, c3#18, t2_c2#19]") + contains("groupByExpr=[t2_c2#19, c1#13, c3#18], outputExpr=[t2_c2#19, c1#13, c3#18]") } explain { @@ -284,6 +284,6 @@ suite("eliminate_gby_key") { select t2_c2, c3, t2_c1, cnt from temp; """) - contains("groupByExpr=[c1#13, c3#18, t2_c2#19], outputExpr=[c1#13, c3#18, t2_c2#19, count(*) AS `cnt`#20]") + contains("groupByExpr=[t2_c2#19, c1#13, c3#18], outputExpr=[t2_c2#19, c1#13, c3#18, count(*) AS `cnt`#20]") } } --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org