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

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


The following commit(s) were added to refs/heads/master by this push:
     new b43c16c9f35 [opt](mtmv) Optimize pre rbo query rewrite by materialized 
view performance to avoid fe oom and slow (#59154)
b43c16c9f35 is described below

commit b43c16c9f35c02701c915daee56de662976389a1
Author: seawinde <[email protected]>
AuthorDate: Tue Jan 27 19:01:08 2026 +0800

    [opt](mtmv) Optimize pre rbo query rewrite by materialized view performance 
to avoid fe oom and slow (#59154)
    
    This PR introduces several performance optimizations for transparent
    query rewriting using materialized views:
    
    1. Optimized structural information caching key:
    When refreshing the structural information of a query, we now use
    tableId as the key instead of more complex combinations. This
    significantly reduces the number of distinct structural information
    entries. Although this may introduce some false positives during initial
    retrieval, subsequent transparent rewriting rules will accurately filter
    and validate only the applicable materialized views for rewriting.
    
    Note: In the preceding RBO (Rule-Based Optimization) rewriting
    phase—when fetching structural information and plans from the memo—we
    still use relationId to ensure correctness in plan retrieval.
    
    2. Early pruning during structural information refresh:
    During refresh, we proactively prune and only update structural
    information that is actually relevant to transparent rewriting, avoiding
    unnecessary computation.
    Improved performance for nested rewrites:
    
    3. We refined the versioning scheme of structural information stored in
    the memo. Instead of maintaining a generic version number, each entry
    now tracks both the version number and the specific materialized view it
    belongs to. This enables more precise invalidation and avoids refreshing
    irrelevant structural information during updates.
---
 .../main/java/org/apache/doris/mtmv/MTMVCache.java |   3 +-
 .../org/apache/doris/nereids/StatementContext.java |  11 +-
 .../java/org/apache/doris/nereids/memo/Memo.java   |  37 ++-
 .../apache/doris/nereids/memo/StructInfoMap.java   | 294 ++++++++++++++++-----
 .../mv/AbstractMaterializedViewAggregateRule.java  |  24 +-
 .../mv/AbstractMaterializedViewJoinRule.java       |   1 -
 .../mv/AbstractMaterializedViewRule.java           |  53 ++--
 .../mv/AbstractMaterializedViewScanRule.java       |   1 -
 .../mv/AbstractMaterializedViewWindowRule.java     |   1 -
 .../exploration/mv/MaterializationContext.java     |  29 +-
 .../exploration/mv/MaterializedViewUtils.java      |  30 ++-
 .../mv/PartitionIncrementMaintainer.java           |  11 +-
 .../mv/PreMaterializedViewRewriter.java            |   4 +-
 .../nereids/rules/exploration/mv/Predicates.java   |   3 +-
 .../nereids/rules/exploration/mv/StructInfo.java   |  34 ++-
 .../rules/rewrite/QueryPartitionCollector.java     |   7 +-
 .../apache/doris/nereids/util/ExpressionUtils.java |   8 +-
 .../doris/nereids/memo/StructInfoMapTest.java      |  75 ++++--
 .../mv/PointQueryShouldNotMvRewriteTest.java       |   5 +-
 .../mv/PreMaterializedViewRewriterTest.java        |   4 +-
 .../apache/doris/nereids/mv/PredicatesTest.java    |  14 +-
 .../rules/exploration/mv/HyperGraphAggTest.java    |   7 +-
 .../apache/doris/nereids/sqltest/SqlTestBase.java  |   9 +-
 .../doris/nereids/util/ExpressionUtilsTest.java    |   3 +-
 .../mv/nested_mtmv/nested_mtmv.groovy              |  12 +-
 25 files changed, 426 insertions(+), 254 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 3a2cb45f51a..391a9546500 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
@@ -48,7 +48,6 @@ import org.apache.logging.log4j.LogManager;
 import org.apache.logging.log4j.Logger;
 
 import java.util.ArrayList;
-import java.util.BitSet;
 import java.util.List;
 import java.util.Optional;
 
@@ -182,7 +181,7 @@ public class MTMVCache {
         }, mvPlan, plan, false);
         // Construct structInfo once for use later
         Optional<StructInfo> structInfoOptional = 
MaterializationContext.constructStructInfo(mvPlan, plan,
-                cascadesContext, new BitSet());
+                cascadesContext);
         return Pair.of(mvPlan, structInfoOptional.orElse(null));
     }
 }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java
index 13a9de3ea58..3e1945965c9 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java
@@ -265,9 +265,10 @@ public class StatementContext implements Closeable {
     private long materializedViewRewriteDuration = 0L;
 
     // Record used table and it's used partitions
-    private final Multimap<List<String>, Pair<RelationId, Set<String>>> 
tableUsedPartitionNameMap = HashMultimap
-            .create();
-    private final Map<Integer, Integer> relationIdToCommonTableIdMap = new 
HashMap<>();
+    private final Multimap<List<String>, Pair<RelationId, Set<String>>> 
tableUsedPartitionNameMap =
+            HashMultimap.create();
+    // Record query common table id to relation id mapping, this is used for 
mv rewrite
+    private final Multimap<Integer, Integer> commonTableIdToRelationIdToMap = 
HashMultimap.create();
 
     // Record mtmv and valid partitions map because this is time-consuming 
behavior
     private final Map<BaseTableInfo, Collection<Partition>> 
mvCanRewritePartitionsMap = new HashMap<>();
@@ -1093,8 +1094,8 @@ public class StatementContext implements Closeable {
         return tableUsedPartitionNameMap;
     }
 
-    public Map<Integer, Integer> getRelationIdToCommonTableIdMap() {
-        return relationIdToCommonTableIdMap;
+    public Multimap<Integer, Integer> getCommonTableIdToRelationIdMap() {
+        return commonTableIdToRelationIdToMap;
     }
 
     public Map<BaseTableInfo, Collection<Partition>> 
getMvCanRewritePartitionsMap() {
diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Memo.java 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Memo.java
index 0ab94348f27..d132966daab 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Memo.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Memo.java
@@ -34,6 +34,7 @@ import org.apache.doris.nereids.trees.expressions.Expression;
 import org.apache.doris.nereids.trees.plans.GroupPlan;
 import org.apache.doris.nereids.trees.plans.LeafPlan;
 import org.apache.doris.nereids.trees.plans.Plan;
+import org.apache.doris.nereids.trees.plans.TableId;
 import org.apache.doris.nereids.trees.plans.algebra.CatalogRelation;
 import org.apache.doris.nereids.trees.plans.algebra.SetOperation;
 import org.apache.doris.nereids.trees.plans.logical.LogicalCatalogRelation;
@@ -51,6 +52,7 @@ import org.apache.logging.log4j.LogManager;
 import org.apache.logging.log4j.Logger;
 
 import java.util.ArrayList;
+import java.util.BitSet;
 import java.util.HashMap;
 import java.util.HashSet;
 import java.util.LinkedHashMap;
@@ -59,7 +61,7 @@ import java.util.Map;
 import java.util.Optional;
 import java.util.PriorityQueue;
 import java.util.Set;
-import java.util.concurrent.atomic.AtomicLong;
+import java.util.concurrent.atomic.AtomicInteger;
 import java.util.stream.Collectors;
 import java.util.stream.Stream;
 import javax.annotation.Nullable;
@@ -74,7 +76,9 @@ public class Memo {
             EventChannel.getDefaultChannel().addConsumers(new 
LogConsumer(GroupMergeEvent.class, EventChannel.LOG)));
     private static long stateId = 0;
     private final ConnectContext connectContext;
-    private final AtomicLong refreshVersion = new AtomicLong(1);
+    // The key is the query tableId, the value is the refresh version when 
last refresh, this is needed
+    // because struct info refresh base on target tableId.
+    private final Map<Integer, AtomicInteger> refreshVersion = new HashMap<>();
     private final Map<Class<? extends AbstractMaterializedViewRule>, 
Set<Long>> materializationCheckSuccessMap =
             new LinkedHashMap<>();
     private final Map<Class<? extends AbstractMaterializedViewRule>, 
Set<Long>> materializationCheckFailMap =
@@ -128,12 +132,28 @@ public class Memo {
         return groupExpressions.size();
     }
 
-    public long getRefreshVersion() {
-        return refreshVersion.get();
+    /** get the refresh version map*/
+    public Map<Integer, AtomicInteger> getRefreshVersion() {
+        return refreshVersion;
     }
 
-    public long incrementAndGetRefreshVersion() {
-        return refreshVersion.incrementAndGet();
+    /** return the incremented refresh version for the given commonTableId*/
+    public long incrementAndGetRefreshVersion(int commonTableId) {
+        return refreshVersion.compute(commonTableId, (k, v) -> {
+            if (v == null) {
+                return new AtomicInteger(1);
+            }
+            v.incrementAndGet();
+            return v;
+        }).get();
+    }
+
+    /** return the incremented refresh version for the given relationId set*/
+    public void incrementAndGetRefreshVersion(BitSet commonTableIdSet) {
+        for (int i = commonTableIdSet.nextSetBit(0); i >= 0;
+                i = commonTableIdSet.nextSetBit(i + 1)) {
+            incrementAndGetRefreshVersion(i);
+        }
     }
 
     /**
@@ -461,13 +481,14 @@ public class Memo {
                     plan.getLogicalProperties(), 
targetGroup.getLogicalProperties());
             throw new IllegalStateException("Insert a plan into targetGroup 
but differ in logicalproperties");
         }
-        // TODO Support sync materialized view in the future
         if (connectContext != null
                 && 
connectContext.getSessionVariable().isEnableMaterializedViewNestRewrite()
                 && plan instanceof LogicalCatalogRelation
                 && ((CatalogRelation) plan).getTable() instanceof MTMV
                 && !plan.getGroupExpression().isPresent()) {
-            incrementAndGetRefreshVersion();
+            TableId mvCommonTableId
+                    = 
this.connectContext.getStatementContext().getTableId(((CatalogRelation) 
plan).getTable());
+            incrementAndGetRefreshVersion(mvCommonTableId.asInt());
         }
         Optional<GroupExpression> groupExpr = plan.getGroupExpression();
         if (groupExpr.isPresent()) {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/StructInfoMap.java 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/StructInfoMap.java
index 2f9e0ca5243..ac567c4bd34 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/StructInfoMap.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/StructInfoMap.java
@@ -43,6 +43,7 @@ import java.util.List;
 import java.util.Map;
 import java.util.Optional;
 import java.util.Set;
+import java.util.concurrent.atomic.AtomicInteger;
 import javax.annotation.Nullable;
 
 /**
@@ -51,80 +52,221 @@ import javax.annotation.Nullable;
 public class StructInfoMap {
 
     public static final Logger LOG = LogManager.getLogger(StructInfoMap.class);
+    // 2166136261
+    private static final int FNV32_OFFSET_BASIS = 0x811C9DC5;
+    // 16777619
+    private static final int FNV32_PRIME = 0x01000193;
+    /**
+     * Strategy for table ID mode
+     */
+    private static final IdModeStrategy TABLE_ID_STRATEGY = new 
IdModeStrategy() {
+        @Override
+        public Map<BitSet, Pair<GroupExpression, List<BitSet>>> 
getGroupExpressionMap(StructInfoMap structInfoMap) {
+            return structInfoMap.groupExpressionMapByTableId;
+        }
+
+        @Override
+        public Map<BitSet, StructInfo> getInfoMap(StructInfoMap structInfoMap) 
{
+            return structInfoMap.infoMapByTableId;
+        }
+
+        @Override
+        public BitSet constructLeaf(GroupExpression groupExpression, 
CascadesContext cascadesContext,
+                                    boolean forceRefresh) {
+            Plan plan = groupExpression.getPlan();
+            BitSet tableMap = new BitSet();
+            if (plan instanceof LogicalCatalogRelation) {
+                LogicalCatalogRelation relation = (LogicalCatalogRelation) 
plan;
+                TableIf table = relation.getTable();
+                if (!forceRefresh && cascadesContext.getStatementContext()
+                        
.getMaterializationRewrittenSuccessSet().contains(table.getFullQualifiers())) {
+                    return tableMap;
+                }
+                
tableMap.set(cascadesContext.getStatementContext().getTableId(table).asInt());
+            }
+            return tableMap;
+        }
+
+        @Override
+        public int computeMemoVersion(BitSet targetIdMap, CascadesContext 
cascadesContext) {
+            return getMemoVersion(targetIdMap, 
cascadesContext.getMemo().getRefreshVersion());
+        }
+    };
+
+    /**
+     * Strategy for relation ID mode
+     */
+    private static final IdModeStrategy RELATION_ID_STRATEGY = new 
IdModeStrategy() {
+        @Override
+        public Map<BitSet, Pair<GroupExpression, List<BitSet>>> 
getGroupExpressionMap(StructInfoMap structInfoMap) {
+            return structInfoMap.groupExpressionMapByRelationId;
+        }
+
+        @Override
+        public Map<BitSet, StructInfo> getInfoMap(StructInfoMap structInfoMap) 
{
+            return structInfoMap.infoMapByRelationId;
+        }
+
+        @Override
+        public BitSet constructLeaf(GroupExpression groupExpression, 
CascadesContext cascadesContext,
+                                    boolean forceRefresh) {
+            Plan plan = groupExpression.getPlan();
+            BitSet tableMap = new BitSet();
+            if (plan instanceof LogicalCatalogRelation) {
+                LogicalCatalogRelation relation = (LogicalCatalogRelation) 
plan;
+                TableIf table = relation.getTable();
+                if (!forceRefresh && cascadesContext.getStatementContext()
+                        
.getMaterializationRewrittenSuccessSet().contains(table.getFullQualifiers())) {
+                    return tableMap;
+                }
+                tableMap.set(relation.getRelationId().asInt());
+            }
+            if (plan instanceof LogicalCTEConsumer || plan instanceof 
LogicalEmptyRelation
+                    || plan instanceof LogicalOneRowRelation) {
+                tableMap.set(((LogicalRelation) plan).getRelationId().asInt());
+            }
+            return tableMap;
+        }
+
+        @Override
+        public int computeMemoVersion(BitSet targetIdMap, CascadesContext 
cascadesContext) {
+            return getMemoVersion(targetIdMap, 
cascadesContext.getMemo().getRefreshVersion());
+        }
+    };
     /**
      * The map key is the relation id bit set to get corresponding plan 
accurately
      */
-    private final Map<BitSet, Pair<GroupExpression, List<BitSet>>> 
groupExpressionMap = new HashMap<>();
+    private final Map<BitSet, Pair<GroupExpression, List<BitSet>>> 
groupExpressionMapByRelationId = new HashMap<>();
     /**
      * The map key is the relation id bit set to get corresponding plan 
accurately
      */
-    private final Map<BitSet, StructInfo> infoMap = new HashMap<>();
-    private long refreshVersion = 0;
+    private final Map<BitSet, StructInfo> infoMapByRelationId = new 
HashMap<>();
+
+    /**
+     * The map key is the common table id bit set to get corresponding plan 
accurately
+     */
+    private final Map<BitSet, Pair<GroupExpression, List<BitSet>>> 
groupExpressionMapByTableId = new HashMap<>();
+    /**
+     * The map key is the common table id bit set to get corresponding plan 
accurately
+     */
+    private final Map<BitSet, StructInfo> infoMapByTableId = new HashMap<>();
+
+    // The key is the tableIds query used, the value is the refresh version 
when last refresh
+    private final Map<BitSet, Integer> refreshVersion = new HashMap<>();
 
     /**
      * get struct info according to table map
      *
-     * @param tableMap the original table map
+     * @param targetIdMap the original table map
      * @param group the group that the mv matched
      * @return struct info or null if not found
      */
-    public @Nullable StructInfo getStructInfo(CascadesContext cascadesContext, 
BitSet tableMap, Group group,
-            Plan originPlan, boolean forceRefresh) {
-        StructInfo structInfo = infoMap.get(tableMap);
+    public @Nullable StructInfo getStructInfo(CascadesContext cascadesContext, 
BitSet targetIdMap, Group group,
+            Plan originPlan, boolean forceRefresh, boolean tableIdMode) {
+        IdModeStrategy strategy = getStrategy(tableIdMode);
+        Map<BitSet, StructInfo> infoMap = strategy.getInfoMap(this);
+        Map<BitSet, Pair<GroupExpression, List<BitSet>>> groupExprMap = 
strategy.getGroupExpressionMap(this);
+
+        StructInfo structInfo = infoMap.get(targetIdMap);
         if (structInfo != null) {
             return structInfo;
         }
-        if (groupExpressionMap.isEmpty() || 
!groupExpressionMap.containsKey(tableMap)) {
-            refresh(group, cascadesContext, tableMap, new HashSet<>(),
-                    forceRefresh);
-            
group.getStructInfoMap().setRefreshVersion(cascadesContext.getMemo().getRefreshVersion());
+        if (groupExprMap.isEmpty() || !groupExprMap.containsKey(targetIdMap)) {
+            int memoVersion = strategy.computeMemoVersion(targetIdMap, 
cascadesContext);
+            refresh(group, cascadesContext, targetIdMap, new HashSet<>(), 
forceRefresh, memoVersion, tableIdMode);
+            group.getStructInfoMap().setRefreshVersion(targetIdMap, 
cascadesContext.getMemo().getRefreshVersion());
         }
-        if (groupExpressionMap.containsKey(tableMap)) {
-            Pair<GroupExpression, List<BitSet>> groupExpressionBitSetPair = 
getGroupExpressionWithChildren(tableMap);
+        if (groupExprMap.containsKey(targetIdMap)) {
+            Pair<GroupExpression, List<BitSet>> groupExpressionBitSetPair =
+                    getGroupExpressionWithChildren(targetIdMap, tableIdMode);
             // NOTICE: During the transition from physicalAggregate to logical 
aggregation,
-            // the original function signature needs to remain unchanged 
because the constructor of LogicalAggregation
-            // will recalculate the signature of the aggregation function.
+            // the original function signature needs to remain unchanged 
because the constructor
+            // of LogicalAggregation will recalculate the signature of the 
aggregation function.
             // When the calculated signature is inconsistent with the original 
signature
             // (e.g. due to the influence of the session variable 
enable_decimal256),
             // a problem will arise where the output type of the rewritten 
plan is inconsistent with
             // the output type of the upper-level operator.
             structInfo = MoreFieldsThread.keepFunctionSignature(() ->
                     constructStructInfo(groupExpressionBitSetPair.first, 
groupExpressionBitSetPair.second,
-                    originPlan, cascadesContext));
-            infoMap.put(tableMap, structInfo);
+                            originPlan, cascadesContext, tableIdMode));
+            infoMap.put(targetIdMap, structInfo);
         }
         return structInfo;
     }
 
-    public Set<BitSet> getTableMaps() {
-        return groupExpressionMap.keySet();
+    public Set<BitSet> getTableMaps(boolean tableIdMode) {
+        return getStrategy(tableIdMode).getGroupExpressionMap(this).keySet();
+    }
+
+    public Pair<GroupExpression, List<BitSet>> 
getGroupExpressionWithChildren(BitSet tableMap, boolean tableIdMode) {
+        return 
getStrategy(tableIdMode).getGroupExpressionMap(this).get(tableMap);
+    }
+
+    // Set the refresh version for the given targetIdSet
+    public void setRefreshVersion(BitSet targetIdSet, Map<Integer, 
AtomicInteger> memoRefreshVersionMap) {
+        this.refreshVersion.put(targetIdSet, getMemoVersion(targetIdSet, 
memoRefreshVersionMap));
     }
 
-    public Pair<GroupExpression, List<BitSet>> 
getGroupExpressionWithChildren(BitSet tableMap) {
-        return groupExpressionMap.get(tableMap);
+    // Set the refresh version for the given targetIdSet
+    public void setRefreshVersion(BitSet targetIdSet, int memoRefreshVersion) {
+        this.refreshVersion.put(targetIdSet, memoRefreshVersion);
     }
 
-    public void setRefreshVersion(long refreshVersion) {
-        this.refreshVersion = refreshVersion;
+    // Get the refresh version for the given targetIdSet, if not exist, return 0
+    public long getRefreshVersion(BitSet targetIdSet) {
+        return refreshVersion.computeIfAbsent(targetIdSet, k -> 0);
+    }
+
+    /**
+     * Compute a compact "version fingerprint" for the given relation id set.
+     * Algorithm:
+     * - Uses a 32-bit FNV-1a-style hash. Start from FNV32_OFFSET_BASIS and 
multiply by FNV32_PRIME.
+     * - Iterate each set bit (target id) in the BitSet:
+     *   - Fetch its current refresh version from memoRefreshVersionMap 
(default 0 if absent).
+     *   - Mix the version into the hash by XOR, then diffuse by multiplying 
the FNV prime.
+     * - Returns the final hash as the memo version for this set of relations.
+     * Benefits:
+     * - Stable fingerprint: any change in any relation's version produces a 
different hash, enabling
+     *   fast cache invalidation checks without scanning all versions every 
time.
+     * - Order-independent: relies on set iteration; the same set yields the 
same hash regardless of order.
+     * - Low memory and CPU overhead: compresses multiple integers into a 
single 32-bit value efficiently.
+     * - Incremental-friendly: new relations/versions can be incorporated by 
re-running on the changed set.
+     * - Good diffusion: XOR + prime multiplication reduces collisions 
compared to simple sums.
+     * Notes:
+     * - The Integer.MAX_VALUE guard prevents potential overflow edge cases in 
BitSet iteration.
+     */
+    public static int getMemoVersion(BitSet targetIdSet, Map<Integer, 
AtomicInteger> memoRefreshVersionMap) {
+        int hash = FNV32_OFFSET_BASIS;
+        for (int id = targetIdSet.nextSetBit(0);
+                id >= 0; id = targetIdSet.nextSetBit(id + 1)) {
+            AtomicInteger ver = memoRefreshVersionMap.get(id);
+            int tmpVer = ver == null ? 0 : ver.get();
+            hash ^= tmpVer;
+            hash *= FNV32_PRIME;
+            if (id == Integer.MAX_VALUE) {
+                break;
+            }
+        }
+        return hash;
     }
 
     private StructInfo constructStructInfo(GroupExpression groupExpression, 
List<BitSet> children,
-            Plan originPlan, CascadesContext cascadesContext) {
+            Plan originPlan, CascadesContext cascadesContext, boolean 
tableIdMode) {
         // this plan is not origin plan, should record origin plan in struct 
info
-        Plan plan = constructPlan(groupExpression, children);
+        Plan plan = constructPlan(groupExpression, children, tableIdMode);
         return originPlan == null ? StructInfo.of(plan, cascadesContext)
                 : StructInfo.of(plan, originPlan, cascadesContext);
     }
 
-    private Plan constructPlan(GroupExpression groupExpression, List<BitSet> 
children) {
+    private Plan constructPlan(GroupExpression groupExpression, List<BitSet> 
children, boolean tableIdMode) {
         List<Plan> childrenPlan = new ArrayList<>();
         for (int i = 0; i < children.size(); i++) {
             StructInfoMap structInfoMap = 
groupExpression.child(i).getStructInfoMap();
             BitSet childMap = children.get(i);
             Pair<GroupExpression, List<BitSet>> groupExpressionBitSetPair
-                    = structInfoMap.getGroupExpressionWithChildren(childMap);
+                    = structInfoMap.getGroupExpressionWithChildren(childMap, 
tableIdMode);
             childrenPlan.add(
-                    constructPlan(groupExpressionBitSetPair.first, 
groupExpressionBitSetPair.second));
+                    constructPlan(groupExpressionBitSetPair.first, 
groupExpressionBitSetPair.second, tableIdMode));
         }
         // need to clear current group expression info by using 
withGroupExpression
         // this plan would copy into memo, if with group expression, would 
cause err
@@ -136,46 +278,53 @@ public class StructInfoMap {
      *
      * @param group the root group
      * @param targetBitSet refreshed group expression table bitset must 
intersect with the targetBitSet
-     *
      */
     public void refresh(Group group, CascadesContext cascadesContext,
-            BitSet targetBitSet,
-            Set<Integer> refreshedGroup,
-            boolean forceRefresh) {
+            BitSet targetBitSet, Set<Integer> refreshedGroup,
+            boolean forceRefresh, int memoVersion, boolean tableIdMode) {
+        IdModeStrategy strategy = getStrategy(tableIdMode);
+        Map<BitSet, Pair<GroupExpression, List<BitSet>>> groupExprMap = 
strategy.getGroupExpressionMap(this);
         StructInfoMap structInfoMap = group.getStructInfoMap();
         refreshedGroup.add(group.getGroupId().asInt());
-        long memoVersion = cascadesContext.getMemo().getRefreshVersion();
-        if (!structInfoMap.getTableMaps().isEmpty() && memoVersion == 
structInfoMap.refreshVersion) {
+        if (!structInfoMap.getTableMaps(tableIdMode).isEmpty()
+                && memoVersion == 
structInfoMap.getRefreshVersion(targetBitSet)) {
             return;
         }
         for (GroupExpression groupExpression : group.getLogicalExpressions()) {
             List<Set<BitSet>> childrenTableMap = new LinkedList<>();
             if (groupExpression.children().isEmpty()) {
-                BitSet leaf = constructLeaf(groupExpression, cascadesContext, 
forceRefresh);
+                BitSet leaf = strategy.constructLeaf(groupExpression, 
cascadesContext, forceRefresh);
                 if (leaf.isEmpty()) {
                     break;
                 }
-                groupExpressionMap.put(leaf, Pair.of(groupExpression, new 
LinkedList<>()));
+                groupExprMap.put(leaf, Pair.of(groupExpression, new 
LinkedList<>()));
                 continue;
             }
+            // this is used for filter group expression whose children's table 
map all not in targetBitSet
+            BitSet filteredTableMaps = new BitSet();
+            // groupExpression self could be pruned
             for (Group child : groupExpression.children()) {
+                // group in expression should all be reserved
                 StructInfoMap childStructInfoMap = child.getStructInfoMap();
                 if (!refreshedGroup.contains(child.getGroupId().asInt())) {
-                    childStructInfoMap.refresh(child, cascadesContext, 
targetBitSet, refreshedGroup, forceRefresh);
-                    childStructInfoMap.setRefreshVersion(memoVersion);
+                    childStructInfoMap.refresh(child, cascadesContext, 
targetBitSet,
+                            refreshedGroup, forceRefresh, memoVersion, 
tableIdMode);
+                    childStructInfoMap.setRefreshVersion(targetBitSet, 
memoVersion);
                 }
-                Set<BitSet> filteredTableMaps = new HashSet<>();
-                for (BitSet tableMaps : 
child.getStructInfoMap().getTableMaps()) {
-                    // filter the tableSet that used intersects with 
targetBitSet
-                    if (!targetBitSet.isEmpty() && 
!tableMaps.intersects(targetBitSet)) {
-                        continue;
-                    }
-                    filteredTableMaps.add(tableMaps);
+                Set<BitSet> groupTableSet = new HashSet<>();
+                for (BitSet tableMaps : 
child.getStructInfoMap().getTableMaps(tableIdMode)) {
+                    groupTableSet.add(tableMaps);
+                    filteredTableMaps.or(tableMaps);
                 }
                 if (!filteredTableMaps.isEmpty()) {
-                    childrenTableMap.add(filteredTableMaps);
+                    childrenTableMap.add(groupTableSet);
                 }
             }
+            // filter the tableSet that used intersects with targetBitSet, 
make sure the at least constructed
+            if (!structInfoMap.getTableMaps(tableIdMode).isEmpty() && 
!targetBitSet.isEmpty()
+                    && !filteredTableMaps.isEmpty() && 
!filteredTableMaps.intersects(targetBitSet)) {
+                continue;
+            }
             if (childrenTableMap.isEmpty()) {
                 continue;
             }
@@ -186,7 +335,7 @@ public class StructInfoMap {
                     eachGroupExpressionTableSet.or(bitSet);
                 }
             }
-            if (groupExpressionMap.containsKey(eachGroupExpressionTableSet)) {
+            if (groupExprMap.containsKey(eachGroupExpressionTableSet)) {
                 // for the group expressions of group, only need to refresh 
any of the group expression
                 // when they have the same group expression table set
                 continue;
@@ -195,36 +344,12 @@ public class StructInfoMap {
             // or current group expression map is empty, should update the 
groupExpressionMap currently
             Collection<Pair<BitSet, List<BitSet>>> bitSetWithChildren = 
cartesianProduct(childrenTableMap);
             for (Pair<BitSet, List<BitSet>> bitSetWithChild : 
bitSetWithChildren) {
-                groupExpressionMap.putIfAbsent(bitSetWithChild.first,
+                groupExprMap.putIfAbsent(bitSetWithChild.first,
                         Pair.of(groupExpression, bitSetWithChild.second));
             }
-
         }
     }
 
-    private BitSet constructLeaf(GroupExpression groupExpression, 
CascadesContext cascadesContext,
-            boolean forceRefresh) {
-        Plan plan = groupExpression.getPlan();
-        BitSet tableMap = new BitSet();
-        if (plan instanceof LogicalCatalogRelation) {
-            LogicalCatalogRelation logicalCatalogRelation = 
(LogicalCatalogRelation) plan;
-            TableIf table = logicalCatalogRelation.getTable();
-            // If disable materialized view nest rewrite, and mv already 
rewritten successfully once, doesn't construct
-            // table id map for nest mv rewrite
-            if (!forceRefresh && cascadesContext.getStatementContext()
-                    
.getMaterializationRewrittenSuccessSet().contains(table.getFullQualifiers())) {
-                return tableMap;
-            }
-            tableMap.set(logicalCatalogRelation.getRelationId().asInt());
-        }
-        // one row relation / CTE consumer
-        if (plan instanceof LogicalCTEConsumer || plan instanceof 
LogicalEmptyRelation
-                || plan instanceof LogicalOneRowRelation) {
-            tableMap.set(((LogicalRelation) plan).getRelationId().asInt());
-        }
-        return tableMap;
-    }
-
     private Collection<Pair<BitSet, List<BitSet>>> 
cartesianProduct(List<Set<BitSet>> childrenTableMap) {
         Set<List<BitSet>> cartesianLists = 
Sets.cartesianProduct(childrenTableMap);
         List<Pair<BitSet, List<BitSet>>> resultPairSet = new LinkedList<>();
@@ -238,8 +363,31 @@ public class StructInfoMap {
         return resultPairSet;
     }
 
+    /**
+     * Strategy interface to handle different ID modes (tableId vs relationId)
+     */
+    private interface IdModeStrategy {
+        Map<BitSet, Pair<GroupExpression, List<BitSet>>> 
getGroupExpressionMap(StructInfoMap structInfoMap);
+
+        Map<BitSet, StructInfo> getInfoMap(StructInfoMap structInfoMap);
+
+        BitSet constructLeaf(GroupExpression groupExpression, CascadesContext 
cascadesContext, boolean forceRefresh);
+
+        int computeMemoVersion(BitSet targetIdMap, CascadesContext 
cascadesContext);
+    }
+
+    private static IdModeStrategy getStrategy(boolean tableIdMode) {
+        return tableIdMode ? TABLE_ID_STRATEGY : RELATION_ID_STRATEGY;
+    }
+
     @Override
     public String toString() {
-        return "StructInfoMap{ groupExpressionMap = " + groupExpressionMap + 
", infoMap = " + infoMap + '}';
+        return "StructInfoMap{"
+                + " groupExpressionMapByRelationId=" + 
groupExpressionMapByRelationId.keySet()
+                + ", infoMapByRelationId=" + infoMapByRelationId.keySet()
+                + ", groupExpressionMapByTableId=" + 
groupExpressionMapByTableId.keySet()
+                + ", infoMapByTableId=" + infoMapByTableId.keySet()
+                + ", refreshVersion=" + refreshVersion
+                + '}';
     }
 }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
index 87ddc7a0ca5..a7e1e171ee2 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
@@ -59,7 +59,6 @@ import com.google.common.collect.ImmutableMap;
 import com.google.common.collect.Sets;
 
 import java.util.ArrayList;
-import java.util.BitSet;
 import java.util.Collections;
 import java.util.HashMap;
 import java.util.HashSet;
@@ -124,7 +123,6 @@ public abstract class AbstractMaterializedViewAggregateRule 
extends AbstractMate
                     queryTopPlan,
                     materializationContext.getShuttledExprToScanExprMapping(),
                     viewToQuerySlotMapping,
-                    queryStructInfo.getTableBitSet(),
                     ImmutableMap.of(), cascadesContext);
             boolean isRewrittenQueryExpressionValid = true;
             if (!rewrittenQueryExpressions.isEmpty()) {
@@ -314,11 +312,10 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
             MaterializationContext materializationContext, String 
summaryIfFail, Supplier<String> detailIfFail) {
         Expression queryFunctionShuttled = 
ExpressionUtils.shuttleExpressionWithLineage(
                 queryExpression,
-                queryStructInfo.getTopPlan(),
-                queryStructInfo.getTableBitSet());
+                queryStructInfo.getTopPlan());
         AggregateExpressionRewriteContext expressionRewriteContext = new 
AggregateExpressionRewriteContext(
                 rewriteMode, mvShuttledExprToMvScanExprQueryBased, 
queryStructInfo.getTopPlan(),
-                queryStructInfo.getTableBitSet(), 
queryStructInfo.getGroupingId());
+                queryStructInfo.getGroupingId());
         Expression rewrittenExpression = 
queryFunctionShuttled.accept(AGGREGATE_EXPRESSION_REWRITER,
                 expressionRewriteContext);
         if (!expressionRewriteContext.isValid()) {
@@ -358,7 +355,7 @@ public abstract class AbstractMaterializedViewAggregateRule 
extends AbstractMate
         boolean canUnionRewrite = false;
         // Check the query plan group by expression contains partition col or 
not
         List<? extends Expression> groupByShuttledExpressions =
-                
ExpressionUtils.shuttleExpressionWithLineage(groupByExpressions, queryPlan, new 
BitSet());
+                
ExpressionUtils.shuttleExpressionWithLineage(groupByExpressions, queryPlan);
         for (Expression expression : groupByShuttledExpressions) {
             canUnionRewrite = !expression.collectToSet(expr -> {
                 if (!(expr instanceof SlotReference) || !((SlotReference) 
expr).isColumnFromTable()) {
@@ -431,13 +428,13 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
         LogicalAggregate<Plan> viewAggregate = viewTopPlanAndAggPair.value();
 
         Set<Expression> queryGroupByShuttledExpression = new 
HashSet<>(ExpressionUtils.shuttleExpressionWithLineage(
-                queryAggregate.getGroupByExpressions(), queryTopPlan, 
queryStructInfo.getTableBitSet()));
+                queryAggregate.getGroupByExpressions(), queryTopPlan));
 
         // try to eliminate group by dimension by function dependency if group 
by expression is not in query
         Map<Expression, Expression> 
viewShuttledExpressionQueryBasedToGroupByExpressionMap = new HashMap<>();
         List<Expression> viewGroupByExpressions = 
viewAggregate.getGroupByExpressions();
         List<? extends Expression> viewGroupByShuttledExpressions = 
ExpressionUtils.shuttleExpressionWithLineage(
-                viewGroupByExpressions, viewTopPlan, 
viewStructInfo.getTableBitSet());
+                viewGroupByExpressions, viewTopPlan);
 
         for (int index = 0; index < viewGroupByExpressions.size(); index++) {
             Expression viewExpression = viewGroupByExpressions.get(index);
@@ -712,8 +709,7 @@ public abstract class AbstractMaterializedViewAggregateRule 
extends AbstractMate
             if 
(ExpressionRewriteMode.EXPRESSION_ROLL_UP.equals(rewriteContext.getExpressionRewriteMode()))
 {
                 Expression queryFunctionShuttled = 
ExpressionUtils.shuttleExpressionWithLineage(
                         aggregateFunction,
-                        rewriteContext.getQueryTopPlan(),
-                        rewriteContext.getQueryTableBitSet());
+                        rewriteContext.getQueryTopPlan());
                 rewrittenFunction = rollup(aggregateFunction, 
queryFunctionShuttled,
                         
rewriteContext.getMvExprToMvScanExprQueryBasedMapping());
                 if (rewrittenFunction == null) {
@@ -788,16 +784,14 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
         private final ExpressionRewriteMode expressionRewriteMode;
         private final Map<Expression, Expression> 
mvExprToMvScanExprQueryBasedMapping;
         private final Plan queryTopPlan;
-        private final BitSet queryTableBitSet;
         private final Optional<SlotReference> groupingId;
 
         public AggregateExpressionRewriteContext(ExpressionRewriteMode 
expressionRewriteMode,
                 Map<Expression, Expression> 
mvExprToMvScanExprQueryBasedMapping, Plan queryTopPlan,
-                BitSet queryTableBitSet, Optional<SlotReference> groupingId) {
+                                                 Optional<SlotReference> 
groupingId) {
             this.expressionRewriteMode = expressionRewriteMode;
             this.mvExprToMvScanExprQueryBasedMapping = 
mvExprToMvScanExprQueryBasedMapping;
             this.queryTopPlan = queryTopPlan;
-            this.queryTableBitSet = queryTableBitSet;
             this.groupingId = groupingId;
         }
 
@@ -821,10 +815,6 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
             return queryTopPlan;
         }
 
-        public BitSet getQueryTableBitSet() {
-            return queryTableBitSet;
-        }
-
         public Optional<SlotReference> getGroupingId() {
             return groupingId;
         }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java
index 511d7d83b68..f56cb00c5d6 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java
@@ -51,7 +51,6 @@ public abstract class AbstractMaterializedViewJoinRule 
extends AbstractMateriali
                 queryStructInfo.getTopPlan(),
                 materializationContext.getShuttledExprToScanExprMapping(),
                 targetToSourceMapping,
-                queryStructInfo.getTableBitSet(),
                 ImmutableMap.of(), cascadesContext
         );
         // Can not rewrite, bail out
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
index 036ffcd2251..c4b18a96b84 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
@@ -79,7 +79,6 @@ import org.apache.logging.log4j.LogManager;
 import org.apache.logging.log4j.Logger;
 
 import java.util.ArrayList;
-import java.util.BitSet;
 import java.util.Collection;
 import java.util.HashMap;
 import java.util.HashSet;
@@ -128,18 +127,18 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                     statementContext.getMaterializedViewRewriteDuration());
             return rewrittenPlans;
         }
-        for (MaterializationContext context : 
cascadesContext.getMaterializationContexts()) {
+        for (MaterializationContext materializationContext : 
cascadesContext.getMaterializationContexts()) {
             statementContext.getMaterializedViewStopwatch().reset().start();
-            if (checkIfRewritten(queryPlan, context)) {
+            if (checkIfRewritten(queryPlan, materializationContext)) {
                 continue;
             }
             // check mv plan is valid or not
-            if (!isMaterializationValid(queryPlan, cascadesContext, context)) {
+            if (!isMaterializationValid(queryPlan, cascadesContext, 
materializationContext)) {
                 continue;
             }
             // get query struct infos according to the view strut info, if 
valid query struct infos is empty, bail out
             List<StructInfo> queryStructInfos = 
getValidQueryStructInfos(queryPlan, cascadesContext,
-                    context.getCommonTableIdSet(statementContext));
+                    materializationContext);
             if (queryStructInfos.isEmpty()) {
                 continue;
             }
@@ -163,11 +162,11 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                 }
                 try {
                     if (rewrittenPlans.size() < 
sessionVariable.getMaterializedViewRewriteSuccessCandidateNum()) {
-                        rewrittenPlans.addAll(doRewrite(queryStructInfo, 
cascadesContext, context));
+                        rewrittenPlans.addAll(doRewrite(queryStructInfo, 
cascadesContext, materializationContext));
                     }
                 } catch (Exception exception) {
                     LOG.warn("Materialized view rule exec fail", exception);
-                    context.recordFailReason(queryStructInfo,
+                    materializationContext.recordFailReason(queryStructInfo,
                             "Materialized view rule exec fail", 
exception::toString);
                 } finally {
                     elapsed = 
statementContext.getMaterializedViewStopwatch().elapsed(TimeUnit.MILLISECONDS);
@@ -185,18 +184,18 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
      * Get valid query struct infos, if invalid record the invalid reason
      */
     protected List<StructInfo> getValidQueryStructInfos(Plan queryPlan, 
CascadesContext cascadesContext,
-            BitSet materializedViewTableSet) {
+            MaterializationContext materializationContext) {
         List<StructInfo> validStructInfos = new ArrayList<>();
         // For every materialized view we should trigger refreshing struct 
info map
-        List<StructInfo> uncheckedStructInfos = 
MaterializedViewUtils.extractStructInfo(queryPlan, queryPlan,
-                cascadesContext, materializedViewTableSet);
-        uncheckedStructInfos.forEach(queryStructInfo -> {
+        List<StructInfo> uncheckedQueryStructInfos = 
MaterializedViewUtils.extractStructInfoFuzzy(queryPlan, queryPlan,
+                cascadesContext, 
materializationContext.getCommonTableIdSet(cascadesContext.getStatementContext()));
+        uncheckedQueryStructInfos.forEach(queryStructInfo -> {
             boolean valid = checkQueryPattern(queryStructInfo, 
cascadesContext) && queryStructInfo.isValid();
             if (!valid) {
                 cascadesContext.getMaterializationContexts().forEach(ctx ->
                         ctx.recordFailReason(queryStructInfo, "Query struct 
info is invalid",
                                 () -> String.format("query table bitmap is %s, 
plan is %s",
-                                        queryStructInfo.getTableBitSet(), 
queryPlan.treeString())
+                                        queryStructInfo.getRelations(), 
queryPlan.treeString())
                         ));
             } else {
                 validStructInfos.add(queryStructInfo);
@@ -277,9 +276,7 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                 // Try to rewrite compensate predicates by using mv scan
                 List<Expression> rewriteCompensatePredicates = 
rewriteExpression(compensatePredicates.toList(),
                         queryPlan, 
materializationContext.getShuttledExprToScanExprMapping(),
-                        viewToQuerySlotMapping, 
queryStructInfo.getTableBitSet(),
-                        compensatePredicates.getRangePredicateMap(),
-                        cascadesContext);
+                        viewToQuerySlotMapping, 
compensatePredicates.getRangePredicateMap(), cascadesContext);
                 if (rewriteCompensatePredicates.isEmpty()) {
                     materializationContext.recordFailReason(queryStructInfo,
                             "Rewrite compensate predicate by view fail",
@@ -320,7 +317,7 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                     && sessionVariable.isEnableMaterializedViewUnionRewrite()) 
{
                 MTMV mtmv = ((AsyncMaterializationContext) 
materializationContext).getMtmv();
                 Map<List<String>, Set<String>> queryUsedPartitions = 
PartitionCompensator.getQueryUsedPartitions(
-                        cascadesContext.getStatementContext(), 
queryStructInfo.getTableBitSet());
+                        cascadesContext.getStatementContext(), 
queryStructInfo.getRelationBitSet());
                 Set<MTMVRelatedTableIf> pctTables = 
mtmv.getMvPartitionInfo().getPctTables();
                 boolean relateTableUsedPartitionsAnyNull = false;
                 boolean relateTableUsedPartitionsAllEmpty = true;
@@ -465,12 +462,18 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                     materializationContext);
             rewriteResults.add(rewrittenPlan);
             recordIfRewritten(queryStructInfo.getOriginalPlan(), 
materializationContext, cascadesContext);
-            // If rewrite successfully, try to clear mv scan currently because 
it maybe used again
-            materializationContext.clearScanPlan(cascadesContext);
+            resetMaterializationContext(materializationContext, 
cascadesContext);
         }
         return rewriteResults;
     }
 
+    // reset some materialization context state after one materialized view 
written successfully
+    private void resetMaterializationContext(MaterializationContext 
currentContext,
+                                             CascadesContext cascadesContext) {
+        // If rewrite successfully, try to clear mv scan currently because it 
maybe used again
+        currentContext.clearScanPlan(cascadesContext);
+    }
+
     // Set materialization context statistics to statementContext for cost 
estimate later
     // this should be called before MaterializationContext.clearScanPlan 
because clearScanPlan change the
     // mv scan plan relation id
@@ -563,14 +566,14 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
      *         then use the corresponding value of mapping to replace it
      */
     protected List<Expression> rewriteExpression(List<? extends Expression> 
sourceExpressionsToWrite, Plan sourcePlan,
-            ExpressionMapping targetExpressionMapping, SlotMapping 
targetToSourceMapping, BitSet sourcePlanBitSet,
+            ExpressionMapping targetExpressionMapping, SlotMapping 
targetToSourceMapping,
             Map<Expression, ExpressionInfo> queryExprToInfoMap, 
CascadesContext cascadesContext) {
         // Firstly, rewrite the target expression using source with inverse 
mapping
         // then try to use the target expression to represent the query. if 
any of source expressions
         // could not be represented by target expressions, return null.
         // generate target to target replacement expression mapping, and 
change target expression to source based
         List<? extends Expression> sourceShuttledExpressions = 
ExpressionUtils.shuttleExpressionWithLineage(
-                sourceExpressionsToWrite, sourcePlan, sourcePlanBitSet);
+                sourceExpressionsToWrite, sourcePlan);
         ExpressionMapping expressionMappingKeySourceBased = 
targetExpressionMapping.keyPermute(targetToSourceMapping);
         // target to target replacement expression mapping, because mv is 1:1 
so get the first element
         List<Map<Expression, Expression>> flattenExpressionMap = 
expressionMappingKeySourceBased.flattenMap();
@@ -892,7 +895,7 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
         }
         // query slot need shuttle to use table slot, avoid alias influence
         Set<Expression> queryUsedNeedRejectNullSlotsViewBased = 
ExpressionUtils.shuttleExpressionWithLineage(
-                        new ArrayList<>(queryNullRejectSlotSet), 
queryStructInfo.getTopPlan(), new BitSet()).stream()
+                        new ArrayList<>(queryNullRejectSlotSet), 
queryStructInfo.getTopPlan()).stream()
                 .map(expr -> ExpressionUtils.replace(expr, 
queryToViewMapping.toSlotReferenceMap()))
                 .collect(Collectors.toSet());
         // view slot need shuttle to use table slot, avoid alias influence
@@ -900,7 +903,7 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
         for (Set<Slot> requireNullableSlots : requireNoNullableViewSlot) {
             shuttledRequireNoNullableViewSlot.add(
                     ExpressionUtils.shuttleExpressionWithLineage(new 
ArrayList<>(requireNullableSlots),
-                                    viewStructInfo.getTopPlan(), new 
BitSet()).stream().map(Slot.class::cast)
+                                    
viewStructInfo.getTopPlan()).stream().map(Slot.class::cast)
                             .collect(Collectors.toSet()));
         }
         // query pulledUp predicates should have null reject predicates and 
contains any require noNullable slot
@@ -1076,7 +1079,7 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
         List<Expression> queryOrderKeysExpressions = queryOrderKeys.stream()
                 .map(OrderKey::getExpr).collect(Collectors.toList());
         List<? extends Expression> queryOrderByExpressionsShuttled = 
ExpressionUtils.shuttleExpressionWithLineage(
-                queryOrderKeysExpressions, queryStructInfo.getTopPlan(), 
queryStructInfo.getTableBitSet());
+                queryOrderKeysExpressions, queryStructInfo.getTopPlan());
 
         List<OrderKey> queryShuttledOrderKeys = new ArrayList<>();
         for (int i = 0; i < queryOrderKeys.size(); i++) {
@@ -1087,7 +1090,7 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
         List<OrderKey> viewShuttledOrderKeys = new ArrayList<>();
         List<? extends Expression> viewOrderByExpressionsShuttled = 
ExpressionUtils.shuttleExpressionWithLineage(
                 
viewOrderKeys.stream().map(OrderKey::getExpr).collect(Collectors.toList()),
-                viewStructInfo.getTopPlan(), new BitSet());
+                viewStructInfo.getTopPlan());
         List<Expression> viewOrderByExpressionsQueryBasedSet = 
ExpressionUtils.replace(
                 
viewOrderByExpressionsShuttled.stream().map(Expression.class::cast).collect(Collectors.toList()),
                 viewToQuerySlotMapping.toSlotReferenceMap());
@@ -1107,7 +1110,7 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
         // try to rewrite the order by expressions using the mv scan slot
         List<Expression> rewrittenExpressions = 
rewriteExpression(queryOrderKeysExpressions,
                 queryStructInfo.getTopPlan(), 
materializationContext.shuttledExprToScanExprMapping,
-                viewToQuerySlotMapping, queryStructInfo.getTableBitSet(), 
ImmutableMap.of(), cascadesContext);
+                viewToQuerySlotMapping, ImmutableMap.of(), cascadesContext);
         if (rewrittenExpressions.isEmpty()) {
             materializationContext.recordFailReason(queryStructInfo,
                     "query topN order keys rewrite fail, query topN order keys 
is not consistent "
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewScanRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewScanRule.java
index 24c63361f69..e542194f508 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewScanRule.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewScanRule.java
@@ -51,7 +51,6 @@ public abstract class AbstractMaterializedViewScanRule 
extends AbstractMateriali
                 queryStructInfo.getTopPlan(),
                 materializationContext.getShuttledExprToScanExprMapping(),
                 targetToSourceMapping,
-                queryStructInfo.getTableBitSet(),
                 ImmutableMap.of(), cascadesContext
         );
         // Can not rewrite, bail out
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewWindowRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewWindowRule.java
index 9a2913aca43..5be68758ef9 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewWindowRule.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewWindowRule.java
@@ -102,7 +102,6 @@ public abstract class AbstractMaterializedViewWindowRule 
extends AbstractMateria
                 queryStructInfo.getTopPlan(),
                 materializationContext.getShuttledExprToScanExprMapping(),
                 viewToQuerySlotMapping,
-                queryStructInfo.getTableBitSet(),
                 ImmutableMap.of(), cascadesContext
         );
         // Can not rewrite, bail out
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java
index 3386e4cd603..2ab84c2108b 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java
@@ -108,6 +108,8 @@ public abstract class MaterializationContext {
     // for one materialization query may be multi when nested materialized 
view.
     protected final Multimap<ObjectId, Pair<String, String>> failReason = 
HashMultimap.create();
     protected List<String> identifier;
+    // The common table id set which is used in materialization, added for 
performance consideration
+    private BitSet commonTableIdSet;
 
     /**
      * MaterializationContext, this contains necessary info for query 
rewriting by materialization
@@ -121,7 +123,7 @@ public abstract class MaterializationContext {
                 && ExplainLevel.MEMO_PLAN == 
parsedStatement.getExplainOptions().getExplainLevel();
         // Construct materialization struct info, catch exception which may 
cause planner roll back
         this.structInfo = structInfo == null
-                ? constructStructInfo(plan, originalPlan, cascadesContext, new 
BitSet()).orElseGet(() -> null)
+                ? constructStructInfo(plan, originalPlan, 
cascadesContext).orElseGet(() -> null)
                 : structInfo;
         this.available = this.structInfo != null;
         if (available) {
@@ -135,22 +137,14 @@ public abstract class MaterializationContext {
      * @param originalPlan original plan, the output is right
      */
     public static Optional<StructInfo> constructStructInfo(Plan plan, Plan 
originalPlan,
-            CascadesContext cascadesContext, BitSet expectedTableBitSet) {
-        List<StructInfo> viewStructInfos;
+                                                           CascadesContext 
cascadesContext) {
         try {
-            viewStructInfos = MaterializedViewUtils.extractStructInfo(plan, 
originalPlan,
-                    cascadesContext, expectedTableBitSet);
-            if (viewStructInfos.size() > 1) {
-                // view struct info should only have one, log error and use 
the first struct info
-                LOG.warn(String.format("view strut info is more than one, 
materialization plan is %s",
-                        plan.treeString()));
-            }
+            return Optional.of(StructInfo.of(plan, originalPlan, 
cascadesContext));
         } catch (Exception exception) {
             LOG.warn(String.format("construct materialization struct info 
fail, materialization plan is %s",
                     plan.treeString()), exception);
             return Optional.empty();
         }
-        return Optional.of(viewStructInfos.get(0));
     }
 
     public boolean alreadyRewrite(GroupId groupId) {
@@ -382,16 +376,19 @@ public abstract class MaterializationContext {
     }
 
     /**
-     * get materialization context common table id by current statementContext
+     * get materialization context common table id by current 
currentQueryStatementContext
      */
-    public BitSet getCommonTableIdSet(StatementContext statementContext) {
-        BitSet commonTableId = new BitSet();
+    public BitSet getCommonTableIdSet(StatementContext 
currentQueryStatementContext) {
+        if (commonTableIdSet != null) {
+            return commonTableIdSet;
+        }
+        commonTableIdSet = new BitSet();
         for (StructInfoNode node : 
structInfo.getRelationIdStructInfoNodeMap().values()) {
             for (CatalogRelation catalogRelation : node.getCatalogRelation()) {
-                
commonTableId.set(statementContext.getTableId(catalogRelation.getTable()).asInt());
+                
commonTableIdSet.set(currentQueryStatementContext.getTableId(catalogRelation.getTable()).asInt());
             }
         }
-        return commonTableId;
+        return commonTableIdSet;
     }
 
     @Override
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 319703cfa07..4c3194703f5 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
@@ -271,31 +271,33 @@ public class MaterializedViewUtils {
      * Extract struct info from plan, support to get struct info from logical 
plan or plan in group.
      * @param plan maybe remove unnecessary plan node, and the logical output 
maybe wrong
      * @param originalPlan original plan, the output is right
+     * @param cascadesContext the cascadesContext when extractStructInfo
+     * @param targetTableIdSet the target relation id set which used to filter 
struct info,
+     *                            empty means no struct info match
      */
-    public static List<StructInfo> extractStructInfo(Plan plan, Plan 
originalPlan, CascadesContext cascadesContext,
-            BitSet materializedViewTableSet) {
+    public static List<StructInfo> extractStructInfoFuzzy(Plan plan, Plan 
originalPlan,
+                                                          CascadesContext 
cascadesContext, BitSet targetTableIdSet) {
         // If plan belong to some group, construct it with group struct info
         if (plan.getGroupExpression().isPresent()) {
             Group ownerGroup = plan.getGroupExpression().get().getOwnerGroup();
             StructInfoMap structInfoMap = ownerGroup.getStructInfoMap();
             // Refresh struct info in current level plan from top to bottom
             SessionVariable sessionVariable = 
cascadesContext.getConnectContext().getSessionVariable();
-            structInfoMap.refresh(ownerGroup, cascadesContext, new BitSet(), 
new HashSet<>(),
-                    sessionVariable.isEnableMaterializedViewNestRewrite());
-            
structInfoMap.setRefreshVersion(cascadesContext.getMemo().getRefreshVersion());
-            Set<BitSet> queryTableSets = structInfoMap.getTableMaps();
+            int memoVersion = StructInfoMap.getMemoVersion(targetTableIdSet,
+                    cascadesContext.getMemo().getRefreshVersion());
+            structInfoMap.refresh(ownerGroup, cascadesContext, 
targetTableIdSet, new HashSet<>(),
+                    sessionVariable.isEnableMaterializedViewNestRewrite(), 
memoVersion, true);
+            structInfoMap.setRefreshVersion(targetTableIdSet, 
cascadesContext.getMemo().getRefreshVersion());
+            Set<BitSet> queryTableIdSets = structInfoMap.getTableMaps(true);
             ImmutableList.Builder<StructInfo> structInfosBuilder = 
ImmutableList.builder();
-            if (!queryTableSets.isEmpty()) {
-                for (BitSet queryTableSet : queryTableSets) {
-                    BitSet queryCommonTableSet = 
MaterializedViewUtils.transformToCommonTableId(queryTableSet,
-                            
cascadesContext.getStatementContext().getRelationIdToCommonTableIdMap());
+            if (!queryTableIdSets.isEmpty()) {
+                for (BitSet queryTableIdSet : queryTableIdSets) {
                     // compare relation id corresponding table id
-                    if (!materializedViewTableSet.isEmpty()
-                            && !containsAll(materializedViewTableSet, 
queryCommonTableSet)) {
+                    if (!containsAll(targetTableIdSet, queryTableIdSet)) {
                         continue;
                     }
-                    StructInfo structInfo = 
structInfoMap.getStructInfo(cascadesContext, queryTableSet, ownerGroup,
-                            originalPlan, 
sessionVariable.isEnableMaterializedViewNestRewrite());
+                    StructInfo structInfo = 
structInfoMap.getStructInfo(cascadesContext, queryTableIdSet, ownerGroup,
+                            originalPlan, 
sessionVariable.isEnableMaterializedViewNestRewrite(), true);
                     if (structInfo != null) {
                         structInfosBuilder.add(structInfo);
                     }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/PartitionIncrementMaintainer.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/PartitionIncrementMaintainer.java
index 46045ce50f3..4b29f95aeef 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/PartitionIncrementMaintainer.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/PartitionIncrementMaintainer.java
@@ -66,7 +66,6 @@ import com.google.common.collect.ImmutableSet;
 import com.google.common.collect.Sets;
 
 import java.util.ArrayList;
-import java.util.BitSet;
 import java.util.Collection;
 import java.util.HashMap;
 import java.util.HashSet;
@@ -200,7 +199,7 @@ public class PartitionIncrementMaintainer {
                 for (Set<Slot> equalSlotSet : shuttledEqualSlotSet) {
                     if (equalSlotSet.contains(consumerSlot)) {
                         Expression shuttledSlot = 
ExpressionUtils.shuttleExpressionWithLineage(
-                                producerSlot, producerPlan, new BitSet());
+                                producerSlot, producerPlan);
                         if (shuttledSlot instanceof Slot) {
                             equalSlotSet.add((Slot) shuttledSlot);
                         }
@@ -537,9 +536,9 @@ public class PartitionIncrementMaintainer {
                 Optional<Expression> partitionExpressionOpt = 
partitionTableColumnInfo.getPartitionExpression();
                 Expression partitionExpressionActual = partitionExpressionOpt
                         .map(expr -> 
ExpressionUtils.shuttleExpressionWithLineage(expr,
-                                context.getOriginalPlan(), new BitSet()))
+                                context.getOriginalPlan()))
                         .orElseGet(() -> 
ExpressionUtils.shuttleExpressionWithLineage(partitionNamedExpression,
-                                context.getOriginalPlan(), new BitSet()));
+                                context.getOriginalPlan()));
                 // merge date_trunc
                 partitionExpressionActual = new 
ExpressionNormalization().rewrite(partitionExpressionActual,
                         new 
ExpressionRewriteContext(context.getCascadesContext()));
@@ -547,7 +546,7 @@ public class PartitionIncrementMaintainer {
                 for (Expression projectSlotToCheck : expressionsToCheck) {
                     Expression expressionShuttledToCheck =
                             
ExpressionUtils.shuttleExpressionWithLineage(projectSlotToCheck,
-                                    context.getOriginalPlan(), new BitSet());
+                                    context.getOriginalPlan());
                     // merge date_trunc
                     expressionShuttledToCheck = new 
ExpressionNormalization().rewrite(expressionShuttledToCheck,
                             new 
ExpressionRewriteContext(context.getCascadesContext()));
@@ -831,7 +830,7 @@ public class PartitionIncrementMaintainer {
         List<Expression> extendedPartitionEqualSlotSet = new 
ArrayList<>(partitionEqualSlotSet);
         extendedPartitionEqualSlotSet.add(slot);
         List<? extends Expression> shuttledEqualExpressions = 
ExpressionUtils.shuttleExpressionWithLineage(
-                extendedPartitionEqualSlotSet, join, new BitSet());
+                extendedPartitionEqualSlotSet, join);
         for (Expression shuttledEqualExpression : shuttledEqualExpressions) {
             Set<Slot> objects = shuttledEqualExpression.collectToSet(expr -> 
expr instanceof SlotReference);
             if (objects.size() != 1 || !(shuttledEqualExpression instanceof 
SlotReference)) {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/PreMaterializedViewRewriter.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/PreMaterializedViewRewriter.java
index 5eb4f82f0e3..d098c00e40b 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/PreMaterializedViewRewriter.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/PreMaterializedViewRewriter.java
@@ -88,11 +88,9 @@ public class PreMaterializedViewRewriter {
         Pair<Map<List<String>, MaterializationContext>, BitSet> 
chosenMaterializationAndUsedTable
                 = 
MaterializedViewUtils.getChosenMaterializationAndUsedTable(physicalPlan,
                 cascadesContext.getAllMaterializationContexts());
-        // Calc the table id set which is used by physical plan
-        cascadesContext.getMemo().incrementAndGetRefreshVersion();
         // Extract logical plan by table id set by the corresponding best 
physical plan
         StructInfo structInfo = 
root.getStructInfoMap().getStructInfo(cascadesContext,
-                chosenMaterializationAndUsedTable.value(), root, null, true);
+                chosenMaterializationAndUsedTable.value(), root, null, true, 
false);
         if (structInfo == null) {
             LOG.error("preMaterializedViewRewriter rewrite structInfo is null, 
query id is {}",
                     cascadesContext.getConnectContext().getQueryIdentifier());
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/Predicates.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/Predicates.java
index 5ccf624c510..bbd9f6181da 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/Predicates.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/Predicates.java
@@ -41,7 +41,6 @@ import com.google.common.collect.Lists;
 import com.google.common.collect.Sets;
 
 import java.util.ArrayList;
-import java.util.BitSet;
 import java.util.Collection;
 import java.util.HashMap;
 import java.util.HashSet;
@@ -117,7 +116,7 @@ public class Predicates {
 
         List<? extends Expression> viewPredicatesShuttled = 
ExpressionUtils.shuttleExpressionWithLineage(
                 
Lists.newArrayList(viewStructInfoPredicates.getCouldNotPulledUpPredicates()),
-                viewStructInfo.getTopPlan(), new BitSet());
+                viewStructInfo.getTopPlan());
         List<Expression> viewPredicatesQueryBased = 
ExpressionUtils.replace((List<Expression>) viewPredicatesShuttled,
                 viewToQuerySlotMapping.toSlotReferenceMap());
         // could not be pulled up predicates in query and view should be same
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 79c6d2ee119..a45863aa32c 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
@@ -107,8 +107,8 @@ public class StructInfo {
     private final List<CatalogRelation> relations;
     // This is generated by cascadesContext, this may be different in 
different cascadesContext
     // So if the cascadesContext currently is different form the 
cascadesContext which generated it.
-    // Should regenerate the tableBitSet by current cascadesContext and call 
withTableBitSet method
-    private final BitSet tableBitSet;
+    // Should regenerate the relationBitSet by current cascadesContext and 
call withTableBitSet method
+    private final BitSet relationBitSet;
     // this is for LogicalCompatibilityContext later
     private final Map<RelationId, StructInfoNode> relationIdStructInfoNodeMap;
     // this recorde the predicates which can pull up, not shuttled
@@ -148,7 +148,7 @@ public class StructInfo {
             Map<ExpressionPosition, Multimap<Expression, Pair<Expression, 
HyperElement>>>
                     shuttledExpressionsToExpressionsMap,
             Map<ExpressionPosition, Map<Expression, Expression>> 
expressionToShuttledExpressionToMap,
-            BitSet tableIdSet,
+            BitSet relationIdSet,
             SplitPredicate splitPredicate,
             EquivalenceClass equivalenceClass,
             List<? extends Expression> planOutputShuttledExpressions) {
@@ -159,7 +159,7 @@ public class StructInfo {
         this.topPlan = topPlan;
         this.bottomPlan = bottomPlan;
         this.relations = relations;
-        this.tableBitSet = tableIdSet;
+        this.relationBitSet = relationIdSet;
         this.relationIdStructInfoNodeMap = relationIdStructInfoNodeMap;
         this.predicates = predicates;
         this.groupingId = groupingId;
@@ -177,7 +177,7 @@ public class StructInfo {
         return new StructInfo(this.originalPlan, this.originalPlanId, 
this.hyperGraph, this.valid, this.topPlan,
                 this.bottomPlan, this.relations, 
this.relationIdStructInfoNodeMap, predicates, this.groupingId,
                 this.shuttledExpressionsToExpressionsMap, 
this.expressionToShuttledExpressionToMap,
-                this.tableBitSet, null, null, 
this.planOutputShuttledExpressions);
+                this.relationBitSet, null, null, 
this.planOutputShuttledExpressions);
     }
 
     private static boolean collectStructInfoFromGraph(HyperGraph hyperGraph,
@@ -202,8 +202,7 @@ public class StructInfo {
             List<Expression> nodeExpressions = 
structInfoNode.getCouldMoveExpressions();
             if (nodeExpressions != null) {
                 List<? extends Expression> shuttledExpressions = 
ExpressionUtils.shuttleExpressionWithLineage(
-                        nodeExpressions, structInfoNode.getPlan(),
-                        new BitSet());
+                        nodeExpressions, structInfoNode.getPlan());
                 for (int index = 0; index < nodeExpressions.size(); index++) {
                     
putShuttledExpressionToExpressionsMap(shuttledExpressionsToExpressionsMap,
                             expressionToShuttledExpressionToMap, 
ExpressionPosition.NODE_COULD_MOVE,
@@ -213,8 +212,7 @@ public class StructInfo {
             nodeExpressions = structInfoNode.getCouldNotMoveExpressions();
             if (nodeExpressions != null) {
                 List<? extends Expression> shuttledExpressions = 
ExpressionUtils.shuttleExpressionWithLineage(
-                        nodeExpressions, structInfoNode.getPlan(),
-                        new BitSet());
+                        nodeExpressions, structInfoNode.getPlan());
                 for (int index = 0; index < nodeExpressions.size(); index++) {
                     
putShuttledExpressionToExpressionsMap(shuttledExpressionsToExpressionsMap,
                             expressionToShuttledExpressionToMap, 
ExpressionPosition.NODE_COULD_NOT_MOVE,
@@ -233,7 +231,7 @@ public class StructInfo {
             // Record the exprId to expr map in the processing to strut info
             // Replace expressions by expression map
             List<? extends Expression> shuttledExpressions = 
ExpressionUtils.shuttleExpressionWithLineage(
-                    joinConjunctExpressions, topPlan, new BitSet());
+                    joinConjunctExpressions, topPlan);
             for (int i = 0; i < shuttledExpressions.size(); i++) {
                 
putShuttledExpressionToExpressionsMap(shuttledExpressionsToExpressionsMap,
                         expressionToShuttledExpressionToMap,
@@ -245,7 +243,7 @@ public class StructInfo {
         hyperGraph.getFilterEdges().forEach(filterEdge -> {
             List<? extends Expression> filterExpressions = 
filterEdge.getExpressions();
             List<? extends Expression> shuttledExpressions = 
ExpressionUtils.shuttleExpressionWithLineage(
-                    filterExpressions, topPlan, new BitSet());
+                    filterExpressions, topPlan);
             for (int i = 0; i < shuttledExpressions.size(); i++) {
                 
putShuttledExpressionToExpressionsMap(shuttledExpressionsToExpressionsMap,
                         expressionToShuttledExpressionToMap,
@@ -260,7 +258,7 @@ public class StructInfo {
     private static Pair<SplitPredicate, EquivalenceClass> 
predicatesDerive(Predicates predicates, Plan originalPlan) {
         // construct equivalenceClass according to equals predicates
         List<Expression> shuttledExpression = 
ExpressionUtils.shuttleExpressionWithLineage(
-                        new ArrayList<>(predicates.getPulledUpPredicates()), 
originalPlan, new BitSet()).stream()
+                        new ArrayList<>(predicates.getPulledUpPredicates()), 
originalPlan).stream()
                 .map(Expression.class::cast)
                 .collect(Collectors.toList());
         SplitPredicate splitPredicate = 
Predicates.splitPredicates(ExpressionUtils.and(shuttledExpression));
@@ -311,13 +309,13 @@ public class StructInfo {
         Map<RelationId, StructInfoNode> relationIdStructInfoNodeMap = new 
LinkedHashMap<>();
         Map<ExpressionPosition, Multimap<Expression, Pair<Expression, 
HyperElement>>>
                 shuttledHashConjunctsToConjunctsMap = new LinkedHashMap<>();
-        BitSet tableBitSet = new BitSet();
+        BitSet relationBitSet = new BitSet();
         Map<ExpressionPosition, Map<Expression, Expression>> 
expressionToShuttledExpressionToMap = new HashMap<>();
         boolean valid = collectStructInfoFromGraph(hyperGraph, topPlan, 
shuttledHashConjunctsToConjunctsMap,
                 expressionToShuttledExpressionToMap,
                 relationList,
                 relationIdStructInfoNodeMap,
-                tableBitSet,
+                relationBitSet,
                 cascadesContext);
         valid = valid
                 && hyperGraph.getNodes().stream().allMatch(n -> 
((StructInfoNode) n).getExpressions() != null);
@@ -333,11 +331,11 @@ public class StructInfo {
                 predicateCollectorContext.getCouldNotPullUpPredicates());
         // this should use the output of originalPlan to make sure the output 
right order
         List<? extends Expression> planOutputShuttledExpressions =
-                
ExpressionUtils.shuttleExpressionWithLineage(originalPlan.getOutput(), 
originalPlan, new BitSet());
+                
ExpressionUtils.shuttleExpressionWithLineage(originalPlan.getOutput(), 
originalPlan);
         return new StructInfo(originalPlan, originalPlanId, hyperGraph, valid, 
topPlan, bottomPlan,
                 relationList, relationIdStructInfoNodeMap, predicates, 
planSplitContext.getGroupingId(),
                 shuttledHashConjunctsToConjunctsMap, 
expressionToShuttledExpressionToMap,
-                tableBitSet, null, null, planOutputShuttledExpressions);
+                relationBitSet, null, null, planOutputShuttledExpressions);
     }
 
     public List<CatalogRelation> getRelations() {
@@ -441,8 +439,8 @@ public class StructInfo {
         return originalPlanId;
     }
 
-    public BitSet getTableBitSet() {
-        return tableBitSet;
+    public BitSet getRelationBitSet() {
+        return relationBitSet;
     }
 
     public List<? extends Expression> getPlanOutputShuttledExpressions() {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/QueryPartitionCollector.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/QueryPartitionCollector.java
index 9907f5270e7..e0c833f95b1 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/QueryPartitionCollector.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/QueryPartitionCollector.java
@@ -36,7 +36,6 @@ import org.apache.logging.log4j.Logger;
 
 import java.util.HashSet;
 import java.util.List;
-import java.util.Map;
 import java.util.Set;
 
 /**
@@ -56,9 +55,9 @@ public class QueryPartitionCollector extends 
DefaultPlanVisitor<Void, CascadesCo
         }
         StatementContext statementContext = context.getStatementContext();
         // Collect relationId to tableId mapping
-        Map<Integer, Integer> relationIdToTableId = 
statementContext.getRelationIdToCommonTableIdMap();
-        relationIdToTableId.put(catalogRelation.getRelationId().asInt(),
-                
statementContext.getTableId(catalogRelation.getTable()).asInt());
+        Multimap<Integer, Integer> relationIdToTableId = 
statementContext.getCommonTableIdToRelationIdMap();
+        
relationIdToTableId.put(statementContext.getTableId(catalogRelation.getTable()).asInt(),
+                catalogRelation.getRelationId().asInt());
         // Collect table used partition mapping
         Multimap<List<String>, Pair<RelationId, Set<String>>> 
tableUsedPartitionNameMap = statementContext
                 .getTableUsedPartitionNameMap();
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java
index dadfc9fa0f4..f131128a088 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java
@@ -319,15 +319,10 @@ public class ExpressionUtils {
         }
     }
 
-    public static Expression shuttleExpressionWithLineage(Expression 
expression, Plan plan, BitSet tableBitSet) {
+    public static Expression shuttleExpressionWithLineage(Expression 
expression, Plan plan) {
         return shuttleExpressionWithLineage(Lists.newArrayList(expression), 
plan).get(0);
     }
 
-    public static List<? extends Expression> 
shuttleExpressionWithLineage(List<? extends Expression> expressions,
-            Plan plan, BitSet tableBitSet) {
-        return shuttleExpressionWithLineage(expressions, plan);
-    }
-
     /**
      * Replace the slot in expressions with the lineage identifier from 
specifiedbaseTable sets or target table types
      * example as following:
@@ -335,7 +330,6 @@ public class ExpressionUtils {
      * select b - 5 as a, d from table
      * );
      * op expression before is: a + 10 as a1, d. after is: b - 5 + 10, d
-     * todo to get from plan struct info
      */
     public static List<? extends Expression> 
shuttleExpressionWithLineage(List<? extends Expression> expressions,
             Plan plan) {
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/memo/StructInfoMapTest.java 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/memo/StructInfoMapTest.java
index d1ba92c7d1c..6be5527341b 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/memo/StructInfoMapTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/memo/StructInfoMapTest.java
@@ -27,6 +27,7 @@ import org.apache.doris.nereids.util.PlanChecker;
 import org.apache.doris.qe.ConnectContext;
 import org.apache.doris.qe.SessionVariable;
 
+import com.google.common.collect.Multimap;
 import mockit.Mock;
 import mockit.MockUp;
 import org.junit.jupiter.api.Assertions;
@@ -61,10 +62,19 @@ class StructInfoMapTest extends SqlTestBase {
                 .rewrite()
                 .optimize();
         Group root = c1.getMemo().getRoot();
-        Set<BitSet> tableMaps = root.getStructInfoMap().getTableMaps();
+        Set<BitSet> tableMaps = root.getStructInfoMap().getTableMaps(true);
         Assertions.assertTrue(tableMaps.isEmpty());
-        root.getStructInfoMap().refresh(root, c1, new BitSet(), new 
HashSet<>(),
-                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite);
+
+        Multimap<Integer, Integer> commonTableIdToRelationIdMap
+                = c1.getStatementContext().getCommonTableIdToRelationIdMap();
+        BitSet targetBitSet = new BitSet();
+        for (Integer tableId : commonTableIdToRelationIdMap.keys()) {
+            targetBitSet.set(tableId);
+        }
+        c1.getMemo().incrementAndGetRefreshVersion(targetBitSet);
+        int memoVersion = StructInfoMap.getMemoVersion(targetBitSet, 
c1.getMemo().getRefreshVersion());
+        root.getStructInfoMap().refresh(root, c1, targetBitSet, new 
HashSet<>(),
+                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite, 
memoVersion, true);
         Assertions.assertEquals(1, tableMaps.size());
         new MockUp<MTMVRelationManager>() {
             @Mock
@@ -81,6 +91,7 @@ class StructInfoMapTest extends SqlTestBase {
         };
         connectContext.getSessionVariable().enableMaterializedViewRewrite = 
true;
         connectContext.getSessionVariable().enableMaterializedViewNestRewrite 
= true;
+        
connectContext.getSessionVariable().materializedViewRewriteDurationThresholdMs 
= 1000000;
 
         dropMvByNereids("drop materialized view if exists mv1");
         createMvByNereids("create materialized view mv1 BUILD IMMEDIATE 
REFRESH COMPLETE ON MANUAL\n"
@@ -102,9 +113,19 @@ class StructInfoMapTest extends SqlTestBase {
                 .optimize()
                 .printlnBestPlanTree();
         root = c1.getMemo().getRoot();
-        root.getStructInfoMap().refresh(root, c1, new BitSet(), new 
HashSet<>(),
-                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite);
-        tableMaps = root.getStructInfoMap().getTableMaps();
+        // because refresh struct info by targetBitSet when 
getValidQueryStructInfos, this would cause
+        // query struct info version increase twice. so need increase the memo 
version manually.
+        commonTableIdToRelationIdMap
+                = c1.getStatementContext().getCommonTableIdToRelationIdMap();
+        targetBitSet = new BitSet();
+        for (Integer tableId : commonTableIdToRelationIdMap.keys()) {
+            targetBitSet.set(tableId);
+        }
+        c1.getMemo().incrementAndGetRefreshVersion(targetBitSet);
+        memoVersion = StructInfoMap.getMemoVersion(targetBitSet, 
c1.getMemo().getRefreshVersion());
+        root.getStructInfoMap().refresh(root, c1, targetBitSet, new 
HashSet<>(),
+                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite, 
memoVersion, true);
+        tableMaps = root.getStructInfoMap().getTableMaps(true);
         Assertions.assertEquals(2, tableMaps.size());
         dropMvByNereids("drop materialized view mv1");
     }
@@ -130,12 +151,12 @@ class StructInfoMapTest extends SqlTestBase {
                 .rewrite()
                 .optimize();
         Group root = c1.getMemo().getRoot();
-        Set<BitSet> tableMaps = root.getStructInfoMap().getTableMaps();
+        Set<BitSet> tableMaps = root.getStructInfoMap().getTableMaps(true);
         Assertions.assertTrue(tableMaps.isEmpty());
         root.getStructInfoMap().refresh(root, c1, new BitSet(), new 
HashSet<>(),
-                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite);
+                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite, 0, true);
         root.getStructInfoMap().refresh(root, c1, new BitSet(), new 
HashSet<>(),
-                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite);
+                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite, 0, true);
         Assertions.assertEquals(1, tableMaps.size());
         new MockUp<MTMVRelationManager>() {
             @Mock
@@ -172,9 +193,19 @@ class StructInfoMapTest extends SqlTestBase {
                 .optimize()
                 .printlnBestPlanTree();
         root = c1.getMemo().getRoot();
-        root.getStructInfoMap().refresh(root, c1, new BitSet(), new 
HashSet<>(),
-                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite);
-        tableMaps = root.getStructInfoMap().getTableMaps();
+        // because refresh struct info by targetBitSet when 
getValidQueryStructInfos, this would cause
+        // query struct info version increase twice. so need increase the memo 
version manually.
+        Multimap<Integer, Integer> commonTableIdToRelationIdMap
+                = c1.getStatementContext().getCommonTableIdToRelationIdMap();
+        BitSet targetBitSet = new BitSet();
+        for (Integer relationId : commonTableIdToRelationIdMap.values()) {
+            targetBitSet.set(relationId);
+        }
+        c1.getMemo().incrementAndGetRefreshVersion(targetBitSet);
+        int memoVersion = StructInfoMap.getMemoVersion(targetBitSet, 
c1.getMemo().getRefreshVersion());
+        root.getStructInfoMap().refresh(root, c1, targetBitSet, new 
HashSet<>(),
+                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite, 
memoVersion, true);
+        tableMaps = root.getStructInfoMap().getTableMaps(true);
         Assertions.assertEquals(2, tableMaps.size());
         dropMvByNereids("drop materialized view mv1");
     }
@@ -229,15 +260,25 @@ class StructInfoMapTest extends SqlTestBase {
                 .preMvRewrite()
                 .optimize();
         Group root = c1.getMemo().getRoot();
-        root.getStructInfoMap().refresh(root, c1, new BitSet(), new 
HashSet<>(),
-                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite);
+        // because refresh struct info by targetBitSet when 
getValidQueryStructInfos, this would cause
+        // query struct info version increase twice. so need increase the memo 
version manually.
+        Multimap<Integer, Integer> commonTableIdToRelationIdMap
+                = c1.getStatementContext().getCommonTableIdToRelationIdMap();
+        BitSet targetBitSet = new BitSet();
+        for (Integer relationId : commonTableIdToRelationIdMap.values()) {
+            targetBitSet.set(relationId);
+        }
+        c1.getMemo().incrementAndGetRefreshVersion(targetBitSet);
+        int memoVersion = StructInfoMap.getMemoVersion(targetBitSet, 
c1.getMemo().getRefreshVersion());
+        root.getStructInfoMap().refresh(root, c1, targetBitSet, new 
HashSet<>(),
+                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite, 
memoVersion, true);
         StructInfoMap structInfoMap = root.getStructInfoMap();
-        Assertions.assertEquals(2, structInfoMap.getTableMaps().size());
-        BitSet mvMap = structInfoMap.getTableMaps().stream()
+        Assertions.assertEquals(2, structInfoMap.getTableMaps(true).size());
+        BitSet mvMap = structInfoMap.getTableMaps(true).stream()
                 .filter(b -> b.cardinality() == 2)
                 .collect(Collectors.toList()).get(0);
         StructInfo structInfo = structInfoMap.getStructInfo(c1, mvMap, root, 
null,
-                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite);
+                
connectContext.getSessionVariable().enableMaterializedViewNestRewrite, true);
         System.out.println(structInfo.getOriginalPlan().treeString());
         BitSet bitSet = new BitSet();
         for (CatalogRelation relation : structInfo.getRelations()) {
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PointQueryShouldNotMvRewriteTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PointQueryShouldNotMvRewriteTest.java
index 423c996a26b..0b8881cc63d 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PointQueryShouldNotMvRewriteTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PointQueryShouldNotMvRewriteTest.java
@@ -85,8 +85,9 @@ public class PointQueryShouldNotMvRewriteTest extends 
SqlTestBase {
                 .optimize()
                 .printlnBestPlanTree();
         Group root = c1.getMemo().getRoot();
-        root.getStructInfoMap().refresh(root, c1, new BitSet(), new 
HashSet<>(), false);
-        Set<BitSet> tableMaps = root.getStructInfoMap().getTableMaps();
+        root.getStructInfoMap().refresh(root, c1, new BitSet(), new 
HashSet<>(), false, 0,
+                false);
+        Set<BitSet> tableMaps = root.getStructInfoMap().getTableMaps(false);
         Assertions.assertEquals(1, tableMaps.size());
         dropMvByNereids("drop materialized view mv1");
     }
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PreMaterializedViewRewriterTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PreMaterializedViewRewriterTest.java
index 214f5d9a7f6..9385ac2cd5d 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PreMaterializedViewRewriterTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PreMaterializedViewRewriterTest.java
@@ -2967,9 +2967,9 @@ public class PreMaterializedViewRewriterTest extends 
SqlTestBase {
         // extract plan from memo and check is equals or not
         Memo memo = cascadesContext.getMemo();
         for (Map.Entry<BitSet, LogicalPlan> planEntry : 
bitSetLogicalPlanMap.entrySet()) {
-            memo.incrementAndGetRefreshVersion();
+            memo.incrementAndGetRefreshVersion(planEntry.getKey());
             StructInfo structInfo = 
memo.getRoot().getStructInfoMap().getStructInfo(cascadesContext,
-                    planEntry.getKey(), memo.getRoot(), null, true);
+                    planEntry.getKey(), memo.getRoot(), null, true, false);
             Assertions.assertNotNull(structInfo);
             
Assertions.assertTrue(structInfo.getOriginalPlan().deepEquals(planEntry.getValue()));
         }
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PredicatesTest.java 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PredicatesTest.java
index 6b5e0596205..473ebb4908f 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PredicatesTest.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/PredicatesTest.java
@@ -20,7 +20,6 @@ package org.apache.doris.nereids.mv;
 import org.apache.doris.nereids.CascadesContext;
 import org.apache.doris.nereids.rules.exploration.mv.ComparisonResult;
 import org.apache.doris.nereids.rules.exploration.mv.HyperGraphComparator;
-import org.apache.doris.nereids.rules.exploration.mv.MaterializedViewUtils;
 import org.apache.doris.nereids.rules.exploration.mv.Predicates;
 import org.apache.doris.nereids.rules.exploration.mv.Predicates.ExpressionInfo;
 import org.apache.doris.nereids.rules.exploration.mv.StructInfo;
@@ -34,7 +33,6 @@ import org.apache.doris.nereids.util.PlanChecker;
 import org.junit.jupiter.api.Assertions;
 import org.junit.jupiter.api.Test;
 
-import java.util.BitSet;
 import java.util.Map;
 
 /** Test the method in Predicates*/
@@ -107,10 +105,8 @@ public class PredicatesTest extends SqlTestBase {
                 .rewrite()
                 .getAllPlan().get(0).child(0);
 
-        StructInfo mvStructInfo = 
MaterializedViewUtils.extractStructInfo(mvPlan, mvPlan,
-                mvContext, new BitSet()).get(0);
-        StructInfo queryStructInfo = 
MaterializedViewUtils.extractStructInfo(queryPlan, queryPlan,
-                queryContext, new BitSet()).get(0);
+        StructInfo mvStructInfo = StructInfo.of(mvPlan, mvPlan, mvContext);
+        StructInfo queryStructInfo = StructInfo.of(queryPlan, queryPlan, 
queryContext);
         RelationMapping relationMapping = 
RelationMapping.generate(mvStructInfo.getRelations(),
                         queryStructInfo.getRelations(), 16).get(0);
 
@@ -163,10 +159,8 @@ public class PredicatesTest extends SqlTestBase {
                 .rewrite()
                 .getAllPlan().get(0).child(0);
 
-        StructInfo mvStructInfo = 
MaterializedViewUtils.extractStructInfo(mvPlan, mvPlan,
-                mvContext, new BitSet()).get(0);
-        StructInfo queryStructInfo = 
MaterializedViewUtils.extractStructInfo(queryPlan, queryPlan,
-                queryContext, new BitSet()).get(0);
+        StructInfo mvStructInfo = StructInfo.of(mvPlan, mvPlan, mvContext);
+        StructInfo queryStructInfo = StructInfo.of(queryPlan, queryPlan, 
queryContext);
         RelationMapping relationMapping = 
RelationMapping.generate(mvStructInfo.getRelations(),
                 queryStructInfo.getRelations(), 16).get(0);
 
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphAggTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphAggTest.java
index b62c41db00d..af36afe7151 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphAggTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphAggTest.java
@@ -31,7 +31,6 @@ import org.junit.jupiter.api.Assertions;
 import org.junit.jupiter.api.Disabled;
 import org.junit.jupiter.api.Test;
 
-import java.util.BitSet;
 import java.util.Objects;
 
 class HyperGraphAggTest extends SqlTestBase {
@@ -89,10 +88,8 @@ class HyperGraphAggTest extends SqlTestBase {
     }
 
     LogicalCompatibilityContext constructContext(Plan p1, Plan p2) {
-        StructInfo st1 = MaterializedViewUtils.extractStructInfo(p1, p1,
-                null, new BitSet()).get(0);
-        StructInfo st2 = MaterializedViewUtils.extractStructInfo(p2, p2,
-                null, new BitSet()).get(0);
+        StructInfo st1 = StructInfo.of(p1, p1, null);
+        StructInfo st2 = StructInfo.of(p2, p2, null);
         RelationMapping rm = RelationMapping.generate(st1.getRelations(), 
st2.getRelations(), 8)
                 .get(0);
         SlotMapping sm = SlotMapping.generate(rm);
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/sqltest/SqlTestBase.java 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/sqltest/SqlTestBase.java
index a34ec7efb15..1be34d85bef 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/nereids/sqltest/SqlTestBase.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/sqltest/SqlTestBase.java
@@ -21,7 +21,6 @@ import org.apache.doris.catalog.Env;
 import org.apache.doris.catalog.MetaIdGenerator.IdGeneratorBuffer;
 import org.apache.doris.nereids.CascadesContext;
 import 
org.apache.doris.nereids.rules.exploration.mv.LogicalCompatibilityContext;
-import org.apache.doris.nereids.rules.exploration.mv.MaterializedViewUtils;
 import org.apache.doris.nereids.rules.exploration.mv.StructInfo;
 import org.apache.doris.nereids.rules.exploration.mv.mapping.RelationMapping;
 import org.apache.doris.nereids.rules.exploration.mv.mapping.SlotMapping;
@@ -30,8 +29,6 @@ import org.apache.doris.nereids.trees.plans.Plan;
 import org.apache.doris.nereids.util.MemoPatternMatchSupported;
 import org.apache.doris.utframe.TestWithFeService;
 
-import java.util.BitSet;
-
 public abstract class SqlTestBase extends TestWithFeService implements 
MemoPatternMatchSupported {
     @Override
     protected void runBeforeAll() throws Exception {
@@ -840,10 +837,8 @@ public abstract class SqlTestBase extends 
TestWithFeService implements MemoPatte
     }
 
     protected LogicalCompatibilityContext constructContext(Plan p1, Plan p2, 
CascadesContext context) {
-        StructInfo st1 = MaterializedViewUtils.extractStructInfo(p1, p1,
-                context, new BitSet()).get(0);
-        StructInfo st2 = MaterializedViewUtils.extractStructInfo(p2, p2,
-                context, new BitSet()).get(0);
+        StructInfo st1 = StructInfo.of(p1, p1, context);
+        StructInfo st2 = StructInfo.of(p2, p2, context);
         RelationMapping rm = RelationMapping.generate(st1.getRelations(), 
st2.getRelations(), 8)
                 .get(0);
         SlotMapping sm = SlotMapping.generate(rm);
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/util/ExpressionUtilsTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/util/ExpressionUtilsTest.java
index 65e6399e67f..a0016dfdf5d 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/util/ExpressionUtilsTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/util/ExpressionUtilsTest.java
@@ -56,7 +56,6 @@ import org.junit.jupiter.api.Assertions;
 import org.junit.jupiter.api.Test;
 
 import java.util.Arrays;
-import java.util.BitSet;
 import java.util.HashMap;
 import java.util.List;
 import java.util.Map;
@@ -241,7 +240,7 @@ public class ExpressionUtilsTest extends TestWithFeService {
         LogicalProject<LogicalOdbcScan> project = new 
LogicalProject<>(ImmutableList.of(a, bAlias),
                 new LogicalOdbcScan(new RelationId(0), olapTable, 
ImmutableList.of("test")));
         List<? extends Expression> expressions = 
ExpressionUtils.shuttleExpressionWithLineage(project.getOutput(),
-                project, new BitSet());
+                project);
         // should not loop, should break out loop
         Assertions.assertEquals(expressions, ImmutableList.of(a, 
bAlias.toSlot()));
     }
diff --git 
a/regression-test/suites/nereids_rules_p0/mv/nested_mtmv/nested_mtmv.groovy 
b/regression-test/suites/nereids_rules_p0/mv/nested_mtmv/nested_mtmv.groovy
index 30ea11ec956..89e927c3f93 100644
--- a/regression-test/suites/nereids_rules_p0/mv/nested_mtmv/nested_mtmv.groovy
+++ b/regression-test/suites/nereids_rules_p0/mv/nested_mtmv/nested_mtmv.groovy
@@ -214,19 +214,19 @@ suite("nested_mtmv") {
         group by l_orderkey, l_linenumber, l_partkey, o_orderkey, o_custkey, 
ps_partkey
         """
     def mv_level1_name = "mv_level1_name"
-    def mv_stmt_5 = """
+    def mv_level2_stmt = """
         select l_orderkey, l_linenumber, l_partkey, o_orderkey, o_custkey, 
ps_partkey, col1
         from ${mv_level1_name}
         """
     def mv_level2_name = "mv_level2_name"
-    def mv_stmt_6 = """
+    def mv_level3_stmt = """
         select t1.l_orderkey, t2.l_linenumber, t1.l_partkey, t2.o_orderkey, 
t1.o_custkey, t2.ps_partkey, t1.col1
         from ${mv_level1_name} as t1
         left join ${mv_level1_name} as t2 
         on t1.l_orderkey = t2.l_orderkey
         """
     def mv_level3_name = "mv_level3_name"
-    def mv_stmt_7 = """
+    def mv_level4_stmt = """
         select t1.l_orderkey, t2.l_linenumber, t1.l_partkey, t2.o_orderkey, 
t1.o_custkey, t2.ps_partkey, t1.col1
         from ${mv_level2_name} as t1
         left join ${mv_level2_name} as t2 
@@ -236,11 +236,11 @@ suite("nested_mtmv") {
 
     create_async_mv(db, mv_level1_name, mv_stmt_4)
 
-    create_async_mv(db, mv_level2_name, mv_stmt_5)
+    create_async_mv(db, mv_level2_name, mv_level2_stmt)
 
-    create_async_mv(db, mv_level3_name, mv_stmt_6)
+    create_async_mv(db, mv_level3_name, mv_level3_stmt)
 
-    create_async_mv(db, mv_level4_name, mv_stmt_7)
+    create_async_mv(db, mv_level4_name, mv_level4_stmt)
 
     def query_stmt_2 = """
         select t1.l_orderkey, t2.l_linenumber, t1.l_partkey, t2.o_orderkey, 
t1.o_custkey, t2.ps_partkey, t1.col1


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to