This is an automated email from the ASF dual-hosted git repository.

morrysnow pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/doris.git


The following commit(s) were added to refs/heads/master by this push:
     new b51f6ae050 [feature](Nereids)add rule: PruneOlapScanTablet (#14378)
b51f6ae050 is described below

commit b51f6ae0500f4bab7971c8557d3ac039ef8b3e9a
Author: mch_ucchi <41606806+sohardforan...@users.noreply.github.com>
AuthorDate: Tue Nov 29 01:06:14 2022 +0800

    [feature](Nereids)add rule: PruneOlapScanTablet (#14378)
---
 .../glue/translator/PhysicalPlanTranslator.java    |   1 +
 .../jobs/batch/NereidsRewriteJobExecutor.java      |   2 +
 .../org/apache/doris/nereids/rules/RuleType.java   |   1 +
 .../rewrite/logical/PruneOlapScanPartition.java    |   2 +-
 .../rules/rewrite/logical/PruneOlapScanTablet.java |  84 +++++++++++
 .../visitor/ExpressionColumnFilterConverter.java   | 134 +++++++++++++++++
 .../trees/plans/logical/LogicalOlapScan.java       |  50 +++++--
 .../apache/doris/nereids/util/ExpressionUtils.java |  30 ++++
 .../doris/planner/HashDistributionPruner.java      |   2 +-
 .../rewrite/logical/PruneOlapScanTabletTest.java   | 164 +++++++++++++++++++++
 10 files changed, 452 insertions(+), 18 deletions(-)

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 3faf6b2598..ea368bfedf 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
@@ -393,6 +393,7 @@ public class PhysicalPlanTranslator extends 
DefaultPlanVisitor<PlanFragment, Pla
         BaseTableRef tableRef = new BaseTableRef(ref, olapTable, tableName);
         tupleDescriptor.setRef(tableRef);
         
olapScanNode.setSelectedPartitionIds(olapScan.getSelectedPartitionIds());
+        olapScanNode.setSampleTabletIds(olapScan.getSelectedTabletIds());
 
         switch (olapScan.getTable().getKeysType()) {
             case AGG_KEYS:
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/batch/NereidsRewriteJobExecutor.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/batch/NereidsRewriteJobExecutor.java
index 0ebe87f22e..cfd777ad2a 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/batch/NereidsRewriteJobExecutor.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/batch/NereidsRewriteJobExecutor.java
@@ -38,6 +38,7 @@ import 
org.apache.doris.nereids.rules.rewrite.logical.LimitPushDown;
 import org.apache.doris.nereids.rules.rewrite.logical.MergeProjects;
 import org.apache.doris.nereids.rules.rewrite.logical.NormalizeAggregate;
 import org.apache.doris.nereids.rules.rewrite.logical.PruneOlapScanPartition;
+import org.apache.doris.nereids.rules.rewrite.logical.PruneOlapScanTablet;
 import org.apache.doris.nereids.rules.rewrite.logical.PushFilterInsideJoin;
 import org.apache.doris.nereids.rules.rewrite.logical.ReorderJoin;
 
@@ -88,6 +89,7 @@ public class NereidsRewriteJobExecutor extends BatchRulesJob {
                 .add(topDownBatch(ImmutableList.of(new 
PruneOlapScanPartition())))
                 .add(topDownBatch(ImmutableList.of(new 
SelectMaterializedIndexWithAggregate())))
                 .add(topDownBatch(ImmutableList.of(new 
SelectMaterializedIndexWithoutAggregate())))
+                .add(topDownBatch(ImmutableList.of(new PruneOlapScanTablet())))
                 // we need to execute this rule at the end of rewrite
                 // to avoid two consecutive same project appear when we do 
optimization.
                 .add(topDownBatch(ImmutableList.of(new 
EliminateGroupByConstant())))
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 5160fc2b4d..32bb60b8b3 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
@@ -136,6 +136,7 @@ public enum RuleType {
     MATERIALIZED_INDEX_PROJECT_FILTER_SCAN(RuleTypeClass.REWRITE),
     MATERIALIZED_INDEX_FILTER_PROJECT_SCAN(RuleTypeClass.REWRITE),
     OLAP_SCAN_PARTITION_PRUNE(RuleTypeClass.REWRITE),
+    OLAP_SCAN_TABLET_PRUNE(RuleTypeClass.REWRITE),
     EXTRACT_SINGLE_TABLE_EXPRESSION_FROM_DISJUNCTION(RuleTypeClass.REWRITE),
     REWRITE_SENTINEL(RuleTypeClass.REWRITE),
 
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/logical/PruneOlapScanPartition.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/logical/PruneOlapScanPartition.java
index 018074dcfd..b6e8887f83 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/logical/PruneOlapScanPartition.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/logical/PruneOlapScanPartition.java
@@ -89,7 +89,7 @@ public class PruneOlapScanPartition extends 
OneRewriteRuleFactory {
                     partitionInfo.getPartitionColumns(), columnNameToRange);
             Collection<Long> selectedPartitionId = 
Utils.execWithReturnVal(partitionPruner::prune);
             LogicalOlapScan rewrittenScan =
-                    scan.withSelectedPartitionId(new 
ArrayList<>(selectedPartitionId));
+                    scan.withSelectedPartitionIds(new 
ArrayList<>(selectedPartitionId));
             return new LogicalFilter<>(filter.getPredicates(), rewrittenScan);
         }).toRule(RuleType.OLAP_SCAN_PARTITION_PRUNE);
     }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/logical/PruneOlapScanTablet.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/logical/PruneOlapScanTablet.java
new file mode 100644
index 0000000000..0251a8ccf5
--- /dev/null
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/logical/PruneOlapScanTablet.java
@@ -0,0 +1,84 @@
+// 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.logical;
+
+import org.apache.doris.catalog.DistributionInfo;
+import org.apache.doris.catalog.DistributionInfo.DistributionInfoType;
+import org.apache.doris.catalog.HashDistributionInfo;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.nereids.rules.Rule;
+import org.apache.doris.nereids.rules.RuleType;
+import org.apache.doris.nereids.rules.rewrite.OneRewriteRuleFactory;
+import org.apache.doris.nereids.trees.expressions.Expression;
+import 
org.apache.doris.nereids.trees.expressions.visitor.ExpressionColumnFilterConverter;
+import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan;
+import org.apache.doris.nereids.util.ExpressionUtils;
+import org.apache.doris.planner.HashDistributionPruner;
+import org.apache.doris.planner.PartitionColumnFilter;
+
+import com.google.common.collect.ImmutableList;
+import com.google.common.collect.Lists;
+import com.google.common.collect.Maps;
+
+import java.util.Collection;
+import java.util.List;
+import java.util.Map;
+import java.util.Optional;
+
+/**
+ * prune bucket
+ */
+public class PruneOlapScanTablet extends OneRewriteRuleFactory {
+
+    @Override
+    public Rule build() {
+        return logicalFilter(logicalOlapScan())
+                .when(filter -> !filter.child().isTabletPruned())
+                .then(filter -> {
+                    LogicalOlapScan olapScan = filter.child();
+                    OlapTable table = olapScan.getTable();
+                    List<Long> selectedTabletIds = Lists.newArrayList();
+                    for (Long id : olapScan.getSelectedPartitionIds()) {
+                        Partition partition = table.getPartition(id);
+                        MaterializedIndex index = 
partition.getIndex(olapScan.getSelectedIndexId());
+                        
selectedTabletIds.addAll(getSelectedTabletIds(filter.getConjuncts(),
+                                index, partition.getDistributionInfo()));
+                    }
+                    return 
filter.withChildren(olapScan.withSelectedTabletIds(ImmutableList.copyOf(selectedTabletIds)));
+                }).toRule(RuleType.OLAP_SCAN_TABLET_PRUNE);
+    }
+
+    private Collection<Long> getSelectedTabletIds(List<Expression> expressions,
+            MaterializedIndex index, DistributionInfo info) {
+        if (info.getType() != DistributionInfoType.HASH) {
+            return index.getTabletIdsInOrder();
+        }
+        HashDistributionInfo hashInfo = (HashDistributionInfo) info;
+        Map<String, PartitionColumnFilter> filterMap = Maps.newHashMap();
+        
expressions.stream().map(ExpressionUtils::checkAndMaybeCommute).filter(Optional::isPresent)
+                        .forEach(expr -> new 
ExpressionColumnFilterConverter(filterMap).convert(expr.get()));
+        return new HashDistributionPruner(index.getTabletIdsInOrder(),
+                hashInfo.getDistributionColumns(),
+                filterMap,
+                hashInfo.getBucketNum()
+                ).prune();
+    }
+}
+
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/visitor/ExpressionColumnFilterConverter.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/visitor/ExpressionColumnFilterConverter.java
new file mode 100644
index 0000000000..be575f73c9
--- /dev/null
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/visitor/ExpressionColumnFilterConverter.java
@@ -0,0 +1,134 @@
+// 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.visitor;
+
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.LiteralExpr;
+import org.apache.doris.analysis.NullLiteral;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.nereids.trees.expressions.ComparisonPredicate;
+import org.apache.doris.nereids.trees.expressions.EqualTo;
+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.InPredicate;
+import org.apache.doris.nereids.trees.expressions.IsNull;
+import org.apache.doris.nereids.trees.expressions.LessThan;
+import org.apache.doris.nereids.trees.expressions.LessThanEqual;
+import org.apache.doris.nereids.trees.expressions.NullSafeEqual;
+import org.apache.doris.nereids.trees.expressions.Slot;
+import org.apache.doris.nereids.trees.expressions.literal.Literal;
+import org.apache.doris.planner.PartitionColumnFilter;
+
+import java.util.List;
+import java.util.Map;
+import java.util.stream.Collectors;
+
+/**
+ * expression column filter converter
+ */
+public class ExpressionColumnFilterConverter
+        extends DefaultExpressionVisitor<Expression, Void> {
+    private final Map<String, PartitionColumnFilter> columnFilterMap;
+
+    private class FilterParam {
+        public LiteralExpr lowerBound = null;
+        public boolean lowerBoundInclusive = false;
+        public LiteralExpr upperBound = null;
+        public boolean upperBoundInclusive = false;
+        public org.apache.doris.analysis.InPredicate inPredicate = null;
+
+        public void setValues(LiteralExpr lowerBound, boolean lowerInclusive,
+                LiteralExpr upperBound, boolean upperInclusive) {
+            this.lowerBound = lowerBound;
+            this.lowerBoundInclusive = lowerInclusive;
+            this.upperBound = upperBound;
+            this.upperBoundInclusive = upperInclusive;
+        }
+
+        public void setInPredicate(org.apache.doris.analysis.InPredicate 
inPredicate) {
+            this.inPredicate = inPredicate;
+        }
+    }
+
+    private final FilterParam param = new FilterParam();
+
+    public ExpressionColumnFilterConverter(Map<String, PartitionColumnFilter> 
filterMap) {
+        this.columnFilterMap = filterMap;
+    }
+
+    public void convert(Expression expr) {
+        expr.accept(this, null);
+    }
+
+    @Override
+    public Expression visitComparisonPredicate(ComparisonPredicate predicate, 
Void unused) {
+        if (predicate instanceof NullSafeEqual) {
+            return null;
+        }
+        LiteralExpr literal = ((Literal) predicate.right()).toLegacyLiteral();
+        if (predicate instanceof EqualTo) {
+            param.setValues(literal, true, literal, true);
+        } else if (predicate instanceof GreaterThan) {
+            param.setValues(literal, false, null, false);
+        } else if (predicate instanceof GreaterThanEqual) {
+            param.setValues(literal, true, null, false);
+        } else if (predicate instanceof LessThan) {
+            param.setValues(null, false, literal, false);
+        } else if (predicate instanceof LessThanEqual) {
+            param.setValues(null, false, literal, true);
+        }
+        setOrUpdateFilter(((Slot) predicate.left()).getName());
+        return null;
+    }
+
+    @Override
+    public Expression visitInPredicate(InPredicate predicate, Void unused) {
+        List<Expr> literals = predicate.getOptions().stream()
+                .map(expr -> ((Expr) ((Literal) expr).toLegacyLiteral()))
+                .collect(Collectors.toList());
+        param.setInPredicate(new org.apache.doris.analysis.InPredicate(new 
SlotRef(null, ""), literals, false));
+        setOrUpdateFilter(((Slot) predicate.getCompareExpr()).getName());
+        return null;
+    }
+
+    @Override
+    public Expression visitIsNull(IsNull predicate, Void unused) {
+        param.setValues(new NullLiteral(), true, new NullLiteral(), true);
+        setOrUpdateFilter(((Slot) predicate.child()).getName());
+        return null;
+    }
+
+    private void setOrUpdateFilter(String columnName) {
+        PartitionColumnFilter filter = 
columnFilterMap.computeIfAbsent(columnName,
+                k -> new PartitionColumnFilter());
+        if (param.lowerBound != null) {
+            filter.setLowerBound(param.lowerBound, param.lowerBoundInclusive);
+        }
+        if (param.upperBound != null) {
+            filter.setUpperBound(param.upperBound, param.upperBoundInclusive);
+        }
+        if (param.inPredicate != null) {
+            if (filter.getInPredicate() == null) {
+                filter.setInPredicate(param.inPredicate);
+            } else {
+                
filter.getInPredicate().getChildren().addAll(param.inPredicate.getListChildren());
+            }
+        }
+    }
+}
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 79f0b6e02e..dcf4aeeb93 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
@@ -49,6 +49,7 @@ public class LogicalOlapScan extends LogicalRelation 
implements CatalogRelation
     private final long selectedIndexId;
     private final ImmutableList<Long> selectedTabletId;
     private final boolean partitionPruned;
+    private final boolean tabletPruned;
 
     private final ImmutableList<Long> candidateIndexIds;
     private final boolean indexSelected;
@@ -61,12 +62,14 @@ public class LogicalOlapScan extends LogicalRelation 
implements CatalogRelation
 
     public LogicalOlapScan(RelationId id, OlapTable table, List<String> 
qualifier) {
         this(id, table, qualifier, Optional.empty(), Optional.empty(),
-                table.getPartitionIds(), false, ImmutableList.of(), false, 
PreAggStatus.on());
+                table.getPartitionIds(), false, ImmutableList.of(), false,
+                ImmutableList.of(), false, PreAggStatus.on());
     }
 
     public LogicalOlapScan(RelationId id, Table table, List<String> qualifier) 
{
         this(id, table, qualifier, Optional.empty(), Optional.empty(),
-                ((OlapTable) table).getPartitionIds(), false, 
ImmutableList.of(), false, PreAggStatus.on());
+                ((OlapTable) table).getPartitionIds(), false, 
ImmutableList.of(), false,
+                ImmutableList.of(), false, PreAggStatus.on());
     }
 
     /**
@@ -74,18 +77,18 @@ public class LogicalOlapScan extends LogicalRelation 
implements CatalogRelation
      */
     public LogicalOlapScan(RelationId id, Table table, List<String> qualifier,
             Optional<GroupExpression> groupExpression, 
Optional<LogicalProperties> logicalProperties,
-            List<Long> selectedPartitionIdList, boolean partitionPruned, 
List<Long> candidateIndexIds,
-            boolean indexSelected, PreAggStatus preAggStatus) {
+            List<Long> selectedPartitionIds, boolean partitionPruned,
+            ImmutableList<Long> selectedTabletIds, boolean tabletPruned,
+            List<Long> candidateIndexIds, boolean indexSelected, PreAggStatus 
preAggStatus) {
         super(id, PlanType.LOGICAL_OLAP_SCAN, table, qualifier,
-                groupExpression, logicalProperties, selectedPartitionIdList);
+                groupExpression, logicalProperties, selectedPartitionIds);
         // TODO: use CBO manner to select best index id, according to index's 
statistics info,
         //   revisit this after rollup and materialized view selection are 
fully supported.
         this.selectedIndexId = CollectionUtils.isEmpty(candidateIndexIds)
                 ? getTable().getBaseIndexId() : candidateIndexIds.get(0);
-        this.selectedTabletId = getTable().getAllPartitions().stream()
-                .flatMap(partition -> 
partition.getBaseIndex().getTabletIdsInOrder().stream())
-                .collect(ImmutableList.toImmutableList());
+        this.selectedTabletId = selectedTabletIds;
         this.partitionPruned = partitionPruned;
+        this.tabletPruned = tabletPruned;
         this.candidateIndexIds = ImmutableList.copyOf(candidateIndexIds);
         this.indexSelected = indexSelected;
         this.preAggStatus = preAggStatus;
@@ -124,34 +127,45 @@ public class LogicalOlapScan extends LogicalRelation 
implements CatalogRelation
             return false;
         }
         return Objects.equals(selectedPartitionIds, ((LogicalOlapScan) 
o).selectedPartitionIds)
-                && Objects.equals(candidateIndexIds, ((LogicalOlapScan) 
o).candidateIndexIds);
+                && Objects.equals(candidateIndexIds, ((LogicalOlapScan) 
o).candidateIndexIds)
+                && Objects.equals(selectedTabletId, ((LogicalOlapScan) 
o).selectedTabletId);
     }
 
     @Override
     public int hashCode() {
-        return Objects.hash(id, selectedPartitionIds, candidateIndexIds);
+        return Objects.hash(id, selectedPartitionIds, candidateIndexIds, 
selectedTabletId);
     }
 
     @Override
     public Plan withGroupExpression(Optional<GroupExpression> groupExpression) 
{
         return new LogicalOlapScan(id, table, qualifier, groupExpression, 
Optional.of(getLogicalProperties()),
-                selectedPartitionIds, partitionPruned, candidateIndexIds, 
indexSelected, preAggStatus);
+                selectedPartitionIds, partitionPruned, selectedTabletId, 
tabletPruned,
+                candidateIndexIds, indexSelected, preAggStatus);
     }
 
     @Override
     public LogicalOlapScan withLogicalProperties(Optional<LogicalProperties> 
logicalProperties) {
-        return new LogicalOlapScan(id, table, qualifier, Optional.empty(), 
logicalProperties, selectedPartitionIds,
-                partitionPruned, candidateIndexIds, indexSelected, 
preAggStatus);
+        return new LogicalOlapScan(id, table, qualifier, Optional.empty(), 
logicalProperties,
+                selectedPartitionIds, partitionPruned, selectedTabletId, 
tabletPruned,
+                candidateIndexIds, indexSelected, preAggStatus);
     }
 
-    public LogicalOlapScan withSelectedPartitionId(List<Long> 
selectedPartitionId) {
+    public LogicalOlapScan withSelectedPartitionIds(List<Long> 
selectedPartitionIds) {
         return new LogicalOlapScan(id, table, qualifier, Optional.empty(), 
Optional.of(getLogicalProperties()),
-                selectedPartitionId, true, candidateIndexIds, indexSelected, 
preAggStatus);
+                selectedPartitionIds, true, selectedTabletId, tabletPruned,
+                candidateIndexIds, indexSelected, preAggStatus);
     }
 
     public LogicalOlapScan withMaterializedIndexSelected(PreAggStatus preAgg, 
List<Long> candidateIndexIds) {
         return new LogicalOlapScan(id, table, qualifier, Optional.empty(), 
Optional.of(getLogicalProperties()),
-                selectedPartitionIds, partitionPruned, candidateIndexIds, 
true, preAgg);
+                selectedPartitionIds, partitionPruned, selectedTabletId, 
tabletPruned,
+                candidateIndexIds, true, preAgg);
+    }
+
+    public LogicalOlapScan withSelectedTabletIds(ImmutableList<Long> 
selectedTabletIds) {
+        return new LogicalOlapScan(id, table, qualifier, Optional.empty(), 
Optional.of(getLogicalProperties()),
+                selectedPartitionIds, partitionPruned, selectedTabletIds, true,
+                candidateIndexIds, indexSelected, preAggStatus);
     }
 
     @Override
@@ -163,6 +177,10 @@ public class LogicalOlapScan extends LogicalRelation 
implements CatalogRelation
         return partitionPruned;
     }
 
+    public boolean isTabletPruned() {
+        return tabletPruned;
+    }
+
     public List<Long> getSelectedTabletId() {
         return selectedTabletId;
     }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java
index baa91b25a8..6d85eec105 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java
@@ -20,9 +20,13 @@ package org.apache.doris.nereids.util;
 import org.apache.doris.nereids.trees.TreeNode;
 import org.apache.doris.nereids.trees.expressions.And;
 import org.apache.doris.nereids.trees.expressions.Cast;
+import org.apache.doris.nereids.trees.expressions.ComparisonPredicate;
 import org.apache.doris.nereids.trees.expressions.CompoundPredicate;
 import org.apache.doris.nereids.trees.expressions.ExprId;
 import org.apache.doris.nereids.trees.expressions.Expression;
+import org.apache.doris.nereids.trees.expressions.InPredicate;
+import org.apache.doris.nereids.trees.expressions.IsNull;
+import org.apache.doris.nereids.trees.expressions.Not;
 import org.apache.doris.nereids.trees.expressions.Or;
 import org.apache.doris.nereids.trees.expressions.Slot;
 import org.apache.doris.nereids.trees.expressions.SlotReference;
@@ -355,6 +359,32 @@ public class ExpressionUtils {
         return groupingSets;
     }
 
+    /**
+     * check and maybe commute for predications except not pred.
+     */
+    public static Optional<Expression> checkAndMaybeCommute(Expression 
expression) {
+        if (expression instanceof Not) {
+            return Optional.empty();
+        }
+        if (expression instanceof InPredicate) {
+            InPredicate predicate = ((InPredicate) expression);
+            if (!predicate.getCompareExpr().isSlot()) {
+                return Optional.empty();
+            }
+            return Optional.ofNullable(predicate.getOptions().stream()
+                    .allMatch(Expression::isLiteral) ? expression : null);
+        } else if (expression instanceof ComparisonPredicate) {
+            ComparisonPredicate predicate = ((ComparisonPredicate) expression);
+            if (predicate.left() instanceof Literal) {
+                predicate = predicate.commute();
+            }
+            return Optional.ofNullable(predicate.left().isSlot() && 
predicate.right().isLiteral() ? predicate : null);
+        } else if (expression instanceof IsNull) {
+            return Optional.ofNullable(((IsNull) expression).child().isSlot() 
? expression : null);
+        }
+        return Optional.empty();
+    }
+
     public static List<List<Expression>> cubeToGroupingSets(List<Expression> 
cubeExpressions) {
         List<List<Expression>> groupingSets = Lists.newArrayList();
         cubeToGroupingSets(cubeExpressions, 0, Lists.newArrayList(), 
groupingSets);
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/planner/HashDistributionPruner.java 
b/fe/fe-core/src/main/java/org/apache/doris/planner/HashDistributionPruner.java
index 7ec05c002f..7314fe0564 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/planner/HashDistributionPruner.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/planner/HashDistributionPruner.java
@@ -58,7 +58,7 @@ public class HashDistributionPruner implements 
DistributionPruner {
     private Map<String, PartitionColumnFilter> distributionColumnFilters;
     private int                                hashMod;
 
-    HashDistributionPruner(List<Long> bucketsList, List<Column> columns,
+    public HashDistributionPruner(List<Long> bucketsList, List<Column> columns,
                            Map<String, PartitionColumnFilter> filters, int 
hashMod) {
         this.bucketsList = bucketsList;
         this.distributionColumns = columns;
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/logical/PruneOlapScanTabletTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/logical/PruneOlapScanTabletTest.java
new file mode 100644
index 0000000000..e5745658be
--- /dev/null
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/logical/PruneOlapScanTabletTest.java
@@ -0,0 +1,164 @@
+// 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.logical;
+
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.IntLiteral;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.StringLiteral;
+import org.apache.doris.catalog.Column;
+import org.apache.doris.catalog.DistributionInfo;
+import org.apache.doris.catalog.HashDistributionInfo;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.PrimitiveType;
+import org.apache.doris.catalog.Type;
+import org.apache.doris.nereids.CascadesContext;
+import org.apache.doris.nereids.trees.expressions.EqualTo;
+import org.apache.doris.nereids.trees.expressions.Expression;
+import org.apache.doris.nereids.trees.expressions.GreaterThanEqual;
+import org.apache.doris.nereids.trees.expressions.InPredicate;
+import org.apache.doris.nereids.trees.expressions.LessThanEqual;
+import org.apache.doris.nereids.trees.expressions.SlotReference;
+import org.apache.doris.nereids.trees.expressions.literal.DateLiteral;
+import org.apache.doris.nereids.trees.expressions.literal.IntegerLiteral;
+import org.apache.doris.nereids.trees.plans.RelationId;
+import org.apache.doris.nereids.trees.plans.logical.LogicalFilter;
+import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan;
+import org.apache.doris.nereids.types.DataType;
+import org.apache.doris.nereids.util.ExpressionUtils;
+import org.apache.doris.nereids.util.MemoTestUtils;
+import org.apache.doris.planner.PartitionColumnFilter;
+
+import com.google.common.collect.ImmutableList;
+import com.google.common.collect.Lists;
+import mockit.Expectations;
+import mockit.Mocked;
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.Test;
+
+import java.util.List;
+
+public class PruneOlapScanTabletTest {
+
+    @Test
+    public void testPruneOlapScanTablet(@Mocked OlapTable olapTable,
+            @Mocked Partition partition, @Mocked MaterializedIndex index,
+            @Mocked HashDistributionInfo distributionInfo) {
+        List<Long> tabletIds = Lists.newArrayListWithExpectedSize(300);
+        for (long i = 0; i < 300; i++) {
+            tabletIds.add(i);
+        }
+
+        List<Column> columns = Lists.newArrayList(
+                new Column("k0", PrimitiveType.DATE, false),
+                new Column("k1", PrimitiveType.INT, false),
+                new Column("k2", PrimitiveType.INT, false),
+                new Column("k3", PrimitiveType.INT, false),
+                new Column("k4", PrimitiveType.INT, false)
+        );
+
+        PartitionColumnFilter k0Filter = new PartitionColumnFilter();
+        k0Filter.setLowerBound(new StringLiteral("2019-08-22"), true);
+        k0Filter.setUpperBound(new StringLiteral("2019-08-22"), true);
+
+        PartitionColumnFilter k1Filter = new PartitionColumnFilter();
+        List<Expr> inList = Lists.newArrayList();
+        inList.add(new IntLiteral(100));
+        inList.add(new IntLiteral(200));
+        inList.add(new IntLiteral(300));
+        inList.add(new IntLiteral(400));
+        inList.add(new IntLiteral(500));
+        k1Filter.setInPredicate(new org.apache.doris.analysis.InPredicate(new 
SlotRef(null, "k1"), inList, false));
+
+        PartitionColumnFilter k2Filter = new PartitionColumnFilter();
+        List<Expr> inList2 = Lists.newArrayList();
+        inList2.add(new IntLiteral(900));
+        inList2.add(new IntLiteral(1100));
+        k2Filter.setInPredicate(new org.apache.doris.analysis.InPredicate(new 
SlotRef(null, "k2"), inList2, false));
+
+        PartitionColumnFilter k3Filter = new PartitionColumnFilter();
+        List<Expr> inList3 = Lists.newArrayList();
+        inList3.add(new IntLiteral(1));
+        inList3.add(new IntLiteral(3));
+        k3Filter.setInPredicate(new org.apache.doris.analysis.InPredicate(new 
SlotRef(null, "k3"), inList3, false));
+
+        PartitionColumnFilter k4Filter = new PartitionColumnFilter();
+        List<Expr> inList4 = Lists.newArrayList();
+        inList4.add(new IntLiteral(2));
+        k4Filter.setInPredicate(new org.apache.doris.analysis.InPredicate(new 
SlotRef(null, "k4"), inList4, false));
+
+        SlotReference k0 = new SlotReference("k0", 
DataType.convertFromCatalogDataType(Type.INT), false, ImmutableList.of());
+        SlotReference k1 = new SlotReference("k1", 
DataType.convertFromCatalogDataType(Type.INT), false, ImmutableList.of());
+        SlotReference k2 = new SlotReference("k2", 
DataType.convertFromCatalogDataType(Type.INT), false, ImmutableList.of());
+        SlotReference k3 = new SlotReference("k3", 
DataType.convertFromCatalogDataType(Type.INT), false, ImmutableList.of());
+        SlotReference k4 = new SlotReference("k4", 
DataType.convertFromCatalogDataType(Type.INT), false, ImmutableList.of());
+
+        GreaterThanEqual greaterThanEqual = new GreaterThanEqual(k0, new 
DateLiteral("2019-08-22"));
+        LessThanEqual lessThanEqual = new LessThanEqual(k0, new 
DateLiteral("2019-08-22"));
+
+        InPredicate inPredicate1 = new InPredicate(k1, ImmutableList.of(new 
IntegerLiteral(101),
+                new IntegerLiteral(201),
+                new IntegerLiteral(301),
+                new IntegerLiteral(401),
+                new IntegerLiteral(500)));
+        InPredicate inPredicate2 = new InPredicate(k2, ImmutableList.of(new 
IntegerLiteral(901),
+                new IntegerLiteral(1101)));
+        InPredicate inPredicate3 = new InPredicate(k3, ImmutableList.of(new 
IntegerLiteral(1),
+                new IntegerLiteral(3)));
+        EqualTo equalTo = new EqualTo(k4, new IntegerLiteral(10));
+
+        new Expectations() {
+            {
+                olapTable.getPartitionIds();
+                result = ImmutableList.of(1L);
+
+                olapTable.getName();
+                result = "t1";
+                olapTable.getPartition(anyLong);
+                result = partition;
+                partition.getIndex(anyLong);
+                result = index;
+                partition.getDistributionInfo();
+                result = distributionInfo;
+                index.getTabletIdsInOrder();
+                result = tabletIds;
+                distributionInfo.getDistributionColumns();
+                result = columns;
+                distributionInfo.getType();
+                result = DistributionInfo.DistributionInfoType.HASH;
+                distributionInfo.getBucketNum();
+                result = tabletIds.size();
+            }
+        };
+
+        Expression expr = ExpressionUtils.and(greaterThanEqual, lessThanEqual, 
inPredicate1, inPredicate2, inPredicate3, equalTo);
+        LogicalFilter<LogicalOlapScan> filter = new LogicalFilter<>(expr,
+                new LogicalOlapScan(RelationId.createGenerator().getNextId(), 
olapTable));
+
+        Assertions.assertEquals(0, 
filter.child().getSelectedTabletId().size());
+
+        CascadesContext context = MemoTestUtils.createCascadesContext(filter);
+        context.topDownRewrite(ImmutableList.of(new 
PruneOlapScanTablet().build()));
+
+        LogicalFilter<LogicalOlapScan> filter1 = 
((LogicalFilter<LogicalOlapScan>) context.getMemo().copyOut());
+        LogicalOlapScan olapScan = filter1.child();
+        Assertions.assertEquals(19, olapScan.getSelectedTabletId().size());
+    }
+}


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org
For additional commands, e-mail: commits-h...@doris.apache.org


Reply via email to