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

starocean999 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 d7e5d461157 [fix](mtmv) Fix compensate union all wrongly when query 
rewrite by materialized view (#40803)
d7e5d461157 is described below

commit d7e5d46115729213c588e4885f5e307eee0c597f
Author: seawinde <149132972+seawi...@users.noreply.github.com>
AuthorDate: Fri Sep 20 18:48:39 2024 +0800

    [fix](mtmv) Fix compensate union all wrongly when query rewrite by 
materialized view (#40803)
    
    ## Proposed changes
    
    This is brought by https://github.com/apache/doris/pull/36056
    
    Not all query after rewritten successfully can compensate union all
    Such as:
    mv def sql is as following, partition column is a
    ```sql
    select a, b, count(*) from t1 group by a, b
    ```
    Query is as following:
    ```sq
    select count(*) from t1
    ```
    the result is
    +----------+
    | count(*) |
    +----------+
    |       24 |
    +----------+
    
    after rewritten by materialized view successfully
    If mv part partition is invalid, can not compensate union all, because
    result is wrong after
    compensate union all.
    
    +----------+
    | count(*) |
    +----------+
    |       24 |
    |       3 |
    +----------+
    
    This pr fix this.
---
 .../mv/AbstractMaterializedViewAggregateRule.java  |  50 +++
 .../mv/AbstractMaterializedViewRule.java           |  25 ++
 .../union_all_compensate/union_all_compensate.out  | 201 ++++++++++++
 .../org/apache/doris/regression/suite/Suite.groovy |  11 +-
 .../availability/materialized_view_switch.groovy   |   4 +-
 .../union_all_compensate.groovy                    | 345 +++++++++++++++++++++
 6 files changed, 631 insertions(+), 5 deletions(-)

diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
index 909f67de204..1a66eda2ad3 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
@@ -17,7 +17,10 @@
 
 package org.apache.doris.nereids.rules.exploration.mv;
 
+import org.apache.doris.catalog.Column;
+import org.apache.doris.catalog.MTMV;
 import org.apache.doris.common.Pair;
+import org.apache.doris.mtmv.BaseTableInfo;
 import org.apache.doris.nereids.CascadesContext;
 import org.apache.doris.nereids.jobs.executor.Rewriter;
 import org.apache.doris.nereids.properties.DataTrait;
@@ -38,6 +41,7 @@ import org.apache.doris.nereids.trees.expressions.ExprId;
 import org.apache.doris.nereids.trees.expressions.Expression;
 import org.apache.doris.nereids.trees.expressions.NamedExpression;
 import org.apache.doris.nereids.trees.expressions.Slot;
+import org.apache.doris.nereids.trees.expressions.SlotReference;
 import org.apache.doris.nereids.trees.expressions.VirtualSlotReference;
 import org.apache.doris.nereids.trees.expressions.functions.Function;
 import 
org.apache.doris.nereids.trees.expressions.functions.agg.AggregateFunction;
@@ -63,6 +67,7 @@ import java.util.HashMap;
 import java.util.HashSet;
 import java.util.List;
 import java.util.Map;
+import java.util.Objects;
 import java.util.Optional;
 import java.util.Set;
 import java.util.function.Supplier;
@@ -324,6 +329,51 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
         return rewrittenExpression;
     }
 
+    /**
+     * Not all query after rewritten successfully can compensate union all
+     * Such as:
+     * mv def sql is as following, partition column is a
+     * select a, b, count(*) from t1 group by a, b
+     * Query is as following:
+     * select b, count(*) from t1 group by b, after rewritten by materialized 
view successfully
+     * If mv part partition is invalid, can not compensate union all, because 
result is wrong after
+     * compensate union all.
+     */
+    @Override
+    protected boolean canUnionRewrite(Plan queryPlan, MTMV mtmv, 
CascadesContext cascadesContext) {
+        // Check query plan is contain the partition column
+        // Query plan in the current rule must contain aggregate node, because 
the rule pattern is
+        //
+        Optional<LogicalAggregate<Plan>> logicalAggregateOptional =
+                queryPlan.collectFirst(planTreeNode -> planTreeNode instanceof 
LogicalAggregate);
+        if (!logicalAggregateOptional.isPresent()) {
+            return true;
+        }
+        List<Expression> groupByExpressions = 
logicalAggregateOptional.get().getGroupByExpressions();
+        if (groupByExpressions.isEmpty()) {
+            // Scalar aggregate can not compensate union all
+            return false;
+        }
+        final String relatedCol = mtmv.getMvPartitionInfo().getRelatedCol();
+        final BaseTableInfo relatedTableInfo = 
mtmv.getMvPartitionInfo().getRelatedTableInfo();
+        boolean canUnionRewrite = false;
+        // Check the query plan group by expression contains partition col or 
not
+        List<? extends Expression> groupByShuttledExpressions =
+                
ExpressionUtils.shuttleExpressionWithLineage(groupByExpressions, queryPlan, new 
BitSet());
+        for (Expression expression : groupByShuttledExpressions) {
+            canUnionRewrite = !expression.collectToSet(expr -> expr instanceof 
SlotReference
+                    && ((SlotReference) expr).isColumnFromTable()
+                    && Objects.equals(((SlotReference) 
expr).getColumn().map(Column::getName).orElse(null),
+                    relatedCol)
+                    && Objects.equals(((SlotReference) 
expr).getTable().map(BaseTableInfo::new).orElse(null),
+                    relatedTableInfo)).isEmpty();
+            if (canUnionRewrite) {
+                break;
+            }
+        }
+        return canUnionRewrite;
+    }
+
     /**
      * Check query and view aggregate compatibility
      */
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
index 9fef549c0a7..7d84b8ab36b 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
@@ -292,6 +292,17 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                     return rewriteResults;
                 }
                 boolean partitionNeedUnion = 
needUnionRewrite(invalidPartitions, cascadesContext);
+                boolean canUnionRewrite = canUnionRewrite(queryPlan,
+                        ((AsyncMaterializationContext) 
materializationContext).getMtmv(),
+                        cascadesContext);
+                if (partitionNeedUnion && !canUnionRewrite) {
+                    materializationContext.recordFailReason(queryStructInfo,
+                            "need compensate union all, but can not, because 
the query structInfo",
+                            () -> String.format("mv partition info is %s, and 
the query plan is %s",
+                                    ((AsyncMaterializationContext) 
materializationContext).getMtmv()
+                                            .getMvPartitionInfo(), 
queryPlan.treeString()));
+                    return rewriteResults;
+                }
                 final Pair<Map<BaseTableInfo, Set<String>>, Map<BaseTableInfo, 
Set<String>>> finalInvalidPartitions =
                         invalidPartitions;
                 if (partitionNeedUnion) {
@@ -377,6 +388,20 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                 && (!invalidPartitions.key().isEmpty() || 
!invalidPartitions.value().isEmpty());
     }
 
+    /**
+     * Not all query after rewritten successfully can compensate union all
+     * Such as:
+     * mv def sql is as following, partition column is a
+     * select a, b, count(*) from t1 group by a, b
+     * Query is as following:
+     * select b, count(*) from t1 group by b, after rewritten by materialized 
view successfully
+     * If mv part partition is invalid, can not compensate union all, because 
result is wrong after
+     * compensate union all.
+     */
+    protected boolean canUnionRewrite(Plan queryPlan, MTMV mtmv, 
CascadesContext cascadesContext) {
+        return true;
+    }
+
     // Normalize expression such as nullable property and output slot id
     protected Plan normalizeExpressions(Plan rewrittenPlan, Plan originPlan) {
         if (rewrittenPlan.getOutput().size() != originPlan.getOutput().size()) 
{
diff --git 
a/regression-test/data/nereids_rules_p0/mv/union_all_compensate/union_all_compensate.out
 
b/regression-test/data/nereids_rules_p0/mv/union_all_compensate/union_all_compensate.out
new file mode 100644
index 00000000000..7ff775063fb
--- /dev/null
+++ 
b/regression-test/data/nereids_rules_p0/mv/union_all_compensate/union_all_compensate.out
@@ -0,0 +1,201 @@
+-- This file is automatically generated. You should know what you did if you 
want to edit this
+-- !query1_0_before --
+28
+
+-- !query1_0_after --
+28
+
+-- !query1_1_before --
+32
+
+-- !query1_1_after --
+32
+
+-- !query2_0_before --
+a      4
+b      28
+
+-- !query2_0_after --
+a      2
+b      26
+
+-- !query3_0_before --
+a      4
+b      28
+
+-- !query3_0_after --
+a      4
+b      28
+
+-- !query4_0_before --
+2024-09-12     8
+2024-09-13     8
+2024-09-14     8
+2024-09-15     8
+
+-- !query4_0_after --
+2024-09-12     4
+2024-09-13     8
+2024-09-14     8
+2024-09-15     8
+
+-- !query5_0_before --
+2024-09-12     8
+2024-09-13     8
+2024-09-14     8
+2024-09-15     8
+
+-- !query5_0_after --
+2024-09-12     8
+2024-09-13     8
+2024-09-14     8
+2024-09-15     8
+
+-- !query6_0_before --
+a      1
+a      1
+a      1
+a      1
+a      1
+a      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+
+-- !query6_0_after --
+a      1
+a      1
+a      1
+a      1
+a      1
+a      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+
+-- !query7_0_before --
+a      1
+a      1
+a      1
+a      1
+a      1
+a      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+
+-- !query7_0_after --
+a      1
+a      1
+a      1
+a      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+b      1
+
diff --git 
a/regression-test/framework/src/main/groovy/org/apache/doris/regression/suite/Suite.groovy
 
b/regression-test/framework/src/main/groovy/org/apache/doris/regression/suite/Suite.groovy
index 73b2cf9b5bc..65c65f85c03 100644
--- 
a/regression-test/framework/src/main/groovy/org/apache/doris/regression/suite/Suite.groovy
+++ 
b/regression-test/framework/src/main/groovy/org/apache/doris/regression/suite/Suite.groovy
@@ -1663,7 +1663,9 @@ class Suite implements GroovyInterceptable {
     def mv_rewrite_success_without_check_chosen = { query_sql, mv_name ->
         explain {
             sql(" memo plan ${query_sql}")
-            contains("${mv_name} not chose")
+            check { result ->
+                result.contains("${mv_name} chose") || 
result.contains("${mv_name} not chose")
+            }
         }
     }
 
@@ -1721,7 +1723,9 @@ class Suite implements GroovyInterceptable {
 
         explain {
             sql(" memo plan ${query_sql}")
-            notContains("${mv_name} fail")
+            check { result ->
+                result.contains("${mv_name} chose") || 
result.contains("${mv_name} not chose")
+            }
         }
     }
 
@@ -1744,8 +1748,7 @@ class Suite implements GroovyInterceptable {
 
         explain {
             sql(" memo plan ${query_sql}")
-            notContains("${mv_name} chose")
-            notContains("${mv_name} not chose")
+            contains("${mv_name} fail")
         }
     }
 
diff --git 
a/regression-test/suites/nereids_rules_p0/mv/availability/materialized_view_switch.groovy
 
b/regression-test/suites/nereids_rules_p0/mv/availability/materialized_view_switch.groovy
index 1012d84434e..97d9325d959 100644
--- 
a/regression-test/suites/nereids_rules_p0/mv/availability/materialized_view_switch.groovy
+++ 
b/regression-test/suites/nereids_rules_p0/mv/availability/materialized_view_switch.groovy
@@ -151,8 +151,10 @@ suite("materialized_view_switch") {
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv_name_1"""
 
     sql "SET enable_materialized_view_rewrite=false"
-    async_mv_rewrite_fail(db, mv_name, query, "mv_name_2")
+    create_async_mv(db, "mv_name_2", mv_name)
+    mv_not_part_in(query, "mv_name_2")
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv_name_2"""
+
     sql "SET enable_materialized_view_rewrite=true"
     async_mv_rewrite_success(db, mv_name, query, "mv_name_3")
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv_name_3"""
diff --git 
a/regression-test/suites/nereids_rules_p0/mv/union_all_compensate/union_all_compensate.groovy
 
b/regression-test/suites/nereids_rules_p0/mv/union_all_compensate/union_all_compensate.groovy
new file mode 100644
index 00000000000..dbab81ee22a
--- /dev/null
+++ 
b/regression-test/suites/nereids_rules_p0/mv/union_all_compensate/union_all_compensate.groovy
@@ -0,0 +1,345 @@
+package mv.union_all_compensate
+// 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("union_all_compensate") {
+    String db = context.config.getDbNameByFile(context.file)
+    sql "use ${db}"
+    sql "set runtime_filter_mode=OFF";
+    sql "SET ignore_shape_nodes='PhysicalDistribute,PhysicalProject'"
+
+    sql """
+    drop table if exists test_table1
+    """
+    sql """
+    CREATE TABLE `test_table1` (
+      `data_date` datetime NOT NULL COMMENT '', 
+      `slot_id` varchar(255) NULL,
+      `num` int NULL
+    ) ENGINE = OLAP DUPLICATE KEY(
+      `data_date`,
+      `slot_id`
+    ) PARTITION BY RANGE(`data_date`) (
+    FROM ("2024-09-01") TO ("2024-09-30") INTERVAL 1 DAY
+    ) 
+     DISTRIBUTED BY HASH (`data_date`, `slot_id`) BUCKETS 10 
+     PROPERTIES (
+      "file_cache_ttl_seconds" = "0", 
+      "is_being_synced" = "false", 
+      "storage_medium" = "hdd", "storage_format" = "V2", 
+      "inverted_index_storage_format" = "V2", 
+      "light_schema_change" = "true", "disable_auto_compaction" = "false", 
+      "enable_single_replica_compaction" = "false", 
+      "group_commit_interval_ms" = "10000", 
+      "group_commit_data_bytes" = "134217728", 
+      'replication_num' = '1'
+    );
+    """
+
+    sql """
+    drop table if exists test_table2
+    """
+    sql """
+    CREATE TABLE `test_table2` (
+      `data_date` datetime NOT NULL COMMENT '', 
+      `slot_id` varchar(255) NULL,
+      `num` int NULL
+    ) ENGINE = OLAP DUPLICATE KEY(
+      `data_date`,
+      `slot_id`
+    ) PARTITION BY RANGE(`data_date`) (
+        FROM ("2024-09-01") TO ("2024-09-30") INTERVAL 1 DAY
+    ) 
+     DISTRIBUTED BY HASH (`data_date`, `slot_id`) BUCKETS 10 
+     PROPERTIES (
+      "file_cache_ttl_seconds" = "0", "is_being_synced" = "false", 
+      "storage_medium" = "hdd", "storage_format" = "V2", 
+      "inverted_index_storage_format" = "V2", 
+      "light_schema_change" = "true", "disable_auto_compaction" = "false", 
+      "enable_single_replica_compaction" = "false", 
+      "group_commit_interval_ms" = "10000", 
+      "group_commit_data_bytes" = "134217728", 
+      'replication_num' = '1'
+    );
+    """
+
+    sql """
+    insert into test_table1 values 
+    ('2024-09-11 00:10:00', 'a', 1), 
+    ('2024-09-11 00:20:00', 'a', 1),
+    ('2024-09-12 00:20:00', 'a', 1),
+    ('2024-09-12 00:20:00', 'b', 1),
+    ('2024-09-13 00:20:00', 'b', 1),
+    ('2024-09-13 00:30:00', 'b', 1),
+    ('2024-09-13 00:20:00', 'b', 1),
+    ('2024-09-13 00:30:00', 'b', 1),
+    ('2024-09-14 00:20:00', 'b', 1),
+    ('2024-09-14 00:30:00', 'b', 1),
+    ('2024-09-14 00:20:00', 'b', 1),
+    ('2024-09-14 00:30:00', 'b', 1),
+    ('2024-09-15 00:20:00', 'b', 1),
+    ('2024-09-15 00:30:00', 'b', 1),
+    ('2024-09-15 00:20:00', 'b', 1),
+    ('2024-09-15 00:30:00', 'b', 1);
+    """
+
+    sql """
+    insert into test_table2 values 
+    ('2024-09-11 00:10:00', 'a', 1), 
+    ('2024-09-11 00:20:00', 'a', 1),
+    ('2024-09-12 00:20:00', 'a', 1),
+    ('2024-09-12 00:20:00', 'b', 1),
+    ('2024-09-13 00:20:00', 'b', 1),
+    ('2024-09-13 00:30:00', 'b', 1),
+    ('2024-09-13 00:20:00', 'b', 1),
+    ('2024-09-13 00:30:00', 'b', 1),
+    ('2024-09-14 00:20:00', 'b', 1),
+    ('2024-09-14 00:30:00', 'b', 1),
+    ('2024-09-14 00:20:00', 'b', 1),
+    ('2024-09-14 00:30:00', 'b', 1),
+    ('2024-09-15 00:20:00', 'b', 1),
+    ('2024-09-15 00:30:00', 'b', 1),
+    ('2024-09-15 00:20:00', 'b', 1),
+    ('2024-09-15 00:30:00', 'b', 1);
+    """
+
+    sql """analyze table test_table1 with sync"""
+    sql """analyze table test_table2 with sync"""
+
+    // Aggregate, scalar aggregate, should not compensate union all
+    sql """ DROP MATERIALIZED VIEW IF EXISTS test_agg_mv"""
+    sql"""
+            CREATE MATERIALIZED VIEW test_agg_mv
+             BUILD IMMEDIATE REFRESH ON MANUAL
+             partition by(data_date) 
+             DISTRIBUTED BY HASH(data_date) BUCKETS 3 
+             PROPERTIES(
+               "refresh_partition_num" = "1", 'replication_num' = '1'
+             ) 
+            AS
+            SELECT 
+              date_trunc(t1.data_date, 'day') as data_date, 
+              to_date(t1.data_date) as dt, 
+              t2.slot_id,  
+              sum(t1.num) num_sum 
+            FROM 
+              test_table1 t1 
+              inner join
+              test_table2 t2 on t1.data_date = t2.data_date
+            GROUP BY 
+              date_trunc(t1.data_date, 'day'), 
+              to_date(t1.data_date), 
+              t2.slot_id;
+    """
+    waitingMTMVTaskFinishedByMvName("test_agg_mv")
+    sql """analyze table test_agg_mv with sync"""
+
+    def query1_0 =
+            """
+            select sum(t1.num) 
+            FROM 
+              test_table1 t1 
+              inner join
+              test_table2 t2 on t1.data_date = t2.data_date
+            where to_date(t1.data_date) >= '2024-09-12';
+            """
+    sql """set enable_materialized_view_rewrite = false;"""
+    order_qt_query1_0_before "${query1_0}"
+    sql """set enable_materialized_view_rewrite = true;"""
+    mv_rewrite_success(query1_0, "test_agg_mv")
+    order_qt_query1_0_after "${query1_0}"
+
+    // Data modify
+    sql """
+    insert into test_table1 values
+    ('2024-09-11 00:10:00', 'a', 1),
+    ('2024-09-11 00:20:00', 'a', 1),
+    ('2024-09-12 00:20:00', 'a', 1),
+    ('2024-09-12 00:20:00', 'b', 1);
+    """
+    sql """analyze table test_table1 with sync"""
+
+    sql """set enable_materialized_view_rewrite = false;"""
+    order_qt_query1_1_before "${query1_0}"
+    sql """set enable_materialized_view_rewrite = true;"""
+    mv_rewrite_fail(query1_0, "test_agg_mv")
+    order_qt_query1_1_after "${query1_0}"
+
+
+    // Aggregate, if query group by expression doesn't use the partition 
column, but the invalid partition is in the
+    // grace_period, should not compensate union all, but should rewritten 
successfully
+    def query2_0 =
+            """
+            select t2.slot_id,
+            sum(t1.num)
+            FROM
+              test_table1 t1
+              inner join
+              test_table2 t2 on t1.data_date = t2.data_date
+            where to_date(t1.data_date) >= '2024-09-12'
+            group by t2.slot_id;
+            """
+    sql """set enable_materialized_view_rewrite = false;"""
+    order_qt_query2_0_before "${query2_0}"
+    sql """set enable_materialized_view_rewrite = true;"""
+    sql """ALTER MATERIALIZED VIEW test_agg_mv set("grace_period"="100000");"""
+    mv_rewrite_success(query2_0, "test_agg_mv")
+    order_qt_query2_0_after "${query2_0}"
+
+
+    // Aggregate, if query group by expression doesn't use the partition 
column, and the invalid partition is not in the
+    // grace_period, should not compensate union all, and should rewritten fail
+    def query3_0 =
+            """
+            select t2.slot_id,
+            sum(t1.num)
+            FROM
+              test_table1 t1
+              inner join
+              test_table2 t2 on t1.data_date = t2.data_date
+            where to_date(t1.data_date) >= '2024-09-12'
+            group by t2.slot_id;
+            """
+    sql """set enable_materialized_view_rewrite = false;"""
+    order_qt_query3_0_before "${query2_0}"
+    sql """set enable_materialized_view_rewrite = true;"""
+    sql """ALTER MATERIALIZED VIEW test_agg_mv set("grace_period"="0");"""
+    mv_rewrite_fail(query2_0, "test_agg_mv")
+    order_qt_query3_0_after "${query2_0}"
+
+
+    // Aggregate, if query group by expression use the partition column, but 
the invalid partition is in the
+    // grace_period, should not compensate union all but should rewritten 
successfully
+    def query4_0 =
+            """
+            select to_date(t1.data_date),
+            sum(t1.num)
+            FROM
+              test_table1 t1
+              inner join
+              test_table2 t2 on t1.data_date = t2.data_date
+            where to_date(t1.data_date) >= '2024-09-12'
+            group by
+            to_date(t1.data_date);
+            """
+    sql """set enable_materialized_view_rewrite = false;"""
+    order_qt_query4_0_before "${query4_0}"
+    sql """set enable_materialized_view_rewrite = true;"""
+    sql """ALTER MATERIALIZED VIEW test_agg_mv set("grace_period"="100000");"""
+    mv_rewrite_success(query4_0, "test_agg_mv")
+    order_qt_query4_0_after "${query4_0}"
+
+
+    // Aggregate, if query group by expression use the partition column, and 
the invalid partition is not in the
+    // grace_period, should compensate union all, and should rewritten 
successfully
+    def query5_0 =
+            """
+            select to_date(t1.data_date),
+            sum(t1.num)
+            FROM
+              test_table1 t1
+              inner join
+              test_table2 t2 on t1.data_date = t2.data_date
+            where to_date(t1.data_date) >= '2024-09-12'
+            group by
+            to_date(t1.data_date);
+            """
+    sql """set enable_materialized_view_rewrite = false;"""
+    order_qt_query5_0_before "${query4_0}"
+    sql """ALTER MATERIALIZED VIEW test_agg_mv set("grace_period"="0");"""
+    sql """set enable_materialized_view_rewrite = true;"""
+    mv_rewrite_success(query4_0, "test_agg_mv")
+    order_qt_query5_0_after "${query4_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS test_agg_mv"""
+
+
+    sql """ DROP MATERIALIZED VIEW IF EXISTS test_join_mv"""
+    sql """
+    CREATE MATERIALIZED VIEW test_join_mv
+    BUILD IMMEDIATE REFRESH ON MANUAL
+    partition by(data_date)
+    DISTRIBUTED BY HASH(data_date) BUCKETS 3
+    PROPERTIES(
+      "refresh_partition_num" = "1",
+      'replication_num' = '1'
+    )
+    AS
+    SELECT
+      date_trunc(t3.data_date, 'day') as data_date,
+      to_date(t3.data_date) as dt,
+      t4.slot_id,
+      t3.num
+    FROM
+      test_table1 t3
+      left join
+      test_table2 t4 on t3.data_date = t4.data_date
+    """
+    waitingMTMVTaskFinishedByMvName("test_join_mv")
+    sql """analyze table test_table1 with sync"""
+
+    // Data modify
+    sql """
+    insert into test_table1 values
+    ('2024-09-11 00:10:00', 'a', 1),
+    ('2024-09-11 00:20:00', 'a', 1),
+    ('2024-09-12 00:20:00', 'a', 1),
+    ('2024-09-12 00:20:00', 'b', 1);
+    """
+    sql """analyze table test_join_mv with sync"""
+
+    // Join, if select expression not use the partition column, and the 
invalid partition is not in the
+    // grace_period, should union all,and should rewritten successfully
+    def query6_0 =
+            """
+            select
+              t4.slot_id,
+              t3.num
+            FROM
+              test_table1 t3
+              left join
+              test_table2 t4 on t3.data_date = t4.data_date
+            where to_date(t3.data_date) >= '2024-09-12';
+            """
+    sql """set enable_materialized_view_rewrite = false;"""
+    order_qt_query6_0_before "${query6_0}"
+    sql """set enable_materialized_view_rewrite = true;"""
+    mv_rewrite_success(query6_0, "test_join_mv")
+    order_qt_query6_0_after "${query6_0}"
+
+
+    // Join, if select expression not use the partition column, and the 
invalid partition is in the
+    // grace_period, should not compensate union all, and should rewritten 
successfully
+    def query7_0 =
+            """
+            select
+              t4.slot_id,
+              t3.num
+            FROM
+              test_table1 t3
+              left join
+              test_table2 t4 on t3.data_date = t4.data_date
+            where to_date(t3.data_date) >= '2024-09-12';
+            """
+    sql """set enable_materialized_view_rewrite = false;"""
+    order_qt_query7_0_before "${query7_0}"
+    sql """set enable_materialized_view_rewrite = true;"""
+    sql """ALTER MATERIALIZED VIEW test_join_mv 
set("grace_period"="100000");"""
+    mv_rewrite_success(query7_0, "test_join_mv")
+    order_qt_query7_0_after "${query7_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS test_join_mv"""
+
+}


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

Reply via email to