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 ffd70239875 [feature](nereids) Support to get partition related table 
from mv and check the query operator (#28064)
ffd70239875 is described below

commit ffd70239875bdd954f140c9161bd6f61743d78ab
Author: seawinde <149132972+seawi...@users.noreply.github.com>
AuthorDate: Wed Dec 6 19:15:21 2023 +0800

    [feature](nereids) Support to get partition related table from mv and check 
the query operator (#28064)
    
    Function 1:
    check the select query plan is contain the stmt as following or not
    
    SELECT
    [hint_statement, ...]
    [ALL | DISTINCT | DISTINCTROW | ALL EXCEPT ( col_name1 [, col_name2, 
col_name3, ...] )]
    elect_expr [, select_expr ...]
    [FROM table_references
    PARTITION partition_list]
    [TABLET tabletid_list]
    [TABLESAMPLE sample_value [ROWS | PERCENT]
    [REPEATABLE pos_seek]]
    [WHERE where_condition]
    [GROUP BY [GROUPING SETS | ROLLUP | CUBE] {col_name | expr | position}]
    [HAVING where_condition]
    [ORDER BY {col_name | expr | position}
    [ASC | DESC], ...]
    [LIMIT {[offset,] row_count | row_count OFFSET offset}]
    [INTO OUTFILE 'file_name']
    
    if analyzedPlan contains the stmt as following
    
    [PARTITION partition_list]
    [TABLET tabletid_list] or
    [TABLESAMPLE sample_value [ROWS | PERCENT]
    [REPEATABLE pos_seek]]
    
    this method will return true.
    
    Function 2:
    Get related base table info which materialized view plan column reference,
    input param plan should be rewritten plan that sub query should be 
eliminated
---
 .../exploration/mv/MaterializedViewUtils.java      | 335 +++++++++++++++++++++
 .../exploration/mv/MaterializedViewUtilsTest.java  | 274 +++++++++++++++++
 .../doris/nereids/trees/plans/PlanVisitorTest.java |  18 ++
 .../doris/nereids/util/ExpressionUtilsTest.java    |   4 +-
 4 files changed, 628 insertions(+), 3 deletions(-)

diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewUtils.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewUtils.java
new file mode 100644
index 00000000000..48f4cb37fbe
--- /dev/null
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewUtils.java
@@ -0,0 +1,335 @@
+// 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.exploration.mv;
+
+import org.apache.doris.catalog.Column;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.PartitionInfo;
+import org.apache.doris.catalog.PartitionType;
+import org.apache.doris.catalog.TableIf;
+import org.apache.doris.mtmv.BaseTableInfo;
+import org.apache.doris.nereids.trees.expressions.Expression;
+import org.apache.doris.nereids.trees.expressions.NamedExpression;
+import org.apache.doris.nereids.trees.expressions.Slot;
+import org.apache.doris.nereids.trees.expressions.SlotReference;
+import org.apache.doris.nereids.trees.expressions.WindowExpression;
+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.LogicalCatalogRelation;
+import org.apache.doris.nereids.trees.plans.logical.LogicalFileScan;
+import org.apache.doris.nereids.trees.plans.logical.LogicalFilter;
+import org.apache.doris.nereids.trees.plans.logical.LogicalJoin;
+import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan;
+import org.apache.doris.nereids.trees.plans.logical.LogicalProject;
+import org.apache.doris.nereids.trees.plans.logical.LogicalRelation;
+import org.apache.doris.nereids.trees.plans.logical.LogicalResultSink;
+import org.apache.doris.nereids.trees.plans.logical.LogicalWindow;
+import org.apache.doris.nereids.trees.plans.visitor.DefaultPlanVisitor;
+
+import java.util.HashSet;
+import java.util.List;
+import java.util.Optional;
+import java.util.Set;
+
+/**
+ * The common util for materialized view
+ */
+public class MaterializedViewUtils {
+
+    /**
+     * Get related base table info which materialized view plan column 
reference,
+     * input param plan should be rewritten plan that sub query should be 
eliminated
+     *
+     * @param materializedViewPlan this should be rewritten or analyzed plan, 
should not be physical plan.
+     * @param column ref column name.
+     */
+    public static Optional<RelatedTableInfo> getRelatedTableInfo(String 
column, Plan materializedViewPlan) {
+        List<Slot> outputExpressions = materializedViewPlan.getOutput();
+        Slot columnExpr = null;
+        // get column slot
+        for (Slot outputSlot : outputExpressions) {
+            if (outputSlot.getName().equals(column)) {
+                columnExpr = outputSlot;
+                break;
+            }
+        }
+        if (columnExpr == null) {
+            return Optional.empty();
+        }
+        if (!(columnExpr instanceof SlotReference)) {
+            return Optional.empty();
+        }
+        SlotReference columnSlot = (SlotReference) columnExpr;
+        if (!columnSlot.isColumnFromTable()) {
+            return Optional.empty();
+        }
+        // check sql pattern
+        IncrementCheckerContext context = new 
IncrementCheckerContext(columnSlot);
+        materializedViewPlan.accept(MaterializedViewIncrementChecker.INSTANCE, 
context);
+        if (context.getRelatedTable() == null
+                || context.getRelatedTableColumn() == null
+                || !context.isPctPossible()) {
+            return Optional.empty();
+        }
+        return Optional.of(new RelatedTableInfo(new 
BaseTableInfo(context.getRelatedTable()),
+                context.isPctPossible(),
+                context.getRelatedTableColumn().getName()));
+    }
+
+    /**
+     * This method check the select query plan is contain the stmt as 
following or not
+     * <p>
+     * SELECT
+     * [hint_statement, ...]
+     * [ALL | DISTINCT | DISTINCTROW | ALL EXCEPT ( col_name1 [, col_name2, 
col_name3, ...] )]
+     * select_expr [, select_expr ...]
+     * [FROM table_references
+     * [PARTITION partition_list]
+     * [TABLET tabletid_list]
+     * [TABLESAMPLE sample_value [ROWS | PERCENT]
+     * [REPEATABLE pos_seek]]
+     * [WHERE where_condition]
+     * [GROUP BY [GROUPING SETS | ROLLUP | CUBE] {col_name | expr | position}]
+     * [HAVING where_condition]
+     * [ORDER BY {col_name | expr | position}
+     * [ASC | DESC], ...]
+     * [LIMIT {[offset,] row_count | row_count OFFSET offset}]
+     * [INTO OUTFILE 'file_name']
+     * <p>
+     * if analyzedPlan contains the stmt as following:
+     * [PARTITION partition_list]
+     * [TABLET tabletid_list] or
+     * [TABLESAMPLE sample_value [ROWS | PERCENT]
+     * *         [REPEATABLE pos_seek]]
+     * this method will return true.
+     */
+    public static boolean containTableQueryOperator(Plan analyzedPlan) {
+        return analyzedPlan.accept(TableQueryOperatorChecker.INSTANCE, null);
+    }
+
+    private static final class TableQueryOperatorChecker extends 
DefaultPlanVisitor<Boolean, Void> {
+        public static final TableQueryOperatorChecker INSTANCE = new 
TableQueryOperatorChecker();
+
+        @Override
+        public Boolean visitLogicalRelation(LogicalRelation relation, Void 
context) {
+            if (relation instanceof LogicalFileScan && ((LogicalFileScan) 
relation).getTableSample().isPresent()) {
+                return true;
+            }
+            if (relation instanceof LogicalOlapScan) {
+                LogicalOlapScan logicalOlapScan = (LogicalOlapScan) relation;
+                if (logicalOlapScan.getTableSample().isPresent()) {
+                    return true;
+                }
+                if (!logicalOlapScan.getSelectedTabletIds().isEmpty()) {
+                    return true;
+                }
+                if 
(!logicalOlapScan.getManuallySpecifiedPartitions().isEmpty()) {
+                    return true;
+                }
+            }
+            return visit(relation, context);
+        }
+
+        @Override
+        public Boolean visit(Plan plan, Void context) {
+            for (Plan child : plan.children()) {
+                Boolean checkResult = child.accept(this, context);
+                if (checkResult) {
+                    return checkResult;
+                }
+            }
+            return false;
+        }
+    }
+
+    private static final class MaterializedViewIncrementChecker extends
+            DefaultPlanVisitor<Void, IncrementCheckerContext> {
+
+        public static final MaterializedViewIncrementChecker INSTANCE = new 
MaterializedViewIncrementChecker();
+
+        @Override
+        public Void visitLogicalProject(LogicalProject<? extends Plan> 
project, IncrementCheckerContext context) {
+            return visit(project, context);
+        }
+
+        @Override
+        public Void visitLogicalFilter(LogicalFilter<? extends Plan> filter, 
IncrementCheckerContext context) {
+            return visit(filter, context);
+        }
+
+        @Override
+        public Void visitLogicalJoin(LogicalJoin<? extends Plan, ? extends 
Plan> join,
+                IncrementCheckerContext context) {
+            return visit(join, context);
+        }
+
+        @Override
+        public Void visitLogicalRelation(LogicalRelation relation, 
IncrementCheckerContext context) {
+            if (!(relation instanceof LogicalCatalogRelation) || 
context.getRelatedTable() != null) {
+                return visit(relation, context);
+            }
+            LogicalCatalogRelation logicalCatalogRelation = 
(LogicalCatalogRelation) relation;
+            TableIf table = logicalCatalogRelation.getTable();
+            if (!(table instanceof OlapTable)) {
+                return visit(relation, context);
+            }
+            OlapTable olapTable = (OlapTable) table;
+            PartitionInfo partitionInfo = olapTable.getPartitionInfo();
+            Set<Column> partitionColumnSet = new 
HashSet<>(partitionInfo.getPartitionColumns());
+            if (PartitionType.UNPARTITIONED.equals(partitionInfo.getType())) {
+                return visit(relation, context);
+            }
+            Column mvReferenceColumn = 
context.getMvPartitionColumn().getColumn().get();
+            if (partitionColumnSet.contains(mvReferenceColumn)) {
+                context.setRelatedTable(table);
+                context.setRelatedTableColumn(mvReferenceColumn);
+            }
+            return visit(relation, context);
+        }
+
+        @Override
+        public Void visitLogicalAggregate(LogicalAggregate<? extends Plan> 
aggregate,
+                IncrementCheckerContext context) {
+            Set<Expression> groupByExprSet = new 
HashSet<>(aggregate.getGroupByExpressions());
+            if (groupByExprSet.isEmpty()) {
+                return visit(aggregate, context);
+            }
+            Set<Column> originalGroupbyExprSet = new HashSet<>();
+            groupByExprSet.forEach(groupExpr -> {
+                if (groupExpr instanceof SlotReference && 
groupExpr.isColumnFromTable()) {
+                    originalGroupbyExprSet.add(((SlotReference) 
groupExpr).getColumn().get());
+                }
+            });
+            if 
(!originalGroupbyExprSet.contains(context.getMvPartitionColumn().getColumn().get()))
 {
+                context.setPctPossible(false);
+            }
+            return visit(aggregate, context);
+        }
+
+        @Override
+        public Void visitLogicalWindow(LogicalWindow<? extends Plan> window, 
IncrementCheckerContext context) {
+            List<NamedExpression> windowExpressions = 
window.getWindowExpressions();
+            if (windowExpressions.isEmpty()) {
+                return visit(window, context);
+            }
+            windowExpressions.forEach(expr -> checkWindowPartition(expr, 
context));
+            return super.visitLogicalWindow(window, context);
+        }
+
+        @Override
+        public Void visit(Plan plan, IncrementCheckerContext context) {
+            if (!context.isPctPossible()) {
+                return null;
+            }
+            if (plan instanceof LogicalProject
+                    || plan instanceof LogicalFilter
+                    || plan instanceof LogicalJoin
+                    || plan instanceof LogicalAggregate
+                    || plan instanceof LogicalCatalogRelation
+                    || plan instanceof LogicalResultSink
+                    || plan instanceof LogicalWindow) {
+                return super.visit(plan, context);
+            }
+            context.setPctPossible(false);
+            return null;
+        }
+
+        private void checkWindowPartition(Expression expression, 
IncrementCheckerContext context) {
+            expression.collectToList(expressionTreeNode -> expressionTreeNode 
instanceof WindowExpression)
+                    .forEach(windowObj -> {
+                        WindowExpression windowExpression = (WindowExpression) 
windowObj;
+                        List<Expression> partitionKeys = 
windowExpression.getPartitionKeys();
+                        Set<Column> originalPartitionbyExprSet = new 
HashSet<>();
+                        partitionKeys.forEach(groupExpr -> {
+                            if (groupExpr instanceof SlotReference && 
groupExpr.isColumnFromTable()) {
+                                
originalPartitionbyExprSet.add(((SlotReference) groupExpr).getColumn().get());
+                            }
+                        });
+                        if 
(!originalPartitionbyExprSet.contains(context.getMvPartitionColumn().getColumn().get()))
 {
+                            context.setPctPossible(false);
+                        }
+                    });
+        }
+    }
+
+    private static final class IncrementCheckerContext {
+        private final SlotReference mvPartitionColumn;
+        private boolean pctPossible = true;
+        private TableIf relatedTable;
+        private Column relatedTableColumn;
+
+        public IncrementCheckerContext(SlotReference mvPartitionColumn) {
+            this.mvPartitionColumn = mvPartitionColumn;
+        }
+
+        public SlotReference getMvPartitionColumn() {
+            return mvPartitionColumn;
+        }
+
+        public boolean isPctPossible() {
+            return pctPossible;
+        }
+
+        public void setPctPossible(boolean pctPossible) {
+            this.pctPossible = pctPossible;
+        }
+
+        public TableIf getRelatedTable() {
+            return relatedTable;
+        }
+
+        public void setRelatedTable(TableIf relatedTable) {
+            this.relatedTable = relatedTable;
+        }
+
+        public Column getRelatedTableColumn() {
+            return relatedTableColumn;
+        }
+
+        public void setRelatedTableColumn(Column relatedTableColumn) {
+            this.relatedTableColumn = relatedTableColumn;
+        }
+    }
+
+    /**
+     * The related table info that mv relate
+     */
+    public static final class RelatedTableInfo {
+        private BaseTableInfo tableInfo;
+        private boolean pctPossible;
+        private String column;
+
+        public RelatedTableInfo(BaseTableInfo tableInfo, boolean pctPossible, 
String column) {
+            this.tableInfo = tableInfo;
+            this.pctPossible = pctPossible;
+            this.column = column;
+        }
+
+        public BaseTableInfo getTableInfo() {
+            return tableInfo;
+        }
+
+        public boolean isPctPossible() {
+            return pctPossible;
+        }
+
+        public String getColumn() {
+            return column;
+        }
+    }
+}
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewUtilsTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewUtilsTest.java
new file mode 100644
index 00000000000..f6dba539962
--- /dev/null
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewUtilsTest.java
@@ -0,0 +1,274 @@
+// 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.exploration.mv;
+
+import org.apache.doris.catalog.Env;
+import org.apache.doris.catalog.TableIf;
+import org.apache.doris.mtmv.BaseTableInfo;
+import 
org.apache.doris.nereids.rules.exploration.mv.MaterializedViewUtils.RelatedTableInfo;
+import org.apache.doris.nereids.trees.plans.Plan;
+import org.apache.doris.nereids.util.PlanChecker;
+import org.apache.doris.utframe.TestWithFeService;
+
+import org.junit.jupiter.api.Assertions;
+import org.junit.jupiter.api.Test;
+
+import java.util.Optional;
+
+/**
+ * Test for materialized view util
+ */
+public class MaterializedViewUtilsTest extends TestWithFeService {
+
+    @Override
+    protected void runBeforeAll() throws Exception {
+        createDatabase("mv_util_test");
+        useDatabase("mv_util_test");
+
+        createTable("CREATE TABLE IF NOT EXISTS lineitem (\n"
+                + "  L_ORDERKEY    INTEGER NOT NULL,\n"
+                + "  L_PARTKEY     INTEGER NOT NULL,\n"
+                + "  L_SUPPKEY     INTEGER NOT NULL,\n"
+                + "  L_LINENUMBER  INTEGER NOT NULL,\n"
+                + "  L_QUANTITY    DECIMALV3(15,2) NOT NULL,\n"
+                + "  L_EXTENDEDPRICE  DECIMALV3(15,2) NOT NULL,\n"
+                + "  L_DISCOUNT    DECIMALV3(15,2) NOT NULL,\n"
+                + "  L_TAX         DECIMALV3(15,2) NOT NULL,\n"
+                + "  L_RETURNFLAG  CHAR(1) NOT NULL,\n"
+                + "  L_LINESTATUS  CHAR(1) NOT NULL,\n"
+                + "  L_SHIPDATE    DATE NOT NULL,\n"
+                + "  L_COMMITDATE  DATE NOT NULL,\n"
+                + "  L_RECEIPTDATE DATE NOT NULL,\n"
+                + "  L_SHIPINSTRUCT CHAR(25) NOT NULL,\n"
+                + "  L_SHIPMODE     CHAR(10) NOT NULL,\n"
+                + "  L_COMMENT      VARCHAR(44) NOT NULL\n"
+                + ")\n"
+                + "DUPLICATE KEY(L_ORDERKEY, L_PARTKEY, L_SUPPKEY, 
L_LINENUMBER)\n"
+                + "PARTITION BY RANGE(L_SHIPDATE) (PARTITION `day_1` VALUES 
LESS THAN ('2017-02-01'))\n"
+                + "DISTRIBUTED BY HASH(L_ORDERKEY) BUCKETS 3\n"
+                + "PROPERTIES (\n"
+                + "  \"replication_num\" = \"1\"\n"
+                + ")");
+        createTable("CREATE TABLE IF NOT EXISTS orders  (\n"
+                + "  O_ORDERKEY       INTEGER NOT NULL,\n"
+                + "  O_CUSTKEY        INTEGER NOT NULL,\n"
+                + "  O_ORDERSTATUS    CHAR(1) NOT NULL,\n"
+                + "  O_TOTALPRICE     DECIMALV3(15,2) NOT NULL,\n"
+                + "  O_ORDERDATE      DATE NOT NULL,\n"
+                + "  O_ORDERPRIORITY  CHAR(15) NOT NULL,  \n"
+                + "  O_CLERK          CHAR(15) NOT NULL, \n"
+                + "  O_SHIPPRIORITY   INTEGER NOT NULL,\n"
+                + "  O_COMMENT        VARCHAR(79) NOT NULL\n"
+                + ")\n"
+                + "DUPLICATE KEY(O_ORDERKEY, O_CUSTKEY)\n"
+                + "PARTITION BY RANGE(O_ORDERDATE) (PARTITION `day_2` VALUES 
LESS THAN ('2017-03-01'))\n"
+                + "DISTRIBUTED BY HASH(O_ORDERKEY) BUCKETS 3\n"
+                + "PROPERTIES (\n"
+                + "  \"replication_num\" = \"1\"\n"
+                + ")");
+        createTable("CREATE TABLE IF NOT EXISTS partsupp (\n"
+                + "  PS_PARTKEY     INTEGER NOT NULL,\n"
+                + "  PS_SUPPKEY     INTEGER NOT NULL,\n"
+                + "  PS_AVAILQTY    INTEGER NOT NULL,\n"
+                + "  PS_SUPPLYCOST  DECIMALV3(15,2)  NOT NULL,\n"
+                + "  PS_COMMENT     VARCHAR(199) NOT NULL \n"
+                + ")\n"
+                + "DUPLICATE KEY(PS_PARTKEY, PS_SUPPKEY)\n"
+                + "DISTRIBUTED BY HASH(PS_PARTKEY) BUCKETS 3\n"
+                + "PROPERTIES (\n"
+                + "  \"replication_num\" = \"1\"\n"
+                + ")");
+    }
+
+    @Test
+    public void getRelatedTableInfoTestWithoutGroupTest() {
+        PlanChecker.from(connectContext)
+                .checkExplain("SELECT (o.c1_abs + ps.c2_abs) as add_alias, 
l.L_SHIPDATE, l.L_ORDERKEY, o.O_ORDERDATE, "
+                                + "ps.PS_AVAILQTY "
+                                + "FROM "
+                                + "lineitem as l "
+                                + "LEFT JOIN "
+                                + "(SELECT abs(O_TOTALPRICE + 10) as c1_abs, 
O_CUSTKEY, O_ORDERDATE, O_ORDERKEY "
+                                + "FROM orders) as o "
+                                + "ON l.L_ORDERKEY = o.O_ORDERKEY "
+                                + "JOIN "
+                                + "(SELECT abs(sqrt(PS_SUPPLYCOST)) as c2_abs, 
PS_AVAILQTY, PS_PARTKEY, PS_SUPPKEY "
+                                + "FROM partsupp) as ps "
+                                + "ON l.L_PARTKEY = ps.PS_PARTKEY and 
l.L_SUPPKEY = ps.PS_SUPPKEY",
+                        nereidsPlanner -> {
+                            Plan rewrittenPlan = 
nereidsPlanner.getRewrittenPlan();
+                            Optional<RelatedTableInfo> relatedTableInfo =
+                                    
MaterializedViewUtils.getRelatedTableInfo("L_SHIPDATE", rewrittenPlan);
+                            checkRelatedTableInfo(relatedTableInfo,
+                                    "lineitem",
+                                    "L_SHIPDATE",
+                                    true);
+                        });
+    }
+
+    @Test
+    public void getRelatedTableInfoTestWithAliasAndGroupTest() {
+        PlanChecker.from(connectContext)
+                .checkExplain("SELECT l.L_SHIPDATE AS ship_data_alias, 
o.O_ORDERDATE, count(*) "
+                                + "FROM "
+                                + "lineitem as l "
+                                + "LEFT JOIN "
+                                + "(SELECT abs(O_TOTALPRICE + 10) as c1_abs, 
O_CUSTKEY, O_ORDERDATE, O_ORDERKEY "
+                                + "FROM orders) as o "
+                                + "ON l.L_ORDERKEY = o.O_ORDERKEY "
+                                + "JOIN "
+                                + "(SELECT abs(sqrt(PS_SUPPLYCOST)) as c2_abs, 
PS_AVAILQTY, PS_PARTKEY, PS_SUPPKEY "
+                                + "FROM partsupp) as ps "
+                                + "ON l.L_PARTKEY = ps.PS_PARTKEY and 
l.L_SUPPKEY = ps.PS_SUPPKEY "
+                                + "GROUP BY l.L_SHIPDATE, o.O_ORDERDATE ",
+                        nereidsPlanner -> {
+                            Plan rewrittenPlan = 
nereidsPlanner.getRewrittenPlan();
+                            Optional<RelatedTableInfo> relatedTableInfo =
+                                    
MaterializedViewUtils.getRelatedTableInfo("ship_data_alias", rewrittenPlan);
+                            checkRelatedTableInfo(relatedTableInfo,
+                                    "lineitem",
+                                    "L_SHIPDATE",
+                                    true);
+                        });
+    }
+
+    @Test
+    public void getRelatedTableInfoTestWithoutPartitionTest() {
+        PlanChecker.from(connectContext)
+                .checkExplain("SELECT ps_1.PS_SUPPLYCOST "
+                                + "FROM "
+                                + "partsupp as ps_1 "
+                                + "LEFT JOIN "
+                                + "partsupp as ps_2 "
+                                + "ON ps_1.PS_PARTKEY = ps_2.PS_SUPPKEY ",
+                        nereidsPlanner -> {
+                            Plan rewrittenPlan = 
nereidsPlanner.getRewrittenPlan();
+                            Optional<RelatedTableInfo> relatedTableInfo =
+                                    
MaterializedViewUtils.getRelatedTableInfo("PS_SUPPLYCOST", rewrittenPlan);
+                            
Assertions.assertFalse(relatedTableInfo.isPresent());
+                        });
+    }
+
+    @Test
+    public void getRelatedTableInfoTestWithWindowTest() {
+        PlanChecker.from(connectContext)
+                .checkExplain("SELECT (o.c1_abs + ps.c2_abs) as add_alias, 
l.L_SHIPDATE, l.L_ORDERKEY, o.O_ORDERDATE, "
+                                + "count(o.O_ORDERDATE) over (partition by 
l.L_SHIPDATE order by l.L_ORDERKEY  rows between unbounded preceding and 
current row) as window_count "
+                                + "FROM "
+                                + "lineitem as l "
+                                + "LEFT JOIN "
+                                + "(SELECT abs(O_TOTALPRICE + 10) as c1_abs, 
O_CUSTKEY, O_ORDERDATE, O_ORDERKEY "
+                                + "FROM orders) as o "
+                                + "ON l.L_ORDERKEY = o.O_ORDERKEY "
+                                + "JOIN "
+                                + "(SELECT abs(sqrt(PS_SUPPLYCOST)) as c2_abs, 
PS_AVAILQTY, PS_PARTKEY, PS_SUPPKEY "
+                                + "FROM partsupp) as ps "
+                                + "ON l.L_PARTKEY = ps.PS_PARTKEY and 
l.L_SUPPKEY = ps.PS_SUPPKEY",
+                        nereidsPlanner -> {
+                            Plan rewrittenPlan = 
nereidsPlanner.getRewrittenPlan();
+                            Optional<RelatedTableInfo> relatedTableInfo =
+                                    
MaterializedViewUtils.getRelatedTableInfo("L_SHIPDATE", rewrittenPlan);
+                            checkRelatedTableInfo(relatedTableInfo,
+                                    "lineitem",
+                                    "L_SHIPDATE",
+                                    true);
+                        });
+    }
+
+    @Test
+    public void getRelatedTableInfoTestWithWindowButNotPartitionTest() {
+        PlanChecker.from(connectContext)
+                .checkExplain("SELECT (o.c1_abs + ps.c2_abs) as add_alias, 
l.L_SHIPDATE, l.L_ORDERKEY, o.O_ORDERDATE, "
+                                + "count(o.O_ORDERDATE) over (partition by 
l.L_ORDERKEY order by l.L_ORDERKEY  rows between unbounded preceding and 
current row) as window_count "
+                                + "FROM "
+                                + "lineitem as l "
+                                + "LEFT JOIN "
+                                + "(SELECT abs(O_TOTALPRICE + 10) as c1_abs, 
O_CUSTKEY, O_ORDERDATE, O_ORDERKEY "
+                                + "FROM orders) as o "
+                                + "ON l.L_ORDERKEY = o.O_ORDERKEY "
+                                + "JOIN "
+                                + "(SELECT abs(sqrt(PS_SUPPLYCOST)) as c2_abs, 
PS_AVAILQTY, PS_PARTKEY, PS_SUPPKEY "
+                                + "FROM partsupp) as ps "
+                                + "ON l.L_PARTKEY = ps.PS_PARTKEY and 
l.L_SUPPKEY = ps.PS_SUPPKEY",
+                        nereidsPlanner -> {
+                            Plan rewrittenPlan = 
nereidsPlanner.getRewrittenPlan();
+                            Optional<RelatedTableInfo> relatedTableInfo =
+                                    
MaterializedViewUtils.getRelatedTableInfo("L_SHIPDATE", rewrittenPlan);
+                            
Assertions.assertFalse(relatedTableInfo.isPresent());
+                        });
+    }
+
+    @Test
+    public void containTableQueryOperatorWithTabletTest() {
+        PlanChecker.from(connectContext)
+                .checkExplain("select * from orders TABLET(11080)",
+                        nereidsPlanner -> {
+                            Plan analyzedPlan = 
nereidsPlanner.getAnalyzedPlan();
+                            
Assertions.assertTrue(MaterializedViewUtils.containTableQueryOperator(analyzedPlan));
+                        });
+    }
+
+    @Test
+    public void containTableQueryOperatorTableSampleTest() {
+        PlanChecker.from(connectContext)
+                .checkExplain("select * from orders TABLESAMPLE(20 percent)",
+                        nereidsPlanner -> {
+                            Plan analyzedPlan = 
nereidsPlanner.getAnalyzedPlan();
+                            
Assertions.assertTrue(MaterializedViewUtils.containTableQueryOperator(analyzedPlan));
+                        });
+    }
+
+    @Test
+    public void containTableQueryOperatorPartitionTest() {
+        PlanChecker.from(connectContext)
+                .checkExplain("select * from orders PARTITION day_2",
+                        nereidsPlanner -> {
+                            Plan analyzedPlan = 
nereidsPlanner.getAnalyzedPlan();
+                            
Assertions.assertTrue(MaterializedViewUtils.containTableQueryOperator(analyzedPlan));
+                        });
+    }
+
+    @Test
+    public void containTableQueryOperatorWithoutOperatorTest() {
+        PlanChecker.from(connectContext)
+                .checkExplain("select * from orders",
+                        nereidsPlanner -> {
+                            Plan analyzedPlan = 
nereidsPlanner.getAnalyzedPlan();
+                            
Assertions.assertFalse(MaterializedViewUtils.containTableQueryOperator(analyzedPlan));
+                        });
+    }
+
+    private void checkRelatedTableInfo(Optional<RelatedTableInfo> 
relatedTableInfo,
+            String expectTableName,
+            String expectColumnName,
+            boolean pctPossible) {
+        Assertions.assertTrue(relatedTableInfo.isPresent());
+        BaseTableInfo relatedBaseTableInfo = 
relatedTableInfo.get().getTableInfo();
+        try {
+            TableIf tableIf = Env.getCurrentEnv().getCatalogMgr()
+                    
.getCatalogOrAnalysisException(relatedBaseTableInfo.getCtlId())
+                    .getDbOrAnalysisException(relatedBaseTableInfo.getDbId())
+                    
.getTableOrAnalysisException(relatedBaseTableInfo.getTableId());
+            Assertions.assertEquals(tableIf.getName(), expectTableName);
+        } catch (Exception exception) {
+            Assertions.fail();
+        }
+        Assertions.assertEquals(relatedTableInfo.get().getColumn(), 
expectColumnName);
+        Assertions.assertTrue(pctPossible);
+    }
+}
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/trees/plans/PlanVisitorTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/trees/plans/PlanVisitorTest.java
index c6ca20577cc..775bc05e163 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/trees/plans/PlanVisitorTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/trees/plans/PlanVisitorTest.java
@@ -21,6 +21,8 @@ import org.apache.doris.catalog.TableIf;
 import org.apache.doris.catalog.TableIf.TableType;
 import org.apache.doris.nereids.trees.TreeNode;
 import org.apache.doris.nereids.trees.expressions.Expression;
+import org.apache.doris.nereids.trees.expressions.functions.scalar.CurrentDate;
+import org.apache.doris.nereids.trees.expressions.functions.scalar.Now;
 import org.apache.doris.nereids.trees.expressions.functions.scalar.Random;
 import org.apache.doris.nereids.trees.expressions.functions.scalar.Uuid;
 import org.apache.doris.nereids.trees.plans.physical.PhysicalPlan;
@@ -142,4 +144,20 @@ public class PlanVisitorTest extends TestWithFeService {
                                     expectedTables);
                         });
     }
+
+    @Test
+    public void testTimeFunction() {
+        PlanChecker.from(connectContext)
+                .checkExplain("SELECT *, now() FROM table1 "
+                                + "LEFT SEMI JOIN table2 ON table1.c1 = 
table2.c1 "
+                                + "WHERE table1.c1 IN (SELECT c1 FROM table2) 
OR CURDATE() < '2023-01-01'",
+                        nereidsPlanner -> {
+                            List<TreeNode<Expression>> collectResult = new 
ArrayList<>();
+                            // Check nondeterministic collect
+                            
nereidsPlanner.getAnalyzedPlan().accept(NondeterministicFunctionCollector.INSTANCE,
 collectResult);
+                            Assertions.assertEquals(2, collectResult.size());
+                            Assertions.assertTrue(collectResult.get(0) 
instanceof Now);
+                            Assertions.assertTrue(collectResult.get(1) 
instanceof CurrentDate);
+                        });
+    }
 }
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/util/ExpressionUtilsTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/util/ExpressionUtilsTest.java
index 7ab3e8083d1..f68d37a24db 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/util/ExpressionUtilsTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/util/ExpressionUtilsTest.java
@@ -91,8 +91,6 @@ public class ExpressionUtilsTest extends TestWithFeService {
                 + "PROPERTIES (\n"
                 + "  \"replication_num\" = \"1\"\n"
                 + ")");
-        connectContext.getSessionVariable().enableNereidsTimeout = false;
-
     }
 
     @Test
@@ -158,7 +156,7 @@ public class ExpressionUtilsTest extends TestWithFeService {
     }
 
     @Test
-    public void testShuttleExpressionWithLineage1() {
+    public void shuttleExpressionWithLineageTest1() {
         PlanChecker.from(connectContext)
                 .checkExplain("SELECT (o.c1_abs + ps.c2_abs) as add_alias, 
l.L_LINENUMBER, o.O_ORDERSTATUS "
                                 + "FROM "


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


Reply via email to