This is an automated email from the ASF dual-hosted git repository. yiguolei pushed a commit to branch branch-2.1 in repository https://gitbox.apache.org/repos/asf/doris.git
The following commit(s) were added to refs/heads/branch-2.1 by this push: new 40d755a971a [Feat](nereids) add max/min filter push down rewrite rule (#39252) (#43671) 40d755a971a is described below commit 40d755a971aa685052b241a6433367534ef9576e Author: feiniaofeiafei <moail...@selectdb.com> AuthorDate: Wed Nov 13 11:55:39 2024 +0800 [Feat](nereids) add max/min filter push down rewrite rule (#39252) (#43671) cherry-pick #39252 to branch-2.1 --- .../org/apache/doris/nereids/rules/RuleSet.java | 2 + .../org/apache/doris/nereids/rules/RuleType.java | 1 + .../rules/rewrite/MaxMinFilterPushDown.java | 133 ++++++++++ .../rules/rewrite/MaxMinFilterPushDownTest.java | 115 ++++++++ .../eager_aggregate/push_down_max_through_join.out | 20 +- .../max_min_filter_push_down.out | 290 +++++++++++++++++++++ .../max_min_filter_push_down.groovy | 214 +++++++++++++++ 7 files changed, 765 insertions(+), 10 deletions(-) diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java index 26dfa1bfcb7..95f74f01571 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java @@ -97,6 +97,7 @@ import org.apache.doris.nereids.rules.implementation.LogicalWindowToPhysicalWind import org.apache.doris.nereids.rules.rewrite.ConvertOuterJoinToAntiJoin; import org.apache.doris.nereids.rules.rewrite.CreatePartitionTopNFromWindow; import org.apache.doris.nereids.rules.rewrite.EliminateOuterJoin; +import org.apache.doris.nereids.rules.rewrite.MaxMinFilterPushDown; import org.apache.doris.nereids.rules.rewrite.MergeFilters; import org.apache.doris.nereids.rules.rewrite.MergeGenerates; import org.apache.doris.nereids.rules.rewrite.MergeLimits; @@ -145,6 +146,7 @@ public class RuleSet { .build(); public static final List<RuleFactory> PUSH_DOWN_FILTERS = ImmutableList.of( + new MaxMinFilterPushDown(), new CreatePartitionTopNFromWindow(), new PushDownFilterThroughProject(), new PushDownFilterThroughSort(), 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 b2d84679f23..f6445ba5878 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 @@ -177,6 +177,7 @@ public enum RuleType { PUSH_DOWN_FILTER_THROUGH_CTE(RuleTypeClass.REWRITE), PUSH_DOWN_FILTER_THROUGH_CTE_ANCHOR(RuleTypeClass.REWRITE), + MAX_MIN_FILTER_PUSH_DOWN(RuleTypeClass.REWRITE), PUSH_DOWN_DISTINCT_THROUGH_JOIN(RuleTypeClass.REWRITE), diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/MaxMinFilterPushDown.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/MaxMinFilterPushDown.java new file mode 100644 index 00000000000..a54c3785b35 --- /dev/null +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/MaxMinFilterPushDown.java @@ -0,0 +1,133 @@ +// 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.annotation.DependsRules; +import org.apache.doris.nereids.rules.Rule; +import org.apache.doris.nereids.rules.RuleType; +import org.apache.doris.nereids.rules.expression.ExpressionRewrite; +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.GreaterThan; +import org.apache.doris.nereids.trees.expressions.GreaterThanEqual; +import org.apache.doris.nereids.trees.expressions.LessThan; +import org.apache.doris.nereids.trees.expressions.LessThanEqual; +import org.apache.doris.nereids.trees.expressions.NamedExpression; +import org.apache.doris.nereids.trees.expressions.SlotReference; +import org.apache.doris.nereids.trees.expressions.functions.agg.AggregateFunction; +import org.apache.doris.nereids.trees.expressions.functions.agg.Max; +import org.apache.doris.nereids.trees.expressions.functions.agg.Min; +import org.apache.doris.nereids.trees.expressions.literal.Literal; +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.LogicalFilter; +import org.apache.doris.nereids.util.ExpressionUtils; +import org.apache.doris.nereids.util.PlanUtils; + +import com.google.common.base.Preconditions; +import com.google.common.collect.ImmutableList; +import com.google.common.collect.ImmutableSet; + +import java.util.HashSet; +import java.util.List; +import java.util.Optional; +import java.util.Set; + +/** + * select id, max(a) from t group by id having max(a)>10; + * -> + * select id, max(a) from t where a>10 group by id; + * select id, min(a) from t group by id having min(a)<10; + * -> + * select id, min(a) from t where a<10 group by id; + */ +@DependsRules({ + ExpressionRewrite.class +}) +public class MaxMinFilterPushDown extends OneRewriteRuleFactory { + @Override + public Rule build() { + return logicalFilter(logicalAggregate().whenNot(agg -> agg.getGroupByExpressions().isEmpty())) + .then(this::pushDownMaxMinFilter) + .toRule(RuleType.MAX_MIN_FILTER_PUSH_DOWN); + } + + private Plan pushDownMaxMinFilter(LogicalFilter<LogicalAggregate<Plan>> filter) { + Set<Expression> conjuncts = filter.getConjuncts(); + LogicalAggregate<Plan> agg = filter.child(); + Plan aggChild = agg.child(); + List<NamedExpression> aggOutputExpressions = agg.getOutputExpressions(); + Set<Expression> aggFuncs = ExpressionUtils.collect(aggOutputExpressions, + expr -> expr instanceof AggregateFunction); + Set<Expression> maxMinFunc = ExpressionUtils.collect(aggFuncs, + expr -> expr instanceof Max || expr instanceof Min); + // LogicalAggregate only outputs one aggregate function, which is max or min + if (aggFuncs.size() != 1 || maxMinFunc.size() != 1) { + return null; + } + ExprId exprId = null; + Expression func = maxMinFunc.iterator().next(); + for (NamedExpression expr : aggOutputExpressions) { + if (expr instanceof Alias && ((Alias) expr).child().equals(func)) { + Alias alias = (Alias) expr; + exprId = alias.getExprId(); + } + } + // try to find min(a)<10 or max(a)>10 + Expression originConjunct = findMatchingConjunct(conjuncts, func instanceof Max, exprId).orElse(null); + if (null == originConjunct) { + return null; + } + Set<Expression> newUpperConjuncts = new HashSet<>(conjuncts); + newUpperConjuncts.remove(originConjunct); + Expression newPredicate = null; + if (func instanceof Max) { + if (originConjunct instanceof GreaterThan) { + newPredicate = new GreaterThan(func.child(0), originConjunct.child(1)); + } else if (originConjunct instanceof GreaterThanEqual) { + newPredicate = new GreaterThanEqual(func.child(0), originConjunct.child(1)); + } + } else { + if (originConjunct instanceof LessThan) { + newPredicate = new LessThan(func.child(0), originConjunct.child(1)); + } else if (originConjunct instanceof LessThanEqual) { + newPredicate = new LessThanEqual(func.child(0), originConjunct.child(1)); + } + } + Preconditions.checkState(newPredicate != null, "newPredicate is null"); + LogicalFilter<Plan> newPushDownFilter = new LogicalFilter<>(ImmutableSet.of(newPredicate), aggChild); + LogicalAggregate<Plan> newAgg = agg.withChildren(ImmutableList.of(newPushDownFilter)); + return PlanUtils.filterOrSelf(newUpperConjuncts, newAgg); + } + + private Optional<Expression> findMatchingConjunct(Set<Expression> conjuncts, boolean isMax, ExprId exprId) { + for (Expression conjunct : conjuncts) { + if ((isMax && (conjunct instanceof GreaterThan || conjunct instanceof GreaterThanEqual)) + || (!isMax && (conjunct instanceof LessThan || conjunct instanceof LessThanEqual))) { + if (conjunct.child(0) instanceof SlotReference && conjunct.child(1) instanceof Literal) { + SlotReference slot = (SlotReference) conjunct.child(0); + if (slot.getExprId().equals(exprId)) { + return Optional.of(conjunct); + } + } + } + } + return Optional.empty(); + } +} diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/MaxMinFilterPushDownTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/MaxMinFilterPushDownTest.java new file mode 100644 index 00000000000..bc7d32fb3fb --- /dev/null +++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/MaxMinFilterPushDownTest.java @@ -0,0 +1,115 @@ +// 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; + +public class MaxMinFilterPushDownTest extends TestWithFeService implements MemoPatternMatchSupported { + @Override + protected void runBeforeAll() throws Exception { + createDatabase("test"); + connectContext.setDatabase("test"); + createTable("CREATE TABLE IF NOT EXISTS max_t(\n" + + "`id` int(32),\n" + + "`score` int(64) NULL,\n" + + "`name` varchar(64) NULL\n" + + ") properties('replication_num'='1');"); + connectContext.getSessionVariable().setDisableNereidsRules("PRUNE_EMPTY_PARTITION"); + } + + @Test + public void testMaxRewrite() { + String sql = "select id, max(score) from max_t group by id having max(score)>10"; + PlanChecker.from(connectContext).analyze(sql).rewrite() + .matches(logicalFilter(logicalOlapScan()).when(filter -> filter.getConjuncts().size() == 1)); + } + + @Test + public void testMinRewrite() { + String sql = "select id, min(score) from max_t group by id having min(score)<10"; + PlanChecker.from(connectContext).analyze(sql).rewrite() + .matches(logicalFilter(logicalOlapScan()).when(filter -> filter.getConjuncts().size() == 1)); + } + + @Test + public void testNotRewriteBecauseFuncIsMoreThanOne1() { + String sql = "select id, min(score), max(name) from max_t group by id having min(score)<10 and max(name)>'abc'"; + PlanChecker.from(connectContext).analyze(sql).rewrite() + .nonMatch(logicalFilter(logicalOlapScan())); + } + + @Test + public void testNotRewriteBecauseFuncIsMoreThanOne2() { + String sql = "select id, min(score), min(name) from max_t group by id having min(score)<10 and min(name)<'abc'"; + PlanChecker.from(connectContext).analyze(sql).rewrite() + .nonMatch(logicalFilter(logicalOlapScan())); + } + + @Test + public void testMaxNotRewriteBecauseLessThan() { + String sql = "select id, max(score) from max_t group by id having max(score)<10"; + PlanChecker.from(connectContext).analyze(sql).rewrite() + .nonMatch(logicalFilter(logicalOlapScan())); + } + + @Test + public void testMinNotRewriteBecauseGreaterThan() { + String sql = "select id, min(score) from max_t group by id having min(score)>10"; + PlanChecker.from(connectContext).analyze(sql).rewrite() + .nonMatch(logicalFilter(logicalOlapScan())); + } + + @Test + public void testMinNotRewriteBecauseHasMaxFunc() { + String sql = "select id, min(score), max(score) from max_t group by id having min(score)<10"; + PlanChecker.from(connectContext).analyze(sql).rewrite() + .nonMatch(logicalFilter(logicalOlapScan())); + } + + @Test + public void testMinNotRewriteBecauseHasCountFunc() { + String sql = "select id, min(score), count(score) from max_t group by id having min(score)<10"; + PlanChecker.from(connectContext).analyze(sql).rewrite() + .nonMatch(logicalFilter(logicalOlapScan())); + } + + @Test + public void testNotRewriteBecauseConjunctLeftNotSlot() { + String sql = "select id, max(score) from max_t group by id having abs(max(score))>10"; + PlanChecker.from(connectContext).analyze(sql).rewrite() + .nonMatch(logicalFilter(logicalOlapScan())); + } + + @Test + public void testRewriteAggFuncHasExpr() { + String sql = "select id, max(score+1) from max_t group by id having max(score+1)>10"; + PlanChecker.from(connectContext).analyze(sql).rewrite() + .matches(logicalFilter(logicalOlapScan()).when(filter -> filter.getConjuncts().size() == 1)); + } + + @Test + public void testNotRewriteScalarAgg() { + String sql = "select max(score+1) from max_t having max(score+1)>10"; + PlanChecker.from(connectContext).analyze(sql).rewrite() + .nonMatch(logicalFilter(logicalOlapScan())); + } +} diff --git a/regression-test/data/nereids_rules_p0/eager_aggregate/push_down_max_through_join.out b/regression-test/data/nereids_rules_p0/eager_aggregate/push_down_max_through_join.out index 9b4e17b4369..597aa30c509 100644 --- a/regression-test/data/nereids_rules_p0/eager_aggregate/push_down_max_through_join.out +++ b/regression-test/data/nereids_rules_p0/eager_aggregate/push_down_max_through_join.out @@ -91,12 +91,12 @@ PhysicalResultSink -- !groupby_pushdown_having -- PhysicalResultSink ---filter((max(score) > 100)) -----hashAgg[GLOBAL] -------hashAgg[LOCAL] ---------hashJoin[INNER_JOIN] hashCondition=((t1.id = t2.id)) otherCondition=() -----------PhysicalOlapScan[max_t] +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------hashJoin[INNER_JOIN] hashCondition=((t1.id = t2.id)) otherCondition=() +--------filter((t1.score > 100)) ----------PhysicalOlapScan[max_t] +--------PhysicalOlapScan[max_t] -- !groupby_pushdown_mixed_aggregates -- PhysicalResultSink @@ -366,12 +366,12 @@ SyntaxError: -- !with_hint_groupby_pushdown_having -- PhysicalResultSink ---filter((max(score) > 100)) -----hashAgg[GLOBAL] -------hashAgg[LOCAL] ---------hashJoin[INNER_JOIN] hashCondition=((t1.id = t2.id)) otherCondition=() -----------PhysicalOlapScan[max_t] +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------hashJoin[INNER_JOIN] hashCondition=((t1.id = t2.id)) otherCondition=() +--------filter((t1.score > 100)) ----------PhysicalOlapScan[max_t] +--------PhysicalOlapScan[max_t] Hint log: Used: diff --git a/regression-test/data/nereids_rules_p0/max_min_filter_push_down/max_min_filter_push_down.out b/regression-test/data/nereids_rules_p0/max_min_filter_push_down/max_min_filter_push_down.out new file mode 100644 index 00000000000..2e0ac41d5eb --- /dev/null +++ b/regression-test/data/nereids_rules_p0/max_min_filter_push_down/max_min_filter_push_down.out @@ -0,0 +1,290 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !scalar_agg_empty_table -- +PhysicalResultSink +--filter((min(value1) < 20)) +----hashAgg[GLOBAL] +------hashAgg[LOCAL] +--------PhysicalEmptyRelation + +-- !min -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down1.value1 < 20)) +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !max -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down1.value1 > 40)) +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !min_expr -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((cast(value1 as BIGINT) < 19)) +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !max_expr -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((abs(value1) > 39)) +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !min_commute -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down1.value1 < 40)) +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !max -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down1.value1 > 40)) +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !min_equal -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down1.value1 <= 20)) +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !max_equal -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down1.value1 >= 40)) +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !min_commute_equal -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down1.value1 <= 40)) +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !max_commute_equal -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down1.value1 >= 40)) +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !has_other_agg_func -- +PhysicalResultSink +--filter((max(value1) >= 40)) +----hashAgg[GLOBAL] +------hashAgg[LOCAL] +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !min_scalar_agg -- +PhysicalResultSink +--filter((min(value1) < 40)) +----hashAgg[GLOBAL] +------hashAgg[LOCAL] +--------PhysicalStorageLayerAggregate[max_min_filter_push_down1] + +-- !max_scalar_agg -- +PhysicalResultSink +--filter((max(value1) > 40)) +----hashAgg[GLOBAL] +------hashAgg[LOCAL] +--------PhysicalStorageLayerAggregate[max_min_filter_push_down1] + +-- !max_scalar_agg -- +PhysicalResultSink +--filter((max(value1) > 40)) +----hashAgg[GLOBAL] +------hashAgg[LOCAL] +--------PhysicalStorageLayerAggregate[max_min_filter_push_down1] + +-- !min_equal_scalar_agg -- +PhysicalResultSink +--filter((min(value1) <= 20)) +----hashAgg[GLOBAL] +------hashAgg[LOCAL] +--------PhysicalStorageLayerAggregate[max_min_filter_push_down1] + +-- !max_equal_scalar_agg -- +PhysicalResultSink +--filter((max(value1) >= 40)) +----hashAgg[GLOBAL] +------hashAgg[LOCAL] +--------PhysicalStorageLayerAggregate[max_min_filter_push_down1] + +-- !depend_prune_column -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down1.value1 < 10)) +--------PhysicalOlapScan[max_min_filter_push_down1] + +-- !scalar_agg_empty_table_res -- + +-- !min_res -- +1 10 +2 19 + +-- !max_res -- +2 73 +3 61 +4 45 + +-- !min_expr_res -- +1 11 + +-- !max_expr_res -- +2 74 +3 62 +4 46 + +-- !min_commute_res -- +1 10 +2 19 +3 30 + +-- !max_res -- +2 73 +3 61 +4 45 + +-- !min_equal_res -- +1 10 +2 19 + +-- !max_equal_res -- +2 73 +3 61 +4 45 + +-- !min_commute_equal_res -- +1 10 +2 19 +3 30 +4 40 + +-- !max_commute_equal_res -- +2 73 +3 61 +4 45 + +-- !has_other_agg_func_res -- +2 73 19 +3 61 30 +4 45 40 + +-- !min_scalar_agg_res -- +10 + +-- !max_scalar_agg_res -- +73 + +-- !max_scalar_agg_res -- +73 + +-- !min_equal_scalar_agg_res -- +10 + +-- !max_equal_scalar_agg_res -- +73 + +-- !depend_prune_column_res -- +10 +19 + +-- !smallint -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down2.d_smallint > 10)) +--------PhysicalOlapScan[max_min_filter_push_down2] + +-- !tinyint -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down2.d_tinyint < 10)) +--------PhysicalOlapScan[max_min_filter_push_down2] + +-- !char100 -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down2.d_char100 > 'ab')) +--------PhysicalOlapScan[max_min_filter_push_down2] + +-- !char100_cmp_num_cannot_rewrite -- +PhysicalResultSink +--filter((cast(min(d_char100) as DOUBLE) < 10.0)) +----hashAgg[GLOBAL] +------hashAgg[LOCAL] +--------PhysicalOlapScan[max_min_filter_push_down2] + +-- !datetimev2 -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down2.d_datetimev2 < '2020-01-09 00:00:00')) +--------PhysicalOlapScan[max_min_filter_push_down2] + +-- !datev2 -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down2.d_datev2 > '2020-01-09')) +--------PhysicalOlapScan[max_min_filter_push_down2] + +-- !smallint_group_by_key -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down2.d_smallint > 10)) +--------PhysicalOlapScan[max_min_filter_push_down2] + +-- !tinyint_group_by_key -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down2.d_tinyint < 10)) +--------PhysicalOlapScan[max_min_filter_push_down2] + +-- !char100_group_by_key -- +PhysicalResultSink +--hashAgg[GLOBAL] +----hashAgg[LOCAL] +------filter((max_min_filter_push_down2.d_char100 > 'ab')) +--------PhysicalOlapScan[max_min_filter_push_down2] + +-- !smallint_res -- +14 32 + +-- !tinyint_res -- +1 3 + +-- !char100_res -- + +-- !char100_cmp_num_cannot_rewrite_res -- + +-- !datetimev2_res -- +1 2020-01-07T10:00:01 +14 2020-01-07T10:00:01 + +-- !datev2_res -- +1 2020-01-11 +14 2020-01-11 + +-- !smallint_group_by_key_res -- +29 +32 + +-- !tinyint_group_by_key_res -- +3 + +-- !char100_group_by_key_res -- + diff --git a/regression-test/suites/nereids_rules_p0/max_min_filter_push_down/max_min_filter_push_down.groovy b/regression-test/suites/nereids_rules_p0/max_min_filter_push_down/max_min_filter_push_down.groovy new file mode 100644 index 00000000000..47610f2e125 --- /dev/null +++ b/regression-test/suites/nereids_rules_p0/max_min_filter_push_down/max_min_filter_push_down.groovy @@ -0,0 +1,214 @@ +// 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("max_min_filter_push_down") { + sql "SET enable_nereids_planner=true" + sql "SET enable_fallback_to_original_planner=false" + sql "SET ignore_shape_nodes='PhysicalDistribute,PhysicalProject'" + + sql "drop table if exists max_min_filter_push_down1" + sql""" + CREATE TABLE max_min_filter_push_down1 ( + id INT, + value1 INT, + value2 VARCHAR(50) + ) properties("replication_num"="1"); + """ + + sql """ + INSERT INTO max_min_filter_push_down1 (id, value1, value2) VALUES + (1, 10, 'A'),(1, 11, 'A'),(2, 20, 'B'),(2, 73, 'B'),(2, 19, 'B'),(3, 30, 'C'),(3, 61, 'C'),(4, 40, 'D'),(4, 43, 'D'),(4, 45, 'D'); + """ + sql "drop table if exists max_min_filter_push_down_empty" + sql "create table max_min_filter_push_down_empty like max_min_filter_push_down1" + + qt_scalar_agg_empty_table """ + explain shape plan + select min(value1) from max_min_filter_push_down_empty having min(value1) <40 and min(value1) <20; + """ + qt_min """ + explain shape plan + select id,min(value1) from max_min_filter_push_down1 group by id having min(value1) <40 and min(value1) <20; + """ + qt_max """ + explain shape plan + select id,max(value1) from max_min_filter_push_down1 group by id having max(value1) >40; + """ + + qt_min_expr """ + explain shape plan + select id,min(value1+1) from max_min_filter_push_down1 group by id having min(value1+1) <40 and min(value1+1) <20; + """ + qt_max_expr """ + explain shape plan + select id,max(abs(value1)+1) from max_min_filter_push_down1 group by id having max(abs(value1)+1) >40; + """ + + qt_min_commute """ + explain shape plan + select id,min(value1) from max_min_filter_push_down1 group by id having 40>min(value1); + """ + qt_max """ + explain shape plan + select id,max(value1) from max_min_filter_push_down1 group by id having 40<max(value1); + """ + + qt_min_equal """ + explain shape plan + select id,min(value1) from max_min_filter_push_down1 group by id having min(value1) <=40 and min(value1) <=20; + """ + qt_max_equal """ + explain shape plan + select id,max(value1) from max_min_filter_push_down1 group by id having max(value1) >=40; + """ + + qt_min_commute_equal """ + explain shape plan + select id,min(value1) from max_min_filter_push_down1 group by id having 40>=min(value1); + """ + qt_max_commute_equal """ + explain shape plan + select id,max(value1) from max_min_filter_push_down1 group by id having 40<=max(value1); + """ + + qt_has_other_agg_func """ + explain shape plan + select id,max(value1),min(value1) from max_min_filter_push_down1 group by id having 40<=max(value1); + """ + + qt_min_scalar_agg """ + explain shape plan + select min(value1) from max_min_filter_push_down1 having min(value1) <40; + """ + qt_max_scalar_agg """ + explain shape plan + select max(value1) from max_min_filter_push_down1 having max(value1) >40; + """ + qt_max_scalar_agg """ + explain shape plan + select max(value1) from max_min_filter_push_down1 having 40<max(value1); + """ + + qt_min_equal_scalar_agg """ + explain shape plan + select min(value1) from max_min_filter_push_down1 having min(value1) <=40 and min(value1) <=20; + """ + qt_max_equal_scalar_agg """ + explain shape plan + select max(value1) from max_min_filter_push_down1 having max(value1) >=40; + """ + + qt_depend_prune_column """ + explain shape plan + select c1 from (select min(value1) c1,max(value2) from max_min_filter_push_down1 group by id having min(value1)<10) t + """ + + qt_scalar_agg_empty_table_res """ + select min(value1) from max_min_filter_push_down_empty having min(value1) <40 and min(value1) <20; + """ + qt_min_res """ + select id,min(value1) from max_min_filter_push_down1 group by id having min(value1) <40 and min(value1) <20 order by 1,2; + """ + qt_max_res """ + select id,max(value1) from max_min_filter_push_down1 group by id having max(value1) >40 order by 1,2; + """ + qt_min_expr_res """ + select id,min(value1+1) from max_min_filter_push_down1 group by id having min(value1+1) <40 and min(value1+1) <20 order by 1,2; + """ + qt_max_expr_res """ + select id,max(abs(value1)+1) from max_min_filter_push_down1 group by id having max(abs(value1)+1) >40 order by 1,2; + """ + qt_min_commute_res """ + select id,min(value1) from max_min_filter_push_down1 group by id having 40>min(value1) order by 1,2; + """ + qt_max_res """ + select id,max(value1) from max_min_filter_push_down1 group by id having 40<max(value1) order by 1,2; + """ + + qt_min_equal_res """ + select id,min(value1) from max_min_filter_push_down1 group by id having min(value1) <=40 and min(value1) <=20 order by 1,2; + """ + qt_max_equal_res """ + select id,max(value1) from max_min_filter_push_down1 group by id having max(value1) >=40 order by 1,2; + """ + + qt_min_commute_equal_res """ + select id,min(value1) from max_min_filter_push_down1 group by id having 40>=min(value1) order by 1,2; + """ + qt_max_commute_equal_res """ + select id,max(value1) from max_min_filter_push_down1 group by id having 40<=max(value1) order by 1,2; + """ + + qt_has_other_agg_func_res """ + select id,max(value1),min(value1) from max_min_filter_push_down1 group by id having 40<=max(value1) order by 1,2; + """ + + qt_min_scalar_agg_res """ + select min(value1) from max_min_filter_push_down1 having min(value1) <40; + """ + qt_max_scalar_agg_res """ + select max(value1) from max_min_filter_push_down1 having max(value1) >40; + """ + qt_max_scalar_agg_res """ + select max(value1) from max_min_filter_push_down1 having 40<max(value1); + """ + + qt_min_equal_scalar_agg_res """ + select min(value1) from max_min_filter_push_down1 having min(value1) <=40 and min(value1) <=20; + """ + qt_max_equal_scalar_agg_res """ + select max(value1) from max_min_filter_push_down1 having max(value1) >=40; + """ + qt_depend_prune_column_res """ + select c1 from (select min(value1) c1,max(value2) from max_min_filter_push_down1 group by id having min(value1)<20) t order by c1 + """ + + sql "drop table if exists max_min_filter_push_down2" + sql """create table max_min_filter_push_down2(d_int int, d_char100 char(100), d_smallint smallint, d_tinyint tinyint, d_char10 char(10),d_datetimev2 datetimev2, d_datev2 datev2) + properties("replication_num"="1");""" + sql """insert into max_min_filter_push_down2 values(1,'01234567890123456789', 3,3,'0123456789','2020-01-09 10:00:00.99','2020-01-09') + ,(14,'01234567890123456789', 29,23,'0123456789','2020-01-7 10:00:00.99','2020-01-11'),(1,'01234567890123456789', 7,23,'0123456789','2020-01-7 10:00:00.99','2020-01-11') + ,(14,'01234567890123456789', 32,23,'0123456789','2020-01-11 10:00:00.99','2020-01-11'),(1,'01234567890123456789', 8,23,'0123456789','2020-01-11 10:00:00.99','2020-01-11');""" + + qt_smallint """explain shape plan + select d_int,max(d_smallint) from max_min_filter_push_down2 group by d_int having max(d_smallint)>10;""" + qt_tinyint """explain shape plan + select d_int,min(d_tinyint) from max_min_filter_push_down2 group by d_int having min(d_tinyint)<10;""" + qt_char100 """explain shape plan + select d_int,max(d_char100) from max_min_filter_push_down2 group by d_int having max(d_char100)>'ab';""" + qt_char100_cmp_num_cannot_rewrite """explain shape plan + select d_int,min(d_char100) from max_min_filter_push_down2 group by d_int having min(d_char100)<10;""" + qt_datetimev2 """explain shape plan + select d_int,min(d_datetimev2) from max_min_filter_push_down2 group by d_int having min(d_datetimev2)<'2020-01-09';""" + qt_datev2 """explain shape plan + select d_int,max(d_datev2) from max_min_filter_push_down2 group by d_int having max(d_datev2)>'2020-01-09 10:00:00';""" + qt_smallint_group_by_key """explain shape plan + select max(d_smallint) from max_min_filter_push_down2 group by d_smallint having max(d_smallint)>10;""" + qt_tinyint_group_by_key """explain shape plan + select min(d_tinyint) from max_min_filter_push_down2 group by d_tinyint having min(d_tinyint)<10;""" + qt_char100_group_by_key """explain shape plan + select max(d_char100) from max_min_filter_push_down2 group by d_char100 having max(d_char100)>'ab';""" + + qt_smallint_res """select d_int,max(d_smallint) from max_min_filter_push_down2 group by d_int having max(d_smallint)>10 order by 1,2;""" + qt_tinyint_res """select d_int,min(d_tinyint) from max_min_filter_push_down2 group by d_int having min(d_tinyint)<10 order by 1,2;""" + qt_char100_res """select d_int,max(d_char100) from max_min_filter_push_down2 group by d_int having max(d_char100)>'ab' order by 1,2;""" + qt_char100_cmp_num_cannot_rewrite_res """select d_int,min(d_char100) from max_min_filter_push_down2 group by d_int having min(d_char100)<10 order by 1,2;""" + qt_datetimev2_res """select d_int,min(d_datetimev2) from max_min_filter_push_down2 group by d_int having min(d_datetimev2)<'2020-01-09' order by 1,2;""" + qt_datev2_res """select d_int,max(d_datev2) from max_min_filter_push_down2 group by d_int having max(d_datev2)>'2020-01-09 10:00:00' order by 1,2;""" + qt_smallint_group_by_key_res """select max(d_smallint) from max_min_filter_push_down2 group by d_smallint having max(d_smallint)>10 order by 1;""" + qt_tinyint_group_by_key_res """select min(d_tinyint) from max_min_filter_push_down2 group by d_tinyint having min(d_tinyint)<10 order by 1;""" + qt_char100_group_by_key_res """select max(d_char100) from max_min_filter_push_down2 group by d_char100 having max(d_char100)>'ab' order by 1;""" +} \ No newline at end of file --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org