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

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


The following commit(s) were added to refs/heads/branch-3.0 by this push:
     new e70f7f8d0cb [pick](mtmv) Pick three PR: #44779 #44786 #44857  (#45130)
e70f7f8d0cb is described below

commit e70f7f8d0cb44d68d0a62e84857bfbd204b82d71
Author: seawinde <w...@selectdb.com>
AuthorDate: Mon Dec 9 23:59:52 2024 +0800

    [pick](mtmv) Pick three PR: #44779 #44786 #44857  (#45130)
    
    pick from master
    
    5ce76049
    #44779
    
    d350e3c5
    #44786
    
    8ea48afc
    #44857
---
 .../main/java/org/apache/doris/mtmv/MTMVCache.java |  14 +-
 .../java/org/apache/doris/mtmv/MTMVPlanUtil.java   |  31 +-
 .../jobs/joinorder/hypergraph/HyperElement.java    |  27 ++
 .../jobs/joinorder/hypergraph/edge/Edge.java       |   4 +-
 .../joinorder/hypergraph/node/AbstractNode.java    |   8 +-
 .../rules/exploration/mv/HyperGraphComparator.java | 103 +++--
 .../mv/InitMaterializationContextHook.java         |   8 +-
 .../mv/LogicalCompatibilityContext.java            |  74 +++-
 .../exploration/mv/MaterializedViewUtils.java      |   2 +-
 .../nereids/rules/exploration/mv/StructInfo.java   | 107 +++--
 .../plans/commands/UpdateMvByPartitionCommand.java |   9 +-
 .../trees/plans/commands/info/CreateMTMVInfo.java  |  41 +-
 .../commands/info/MTMVPartitionDefinition.java     |  71 ++--
 .../trees/plans/visitor/TableCollector.java        |  34 +-
 .../doris/nereids/trees/plans/PlanVisitorTest.java |  14 +-
 .../aggregate_without_roll_up.out                  |  24 ++
 .../mv/micro_test/micro_test_when_cte.out          | 128 ++++++
 .../aggregate_without_roll_up.groovy               | 463 ++++++++++++++++++++-
 .../range_datetime_part_up_rewrite.groovy          |  20 +-
 .../mv/micro_test/micro_test_when_cte.groovy       | 204 +++++++++
 20 files changed, 1172 insertions(+), 214 deletions(-)

diff --git a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVCache.java 
b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVCache.java
index 56061c75b9c..2895ad73e14 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVCache.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVCache.java
@@ -51,14 +51,18 @@ public class MTMVCache {
     // The materialized view plan which should be optimized by the same rules 
to query
     // and will remove top sink and unused sort
     private final Plan logicalPlan;
-    // The original plan of mv def sql
+    // The original rewritten plan of mv def sql
     private final Plan originalPlan;
+    // The analyzed plan of mv def sql, which is used by tableCollector,should 
not be optimized by rbo
+    private final Plan analyzedPlan;
     private final Statistics statistics;
     private final StructInfo structInfo;
 
-    public MTMVCache(Plan logicalPlan, Plan originalPlan, Statistics 
statistics, StructInfo structInfo) {
+    public MTMVCache(Plan logicalPlan, Plan originalPlan, Plan analyzedPlan,
+            Statistics statistics, StructInfo structInfo) {
         this.logicalPlan = logicalPlan;
         this.originalPlan = originalPlan;
+        this.analyzedPlan = analyzedPlan;
         this.statistics = statistics;
         this.structInfo = structInfo;
     }
@@ -71,6 +75,10 @@ public class MTMVCache {
         return originalPlan;
     }
 
+    public Plan getAnalyzedPlan() {
+        return analyzedPlan;
+    }
+
     public Statistics getStatistics() {
         return statistics;
     }
@@ -117,7 +125,7 @@ public class MTMVCache {
         Optional<StructInfo> structInfoOptional = 
MaterializationContext.constructStructInfo(mvPlan, originPlan,
                 planner.getCascadesContext(),
                 new BitSet());
-        return new MTMVCache(mvPlan, originPlan, needCost
+        return new MTMVCache(mvPlan, originPlan, planner.getAnalyzedPlan(), 
needCost
                 ? 
planner.getCascadesContext().getMemo().getRoot().getStatistics() : null,
                 structInfoOptional.orElseGet(() -> null));
     }
diff --git a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVPlanUtil.java 
b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVPlanUtil.java
index 576e87b44f8..35c06e74d3c 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVPlanUtil.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVPlanUtil.java
@@ -35,12 +35,10 @@ import 
org.apache.doris.nereids.properties.PhysicalProperties;
 import org.apache.doris.nereids.rules.RuleType;
 import org.apache.doris.nereids.trees.plans.Plan;
 import 
org.apache.doris.nereids.trees.plans.commands.ExplainCommand.ExplainLevel;
-import org.apache.doris.nereids.trees.plans.commands.info.CreateMTMVInfo;
 import org.apache.doris.nereids.trees.plans.logical.LogicalPlan;
 import org.apache.doris.nereids.trees.plans.visitor.TableCollector;
 import 
org.apache.doris.nereids.trees.plans.visitor.TableCollector.TableCollectorContext;
 import org.apache.doris.qe.ConnectContext;
-import org.apache.doris.qe.SessionVariable;
 
 import com.google.common.collect.ImmutableSet;
 import com.google.common.collect.Sets;
@@ -104,31 +102,20 @@ public class MTMVPlanUtil {
     public static MTMVRelation generateMTMVRelation(MTMV mtmv, ConnectContext 
ctx) {
         // Should not make table without data to empty relation when analyze 
the related table,
         // so add disable rules
-        SessionVariable sessionVariable = ctx.getSessionVariable();
-        Set<String> tempDisableRules = 
sessionVariable.getDisableNereidsRuleNames();
-        
sessionVariable.setDisableNereidsRules(CreateMTMVInfo.MTMV_PLANER_DISABLE_RULES);
-        if (ctx.getStatementContext() != null) {
-            
ctx.getStatementContext().invalidCache(SessionVariable.DISABLE_NEREIDS_RULES);
-        }
-        Plan plan;
-        try {
-            plan = getPlanBySql(mtmv.getQuerySql(), ctx);
-        } finally {
-            sessionVariable.setDisableNereidsRules(String.join(",", 
tempDisableRules));
-            
ctx.getStatementContext().invalidCache(SessionVariable.DISABLE_NEREIDS_RULES);
-        }
-        return generateMTMVRelation(plan);
+        Plan plan = getAnalyzePlanBySql(mtmv.getQuerySql(), ctx);
+        return generateMTMVRelation(plan, ctx);
     }
 
-    public static MTMVRelation generateMTMVRelation(Plan plan) {
-        return new MTMVRelation(getBaseTables(plan, true), getBaseTables(plan, 
false), getBaseViews(plan));
+    public static MTMVRelation generateMTMVRelation(Plan plan, ConnectContext 
connectContext) {
+        return new MTMVRelation(getBaseTables(plan, true, connectContext),
+                getBaseTables(plan, false, connectContext), 
getBaseViews(plan));
     }
 
-    private static Set<BaseTableInfo> getBaseTables(Plan plan, boolean expand) 
{
+    private static Set<BaseTableInfo> getBaseTables(Plan plan, boolean expand, 
ConnectContext connectContext) {
         TableCollectorContext collectorContext =
                 new TableCollector.TableCollectorContext(
                         com.google.common.collect.Sets
-                                .newHashSet(TableType.values()), expand);
+                                .newHashSet(TableType.values()), expand, 
connectContext);
         plan.accept(TableCollector.INSTANCE, collectorContext);
         Set<TableIf> collectedTables = collectorContext.getCollectedTables();
         return transferTableIfToInfo(collectedTables);
@@ -146,7 +133,7 @@ public class MTMVPlanUtil {
         return result;
     }
 
-    private static Plan getPlanBySql(String querySql, ConnectContext ctx) {
+    private static Plan getAnalyzePlanBySql(String querySql, ConnectContext 
ctx) {
         List<StatementBase> statements;
         try {
             statements = new NereidsParser().parseSQL(querySql);
@@ -159,7 +146,7 @@ public class MTMVPlanUtil {
         ctx.setStatementContext(new StatementContext());
         try {
             NereidsPlanner planner = new 
NereidsPlanner(ctx.getStatementContext());
-            return planner.planWithLock(logicalPlan, PhysicalProperties.ANY, 
ExplainLevel.NONE);
+            return planner.planWithLock(logicalPlan, PhysicalProperties.ANY, 
ExplainLevel.ANALYZED_PLAN);
         } finally {
             ctx.setStatementContext(original);
         }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/HyperElement.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/HyperElement.java
new file mode 100644
index 00000000000..6d8d7c6326c
--- /dev/null
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/HyperElement.java
@@ -0,0 +1,27 @@
+// 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.jobs.joinorder.hypergraph;
+
+/**
+ * This is the common base class for all
+ * */
+public interface HyperElement {
+
+    // Get the references nodes
+    long getReferenceNodes();
+}
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/edge/Edge.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/edge/Edge.java
index 7698d881c66..f75ed832501 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/edge/Edge.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/edge/Edge.java
@@ -18,6 +18,7 @@
 package org.apache.doris.nereids.jobs.joinorder.hypergraph.edge;
 
 import org.apache.doris.common.Pair;
+import org.apache.doris.nereids.jobs.joinorder.hypergraph.HyperElement;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.bitmap.LongBitmap;
 import org.apache.doris.nereids.trees.expressions.Expression;
 import org.apache.doris.nereids.trees.expressions.Slot;
@@ -32,7 +33,7 @@ import java.util.Set;
 /**
  * Edge in HyperGraph
  */
-public abstract class Edge {
+public abstract class Edge implements HyperElement {
     private final int index;
     private final double selectivity;
 
@@ -192,6 +193,7 @@ public abstract class Edge {
         return LongBitmap.isSubset(getReferenceNodes(), otherBitmap);
     }
 
+    @Override
     public long getReferenceNodes() {
         return LongBitmap.newBitmapUnion(leftExtendedNodes, 
rightExtendedNodes);
     }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/node/AbstractNode.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/node/AbstractNode.java
index a4a64e0449d..686576de771 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/node/AbstractNode.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/node/AbstractNode.java
@@ -17,6 +17,7 @@
 
 package org.apache.doris.nereids.jobs.joinorder.hypergraph.node;
 
+import org.apache.doris.nereids.jobs.joinorder.hypergraph.HyperElement;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.bitmap.LongBitmap;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.edge.Edge;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.edge.FilterEdge;
@@ -33,7 +34,7 @@ import java.util.Set;
 /**
  * HyperGraph Node.
  */
-public class AbstractNode {
+public class AbstractNode implements HyperElement {
     protected final int index;
     protected final List<JoinEdge> joinEdges;
     protected final List<FilterEdge> filterEdges;
@@ -65,6 +66,11 @@ public class AbstractNode {
                 .build();
     }
 
+    @Override
+    public long getReferenceNodes() {
+        return getNodeMap();
+    }
+
     public int getIndex() {
         return index;
     }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphComparator.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphComparator.java
index d4594583c31..22282a23516 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphComparator.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphComparator.java
@@ -19,12 +19,14 @@ package org.apache.doris.nereids.rules.exploration.mv;
 
 import org.apache.doris.common.Pair;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.ConflictRulesMaker;
+import org.apache.doris.nereids.jobs.joinorder.hypergraph.HyperElement;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.HyperGraph;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.bitmap.LongBitmap;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.edge.Edge;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.edge.FilterEdge;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.edge.JoinEdge;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.node.StructInfoNode;
+import 
org.apache.doris.nereids.rules.exploration.mv.StructInfo.ExpressionPosition;
 import org.apache.doris.nereids.rules.rewrite.PushDownFilterThroughJoin;
 import org.apache.doris.nereids.trees.expressions.Expression;
 import org.apache.doris.nereids.trees.expressions.NamedExpression;
@@ -51,6 +53,7 @@ import java.util.List;
 import java.util.Map;
 import java.util.Map.Entry;
 import java.util.Objects;
+import java.util.Optional;
 import java.util.Set;
 import java.util.stream.Collectors;
 import javax.annotation.Nullable;
@@ -79,9 +82,9 @@ public class HyperGraphComparator {
     private final Map<Edge, List<? extends Expression>> pullUpViewExprWithEdge 
= new HashMap<>();
     private final LogicalCompatibilityContext logicalCompatibilityContext;
     // this records the slots which needs to reject null
-    // the key is the target join which should reject null, the value is a 
pair, the first value of the pair is the
-    // join type, the second value is also a pair which left represents the 
slots in the left of join that should
-    // reject null, right represents the slots in the right of join that 
should reject null.
+    // the key is the view join edge which should reject null, the value is a 
pair, the first value of the pair is the
+    // query join type, the second value is also a pair which left represents 
the slots in the left of view join that
+    // should reject null, right represents the slots in the right of view 
join that should reject null.
     private final Map<JoinEdge, Pair<JoinType, Pair<Set<Slot>, Set<Slot>>>> 
inferredViewEdgeWithCond = new HashMap<>();
     private List<JoinEdge> viewJoinEdgesAfterInferring;
     private List<FilterEdge> viewFilterEdgesAfterInferring;
@@ -249,9 +252,17 @@ public class HyperGraphComparator {
         }
         int size = queryExprSetList.size();
         for (int i = 0; i < size; i++) {
-            Set<Expression> mappingQueryExprSet = 
queryExprSetList.get(i).stream()
-                    .map(logicalCompatibilityContext::getViewNodeExprFromQuery)
-                    .collect(Collectors.toSet());
+            Set<Expression> queryExpressions = queryExprSetList.get(i);
+            Set<Expression> mappingQueryExprSet = new HashSet<>();
+            for (Expression queryExpression : queryExpressions) {
+                Optional<Expression> mappingViewExprByQueryExpr = 
getMappingViewExprByQueryExpr(queryExpression, query,
+                        this.logicalCompatibilityContext,
+                        ExpressionPosition.NODE);
+                if (!mappingViewExprByQueryExpr.isPresent()) {
+                    return false;
+                }
+                mappingQueryExprSet.add(mappingViewExprByQueryExpr.get());
+            }
             if (!mappingQueryExprSet.equals(viewExprSetList.get(i))) {
                 return false;
             }
@@ -407,7 +418,10 @@ public class HyperGraphComparator {
             if (edgeMap.containsKey(entry.getValue())) {
                 continue;
             }
-            Expression viewExpr = 
logicalCompatibilityContext.getViewJoinExprFromQuery(entry.getKey());
+            Expression viewExpr = getMappingViewExprByQueryExpr(entry.getKey(),
+                    entry.getValue(),
+                    logicalCompatibilityContext,
+                    ExpressionPosition.JOIN_EDGE).orElse(null);
             if (viewExprToEdge.containsKey(viewExpr)) {
                 edgeMap.put(entry.getValue(), 
Objects.requireNonNull(viewExprToEdge.get(viewExpr)));
             }
@@ -441,15 +455,19 @@ public class HyperGraphComparator {
 
         HashMap<Edge, Edge> queryToViewEdgeMap = new HashMap<>();
         for (Entry<Expression, Collection<Edge>> entry : 
queryExprToEdge.asMap().entrySet()) {
-            Expression queryExprViewBased = 
logicalCompatibilityContext.getViewFilterExprFromQuery(entry.getKey());
-            if (queryExprViewBased == null) {
-                continue;
-            }
-            Collection<Edge> viewEdges = 
viewExprToEdge.get(queryExprViewBased);
-            if (viewEdges.isEmpty()) {
-                continue;
-            }
+            Expression queryExprViewBased = null;
             for (Edge queryEdge : entry.getValue()) {
+                queryExprViewBased = 
getMappingViewExprByQueryExpr(entry.getKey(),
+                        queryEdge,
+                        logicalCompatibilityContext,
+                        ExpressionPosition.FILTER_EDGE).orElse(null);
+                if (queryExprViewBased == null) {
+                    continue;
+                }
+                Collection<Edge> viewEdges = 
viewExprToEdge.get(queryExprViewBased);
+                if (viewEdges.isEmpty()) {
+                    continue;
+                }
                 for (Edge viewEdge : viewEdges) {
                     if (!isSubTreeNodesEquals(queryEdge, viewEdge, 
logicalCompatibilityContext)) {
                         // Such as query filter edge is <{1} --FILTER-- {}> 
but view filter edge is
@@ -514,17 +532,17 @@ public class HyperGraphComparator {
     }
 
     private boolean compareFilterEdgeWithNode(FilterEdge query, FilterEdge 
view) {
-        return rewriteQueryNodeMap(query.getReferenceNodes()) == 
view.getReferenceNodes();
+        return getViewNodesByQuery(query.getReferenceNodes()) == 
view.getReferenceNodes();
     }
 
     private boolean compareJoinEdgeWithNode(JoinEdge query, JoinEdge view) {
         boolean res = false;
         if (query.getJoinType().swap() == view.getJoinType()) {
-            res |= rewriteQueryNodeMap(query.getLeftExtendedNodes()) == 
view.getRightExtendedNodes()
-                    && rewriteQueryNodeMap(query.getRightExtendedNodes()) == 
view.getLeftExtendedNodes();
+            res |= getViewNodesByQuery(query.getLeftExtendedNodes()) == 
view.getRightExtendedNodes()
+                    && getViewNodesByQuery(query.getRightExtendedNodes()) == 
view.getLeftExtendedNodes();
         }
-        res |= rewriteQueryNodeMap(query.getLeftExtendedNodes()) == 
view.getLeftExtendedNodes()
-                && rewriteQueryNodeMap(query.getRightExtendedNodes()) == 
view.getRightExtendedNodes();
+        res |= getViewNodesByQuery(query.getLeftExtendedNodes()) == 
view.getLeftExtendedNodes()
+                && getViewNodesByQuery(query.getRightExtendedNodes()) == 
view.getRightExtendedNodes();
         return res;
     }
 
@@ -547,8 +565,8 @@ public class HyperGraphComparator {
     }
 
     private boolean tryInferEdge(JoinEdge query, JoinEdge view) {
-        if (rewriteQueryNodeMap(query.getLeftRequiredNodes()) != 
view.getLeftRequiredNodes()
-                || rewriteQueryNodeMap(query.getRightRequiredNodes()) != 
view.getRightRequiredNodes()) {
+        if (getViewNodesByQuery(query.getLeftRequiredNodes()) != 
view.getLeftRequiredNodes()
+                || getViewNodesByQuery(query.getRightRequiredNodes()) != 
view.getRightRequiredNodes()) {
             return false;
         }
         if (!query.getJoinType().equals(view.getJoinType())) {
@@ -569,7 +587,7 @@ public class HyperGraphComparator {
         return true;
     }
 
-    private long rewriteQueryNodeMap(long bitmap) {
+    private long getViewNodesByQuery(long bitmap) {
         long newBitmap = LongBitmap.newBitmap();
         for (int i : LongBitmap.getIterator(bitmap)) {
             int newIdx = getQueryToViewNodeIdMap().getOrDefault(i, 0);
@@ -578,6 +596,35 @@ public class HyperGraphComparator {
         return newBitmap;
     }
 
+    private Optional<Expression> getMappingViewExprByQueryExpr(Expression 
queryExpression,
+            HyperElement queryExpressionBelongedHyperElement,
+            LogicalCompatibilityContext context,
+            ExpressionPosition expressionPosition) {
+        Expression queryShuttledExpr;
+        Collection<Pair<Expression, HyperElement>> viewExpressions;
+        if (ExpressionPosition.JOIN_EDGE.equals(expressionPosition)) {
+            queryShuttledExpr = 
context.getQueryJoinShuttledExpr(queryExpression);
+            viewExpressions = 
context.getViewJoinExprFromQuery(queryShuttledExpr);
+        } else if (ExpressionPosition.FILTER_EDGE.equals(expressionPosition)) {
+            queryShuttledExpr = 
context.getQueryFilterShuttledExpr(queryExpression);
+            viewExpressions = 
context.getViewFilterExprFromQuery(queryShuttledExpr);
+        } else {
+            queryShuttledExpr = 
context.getQueryNodeShuttledExpr(queryExpression);
+            viewExpressions = 
context.getViewNodeExprFromQuery(queryShuttledExpr);
+        }
+        if (viewExpressions.size() == 1) {
+            return Optional.of(viewExpressions.iterator().next().key());
+        }
+        long queryReferenceNodes = 
queryExpressionBelongedHyperElement.getReferenceNodes();
+        long viewReferenceNodes = getViewNodesByQuery(queryReferenceNodes);
+        for (Pair<Expression, HyperElement> viewExpressionPair : 
viewExpressions) {
+            if (viewExpressionPair.value().getReferenceNodes() == 
viewReferenceNodes) {
+                return Optional.of(viewExpressionPair.key());
+            }
+        }
+        return Optional.empty();
+    }
+
     private void compareJoinEdgeWithExpr(Edge query, Edge view) {
         Set<? extends Expression> queryExprSet = query.getExpressionSet();
         Set<? extends Expression> viewExprSet = view.getExpressionSet();
@@ -585,7 +632,10 @@ public class HyperGraphComparator {
         Set<Expression> exprMappedOfView = new HashSet<>();
         List<Expression> residualQueryExpr = new ArrayList<>();
         for (Expression queryExpr : queryExprSet) {
-            Expression viewExpr = 
logicalCompatibilityContext.getViewJoinExprFromQuery(queryExpr);
+            Expression viewExpr = getMappingViewExprByQueryExpr(queryExpr,
+                    query,
+                    logicalCompatibilityContext,
+                    ExpressionPosition.JOIN_EDGE).orElse(null);
             if (viewExprSet.contains(viewExpr)) {
                 exprMappedOfView.add(viewExpr);
             } else {
@@ -604,7 +654,10 @@ public class HyperGraphComparator {
         Set<Expression> exprMappedOfView = new HashSet<>();
         List<Expression> residualQueryExpr = new ArrayList<>();
         for (Expression queryExpr : queryExprSet) {
-            Expression viewExpr = 
logicalCompatibilityContext.getViewFilterExprFromQuery(queryExpr);
+            Expression viewExpr = getMappingViewExprByQueryExpr(queryExpr,
+                    query,
+                    logicalCompatibilityContext,
+                    ExpressionPosition.FILTER_EDGE).orElse(null);
             if (viewExprSet.contains(viewExpr)) {
                 exprMappedOfView.add(viewExpr);
             } else {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/InitMaterializationContextHook.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/InitMaterializationContextHook.java
index f9ea00e178b..4f8198e0b3c 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/InitMaterializationContextHook.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/InitMaterializationContextHook.java
@@ -85,12 +85,12 @@ public class InitMaterializationContextHook implements 
PlannerHook {
             return;
         }
         // Only collect the table or mv which query use directly, to avoid 
useless mv partition in rewrite
-        TableCollectorContext collectorContext = new 
TableCollectorContext(Sets.newHashSet(), false);
+        // Keep use one connection context when in query, if new connect 
context,
+        // the ConnectionContext.get() will change
+        TableCollectorContext collectorContext = new 
TableCollectorContext(Sets.newHashSet(), false,
+                cascadesContext.getConnectContext());
         try {
             Plan rewritePlan = cascadesContext.getRewritePlan();
-            // Keep use one connection context when in query, if new connect 
context,
-            // the ConnectionContext.get() will change
-            
collectorContext.setConnectContext(cascadesContext.getConnectContext());
             rewritePlan.accept(TableCollector.INSTANCE, collectorContext);
         } catch (Exception e) {
             LOG.warn(String.format("MaterializationContext init table collect 
fail, current queryId is %s",
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java
index ca13c9701da..77ab37873d0 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java
@@ -17,6 +17,8 @@
 
 package org.apache.doris.nereids.rules.exploration.mv;
 
+import org.apache.doris.common.Pair;
+import org.apache.doris.nereids.jobs.joinorder.hypergraph.HyperElement;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.node.StructInfoNode;
 import org.apache.doris.nereids.memo.GroupExpression;
 import 
org.apache.doris.nereids.rules.exploration.mv.StructInfo.ExpressionPosition;
@@ -36,8 +38,10 @@ import org.apache.doris.nereids.util.Utils;
 import com.google.common.base.Suppliers;
 import com.google.common.collect.BiMap;
 import com.google.common.collect.HashBiMap;
+import com.google.common.collect.HashMultimap;
+import com.google.common.collect.Multimap;
 
-import java.util.HashMap;
+import java.util.Collection;
 import java.util.Map;
 import java.util.function.Supplier;
 
@@ -48,11 +52,15 @@ public class LogicalCompatibilityContext {
     private final BiMap<StructInfoNode, StructInfoNode> queryToViewNodeMapping;
     private final BiMap<Integer, Integer> queryToViewNodeIDMapping;
     private final ObjectId planNodeId;
-    private final Supplier<BiMap<Expression, Expression>> 
queryToViewJoinEdgeExpressionMappingSupplier;
-    private final Supplier<BiMap<Expression, Expression>> 
queryToViewNodeExpressionMappingSupplier;
-    private final Supplier<BiMap<Expression, Expression>> 
queryToViewFilterEdgeExpressionMappingSupplier;
-    @Deprecated
-    private BiMap<Expression, Expression> queryToViewAllExpressionMapping;
+    private final Supplier<Multimap<Expression, Pair<Expression, 
HyperElement>>>
+            queryToViewJoinEdgeExpressionMappingSupplier;
+    private final Supplier<Map<Expression, Expression>> 
queryToQueryShuttledJoinExpressionMappingSupplier;
+    private final Supplier<Multimap<Expression, Pair<Expression, 
HyperElement>>>
+            queryToViewNodeExpressionMappingSupplier;
+    private final Supplier<Map<Expression, Expression>> 
queryToQueryShuttledNodeExpressionMappingSupplier;
+    private final Supplier<Multimap<Expression, Pair<Expression, 
HyperElement>>>
+            queryToViewFilterEdgeExpressionMappingSupplier;
+    private final Supplier<Map<Expression, Expression>> 
queryToQueryShuttledFilterExpressionMappingSupplier;
 
     /**
      * LogicalCompatibilityContext
@@ -66,16 +74,25 @@ public class LogicalCompatibilityContext {
                         
queryStructInfo.getShuttledExpressionsToExpressionsMap().get(ExpressionPosition.JOIN_EDGE),
                         
viewStructInfo.getShuttledExpressionsToExpressionsMap().get(ExpressionPosition.JOIN_EDGE)));
 
+        this.queryToQueryShuttledJoinExpressionMappingSupplier = 
Suppliers.memoize(
+                () -> 
queryStructInfo.getExpressionToShuttledExpressionToMap().get(ExpressionPosition.JOIN_EDGE));
+
         this.queryToViewNodeExpressionMappingSupplier =
                 Suppliers.memoize(() -> 
generateExpressionMapping(viewToQuerySlotMapping,
                         
queryStructInfo.getShuttledExpressionsToExpressionsMap().get(ExpressionPosition.NODE),
                         
viewStructInfo.getShuttledExpressionsToExpressionsMap().get(ExpressionPosition.NODE)));
 
+        this.queryToQueryShuttledNodeExpressionMappingSupplier = 
Suppliers.memoize(
+                () -> 
queryStructInfo.getExpressionToShuttledExpressionToMap().get(ExpressionPosition.NODE));
+
         this.queryToViewFilterEdgeExpressionMappingSupplier =
                 Suppliers.memoize(() -> 
generateExpressionMapping(viewToQuerySlotMapping,
                         
queryStructInfo.getShuttledExpressionsToExpressionsMap().get(ExpressionPosition.FILTER_EDGE),
                         
viewStructInfo.getShuttledExpressionsToExpressionsMap().get(ExpressionPosition.FILTER_EDGE)));
 
+        this.queryToQueryShuttledFilterExpressionMappingSupplier = 
Suppliers.memoize(
+                () -> 
queryStructInfo.getExpressionToShuttledExpressionToMap().get(ExpressionPosition.FILTER_EDGE));
+
         this.queryToViewNodeMapping = queryToViewNodeMapping;
         this.queryToViewNodeIDMapping = HashBiMap.create();
         queryToViewNodeMapping.forEach((k, v) -> 
queryToViewNodeIDMapping.put(k.getIndex(), v.getIndex()));
@@ -92,18 +109,30 @@ public class LogicalCompatibilityContext {
         return queryToViewNodeIDMapping;
     }
 
-    public Expression getViewJoinExprFromQuery(Expression queryJoinExpr) {
+    public Collection<Pair<Expression, HyperElement>> 
getViewJoinExprFromQuery(Expression queryJoinExpr) {
         return 
queryToViewJoinEdgeExpressionMappingSupplier.get().get(queryJoinExpr);
     }
 
-    public Expression getViewFilterExprFromQuery(Expression queryJoinExpr) {
+    public Expression getQueryJoinShuttledExpr(Expression queryJoinExpr) {
+        return 
queryToQueryShuttledJoinExpressionMappingSupplier.get().get(queryJoinExpr);
+    }
+
+    public Collection<Pair<Expression, HyperElement>> 
getViewFilterExprFromQuery(Expression queryJoinExpr) {
         return 
queryToViewFilterEdgeExpressionMappingSupplier.get().get(queryJoinExpr);
     }
 
-    public Expression getViewNodeExprFromQuery(Expression queryJoinExpr) {
+    public Expression getQueryFilterShuttledExpr(Expression queryFilterExpr) {
+        return 
queryToQueryShuttledFilterExpressionMappingSupplier.get().get(queryFilterExpr);
+    }
+
+    public Collection<Pair<Expression, HyperElement>> 
getViewNodeExprFromQuery(Expression queryJoinExpr) {
         return 
queryToViewNodeExpressionMappingSupplier.get().get(queryJoinExpr);
     }
 
+    public Expression getQueryNodeShuttledExpr(Expression queryNodeExpr) {
+        return 
queryToQueryShuttledNodeExpressionMappingSupplier.get().get(queryNodeExpr);
+    }
+
     /**
      * Generate logical compatibility context,
      * this make expression mapping between query and view by relation and the 
slot in relation mapping
@@ -134,24 +163,31 @@ public class LogicalCompatibilityContext {
                 viewStructInfo);
     }
 
-    private static BiMap<Expression, Expression> generateExpressionMapping(
+    /**
+     * The result is multimap
+     * the key is shuttled query expr
+     * the value is original view expr collection
+     * */
+    private static Multimap<Expression, Pair<Expression, HyperElement>> 
generateExpressionMapping(
             Map<SlotReference, SlotReference> viewToQuerySlotMapping,
-            Map<Expression, Expression> queryShuttledExprToExprMap,
-            Map<Expression, Expression> viewShuttledExprToExprMap) {
-        final Map<Expression, Expression> viewEdgeToConjunctsMapQueryBased = 
new HashMap<>();
-        BiMap<Expression, Expression> queryToViewEdgeMapping = 
HashBiMap.create();
+            Multimap<Expression, Pair<Expression, HyperElement>> 
queryShuttledExprToExprMap,
+            Multimap<Expression, Pair<Expression, HyperElement>> 
viewShuttledExprToExprMap) {
+        Multimap<Expression, Pair<Expression, HyperElement>> 
queryToViewEdgeMapping = HashMultimap.create();
         if (queryShuttledExprToExprMap == null || viewShuttledExprToExprMap == 
null
                 || queryShuttledExprToExprMap.isEmpty() || 
viewShuttledExprToExprMap.isEmpty()) {
             return queryToViewEdgeMapping;
         }
+        final Multimap<Expression, Pair<Expression, HyperElement>> 
viewShuttledExprToExprMapQueryBased =
+                HashMultimap.create();
         viewShuttledExprToExprMap.forEach((shuttledExpr, expr) -> {
-            viewEdgeToConjunctsMapQueryBased.put(
+            viewShuttledExprToExprMapQueryBased.put(
                     orderSlotAsc(ExpressionUtils.replace(shuttledExpr, 
viewToQuerySlotMapping)), expr);
         });
-        queryShuttledExprToExprMap.forEach((exprSet, edge) -> {
-            Expression viewExpr = 
viewEdgeToConjunctsMapQueryBased.get(orderSlotAsc(exprSet));
-            if (viewExpr != null) {
-                queryToViewEdgeMapping.put(edge, viewExpr);
+        queryShuttledExprToExprMap.forEach((shuttledExpr, expr) -> {
+            Collection<Pair<Expression, HyperElement>> viewExpressions = 
viewShuttledExprToExprMapQueryBased.get(
+                    orderSlotAsc(shuttledExpr));
+            if (viewExpressions != null) {
+                queryToViewEdgeMapping.putAll(shuttledExpr, viewExpressions);
             }
         });
         return queryToViewEdgeMapping;
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
index ee4b002007e..4c5703e2768 100644
--- 
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
@@ -343,7 +343,7 @@ public class MaterializedViewUtils {
                     ImmutableList.of(Rewriter.custom(RuleType.ELIMINATE_SORT, 
EliminateSort::new))).execute();
             return childContext.getRewritePlan();
         }, mvPlan, originPlan);
-        return new MTMVCache(mvPlan, originPlan,
+        return new MTMVCache(mvPlan, originPlan, planner.getAnalyzedPlan(),
                 
planner.getCascadesContext().getMemo().getRoot().getStatistics(), null);
     }
 
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/StructInfo.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/StructInfo.java
index 5a84ab787d7..2e2119efe71 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/StructInfo.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/StructInfo.java
@@ -23,10 +23,10 @@ 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.jobs.joinorder.hypergraph.HyperElement;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.HyperGraph;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.edge.JoinEdge;
 import org.apache.doris.nereids.jobs.joinorder.hypergraph.node.StructInfoNode;
-import org.apache.doris.nereids.memo.Group;
 import org.apache.doris.nereids.memo.GroupExpression;
 import 
org.apache.doris.nereids.rules.exploration.mv.MaterializedViewUtils.TableQueryOperatorChecker;
 import org.apache.doris.nereids.rules.exploration.mv.Predicates.SplitPredicate;
@@ -63,12 +63,15 @@ import 
org.apache.doris.nereids.trees.plans.visitor.DefaultPlanVisitor;
 import org.apache.doris.nereids.trees.plans.visitor.ExpressionLineageReplacer;
 import org.apache.doris.nereids.util.ExpressionUtils;
 
+import com.google.common.collect.HashMultimap;
 import com.google.common.collect.ImmutableSet;
 import com.google.common.collect.Lists;
+import com.google.common.collect.Multimap;
 import com.google.common.collect.Sets;
 
 import java.util.ArrayList;
 import java.util.BitSet;
+import java.util.HashMap;
 import java.util.LinkedHashMap;
 import java.util.LinkedHashSet;
 import java.util.List;
@@ -111,9 +114,23 @@ public class StructInfo {
     // split predicates is shuttled
     private SplitPredicate splitPredicate;
     private EquivalenceClass equivalenceClass;
-    // Key is the expression shuttled and the value is the origin expression
+    // For value of Map, the key is the position of expression
+    // the value is the expressions and the hyper element of expression pair
+    // Key of pair is the expression shuttled and the value is the origin 
expression and the hyper element it belonged
+    // Sometimes origin expressions are different and shuttled expression is 
same
+    // Such as origin expressions are l_partkey#0 > 1 and l_partkey#10 > 1 and 
shuttled expression is l_partkey#10 > 1
     // this is for building LogicalCompatibilityContext later.
-    private final Map<ExpressionPosition, Map<Expression, Expression>> 
shuttledExpressionsToExpressionsMap;
+    private final Map<ExpressionPosition, Multimap<Expression, 
Pair<Expression, HyperElement>>>
+            shuttledExpressionsToExpressionsMap;
+    // For value of Map, the key is the position of expression
+    // the value is the original expression and shuttled expression map
+    // Such as origin expressions are l_partkey#0 > 1 and shuttled expression 
is l_partkey#10 > 1
+    // the map would be {ExpressionPosition.FILTER, {
+    //     l_partkey#0 > 1 : l_partkey#10 > 1
+    // }}
+    // this is for building LogicalCompatibilityContext later.
+    private final Map<ExpressionPosition, Map<Expression, Expression>> 
expressionToShuttledExpressionToMap;
+
     // Record the exprId and the corresponding expr map, this is used by 
expression shuttled
     private final Map<ExprId, Expression> namedExprIdAndExprMapping;
     private final List<? extends Expression> planOutputShuttledExpressions;
@@ -125,7 +142,9 @@ public class StructInfo {
             Plan bottomPlan, List<CatalogRelation> relations,
             Map<RelationId, StructInfoNode> relationIdStructInfoNodeMap,
             @Nullable Predicates predicates,
-            Map<ExpressionPosition, Map<Expression, Expression>> 
shuttledExpressionsToExpressionsMap,
+            Map<ExpressionPosition, Multimap<Expression, Pair<Expression, 
HyperElement>>>
+                    shuttledExpressionsToExpressionsMap,
+            Map<ExpressionPosition, Map<Expression, Expression>> 
expressionToShuttledExpressionToMap,
             Map<ExprId, Expression> namedExprIdAndExprMapping,
             BitSet tableIdSet,
             SplitPredicate splitPredicate,
@@ -144,6 +163,7 @@ public class StructInfo {
         this.splitPredicate = splitPredicate;
         this.equivalenceClass = equivalenceClass;
         this.shuttledExpressionsToExpressionsMap = 
shuttledExpressionsToExpressionsMap;
+        this.expressionToShuttledExpressionToMap = 
expressionToShuttledExpressionToMap;
         this.namedExprIdAndExprMapping = namedExprIdAndExprMapping;
         this.planOutputShuttledExpressions = planOutputShuttledExpressions;
     }
@@ -154,7 +174,8 @@ public class StructInfo {
     public StructInfo withPredicates(Predicates predicates) {
         return new StructInfo(this.originalPlan, this.originalPlanId, 
this.hyperGraph, this.valid, this.topPlan,
                 this.bottomPlan, this.relations, 
this.relationIdStructInfoNodeMap, predicates,
-                this.shuttledExpressionsToExpressionsMap, 
this.namedExprIdAndExprMapping, this.tableBitSet,
+                this.shuttledExpressionsToExpressionsMap, 
this.expressionToShuttledExpressionToMap,
+                this.namedExprIdAndExprMapping, this.tableBitSet,
                 null, null, this.planOutputShuttledExpressions);
     }
 
@@ -164,13 +185,16 @@ public class StructInfo {
     public StructInfo withTableBitSet(BitSet tableBitSet) {
         return new StructInfo(this.originalPlan, this.originalPlanId, 
this.hyperGraph, this.valid, this.topPlan,
                 this.bottomPlan, this.relations, 
this.relationIdStructInfoNodeMap, this.predicates,
-                this.shuttledExpressionsToExpressionsMap, 
this.namedExprIdAndExprMapping, tableBitSet,
+                this.shuttledExpressionsToExpressionsMap, 
this.expressionToShuttledExpressionToMap,
+                this.namedExprIdAndExprMapping, tableBitSet,
                 this.splitPredicate, this.equivalenceClass, 
this.planOutputShuttledExpressions);
     }
 
     private static boolean collectStructInfoFromGraph(HyperGraph hyperGraph,
             Plan topPlan,
-            Map<ExpressionPosition, Map<Expression, Expression>> 
shuttledExpressionsToExpressionsMap,
+            Map<ExpressionPosition, Multimap<Expression, Pair<Expression, 
HyperElement>>>
+                    shuttledExpressionsToExpressionsMap,
+            Map<ExpressionPosition, Map<Expression, Expression>> 
expressionToShuttledExpressionToMap,
             Map<ExprId, Expression> namedExprIdAndExprMapping,
             List<CatalogRelation> relations,
             Map<RelationId, StructInfoNode> relationIdStructInfoNodeMap,
@@ -198,8 +222,9 @@ public class StructInfo {
                     
structInfoNode.getPlan().accept(ExpressionLineageReplacer.INSTANCE, 
replaceContext);
                     // Replace expressions by expression map
                     List<Expression> replacedExpressions = 
replaceContext.getReplacedExpressions();
-                    
putShuttledExpressionsToExpressionsMap(shuttledExpressionsToExpressionsMap,
-                            ExpressionPosition.NODE, 
replacedExpressions.get(0), expression);
+                    
putShuttledExpressionToExpressionsMap(shuttledExpressionsToExpressionsMap,
+                            expressionToShuttledExpressionToMap,
+                            ExpressionPosition.NODE, 
replacedExpressions.get(0), expression, node);
                     // Record this, will be used in top level expression 
shuttle later, see the method
                     // ExpressionLineageReplacer#visitGroupPlan
                     
namedExprIdAndExprMapping.putAll(replaceContext.getExprIdExpressionMap());
@@ -225,8 +250,10 @@ public class StructInfo {
             // Replace expressions by expression map
             List<Expression> replacedExpressions = 
replaceContext.getReplacedExpressions();
             for (int i = 0; i < replacedExpressions.size(); i++) {
-                
putShuttledExpressionsToExpressionsMap(shuttledExpressionsToExpressionsMap,
-                        ExpressionPosition.JOIN_EDGE, 
replacedExpressions.get(i), joinConjunctExpressions.get(i));
+                
putShuttledExpressionToExpressionsMap(shuttledExpressionsToExpressionsMap,
+                        expressionToShuttledExpressionToMap,
+                        ExpressionPosition.JOIN_EDGE, 
replacedExpressions.get(i), joinConjunctExpressions.get(i),
+                        edge);
             }
             // Record this, will be used in top level expression shuttle 
later, see the method
             // ExpressionLineageReplacer#visitGroupPlan
@@ -238,10 +265,11 @@ public class StructInfo {
             filterExpressions.forEach(predicate -> {
                 // this is used for LogicalCompatibilityContext
                 ExpressionUtils.extractConjunction(predicate).forEach(expr ->
-                        
putShuttledExpressionsToExpressionsMap(shuttledExpressionsToExpressionsMap,
+                        
putShuttledExpressionToExpressionsMap(shuttledExpressionsToExpressionsMap,
+                                expressionToShuttledExpressionToMap,
                                 ExpressionPosition.FILTER_EDGE,
                                 
ExpressionUtils.shuttleExpressionWithLineage(predicate, topPlan, new BitSet()),
-                                predicate));
+                                predicate, filterEdge));
             });
         });
         return true;
@@ -313,11 +341,13 @@ public class StructInfo {
         // collect struct info fromGraph
         List<CatalogRelation> relationList = new ArrayList<>();
         Map<RelationId, StructInfoNode> relationIdStructInfoNodeMap = new 
LinkedHashMap<>();
-        Map<ExpressionPosition, Map<Expression, Expression>> 
shuttledHashConjunctsToConjunctsMap =
-                new LinkedHashMap<>();
+        Map<ExpressionPosition, Multimap<Expression, Pair<Expression, 
HyperElement>>>
+                shuttledHashConjunctsToConjunctsMap = new LinkedHashMap<>();
         Map<ExprId, Expression> namedExprIdAndExprMapping = new 
LinkedHashMap<>();
         BitSet tableBitSet = new BitSet();
+        Map<ExpressionPosition, Map<Expression, Expression>> 
expressionToShuttledExpressionToMap = new HashMap<>();
         boolean valid = collectStructInfoFromGraph(hyperGraph, topPlan, 
shuttledHashConjunctsToConjunctsMap,
+                expressionToShuttledExpressionToMap,
                 namedExprIdAndExprMapping,
                 relationList,
                 relationIdStructInfoNodeMap,
@@ -339,19 +369,11 @@ public class StructInfo {
                 
ExpressionUtils.shuttleExpressionWithLineage(originalPlan.getOutput(), 
originalPlan, new BitSet());
         return new StructInfo(originalPlan, originalPlanId, hyperGraph, valid, 
topPlan, bottomPlan,
                 relationList, relationIdStructInfoNodeMap, predicates, 
shuttledHashConjunctsToConjunctsMap,
+                expressionToShuttledExpressionToMap,
                 namedExprIdAndExprMapping, tableBitSet, null, null,
                 planOutputShuttledExpressions);
     }
 
-    /**
-     * Build Struct info from group.
-     * Maybe return multi structInfo when original plan already be rewritten 
by mv
-     */
-    public static StructInfo of(Group group) {
-        // TODO build graph from original plan and get relations and 
predicates from graph
-        return null;
-    }
-
     public List<CatalogRelation> getRelations() {
         return relations;
     }
@@ -408,21 +430,36 @@ public class StructInfo {
         return relationIdStructInfoNodeMap;
     }
 
-    public Map<ExpressionPosition, Map<Expression, Expression>> 
getShuttledExpressionsToExpressionsMap() {
+    public Map<ExpressionPosition, Multimap<Expression, Pair<Expression, 
HyperElement>>>
+            getShuttledExpressionsToExpressionsMap() {
         return shuttledExpressionsToExpressionsMap;
     }
 
-    private static void putShuttledExpressionsToExpressionsMap(
-            Map<ExpressionPosition, Map<Expression, Expression>> 
shuttledExpressionsToExpressionsMap,
+    public Map<ExpressionPosition, Map<Expression, Expression>> 
getExpressionToShuttledExpressionToMap() {
+        return expressionToShuttledExpressionToMap;
+    }
+
+    private static void putShuttledExpressionToExpressionsMap(
+            Map<ExpressionPosition, Multimap<Expression, Pair<Expression, 
HyperElement>>>
+                    shuttledExpressionsToExpressionsMap,
+            Map<ExpressionPosition, Map<Expression, Expression>> 
expressionPositionToExpressionToMap,
             ExpressionPosition expressionPosition,
-            Expression key, Expression value) {
-        Map<Expression, Expression> expressionExpressionMap = 
shuttledExpressionsToExpressionsMap.get(
-                expressionPosition);
-        if (expressionExpressionMap == null) {
-            expressionExpressionMap = new LinkedHashMap<>();
-            shuttledExpressionsToExpressionsMap.put(expressionPosition, 
expressionExpressionMap);
-        }
-        expressionExpressionMap.put(key, value);
+            Expression shuttledExpression, Expression originalExpression, 
HyperElement valueBelongedElement) {
+        Multimap<Expression, Pair<Expression, HyperElement>> 
shuttledExpressionToExpressionMap =
+                shuttledExpressionsToExpressionsMap.get(expressionPosition);
+        if (shuttledExpressionToExpressionMap == null) {
+            shuttledExpressionToExpressionMap = HashMultimap.create();
+            shuttledExpressionsToExpressionsMap.put(expressionPosition, 
shuttledExpressionToExpressionMap);
+        }
+        shuttledExpressionToExpressionMap.put(shuttledExpression, 
Pair.of(originalExpression, valueBelongedElement));
+
+        Map<Expression, Expression> originalExprToShuttledExprMap =
+                expressionPositionToExpressionToMap.get(expressionPosition);
+        if (originalExprToShuttledExprMap == null) {
+            originalExprToShuttledExprMap = new HashMap<>();
+            expressionPositionToExpressionToMap.put(expressionPosition, 
originalExprToShuttledExprMap);
+        }
+        originalExprToShuttledExprMap.put(originalExpression, 
shuttledExpression);
     }
 
     public List<? extends Expression> getExpressions() {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/UpdateMvByPartitionCommand.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/UpdateMvByPartitionCommand.java
index de284bd8377..36cc0f95a77 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/UpdateMvByPartitionCommand.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/UpdateMvByPartitionCommand.java
@@ -246,11 +246,16 @@ public class UpdateMvByPartitionCommand extends 
InsertOverwriteTableCommand {
             if (predicates.isEmpty()) {
                 return cte;
             }
+            List<LogicalSubQueryAlias<Plan>> rewrittenSubQueryAlias = new 
ArrayList<>();
             for (LogicalSubQueryAlias<Plan> subQueryAlias : 
cte.getAliasQueries()) {
+                List<Plan> subQueryAliasChildren = new ArrayList<>();
                 
this.virtualRelationNamePartSet.add(subQueryAlias.getQualifier());
-                subQueryAlias.children().forEach(subQuery -> 
subQuery.accept(this, predicates));
+                subQueryAlias.children().forEach(subQuery ->
+                        subQueryAliasChildren.add(subQuery.accept(this, 
predicates))
+                );
+                
rewrittenSubQueryAlias.add(subQueryAlias.withChildren(subQueryAliasChildren));
             }
-            return super.visitLogicalCTE(cte, predicates);
+            return super.visitLogicalCTE(new 
LogicalCTE<>(rewrittenSubQueryAlias, cte.child()), predicates);
         }
 
         @Override
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java
index fbca38d47a4..3e8e23d4044 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java
@@ -255,9 +255,21 @@ public class CreateMTMVInfo {
         NereidsPlanner planner = new NereidsPlanner(statementContext);
         // this is for expression column name infer when not use alias
         LogicalSink<Plan> logicalSink = new UnboundResultSink<>(logicalQuery);
-        // must disable constant folding by be, because be constant folding 
may return wrong type
-        
ctx.getSessionVariable().setVarOnce(SessionVariable.ENABLE_FOLD_CONSTANT_BY_BE, 
"false");
-        Plan plan = planner.planWithLock(logicalSink, PhysicalProperties.ANY, 
ExplainLevel.ALL_PLAN);
+        // Should not make table without data to empty relation when analyze 
the related table,
+        // so add disable rules
+        Set<String> tempDisableRules = 
ctx.getSessionVariable().getDisableNereidsRuleNames();
+        
ctx.getSessionVariable().setDisableNereidsRules(CreateMTMVInfo.MTMV_PLANER_DISABLE_RULES);
+        
ctx.getStatementContext().invalidCache(SessionVariable.DISABLE_NEREIDS_RULES);
+        Plan plan;
+        try {
+            // must disable constant folding by be, because be constant 
folding may return wrong type
+            
ctx.getSessionVariable().setVarOnce(SessionVariable.ENABLE_FOLD_CONSTANT_BY_BE, 
"false");
+            plan = planner.planWithLock(logicalSink, PhysicalProperties.ANY, 
ExplainLevel.ALL_PLAN);
+        } finally {
+            // after operate, roll back the disable rules
+            ctx.getSessionVariable().setDisableNereidsRules(String.join(",", 
tempDisableRules));
+            
ctx.getStatementContext().invalidCache(SessionVariable.DISABLE_NEREIDS_RULES);
+        }
         // can not contain VIEW or MTMV
         analyzeBaseTables(planner.getAnalyzedPlan());
         // can not contain Random function
@@ -268,8 +280,7 @@ public class CreateMTMVInfo {
             throw new AnalysisException("can not contain invalid expression");
         }
         getRelation(planner);
-        this.mvPartitionInfo = mvPartitionDefinition
-                .analyzeAndTransferToMTMVPartitionInfo(planner, ctx, 
logicalQuery);
+        this.mvPartitionInfo = 
mvPartitionDefinition.analyzeAndTransferToMTMVPartitionInfo(planner, ctx);
         this.partitionDesc = generatePartitionDesc(ctx);
         getColumns(plan, ctx, mvPartitionInfo.getPartitionCol(), distribution);
         analyzeKeys();
@@ -314,24 +325,10 @@ public class CreateMTMVInfo {
         }
     }
 
+    // Should use analyzed plan for collect views and tables
     private void getRelation(NereidsPlanner planner) {
-        // Should not make table without data to empty relation when analyze 
the related table,
-        // so add disable rules
-        ConnectContext ctx = planner.getCascadesContext().getConnectContext();
-        SessionVariable sessionVariable = ctx.getSessionVariable();
-        Set<String> tempDisableRules = 
sessionVariable.getDisableNereidsRuleNames();
-        
sessionVariable.setDisableNereidsRules(CreateMTMVInfo.MTMV_PLANER_DISABLE_RULES);
-        if (ctx.getStatementContext() != null) {
-            
ctx.getStatementContext().invalidCache(SessionVariable.DISABLE_NEREIDS_RULES);
-        }
-        Plan plan;
-        try {
-            plan = planner.planWithLock(logicalQuery, PhysicalProperties.ANY, 
ExplainLevel.NONE);
-        } finally {
-            sessionVariable.setDisableNereidsRules(String.join(",", 
tempDisableRules));
-            
ctx.getStatementContext().invalidCache(SessionVariable.DISABLE_NEREIDS_RULES);
-        }
-        this.relation = MTMVPlanUtil.generateMTMVRelation(plan);
+        this.relation = 
MTMVPlanUtil.generateMTMVRelation(planner.getAnalyzedPlan(),
+                planner.getCascadesContext().getConnectContext());
     }
 
     private PartitionDesc generatePartitionDesc(ConnectContext ctx) {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/MTMVPartitionDefinition.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/MTMVPartitionDefinition.java
index c4117e8608e..a26a97f7240 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/MTMVPartitionDefinition.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/MTMVPartitionDefinition.java
@@ -37,7 +37,6 @@ import org.apache.doris.nereids.NereidsPlanner;
 import org.apache.doris.nereids.analyzer.UnboundFunction;
 import org.apache.doris.nereids.analyzer.UnboundSlot;
 import org.apache.doris.nereids.exceptions.AnalysisException;
-import org.apache.doris.nereids.properties.PhysicalProperties;
 import org.apache.doris.nereids.rules.exploration.mv.MaterializedViewUtils;
 import 
org.apache.doris.nereids.rules.exploration.mv.MaterializedViewUtils.RelatedTableInfo;
 import org.apache.doris.nereids.trees.expressions.Cast;
@@ -45,11 +44,7 @@ import org.apache.doris.nereids.trees.expressions.Expression;
 import org.apache.doris.nereids.trees.expressions.Slot;
 import org.apache.doris.nereids.trees.expressions.functions.scalar.DateTrunc;
 import org.apache.doris.nereids.trees.expressions.literal.Literal;
-import org.apache.doris.nereids.trees.plans.Plan;
-import 
org.apache.doris.nereids.trees.plans.commands.ExplainCommand.ExplainLevel;
-import org.apache.doris.nereids.trees.plans.logical.LogicalPlan;
 import org.apache.doris.qe.ConnectContext;
-import org.apache.doris.qe.SessionVariable;
 
 import com.google.common.collect.Sets;
 
@@ -72,11 +67,9 @@ public class MTMVPartitionDefinition {
      *
      * @param planner planner
      * @param ctx ctx
-     * @param logicalQuery logicalQuery
      * @return MTMVPartitionInfo
      */
-    public MTMVPartitionInfo 
analyzeAndTransferToMTMVPartitionInfo(NereidsPlanner planner, ConnectContext 
ctx,
-            LogicalPlan logicalQuery) {
+    public MTMVPartitionInfo 
analyzeAndTransferToMTMVPartitionInfo(NereidsPlanner planner, ConnectContext 
ctx) {
         MTMVPartitionInfo mtmvPartitionInfo = new 
MTMVPartitionInfo(partitionType);
         if (this.partitionType == MTMVPartitionType.SELF_MANAGE) {
             return mtmvPartitionInfo;
@@ -100,7 +93,7 @@ public class MTMVPartitionDefinition {
             timeUnit = null;
         }
         mtmvPartitionInfo.setPartitionCol(partitionColName);
-        RelatedTableInfo relatedTableInfo = getRelatedTableInfo(planner, ctx, 
logicalQuery, partitionColName, timeUnit);
+        RelatedTableInfo relatedTableInfo = getRelatedTableInfo(planner, ctx, 
partitionColName, timeUnit);
         mtmvPartitionInfo.setRelatedCol(relatedTableInfo.getColumn());
         mtmvPartitionInfo.setRelatedTable(relatedTableInfo.getTableInfo());
         if (relatedTableInfo.getPartitionExpression().isPresent()) {
@@ -125,47 +118,33 @@ public class MTMVPartitionDefinition {
         return mtmvPartitionInfo;
     }
 
-    private RelatedTableInfo getRelatedTableInfo(NereidsPlanner planner, 
ConnectContext ctx, LogicalPlan
-            logicalQuery,
-            String partitionColName,
-            String timeUnit) {
+    // Should use rewritten plan without view and subQuery to get related 
partition table
+    private RelatedTableInfo getRelatedTableInfo(NereidsPlanner planner, 
ConnectContext ctx,
+            String partitionColName, String timeUnit) {
         CascadesContext cascadesContext = planner.getCascadesContext();
-        SessionVariable sessionVariable = 
cascadesContext.getConnectContext().getSessionVariable();
-        Set<String> tempDisableRules = 
sessionVariable.getDisableNereidsRuleNames();
-        // Should not make table without data to empty relation when analyze 
the related table,
-        // so add disable rules
-        
sessionVariable.setDisableNereidsRules(CreateMTMVInfo.MTMV_PLANER_DISABLE_RULES);
-        
cascadesContext.getStatementContext().invalidCache(SessionVariable.DISABLE_NEREIDS_RULES);
+
+        RelatedTableInfo relatedTableInfo = MaterializedViewUtils
+                .getRelatedTableInfo(partitionColName, timeUnit, 
planner.getRewrittenPlan(), cascadesContext);
+        if (!relatedTableInfo.isPctPossible()) {
+            throw new AnalysisException(String.format("Unable to find a 
suitable base table for partitioning,"
+                    + " the fail reason is %s", 
relatedTableInfo.getFailReason()));
+        }
+        MTMVRelatedTableIf mtmvBaseRealtedTable = 
MTMVUtil.getRelatedTable(relatedTableInfo.getTableInfo());
+        Set<String> partitionColumnNames = 
Sets.newTreeSet(String.CASE_INSENSITIVE_ORDER);
         try {
-            Plan mvRewrittenPlan =
-                    planner.planWithLock(logicalQuery, PhysicalProperties.ANY, 
ExplainLevel.REWRITTEN_PLAN);
-            RelatedTableInfo relatedTableInfo = MaterializedViewUtils
-                    .getRelatedTableInfo(partitionColName, timeUnit, 
mvRewrittenPlan, cascadesContext);
-            if (!relatedTableInfo.isPctPossible()) {
-                throw new AnalysisException(String.format("Unable to find a 
suitable base table for partitioning,"
-                        + " the fail reason is %s", 
relatedTableInfo.getFailReason()));
-            }
-            MTMVRelatedTableIf mtmvBaseRealtedTable = 
MTMVUtil.getRelatedTable(relatedTableInfo.getTableInfo());
-            Set<String> partitionColumnNames = 
Sets.newTreeSet(String.CASE_INSENSITIVE_ORDER);
-            try {
-                
partitionColumnNames.addAll(mtmvBaseRealtedTable.getPartitionColumnNames(Optional.empty()));
-            } catch (DdlException e) {
-                throw new AnalysisException(e.getMessage(), e);
-            }
+            
partitionColumnNames.addAll(mtmvBaseRealtedTable.getPartitionColumnNames(Optional.empty()));
+        } catch (DdlException e) {
+            throw new AnalysisException(e.getMessage(), e);
+        }
 
-            if (!partitionColumnNames.contains(relatedTableInfo.getColumn())) {
-                throw new AnalysisException("error related column: " + 
relatedTableInfo.getColumn());
-            }
-            if (!(mtmvBaseRealtedTable instanceof HMSExternalTable)
-                    && partitionColumnNames.size() != 1) {
-                throw new AnalysisException("only hms table support multi 
column partition.");
-            }
-            return relatedTableInfo;
-        } finally {
-            // after operate, roll back the disable rules
-            sessionVariable.setDisableNereidsRules(String.join(",", 
tempDisableRules));
-            
cascadesContext.getStatementContext().invalidCache(SessionVariable.DISABLE_NEREIDS_RULES);
+        if (!partitionColumnNames.contains(relatedTableInfo.getColumn())) {
+            throw new AnalysisException("error related column: " + 
relatedTableInfo.getColumn());
+        }
+        if (!(mtmvBaseRealtedTable instanceof HMSExternalTable)
+                && partitionColumnNames.size() != 1) {
+            throw new AnalysisException("only hms table support multi column 
partition.");
         }
+        return relatedTableInfo;
     }
 
     private static List<Expr> convertToLegacyArguments(List<Expression> 
children) {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/TableCollector.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/TableCollector.java
index 2e2cdb810f0..27ff1e4b68c 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/TableCollector.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/TableCollector.java
@@ -20,8 +20,8 @@ package org.apache.doris.nereids.trees.plans.visitor;
 import org.apache.doris.catalog.MTMV;
 import org.apache.doris.catalog.TableIf;
 import org.apache.doris.catalog.TableIf.TableType;
+import org.apache.doris.common.AnalysisException;
 import org.apache.doris.mtmv.MTMVCache;
-import org.apache.doris.mtmv.MTMVPlanUtil;
 import org.apache.doris.nereids.trees.plans.Plan;
 import org.apache.doris.nereids.trees.plans.logical.LogicalCatalogRelation;
 import org.apache.doris.nereids.trees.plans.physical.PhysicalCatalogRelation;
@@ -70,13 +70,19 @@ public class TableCollector extends 
DefaultPlanVisitor<Plan, TableCollectorConte
     }
 
     private void expandMvAndCollect(MTMV mtmv, TableCollectorContext context) {
-        if (!context.isExpand()) {
+        if (!context.isExpandMaterializedView()) {
             return;
         }
         // Make sure use only one connection context when in query to avoid 
ConnectionContext.get() wrong
-        MTMVCache expandedMv = MTMVCache.from(mtmv, 
context.getConnectContext() == null
-                ? MTMVPlanUtil.createMTMVContext(mtmv) : 
context.getConnectContext(), false);
-        expandedMv.getLogicalPlan().accept(this, context);
+        MTMVCache expandedMvCache;
+        try {
+            expandedMvCache = 
mtmv.getOrGenerateCache(context.getConnectContext());
+        } catch (AnalysisException exception) {
+            LOG.warn(String.format("expandMvAndCollect getOrGenerateCache 
fail, mtmv name is %s", mtmv.getName()),
+                    exception);
+            expandedMvCache = MTMVCache.from(mtmv, 
context.getConnectContext(), false);
+        }
+        expandedMvCache.getAnalyzedPlan().accept(this, context);
     }
 
     /**
@@ -87,12 +93,14 @@ public class TableCollector extends 
DefaultPlanVisitor<Plan, TableCollectorConte
         private final Set<TableIf> collectedTables = new HashSet<>();
         private final Set<TableType> targetTableTypes;
         // if expand the mv or not
-        private final boolean expand;
-        private ConnectContext connectContext;
+        private final boolean expandMaterializedView;
+        private final ConnectContext connectContext;
 
-        public TableCollectorContext(Set<TableType> targetTableTypes, boolean 
expand) {
+        public TableCollectorContext(Set<TableType> targetTableTypes, boolean 
expandMaterializedView,
+                ConnectContext connectContext) {
             this.targetTableTypes = targetTableTypes;
-            this.expand = expand;
+            this.expandMaterializedView = expandMaterializedView;
+            this.connectContext = connectContext;
         }
 
         public Set<TableIf> getCollectedTables() {
@@ -103,16 +111,12 @@ public class TableCollector extends 
DefaultPlanVisitor<Plan, TableCollectorConte
             return targetTableTypes;
         }
 
-        public boolean isExpand() {
-            return expand;
+        public boolean isExpandMaterializedView() {
+            return expandMaterializedView;
         }
 
         public ConnectContext getConnectContext() {
             return connectContext;
         }
-
-        public void setConnectContext(ConnectContext connectContext) {
-            this.connectContext = connectContext;
-        }
     }
 }
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 60f6e19faab..0c54f8fad5a 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
@@ -130,7 +130,7 @@ public class PlanVisitorTest extends TestWithFeService {
                             
Assertions.assertTrue(nondeterministicFunctionSet.get(0) instanceof Random);
                             // Check get tables
                             TableCollectorContext collectorContext = new 
TableCollector.TableCollectorContext(
-                                    Sets.newHashSet(TableType.OLAP), true);
+                                    Sets.newHashSet(TableType.OLAP), true, 
connectContext);
                             physicalPlan.accept(TableCollector.INSTANCE, 
collectorContext);
                             Set<String> expectedTables = new HashSet<>();
                             expectedTables.add("table1");
@@ -159,7 +159,7 @@ public class PlanVisitorTest extends TestWithFeService {
                             
Assertions.assertTrue(nondeterministicFunctionSet.get(1) instanceof Random);
                             // Check get tables
                             TableCollectorContext collectorContext = new 
TableCollector.TableCollectorContext(
-                                    Sets.newHashSet(TableType.OLAP), true);
+                                    Sets.newHashSet(TableType.OLAP), true, 
connectContext);
                             physicalPlan.accept(TableCollector.INSTANCE, 
collectorContext);
                             Set<String> expectedTables = new HashSet<>();
                             expectedTables.add("table1");
@@ -196,7 +196,7 @@ public class PlanVisitorTest extends TestWithFeService {
                             
Assertions.assertTrue(nondeterministicFunctionSet.get(0) instanceof Uuid);
                             // Check get tables
                             TableCollectorContext collectorContext = new 
TableCollector.TableCollectorContext(
-                                    Sets.newHashSet(TableType.OLAP), true);
+                                    Sets.newHashSet(TableType.OLAP), true, 
connectContext);
                             physicalPlan.accept(TableCollector.INSTANCE, 
collectorContext);
                             Set<String> expectedTables = new HashSet<>();
                             expectedTables.add("table1");
@@ -210,7 +210,7 @@ public class PlanVisitorTest extends TestWithFeService {
 
                             TableCollectorContext collectorContextWithNoExpand 
=
                                     new 
TableCollector.TableCollectorContext(Sets.newHashSet(TableType.OLAP),
-                                            false);
+                                            false, connectContext);
                             physicalPlan.accept(TableCollector.INSTANCE, 
collectorContextWithNoExpand);
                             Set<String> expectedTablesWithNoExpand = new 
HashSet<>();
                             expectedTablesWithNoExpand.add("table1");
@@ -222,7 +222,7 @@ public class PlanVisitorTest extends TestWithFeService {
                                     expectedTablesWithNoExpand);
 
                             TableCollectorContext mvCollectorContext = new 
TableCollector.TableCollectorContext(
-                                    
Sets.newHashSet(TableType.MATERIALIZED_VIEW), true);
+                                    
Sets.newHashSet(TableType.MATERIALIZED_VIEW), true, connectContext);
                             physicalPlan.accept(TableCollector.INSTANCE, 
mvCollectorContext);
                             Set<String> expectedMvs = new HashSet<>();
                             expectedMvs.add("mv1");
@@ -234,7 +234,7 @@ public class PlanVisitorTest extends TestWithFeService {
 
                             TableCollectorContext 
mvCollectorContextWithNoExpand =
                                     new TableCollector.TableCollectorContext(
-                                    
Sets.newHashSet(TableType.MATERIALIZED_VIEW), false);
+                                    
Sets.newHashSet(TableType.MATERIALIZED_VIEW), false, connectContext);
                             physicalPlan.accept(TableCollector.INSTANCE, 
mvCollectorContextWithNoExpand);
                             Set<String> expectedMvsWithNoExpand = new 
HashSet<>();
                             expectedMvsWithNoExpand.add("mv1");
@@ -246,7 +246,7 @@ public class PlanVisitorTest extends TestWithFeService {
 
                             TableCollectorContext allTableTypeWithExpand =
                                     new TableCollector.TableCollectorContext(
-                                            
Sets.newHashSet(TableType.values()), true);
+                                            
Sets.newHashSet(TableType.values()), true, connectContext);
                             physicalPlan.accept(TableCollector.INSTANCE, 
allTableTypeWithExpand);
                             // when collect in plan with expand, should 
collect table which is expended
                             Set<String> expectedTablesWithExpand = new 
HashSet<>();
diff --git 
a/regression-test/data/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.out
 
b/regression-test/data/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.out
index 5c9df6b7f92..c400e078daf 100644
--- 
a/regression-test/data/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.out
+++ 
b/regression-test/data/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.out
@@ -323,3 +323,27 @@ c  3       6       c,c,c   5.333333333333333       mi      
3       2
 1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       \N      \N      8       8       1
 1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       1       2       8       8       1
 
+-- !query29_0_before --
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       \N      \N      8       8       1
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       1       2       8       8       1
+
+-- !query29_0_after --
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       \N      \N      8       8       1
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       1       2       8       8       1
+
+-- !query30_0_before --
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       \N      \N      8       8       1
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       1       2       8       8       1
+
+-- !query30_0_after --
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       \N      \N      8       8       1
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       1       2       8       8       1
+
+-- !query31_0_before --
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       \N      \N      8       8       1
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       1       2       8       8       1
+
+-- !query31_0_after --
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       \N      \N      8       8       1
+1      2023-12-09      1       yy      2       2       2       4       3       
\N      2       3       1       2       8       8       1
+
diff --git 
a/regression-test/data/nereids_rules_p0/mv/micro_test/micro_test_when_cte.out 
b/regression-test/data/nereids_rules_p0/mv/micro_test/micro_test_when_cte.out
new file mode 100644
index 00000000000..8abaaf9adec
--- /dev/null
+++ 
b/regression-test/data/nereids_rules_p0/mv/micro_test/micro_test_when_cte.out
@@ -0,0 +1,128 @@
+-- This file is automatically generated. You should know what you did if you 
want to edit this
+-- !query_0_after --
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+
+-- !query_mv_0 --
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+
+-- !query_mv_1 --
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+
+-- !query_0_after --
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+
+-- !query_mv_2 --
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+
+-- !query_mv_3 --
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-08     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-09     4       1       3
+2023-12-10     4       1       3
+2023-12-10     4       1       3
+2023-12-10     4       1       3
+2023-12-10     4       1       3
+2023-12-10     4       1       3
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-10     4       1       4
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-11     4       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+2023-12-12     6       2       3
+
diff --git 
a/regression-test/suites/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.groovy
 
b/regression-test/suites/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.groovy
index 730d6a88c01..f082b3bdefd 100644
--- 
a/regression-test/suites/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.groovy
+++ 
b/regression-test/suites/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.groovy
@@ -45,11 +45,9 @@ suite("aggregate_without_roll_up") {
       "replication_num" = "1"
     );
     """
-
     sql """
     drop table if exists lineitem
     """
-
     sql"""
     CREATE TABLE IF NOT EXISTS lineitem (
       l_orderkey    INTEGER NOT NULL,
@@ -76,11 +74,9 @@ suite("aggregate_without_roll_up") {
       "replication_num" = "1"
     )
     """
-
     sql """
     drop table if exists partsupp
     """
-
     sql """
     CREATE TABLE IF NOT EXISTS partsupp (
       ps_partkey     INTEGER NOT NULL,
@@ -1517,4 +1513,463 @@ suite("aggregate_without_roll_up") {
     order_qt_query28_0_after "${query28_0}"
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv28_0"""
 
+
+
+    // query and mv has the same filter but position is different, should 
rewrite successfully
+    def mv29_0 = """
+      select 
+      o_custkey, 
+      o_orderdate, 
+      o_shippriority, 
+      o_comment, 
+      o_orderkey, 
+      orders.public_col as col1, 
+      l_orderkey, 
+      l_partkey, 
+      l_suppkey, 
+      lineitem.public_col as col2, 
+      ps_partkey, 
+      ps_suppkey, 
+      partsupp.public_col as col3, 
+      partsupp.public_col * 2 as col4, 
+      o_orderkey + l_orderkey + ps_partkey * 2, 
+      sum(
+        o_orderkey + l_orderkey + ps_partkey * 2
+      ), 
+      count() as count_all 
+    from 
+      (
+        select 
+          o_custkey, 
+          o_orderdate, 
+          o_shippriority, 
+          o_comment, 
+          o_orderkey, 
+          orders.public_col as public_col 
+        from 
+          orders
+      ) orders 
+      left join (
+        select 
+          l_orderkey, 
+          l_partkey, 
+          l_suppkey, 
+          lineitem.public_col as public_col 
+        from 
+          lineitem 
+        where 
+          lineitem.public_col is null 
+          or lineitem.public_col <> 1
+      ) lineitem on l_orderkey = o_orderkey 
+      inner join (
+        select 
+          ps_partkey, 
+          ps_suppkey, 
+          partsupp.public_col as public_col 
+        from 
+          partsupp
+      ) partsupp on ps_partkey = o_orderkey
+    where 
+      lineitem.public_col is null 
+      or lineitem.public_col <> 1 
+    group by 
+      1, 
+      2, 
+      3, 
+      4, 
+      5, 
+      6, 
+      7, 
+      8, 
+      9, 
+      10, 
+      11, 
+      12, 
+      13, 
+      14;
+    """
+    def query29_0 = """
+      select 
+      o_custkey, 
+      o_orderdate, 
+      o_shippriority, 
+      o_comment, 
+      o_orderkey, 
+      orders.public_col as col1, 
+      l_orderkey, 
+      l_partkey, 
+      l_suppkey, 
+      lineitem.public_col as col2, 
+      ps_partkey, 
+      ps_suppkey, 
+      partsupp.public_col as col3, 
+      partsupp.public_col * 2 as col4, 
+      o_orderkey + l_orderkey + ps_partkey * 2, 
+      sum(
+        o_orderkey + l_orderkey + ps_partkey * 2
+      ), 
+      count() as count_all 
+    from 
+      (
+        select 
+          o_custkey, 
+          o_orderdate, 
+          o_shippriority, 
+          o_comment, 
+          o_orderkey, 
+          orders.public_col as public_col 
+        from 
+          orders
+      ) orders 
+      left join (
+        select 
+          l_orderkey, 
+          l_partkey, 
+          l_suppkey, 
+          lineitem.public_col as public_col 
+        from 
+          lineitem 
+        where 
+          lineitem.public_col is null 
+          or lineitem.public_col <> 1
+      ) lineitem on l_orderkey = o_orderkey 
+      inner join (
+        select 
+          ps_partkey, 
+          ps_suppkey, 
+          partsupp.public_col as public_col 
+        from 
+          partsupp
+      ) partsupp on ps_partkey = o_orderkey
+    where 
+      lineitem.public_col is null 
+      or lineitem.public_col <> 1 
+    group by 
+      1, 
+      2, 
+      3, 
+      4, 
+      5, 
+      6, 
+      7, 
+      8, 
+      9, 
+      10, 
+      11, 
+      12, 
+      13, 
+      14;
+    """
+    order_qt_query29_0_before "${query29_0}"
+    async_mv_rewrite_success(db, mv29_0, query29_0, "mv29_0")
+    order_qt_query29_0_after "${query29_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv29_0"""
+
+
+    // query and mv has the same filter but position is different, should 
rewrite successfully
+    // mv join condition has alias
+    def mv30_0 = """
+      select 
+      o_custkey, 
+      o_orderdate, 
+      o_shippriority, 
+      o_comment, 
+      o_orderkey_alias, 
+      orders.public_col as col1, 
+      l_orderkey_alias, 
+      l_partkey, 
+      l_suppkey, 
+      lineitem.public_col as col2, 
+      ps_partkey_alias, 
+      ps_suppkey, 
+      partsupp.public_col as col3, 
+      partsupp.public_col * 2 as col4, 
+      o_orderkey_alias + l_orderkey_alias + ps_partkey_alias * 2, 
+      sum(
+        o_orderkey_alias + l_orderkey_alias + ps_partkey_alias * 2
+      ), 
+      count() as count_all 
+    from 
+      (
+        select 
+          o_custkey, 
+          o_orderdate, 
+          o_shippriority, 
+          o_comment, 
+          o_orderkey as o_orderkey_alias, 
+          orders.public_col as public_col 
+        from 
+          orders
+      ) orders 
+      left join (
+        select 
+          l_orderkey as l_orderkey_alias, 
+          l_partkey, 
+          l_suppkey, 
+          lineitem.public_col as public_col 
+        from 
+          lineitem 
+        where 
+          lineitem.public_col is null 
+          or lineitem.public_col <> 1
+      ) lineitem on lineitem.l_orderkey_alias = orders.o_orderkey_alias 
+      inner join (
+        select 
+          ps_partkey as ps_partkey_alias, 
+          ps_suppkey, 
+          partsupp.public_col as public_col 
+        from 
+          partsupp
+      ) partsupp on partsupp.ps_partkey_alias = orders.o_orderkey_alias
+    where 
+      lineitem.public_col is null 
+      or lineitem.public_col <> 1 
+      and o_orderkey_alias = 2
+    group by 
+      1, 
+      2, 
+      3, 
+      4, 
+      5, 
+      6, 
+      7, 
+      8, 
+      9, 
+      10, 
+      11, 
+      12, 
+      13, 
+      14;
+    """
+    def query30_0 = """
+      select 
+      o_custkey, 
+      o_orderdate, 
+      o_shippriority, 
+      o_comment, 
+      o_orderkey, 
+      orders.public_col as col1, 
+      l_orderkey, 
+      l_partkey, 
+      l_suppkey, 
+      lineitem.public_col as col2, 
+      ps_partkey, 
+      ps_suppkey, 
+      partsupp.public_col as col3, 
+      partsupp.public_col * 2 as col4, 
+      o_orderkey + l_orderkey + ps_partkey * 2, 
+      sum(
+        o_orderkey + l_orderkey + ps_partkey * 2
+      ), 
+      count() as count_all 
+    from 
+      (
+        select 
+          o_custkey, 
+          o_orderdate, 
+          o_shippriority, 
+          o_comment, 
+          o_orderkey, 
+          orders.public_col as public_col 
+        from 
+          orders
+      ) orders 
+      left join (
+        select 
+          l_orderkey, 
+          l_partkey, 
+          l_suppkey, 
+          lineitem.public_col as public_col 
+        from 
+          lineitem 
+        where 
+          lineitem.public_col is null 
+          or lineitem.public_col <> 1
+      ) lineitem on l_orderkey = o_orderkey 
+      inner join (
+        select 
+          ps_partkey, 
+          ps_suppkey, 
+          partsupp.public_col as public_col 
+        from 
+          partsupp
+      ) partsupp on ps_partkey = o_orderkey
+    where 
+      lineitem.public_col is null 
+      or lineitem.public_col <> 1 
+      and o_orderkey = 2
+    group by 
+      1, 
+      2, 
+      3, 
+      4, 
+      5, 
+      6, 
+      7, 
+      8, 
+      9, 
+      10, 
+      11, 
+      12, 
+      13, 
+      14;
+    """
+    order_qt_query30_0_before "${query30_0}"
+    async_mv_rewrite_success(db, mv30_0, query30_0, "mv30_0")
+    order_qt_query30_0_after "${query30_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv30_0"""
+
+
+    // query and mv has the same filter but position is different, should 
rewrite successfully
+    // query join condition has alias
+    def mv31_0 = """
+      select 
+      o_custkey, 
+      o_orderdate, 
+      o_shippriority, 
+      o_comment, 
+      o_orderkey, 
+      orders.public_col as col1, 
+      l_orderkey, 
+      l_partkey, 
+      l_suppkey, 
+      lineitem.public_col as col2, 
+      ps_partkey, 
+      ps_suppkey, 
+      partsupp.public_col as col3, 
+      partsupp.public_col * 2 as col4, 
+      o_orderkey + l_orderkey + ps_partkey * 2, 
+      sum(
+        o_orderkey + l_orderkey + ps_partkey * 2
+      ), 
+      count() as count_all 
+    from 
+      (
+        select 
+          o_custkey, 
+          o_orderdate, 
+          o_shippriority, 
+          o_comment, 
+          o_orderkey, 
+          orders.public_col as public_col 
+        from 
+          orders
+      ) orders 
+      left join (
+        select 
+          l_orderkey, 
+          l_partkey, 
+          l_suppkey, 
+          lineitem.public_col as public_col 
+        from 
+          lineitem 
+        where 
+          lineitem.public_col is null 
+          or lineitem.public_col <> 1
+      ) lineitem on l_orderkey = o_orderkey 
+      inner join (
+        select 
+          ps_partkey, 
+          ps_suppkey, 
+          partsupp.public_col as public_col 
+        from 
+          partsupp
+      ) partsupp on ps_partkey = o_orderkey
+    where 
+      lineitem.public_col is null 
+      or lineitem.public_col <> 1 
+      and o_orderkey = 2
+    group by 
+      1, 
+      2, 
+      3, 
+      4, 
+      5, 
+      6, 
+      7, 
+      8, 
+      9, 
+      10, 
+      11, 
+      12, 
+      13, 
+      14;
+    """
+    def query31_0 = """
+select 
+      o_custkey, 
+      o_orderdate, 
+      o_shippriority, 
+      o_comment, 
+      o_orderkey_alias, 
+      orders.public_col as col1, 
+      l_orderkey_alias, 
+      l_partkey, 
+      l_suppkey, 
+      lineitem.public_col as col2, 
+      ps_partkey_alias, 
+      ps_suppkey, 
+      partsupp.public_col as col3, 
+      partsupp.public_col * 2 as col4, 
+      o_orderkey_alias + l_orderkey_alias + ps_partkey_alias * 2, 
+      sum(
+        o_orderkey_alias + l_orderkey_alias + ps_partkey_alias * 2
+      ), 
+      count() as count_all 
+    from 
+      (
+        select 
+          o_custkey, 
+          o_orderdate, 
+          o_shippriority, 
+          o_comment, 
+          o_orderkey as o_orderkey_alias, 
+          orders.public_col as public_col 
+        from 
+          orders
+      ) orders 
+      left join (
+        select 
+          l_orderkey as l_orderkey_alias, 
+          l_partkey, 
+          l_suppkey, 
+          lineitem.public_col as public_col 
+        from 
+          lineitem 
+        where 
+          lineitem.public_col is null 
+          or lineitem.public_col <> 1
+      ) lineitem on lineitem.l_orderkey_alias = orders.o_orderkey_alias 
+      inner join (
+        select 
+          ps_partkey as ps_partkey_alias, 
+          ps_suppkey, 
+          partsupp.public_col as public_col 
+        from 
+          partsupp
+      ) partsupp on partsupp.ps_partkey_alias = orders.o_orderkey_alias
+    where 
+      lineitem.public_col is null 
+      or lineitem.public_col <> 1 
+      and o_orderkey_alias = 2
+    group by 
+      1, 
+      2, 
+      3, 
+      4, 
+      5, 
+      6, 
+      7, 
+      8, 
+      9, 
+      10, 
+      11, 
+      12, 
+      13, 
+      14;
+    """
+    order_qt_query31_0_before "${query31_0}"
+    async_mv_rewrite_success(db, mv31_0, query31_0, "mv31_0")
+    order_qt_query31_0_after "${query31_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv31_0"""
 }
diff --git 
a/regression-test/suites/nereids_rules_p0/mv/create_part_and_up/range_datetime_part_up_rewrite.groovy
 
b/regression-test/suites/nereids_rules_p0/mv/create_part_and_up/range_datetime_part_up_rewrite.groovy
index f8e601e64f5..140a91edd7c 100644
--- 
a/regression-test/suites/nereids_rules_p0/mv/create_part_and_up/range_datetime_part_up_rewrite.groovy
+++ 
b/regression-test/suites/nereids_rules_p0/mv/create_part_and_up/range_datetime_part_up_rewrite.groovy
@@ -168,7 +168,7 @@ suite("mtmv_range_datetime_part_up_rewrite") {
     for (int i = 0; i < mv_name_list.size(); i++) {
         def job_name = getJobName(db, mv_name_list[i])
         waitingMTMVTaskFinished(job_name)
-        mv_rewrite_success(query_stmt_list[i], mv_name_list[i])
+        mv_rewrite_any_success(query_stmt_list[i], mv_name_list)
         compare_res(query_stmt_list[i] + " order by 1,2,3")
     }
 
@@ -178,13 +178,15 @@ suite("mtmv_range_datetime_part_up_rewrite") {
         (1, null, 3, 1, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-10-18', 
'2023-10-18', 'a', 'b', 'yyyyyyyyy', '2023-11-29 03:00:00')"""
     sql """alter table lineitem_range_datetime_union modify column l_comment 
set stats ('row_count'='8');"""
     for (int i = 0; i < mv_name_list.size(); i++) {
-        mv_rewrite_success(query_stmt_list[i], mv_name_list[i])
+        // both mv should rewrite success
+        mv_rewrite_any_success(query_stmt_list[i], mv_name_list)
         compare_res(query_stmt_list[i] + " order by 1,2,3")
     }
 
     for (int i = 0; i < mv_name_list.size(); i++) {
         sql """refresh MATERIALIZED VIEW ${mv_name_list[i]} auto;"""
-        mv_rewrite_success(query_stmt_list[i], mv_name_list[i])
+        // both mv should rewrite success
+        mv_rewrite_any_success(query_stmt_list[i], mv_name_list)
         compare_res(query_stmt_list[i] + " order by 1,2,3")
     }
 
@@ -192,25 +194,29 @@ suite("mtmv_range_datetime_part_up_rewrite") {
         (3, null, 3, 1, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-10-18', 
'2023-10-18', 'a', 'b', 'yyyyyyyyy', '2023-11-29 03:00:00');"""
     sql """alter table lineitem_range_datetime_union modify column l_comment 
set stats ('row_count'='9');"""
     for (int i = 0; i < mv_name_list.size(); i++) {
-        mv_rewrite_success(query_stmt_list[i], mv_name_list[i])
+        // both mv should rewrite success
+        mv_rewrite_any_success(query_stmt_list[i], mv_name_list)
         compare_res(query_stmt_list[i] + " order by 1,2,3")
     }
 
     for (int i = 0; i < mv_name_list.size(); i++) {
         sql """refresh MATERIALIZED VIEW ${mv_name_list[i]} auto;"""
-        mv_rewrite_success(query_stmt_list[i], mv_name_list[i])
+        // both mv should rewrite success
+        mv_rewrite_any_success(query_stmt_list[i], mv_name_list)
         compare_res(query_stmt_list[i] + " order by 1,2,3")
     }
 
     sql """ALTER TABLE lineitem_range_datetime_union DROP PARTITION IF EXISTS 
p4 FORCE"""
     for (int i = 0; i < mv_name_list.size(); i++) {
-        mv_rewrite_success(query_stmt_list[i], mv_name_list[i])
+        // both mv should rewrite success
+        mv_rewrite_any_success(query_stmt_list[i], mv_name_list)
         compare_res(query_stmt_list[i] + " order by 1,2,3")
     }
 
     for (int i = 0; i < mv_name_list.size(); i++) {
         sql """refresh MATERIALIZED VIEW ${mv_name_list[i]} auto;"""
-        mv_rewrite_success(query_stmt_list[i], mv_name_list[i])
+        // both mv should rewrite success
+        mv_rewrite_any_success(query_stmt_list[i], mv_name_list)
         compare_res(query_stmt_list[i] + " order by 1,2,3")
     }
 
diff --git 
a/regression-test/suites/nereids_rules_p0/mv/micro_test/micro_test_when_cte.groovy
 
b/regression-test/suites/nereids_rules_p0/mv/micro_test/micro_test_when_cte.groovy
new file mode 100644
index 00000000000..9e80ea966fc
--- /dev/null
+++ 
b/regression-test/suites/nereids_rules_p0/mv/micro_test/micro_test_when_cte.groovy
@@ -0,0 +1,204 @@
+package mv.micro_test
+// 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("micro_test_when_cte") {
+    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 orders
+    """
+    sql """
+    CREATE TABLE IF NOT EXISTS orders  (
+      o_orderkey       INTEGER NOT NULL,
+      o_custkey        INTEGER NOT NULL,
+      o_orderstatus    CHAR(1) NOT NULL,
+      o_totalprice     DECIMALV3(15,2) NOT NULL,
+      o_orderdate      DATE NOT NULL,
+      o_orderpriority  CHAR(15) NOT NULL,  
+      o_clerk          CHAR(15) NOT NULL, 
+      o_shippriority   INTEGER NOT NULL,
+      O_COMMENT        VARCHAR(79) NOT NULL
+    )
+    DUPLICATE KEY(o_orderkey, o_custkey)
+    PARTITION BY RANGE(o_orderdate)(
+    FROM ('2023-12-01') TO ('2023-12-31') INTERVAL 1 DAY
+    )
+    DISTRIBUTED BY HASH(o_orderkey) BUCKETS 3
+    PROPERTIES (
+      "replication_num" = "1"
+    );
+    """
+
+    sql """
+    drop table if exists lineitem
+    """
+    sql"""
+    CREATE TABLE IF NOT EXISTS lineitem (
+      l_orderkey    INTEGER NOT NULL,
+      l_partkey     INTEGER NOT NULL,
+      l_suppkey     INTEGER NOT NULL,
+      l_linenumber  INTEGER NOT NULL,
+      l_quantity    DECIMALV3(15,2) NOT NULL,
+      l_extendedprice  DECIMALV3(15,2) NOT NULL,
+      l_discount    DECIMALV3(15,2) NOT NULL,
+      l_tax         DECIMALV3(15,2) NOT NULL,
+      l_returnflag  CHAR(1) NOT NULL,
+      l_linestatus  CHAR(1) NOT NULL,
+      l_shipdate    DATE NOT NULL,
+      l_commitdate  DATE NOT NULL,
+      l_receiptdate DATE NOT NULL,
+      l_shipinstruct CHAR(25) NOT NULL,
+      l_shipmode     CHAR(10) NOT NULL,
+      l_comment      VARCHAR(44) NOT NULL
+    )
+    DUPLICATE KEY(l_orderkey, l_partkey, l_suppkey, l_linenumber)
+    PARTITION BY RANGE(l_shipdate) 
+    (FROM ('2023-12-01') TO ('2023-12-31') INTERVAL 1 DAY)
+    DISTRIBUTED BY HASH(l_orderkey) BUCKETS 3
+    PROPERTIES (
+      "replication_num" = "1"
+    )
+    """
+
+    sql """
+    drop table if exists partsupp
+    """
+    sql """
+    CREATE TABLE IF NOT EXISTS partsupp (
+      ps_partkey     INTEGER NOT NULL,
+      ps_suppkey     INTEGER NOT NULL,
+      ps_availqty    INTEGER NOT NULL,
+      ps_supplycost  DECIMALV3(15,2)  NOT NULL,
+      ps_comment     VARCHAR(199) NOT NULL 
+    )
+    DUPLICATE KEY(ps_partkey, ps_suppkey)
+    DISTRIBUTED BY HASH(ps_partkey) BUCKETS 3
+    PROPERTIES (
+      "replication_num" = "1"
+    )
+    """
+
+    sql """ insert into lineitem values
+    (1, 2, 3, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-08', '2023-12-09', 
'2023-12-10', 'a', 'b', 'yyyyyyyyy'),
+    (2, 4, 3, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-09', '2023-12-09', 
'2023-12-10', 'a', 'b', 'yyyyyyyyy'),
+    (3, 2, 4, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-10', '2023-12-09', 
'2023-12-10', 'a', 'b', 'yyyyyyyyy'),
+    (4, 3, 3, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-11', '2023-12-09', 
'2023-12-10', 'a', 'b', 'yyyyyyyyy'),
+    (5, 2, 3, 6, 7.5, 8.5, 9.5, 10.5, 'k', 'o', '2023-12-12', '2023-12-12', 
'2023-12-13', 'c', 'd', 'xxxxxxxxx');
+    """
+
+    sql """
+    insert into orders values
+    (1, 1, 'o', 9.5, '2023-12-08', 'a', 'b', 1, 'yy'),
+    (1, 1, 'o', 10.5, '2023-12-08', 'a', 'b', 1, 'yy'),
+    (1, 1, 'o', 10.5, '2023-12-08', 'a', 'b', 1, 'yy'),
+    (1, 1, 'o', 10.5, '2023-12-08', 'a', 'b', 1, 'yy'),
+    (2, 1, 'o', 11.5, '2023-12-09', 'a', 'b', 1, 'yy'),
+    (2, 1, 'o', 11.5, '2023-12-09', 'a', 'b', 1, 'yy'),
+    (2, 1, 'o', 11.5, '2023-12-09', 'a', 'b', 1, 'yy'),
+    (3, 1, 'o', 12.5, '2023-12-10', 'a', 'b', 1, 'yy'),
+    (3, 1, 'o', 12.5, '2023-12-10', 'a', 'b', 1, 'yy'),
+    (3, 1, 'o', 12.5, '2023-12-10', 'a', 'b', 1, 'yy'),
+    (3, 1, 'o', 33.5, '2023-12-10', 'a', 'b', 1, 'yy'),
+    (4, 2, 'o', 43.2, '2023-12-11', 'c','d',2, 'mm'),
+    (4, 2, 'o', 43.2, '2023-12-11', 'c','d',2, 'mm'),
+    (4, 2, 'o', 43.2, '2023-12-11', 'c','d',2, 'mm'),
+    (5, 2, 'o', 56.2, '2023-12-12', 'c','d',2, 'mi'),
+    (5, 2, 'o', 56.2, '2023-12-12', 'c','d',2, 'mi'),
+    (5, 2, 'o', 56.2, '2023-12-12', 'c','d',2, 'mi'),
+    (5, 2, 'o', 1.2, '2023-12-12', 'c','d',2, 'mi');  
+    """
+
+    sql """
+    insert into partsupp values
+    (2, 3, 9, 10.01, 'supply1'),
+    (2, 3, 10, 11.01, 'supply2');
+    """
+
+    sql """analyze table partsupp with sync"""
+    sql """analyze table lineitem with sync"""
+    sql """analyze table orders with sync"""
+    sql """alter table lineitem modify column l_comment set stats 
('row_count'='5');"""
+    sql """alter table orders modify column O_COMMENT set stats 
('row_count'='18');"""
+    sql """alter table partsupp modify column ps_comment set stats 
('row_count'='2');"""
+
+    def query_sql = """
+    WITH scan_data_cte as (
+        select t1.l_shipdate, t1.L_LINENUMBER, orders.O_CUSTKEY, l_suppkey
+        from (select * from lineitem where L_LINENUMBER > 1) t1
+        left join orders on t1.L_ORDERKEY = orders.O_ORDERKEY
+    )
+    SELECT *  FROM scan_data_cte; 
+    """
+    def mv_sql = """
+    WITH scan_data_cte as (
+        select t1.l_shipdate, t1.L_LINENUMBER, orders.O_CUSTKEY, l_suppkey
+        from (select * from lineitem where L_LINENUMBER > 1) t1
+        left join orders on t1.L_ORDERKEY = orders.O_ORDERKEY
+    )
+    SELECT *  FROM scan_data_cte; 
+    """
+    def mv_name = """mv_with_cte_test"""
+
+    // query directly
+    order_qt_query_0_after "${query_sql}"
+
+    // create and build complete mv
+    create_async_mv(db, mv_name, mv_sql)
+    // refresh mv complete
+    sql """refresh materialized view ${mv_name} complete"""
+    // query mv directly
+    waitingMTMVTaskFinishedByMvName(mv_name)
+    order_qt_query_mv_0 "select * from ${mv_name}"
+
+    // create and build partition mv
+    create_async_partition_mv(db, mv_name, mv_sql, "(l_shipdate)")
+
+    // refresh mv partly
+    sql """refresh materialized view ${mv_name} 
partitions(p_20231208_20231209)"""
+    // query mv directly
+    waitingMTMVTaskFinishedByMvName(mv_name)
+    order_qt_query_mv_1 "select * from ${mv_name}"
+
+    // query rewrite
+    mv_rewrite_success(mv_sql, mv_name)
+    order_qt_query_0_after "${query_sql}"
+
+    // DML
+    // base table insert into data when not partition table
+    sql """
+    insert into orders values
+    (1, 1, 'o', 9.5, '2023-12-08', 'a', 'b', 1, 'yy');
+    """
+    sql """refresh materialized view ${mv_name} complete"""
+    // query mv directly
+    waitingMTMVTaskFinishedByMvName(mv_name)
+    order_qt_query_mv_2 "select * from ${mv_name}"
+
+    // base table insert into data when partition table
+    sql """
+    insert into lineitem values
+    (1, 2, 3, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-10', '2023-12-09', 
'2023-12-10', 'a', 'b', 'yyyyyyyyy');
+    """
+    sql """refresh materialized view ${mv_name} 
partitions(p_20231210_20231211)"""
+    // query mv directly
+    waitingMTMVTaskFinishedByMvName(mv_name)
+    order_qt_query_mv_3 "select * from ${mv_name}"
+}


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

Reply via email to