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

yiguolei 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 99ad5a83482 [Fix](nereids) Fix cte rewrite by mv failure and 
predicates compensation by mistake (#29820)
99ad5a83482 is described below

commit 99ad5a8348274e41fa136ff80ce1c332dd74c99f
Author: seawinde <149132972+seawi...@users.noreply.github.com>
AuthorDate: Fri Jan 12 16:30:35 2024 +0800

    [Fix](nereids) Fix cte rewrite by mv failure and predicates compensation by 
mistake (#29820)
    
    Fix cte rewrite by mv wrongly when query has scalar aggregate but view no
    For example as following, it should not be rewritten by materialized view 
successfully
    
    // materialzied view define
    def mv20_1 = """
    select
    l_shipmode,
    l_shipinstruct,
    sum(l_extendedprice),
    count()
    from lineitem
    left join
    orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY
    group by
    l_shipmode,
    l_shipinstruct;
    """
    // query sql
    def query20_1 =
    """
    select
    sum(l_extendedprice),
    count()
    from lineitem
    left join
    orders
    on lineitem.L_ORDERKEY = orders.O_ORDERKEY
    """
    
    Fix predicates compensation by mistake
    For example as following, it can return right result, but it's wrong 
earlier.
    
    // materialzied view define
    def mv7_1 = """
    select l_shipdate, o_orderdate, l_partkey, l_suppkey
    from lineitem
    left join orders
    on lineitem.l_orderkey = orders.o_orderkey
    where l_shipdate = '2023-12-08' and o_orderdate = '2023-12-08';
    """
    // query sql
    def query7_1 = """
    select l_shipdate, o_orderdate, l_partkey, l_suppkey
    from (select * from lineitem where l_shipdate = '2023-10-17' ) t1
    left join orders
    on t1.l_orderkey = orders.o_orderkey;
    """
    
    and optimize some code usage and add more comment for method
---
 .../query-async-materialized-view.md               |   1 -
 .../query-async-materialized-view.md               |   1 -
 .../mv/AbstractMaterializedViewAggregateRule.java  |  28 ++-
 .../mv/AbstractMaterializedViewRule.java           | 215 ++++++---------------
 .../mv/LogicalCompatibilityContext.java            |   8 +-
 .../exploration/mv/MaterializedViewUtils.java      |  18 ++
 .../nereids/rules/exploration/mv/Predicates.java   | 178 +++++++++++++++--
 .../nereids/rules/exploration/mv/StructInfo.java   |  32 ++-
 .../plans/visitor/ExpressionLineageReplacer.java   |  25 ++-
 .../joinorder/hypergraph/CompareOuterJoinTest.java |   6 +-
 .../jobs/joinorder/hypergraph/InferJoinTest.java   |   6 +-
 .../joinorder/hypergraph/InferPredicateTest.java   |   6 +-
 .../joinorder/hypergraph/PullupExpressionTest.java |   6 +-
 .../exploration/mv/HyperGraphComparatorTest.java   |   4 +-
 .../mv/agg_with_roll_up/aggregate_with_roll_up.out |  36 ++++
 .../aggregate_without_roll_up.out                  |  28 +++
 .../nereids_rules_p0/mv/join/inner/inner_join.out  |  16 ++
 .../mv/join/left_outer/outer_join.out              |  44 +++++
 .../agg_with_roll_up/aggregate_with_roll_up.groovy |  90 +++++++--
 .../aggregate_without_roll_up.groovy               | 107 +++++++++-
 .../mv/join/inner/inner_join.groovy                |  35 ++--
 .../mv/join/left_outer/outer_join.groovy           |  82 ++++++--
 22 files changed, 716 insertions(+), 256 deletions(-)

diff --git 
a/docs/en/docs/query-acceleration/async-materialized-view/query-async-materialized-view.md
 
b/docs/en/docs/query-acceleration/async-materialized-view/query-async-materialized-view.md
index 4c6418e7606..8cd13b4e43a 100644
--- 
a/docs/en/docs/query-acceleration/async-materialized-view/query-async-materialized-view.md
+++ 
b/docs/en/docs/query-acceleration/async-materialized-view/query-async-materialized-view.md
@@ -398,7 +398,6 @@ If you want to know the detailed information about 
materialized view candidates,
 | SET enable_nereids_planner = true;                                        | 
Asynchronous materialized views are only supported under the new optimizer, so 
the new optimizer needs to be enabled.       |
 | SET enable_materialized_view_rewrite = true;                              | 
Enable or disable query transparent rewriting, default is disabled              
       |
 | SET materialized_view_rewrite_enable_contain_external_table = true;       | 
Whether materialized views participating in transparent rewriting are allowed 
to contain external tables, default is not allowed           |
-| SET disable_nereids_rules = 'ELIMINATE_OUTER_JOIN';                       | 
Currently, outer join elimination has an impact on transparent  |
 
 
 ## Limitations
diff --git 
a/docs/zh-CN/docs/query-acceleration/async-materialized-view/query-async-materialized-view.md
 
b/docs/zh-CN/docs/query-acceleration/async-materialized-view/query-async-materialized-view.md
index d35a5383dc8..1921498f85c 100644
--- 
a/docs/zh-CN/docs/query-acceleration/async-materialized-view/query-async-materialized-view.md
+++ 
b/docs/zh-CN/docs/query-acceleration/async-materialized-view/query-async-materialized-view.md
@@ -380,7 +380,6 @@ WHERE o_orderkey > 5 AND o_orderkey <= 10;
 | SET enable_nereids_planner = true;                                     | 
异步物化视图只有在新优化器下才支持,所以需要开启新优化器           |
 | SET enable_materialized_view_rewrite = true;                           | 
开启或者关闭查询透明改写,默认关闭                      |
 | SET materialized_view_rewrite_enable_contain_external_table = true;    | 
参与透明改写的物化视图是否允许包含外表,默认不允许              |
-| SET disable_nereids_rules = 'ELIMINATE_OUTER_JOIN';                    | 目前 
outer join 消除会对透明改写有影响,暂时需要关闭,后面会优化 |
 
 
 ## 限制
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 2804a0814f6..e47e15dd56b 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
@@ -48,8 +48,6 @@ import com.google.common.collect.ArrayListMultimap;
 import com.google.common.collect.HashMultimap;
 import com.google.common.collect.Multimap;
 import com.google.common.collect.Sets;
-import org.apache.logging.log4j.LogManager;
-import org.apache.logging.log4j.Logger;
 
 import java.util.ArrayList;
 import java.util.Collection;
@@ -68,8 +66,6 @@ public abstract class AbstractMaterializedViewAggregateRule 
extends AbstractMate
 
     protected static final Multimap<Expression, Expression>
             AGGREGATE_ROLL_UP_EQUIVALENT_FUNCTION_MAP = 
ArrayListMultimap.create();
-    protected final String currentClassName = this.getClass().getSimpleName();
-    private final Logger logger = LogManager.getLogger(this.getClass());
 
     static {
         // support count distinct roll up
@@ -142,6 +138,18 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
                                     
materializationContext.getMvExprToMvScanExprMapping(),
                                     queryToViewSlotMapping)));
         }
+        // if view is scalar aggregate but query is not. Or if query is scalar 
aggregate but view is not
+        // Should not rewrite
+        if (queryTopPlanAndAggPair.value().getGroupByExpressions().isEmpty()
+                || 
viewTopPlanAndAggPair.value().getGroupByExpressions().isEmpty()) {
+            
materializationContext.recordFailReason(queryStructInfo.getOriginalPlanId(),
+                    Pair.of("only one the of query or view is scalar aggregate 
and "
+                                    + "can not rewrite expression meanwhile",
+                            String.format("query aggregate = %s,\n view 
aggregate = %s,\n",
+                                    
queryTopPlanAndAggPair.value().treeString(),
+                                    
viewTopPlanAndAggPair.value().treeString())));
+            return null;
+        }
         // try to roll up.
         // split the query top plan expressions to group expressions and 
functions, if can not, bail out.
         Pair<Set<? extends Expression>, Set<? extends Expression>> 
queryGroupAndFunctionPair
@@ -268,15 +276,15 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
         Plan viewTopPlan = viewTopPlanAndAggPair.key();
         LogicalAggregate<Plan> queryAggregate = queryTopPlanAndAggPair.value();
         LogicalAggregate<Plan> viewAggregate = viewTopPlanAndAggPair.value();
-        List<? extends Expression> queryGroupShuttledExpression = 
ExpressionUtils.shuttleExpressionWithLineage(
-                queryAggregate.getGroupByExpressions(), queryTopPlan);
-        List<? extends Expression> viewGroupShuttledExpression = 
ExpressionUtils.shuttleExpressionWithLineage(
+        Set<? extends Expression> queryGroupShuttledExpression = new HashSet<>(
+                ExpressionUtils.shuttleExpressionWithLineage(
+                        queryAggregate.getGroupByExpressions(), queryTopPlan));
+        Set<? extends Expression> viewGroupShuttledExpressionQueryBased = 
ExpressionUtils.shuttleExpressionWithLineage(
                         viewAggregate.getGroupByExpressions(), viewTopPlan)
                 .stream()
                 .map(expr -> ExpressionUtils.replace(expr, 
viewToQurySlotMapping.toSlotReferenceMap()))
-                .collect(Collectors.toList());
-        return queryAggregate.getGroupByExpressions().size() == 
viewAggregate.getGroupByExpressions().size()
-                && 
queryGroupShuttledExpression.equals(viewGroupShuttledExpression);
+                .collect(Collectors.toSet());
+        return 
queryGroupShuttledExpression.equals(viewGroupShuttledExpressionQueryBased);
     }
 
     /**
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 7e87fd4c15a..352bccc019b 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
@@ -31,16 +31,13 @@ import org.apache.doris.nereids.jobs.executor.Rewriter;
 import org.apache.doris.nereids.memo.GroupExpression;
 import org.apache.doris.nereids.rules.exploration.ExplorationRuleFactory;
 import org.apache.doris.nereids.rules.exploration.mv.Predicates.SplitPredicate;
-import 
org.apache.doris.nereids.rules.exploration.mv.mapping.EquivalenceClassSetMapping;
 import org.apache.doris.nereids.rules.exploration.mv.mapping.ExpressionMapping;
 import org.apache.doris.nereids.rules.exploration.mv.mapping.RelationMapping;
 import org.apache.doris.nereids.rules.exploration.mv.mapping.SlotMapping;
 import org.apache.doris.nereids.trees.expressions.Alias;
-import org.apache.doris.nereids.trees.expressions.EqualTo;
 import org.apache.doris.nereids.trees.expressions.Expression;
 import org.apache.doris.nereids.trees.expressions.NamedExpression;
 import org.apache.doris.nereids.trees.expressions.Slot;
-import org.apache.doris.nereids.trees.expressions.SlotReference;
 import org.apache.doris.nereids.trees.expressions.functions.scalar.NonNullable;
 import org.apache.doris.nereids.trees.expressions.functions.scalar.Nullable;
 import org.apache.doris.nereids.trees.expressions.literal.BooleanLiteral;
@@ -61,7 +58,6 @@ import com.google.common.collect.Sets;
 import java.util.ArrayList;
 import java.util.Collection;
 import java.util.HashSet;
-import java.util.Iterator;
 import java.util.List;
 import java.util.Map;
 import java.util.Objects;
@@ -85,7 +81,7 @@ public abstract class AbstractMaterializedViewRule implements 
ExplorationRuleFac
         if (materializationContexts.isEmpty()) {
             return rewriteResults;
         }
-        List<StructInfo> queryStructInfos = extractStructInfo(queryPlan, 
cascadesContext);
+        List<StructInfo> queryStructInfos = 
MaterializedViewUtils.extractStructInfo(queryPlan, cascadesContext);
         // TODO Just Check query queryPlan firstly, support multi later.
         StructInfo queryStructInfo = queryStructInfos.get(0);
         if (!checkPattern(queryStructInfo)) {
@@ -99,7 +95,8 @@ public abstract class AbstractMaterializedViewRule implements 
ExplorationRuleFac
             if (checkIfRewritten(queryPlan, materializationContext)) {
                 continue;
             }
-            List<StructInfo> viewStructInfos = 
extractStructInfo(materializationContext.getMvPlan(), cascadesContext);
+            List<StructInfo> viewStructInfos = 
MaterializedViewUtils.extractStructInfo(
+                    materializationContext.getMvPlan(), cascadesContext);
             if (viewStructInfos.size() > 1) {
                 // view struct info should only have one
                 
materializationContext.recordFailReason(queryStructInfo.getOriginalPlanId(),
@@ -145,16 +142,10 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                                     comparisonResult.getErrorMessage()));
                     continue;
                 }
-                // TODO: Use set of list? And consider view expr
-                List<Expression> pulledUpExpressions = 
ImmutableList.copyOf(comparisonResult.getQueryExpressions());
-                // set pulled up expression to queryStructInfo predicates and 
update related predicates
-                if (!pulledUpExpressions.isEmpty()) {
-                    queryStructInfo.addPredicates(pulledUpExpressions);
-                }
                 SplitPredicate compensatePredicates = 
predicatesCompensate(queryStructInfo, viewStructInfo,
                         queryToViewSlotMapping, comparisonResult, 
cascadesContext);
                 // Can not compensate, bail out
-                if (compensatePredicates.isEmpty()) {
+                if (compensatePredicates.isInvalid()) {
                     
materializationContext.recordFailReason(queryStructInfo.getOriginalPlanId(),
                             Pair.of("Predicate compensate fail",
                                     String.format("query predicates = %s,\n 
query equivalenceClass = %s, \n"
@@ -222,6 +213,9 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
         return rewriteResults;
     }
 
+    /**
+     * Check the logical properties of rewritten plan by mv is the same with 
source plan
+     */
     protected boolean checkOutput(Plan sourcePlan, Plan rewrittenPlan, 
MaterializationContext materializationContext) {
         if (sourcePlan.getGroupExpression().isPresent() && 
!rewrittenPlan.getLogicalProperties()
                 
.equals(sourcePlan.getGroupExpression().get().getOwnerGroup().getLogicalProperties()))
 {
@@ -238,7 +232,7 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
     }
 
     /**
-     * Partition will be pruned in query then add the record the partitions to 
select partitions on
+     * Partition will be pruned in query then add the pruned partitions to 
select partitions field of
      * catalog relation.
      * Maybe only just some partitions is valid in materialized view, so we 
should check if the mv can
      * offer the partitions which query used or not.
@@ -285,10 +279,23 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
      * Use target expression to represent the source expression. Visit the 
source expression,
      * try to replace the source expression with target expression in 
targetExpressionMapping, if found then
      * replace the source expression by target expression mapping value.
-     * Note: make the target expression map key to source based according to 
targetExpressionNeedSourceBased,
-     * if targetExpressionNeedSourceBased is true, we should make it source 
based.
-     * the key expression in targetExpressionMapping should be shuttled. with 
the method
-     * ExpressionUtils.shuttleExpressionWithLineage.
+     *
+     * @param sourceExpressionsToWrite the source expression to write by 
target expression
+     * @param sourcePlan the source plan witch the source expression belong to
+     * @param targetExpressionMapping target expression mapping, if finding 
the expression in key set of the mapping
+     *         then use the corresponding value of mapping to replace it
+     * @param targetExpressionNeedSourceBased if 
targetExpressionNeedSourceBased is true,
+     *         we should make the target expression map key to source based,
+     *         Note: the key expression in targetExpressionMapping should be 
shuttled. with the method
+     *         ExpressionUtils.shuttleExpressionWithLineage.
+     *         example as following:
+     *         source                           target
+     *         project(slot 1, 2)              project(slot 3, 2, 1)
+     *         scan(table)                        scan(table)
+     *         then
+     *         transform source to:
+     *         project(slot 2, 1)
+     *         target
      */
     protected List<Expression> rewriteExpression(List<? extends Expression> 
sourceExpressionsToWrite, Plan sourcePlan,
             ExpressionMapping targetExpressionMapping, SlotMapping 
sourceToTargetMapping,
@@ -296,15 +303,6 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
         // 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
         // can not be represented by target expressions, return null.
-        //
-        // example as following:
-        //     source                           target
-        //        project(slot 1, 2)              project(slot 3, 2, 1)
-        //          scan(table)                        scan(table)
-        //
-        //     transform source to:
-        //        project(slot 2, 1)
-        //            target
         // generate target to target replacement expression mapping, and 
change target expression to source based
         List<? extends Expression> sourceShuttledExpressions = 
ExpressionUtils.shuttleExpressionWithLineage(
                 sourceExpressionsToWrite, sourcePlan);
@@ -348,6 +346,9 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
         return rewrittenExpressions;
     }
 
+    /**
+     * Rewrite single expression, the logic is the same with above
+     */
     protected Expression rewriteExpression(Expression 
sourceExpressionsToWrite, Plan sourcePlan,
             ExpressionMapping targetExpressionMapping, SlotMapping 
sourceToTargetMapping,
             boolean targetExpressionNeedSourceBased) {
@@ -374,27 +375,43 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
             ComparisonResult comparisonResult,
             CascadesContext cascadesContext
     ) {
+        // TODO: Use set of list? And consider view expr
+        List<Expression> queryPulledUpExpressions = 
ImmutableList.copyOf(comparisonResult.getQueryExpressions());
+        // set pulled up expression to queryStructInfo predicates and update 
related predicates
+        if (!queryPulledUpExpressions.isEmpty()) {
+            queryStructInfo.addPredicates(queryPulledUpExpressions);
+        }
+        List<Expression> viewPulledUpExpressions = 
ImmutableList.copyOf(comparisonResult.getViewExpressions());
+        // set pulled up expression to viewStructInfo predicates and update 
related predicates
+        if (!viewPulledUpExpressions.isEmpty()) {
+            viewStructInfo.addPredicates(viewPulledUpExpressions);
+        }
         // viewEquivalenceClass to query based
         SlotMapping viewToQuerySlotMapping = queryToViewSlotMapping.inverse();
-        final Set<Expression> equalCompensateConjunctions = 
compensateEquivalence(
+        // equal predicate compensate
+        final Set<Expression> equalCompensateConjunctions = 
Predicates.compensateEquivalence(
                 queryStructInfo,
                 viewStructInfo,
                 viewToQuerySlotMapping,
                 comparisonResult);
         // range compensate
-        final Set<Expression> rangeCompensatePredicates = 
compensateRangePredicate(
+        final Set<Expression> rangeCompensatePredicates = 
Predicates.compensateRangePredicate(
                 queryStructInfo,
                 viewStructInfo,
                 viewToQuerySlotMapping,
                 comparisonResult);
         // residual compensate
-        final Set<Expression> residualCompensatePredicates = 
compensateResidualPredicate(
+        final Set<Expression> residualCompensatePredicates = 
Predicates.compensateResidualPredicate(
                 queryStructInfo,
                 viewStructInfo,
                 viewToQuerySlotMapping,
                 comparisonResult);
-        // if the join type in query and mv plan is different, we should check 
and add filter on mv to make
-        // the mv join type is accord with query
+        if (equalCompensateConjunctions == null || rangeCompensatePredicates 
== null
+                || residualCompensatePredicates == null) {
+            return SplitPredicate.INVALID_INSTANCE;
+        }
+        // if the join type in query and mv plan is different, we should check 
query is have the
+        // filters which rejects null
         Set<Set<Slot>> requireNoNullableViewSlot = 
comparisonResult.getViewNoNullableSlot();
         // check query is use the null reject slot which view comparison need
         if (!requireNoNullableViewSlot.isEmpty()) {
@@ -403,7 +420,7 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                     cascadesContext);
             if (nullRejectPredicates.isEmpty() || 
queryPulledUpPredicates.containsAll(nullRejectPredicates)) {
                 // query has not null reject predicates, so return
-                return SplitPredicate.invalid();
+                return SplitPredicate.INVALID_INSTANCE;
             }
             Set<Expression> queryUsedNeedRejectNullSlotsViewBased = 
nullRejectPredicates.stream()
                     .map(expression -> 
TypeUtils.isNotNull(expression).orElse(null))
@@ -413,122 +430,17 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                     .collect(Collectors.toSet());
             if (requireNoNullableViewSlot.stream().anyMatch(
                     set -> Sets.intersection(set, 
queryUsedNeedRejectNullSlotsViewBased).isEmpty())) {
-                return SplitPredicate.invalid();
+                return SplitPredicate.INVALID_INSTANCE;
             }
         }
-        return 
SplitPredicate.of(ExpressionUtils.and(equalCompensateConjunctions),
-                rangeCompensatePredicates.isEmpty() ? BooleanLiteral.of(true)
+        return SplitPredicate.of(equalCompensateConjunctions.isEmpty() ? 
BooleanLiteral.TRUE
+                        : ExpressionUtils.and(equalCompensateConjunctions),
+                rangeCompensatePredicates.isEmpty() ? BooleanLiteral.TRUE
                         : ExpressionUtils.and(rangeCompensatePredicates),
-                residualCompensatePredicates.isEmpty() ? 
BooleanLiteral.of(true)
+                residualCompensatePredicates.isEmpty() ? BooleanLiteral.TRUE
                         : ExpressionUtils.and(residualCompensatePredicates));
     }
 
-    protected Set<Expression> compensateEquivalence(StructInfo queryStructInfo,
-            StructInfo viewStructInfo,
-            SlotMapping viewToQuerySlotMapping,
-            ComparisonResult comparisonResult) {
-        EquivalenceClass queryEquivalenceClass = 
queryStructInfo.getEquivalenceClass();
-        EquivalenceClass viewEquivalenceClass = 
viewStructInfo.getEquivalenceClass();
-        Map<SlotReference, SlotReference> viewToQuerySlotMap = 
viewToQuerySlotMapping.toSlotReferenceMap();
-        EquivalenceClass viewEquivalenceClassQueryBased = 
viewEquivalenceClass.permute(viewToQuerySlotMap);
-        if (viewEquivalenceClassQueryBased == null) {
-            return ImmutableSet.of();
-        }
-        final Set<Expression> equalCompensateConjunctions = new HashSet<>();
-        if (queryEquivalenceClass.isEmpty() && viewEquivalenceClass.isEmpty()) 
{
-            equalCompensateConjunctions.add(BooleanLiteral.of(true));
-        }
-        if (queryEquivalenceClass.isEmpty()
-                && !viewEquivalenceClass.isEmpty()) {
-            return ImmutableSet.of();
-        }
-        EquivalenceClassSetMapping queryToViewEquivalenceMapping =
-                EquivalenceClassSetMapping.generate(queryEquivalenceClass, 
viewEquivalenceClassQueryBased);
-        // can not map all target equivalence class, can not compensate
-        if (queryToViewEquivalenceMapping.getEquivalenceClassSetMap().size()
-                < viewEquivalenceClass.getEquivalenceSetList().size()) {
-            return ImmutableSet.of();
-        }
-        // do equal compensate
-        Set<Set<SlotReference>> mappedQueryEquivalenceSet =
-                
queryToViewEquivalenceMapping.getEquivalenceClassSetMap().keySet();
-        queryEquivalenceClass.getEquivalenceSetList().forEach(
-                queryEquivalenceSet -> {
-                    // compensate the equivalence in query but not in view
-                    if 
(!mappedQueryEquivalenceSet.contains(queryEquivalenceSet)) {
-                        Iterator<SlotReference> iterator = 
queryEquivalenceSet.iterator();
-                        SlotReference first = iterator.next();
-                        while (iterator.hasNext()) {
-                            Expression equals = new EqualTo(first, 
iterator.next());
-                            equalCompensateConjunctions.add(equals);
-                        }
-                    } else {
-                        // compensate the equivalence both in query and view, 
but query has more equivalence
-                        Set<SlotReference> viewEquivalenceSet =
-                                
queryToViewEquivalenceMapping.getEquivalenceClassSetMap().get(queryEquivalenceSet);
-                        Set<SlotReference> copiedQueryEquivalenceSet = new 
HashSet<>(queryEquivalenceSet);
-                        
copiedQueryEquivalenceSet.removeAll(viewEquivalenceSet);
-                        SlotReference first = 
viewEquivalenceSet.iterator().next();
-                        for (SlotReference slotReference : 
copiedQueryEquivalenceSet) {
-                            Expression equals = new EqualTo(first, 
slotReference);
-                            equalCompensateConjunctions.add(equals);
-                        }
-                    }
-                }
-        );
-        return equalCompensateConjunctions;
-    }
-
-    protected Set<Expression> compensateResidualPredicate(StructInfo 
queryStructInfo,
-            StructInfo viewStructInfo,
-            SlotMapping viewToQuerySlotMapping,
-            ComparisonResult comparisonResult) {
-        SplitPredicate querySplitPredicate = 
queryStructInfo.getSplitPredicate();
-        SplitPredicate viewSplitPredicate = viewStructInfo.getSplitPredicate();
-        Expression queryResidualPredicate = 
querySplitPredicate.getResidualPredicate();
-        Expression viewResidualPredicate = 
viewSplitPredicate.getResidualPredicate();
-        Expression viewResidualPredicateQueryBased =
-                ExpressionUtils.replace(viewResidualPredicate, 
viewToQuerySlotMapping.toSlotReferenceMap());
-        Set<Expression> queryResidualSet =
-                
Sets.newHashSet(ExpressionUtils.extractConjunction(queryResidualPredicate));
-        Set<Expression> viewResidualQueryBasedSet =
-                
Sets.newHashSet(ExpressionUtils.extractConjunction(viewResidualPredicateQueryBased));
-        // query residual predicate can not contain all view residual 
predicate when view have residual predicate,
-        // bail out
-        if (!viewResidualPredicateQueryBased.equals(BooleanLiteral.TRUE)
-                && !queryResidualSet.containsAll(viewResidualQueryBasedSet)) {
-            return ImmutableSet.of();
-        }
-        queryResidualSet.removeAll(viewResidualQueryBasedSet);
-        return queryResidualSet;
-    }
-
-    protected Set<Expression> compensateRangePredicate(StructInfo 
queryStructInfo,
-            StructInfo viewStructInfo,
-            SlotMapping viewToQuerySlotMapping,
-            ComparisonResult comparisonResult) {
-        // TODO range predicates and residual predicates compensate, Simplify 
implementation.
-        SplitPredicate querySplitPredicate = 
queryStructInfo.getSplitPredicate();
-        SplitPredicate viewSplitPredicate = viewStructInfo.getSplitPredicate();
-
-        Expression queryRangePredicate = 
querySplitPredicate.getRangePredicate();
-        Expression viewRangePredicate = viewSplitPredicate.getRangePredicate();
-        Expression viewRangePredicateQueryBased =
-                ExpressionUtils.replace(viewRangePredicate, 
viewToQuerySlotMapping.toSlotReferenceMap());
-
-        Set<Expression> queryRangeSet =
-                
Sets.newHashSet(ExpressionUtils.extractConjunction(queryRangePredicate));
-        Set<Expression> viewRangeQueryBasedSet =
-                
Sets.newHashSet(ExpressionUtils.extractConjunction(viewRangePredicateQueryBased));
-        // query range predicate can not contain all view range predicate when 
view have range predicate, bail out
-        if (!viewRangePredicateQueryBased.equals(BooleanLiteral.TRUE)
-                && !queryRangeSet.containsAll(viewRangeQueryBasedSet)) {
-            return ImmutableSet.of();
-        }
-        queryRangeSet.removeAll(viewRangeQueryBasedSet);
-        return queryRangeSet;
-    }
-
     /**
      * Decide the match mode
      *
@@ -554,23 +466,6 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
         return MatchMode.NOT_MATCH;
     }
 
-    /**
-     * Extract struct info from plan, support to get struct info from logical 
plan or plan in group.
-     */
-    public static List<StructInfo> extractStructInfo(Plan plan, 
CascadesContext cascadesContext) {
-        if (plan.getGroupExpression().isPresent() && 
!plan.getGroupExpression().get().getOwnerGroup().getStructInfos()
-                .isEmpty()) {
-            return 
plan.getGroupExpression().get().getOwnerGroup().getStructInfos();
-        } else {
-            // build struct info and add them to current group
-            List<StructInfo> structInfos = StructInfo.of(plan);
-            if (plan.getGroupExpression().isPresent()) {
-                
plan.getGroupExpression().get().getOwnerGroup().addStructInfo(structInfos);
-            }
-            return structInfos;
-        }
-    }
-
     /**
      * Check the pattern of query or materializedView is supported or not.
      */
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java
index 9c494ae80b2..3cd3000a670 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/LogicalCompatibilityContext.java
@@ -78,10 +78,11 @@ public class LogicalCompatibilityContext {
     }
 
     /**
-     * generate logical compatibility context
+     * Generate logical compatibility context,
+     * this make expression mapping between query and view by relation and the 
slot in relation mapping
      */
     public static LogicalCompatibilityContext from(RelationMapping 
relationMapping,
-            SlotMapping slotMapping,
+            SlotMapping queryToViewSlotMapping,
             StructInfo queryStructInfo,
             StructInfo viewStructInfo) {
         // init node mapping
@@ -101,7 +102,8 @@ public class LogicalCompatibilityContext {
             }
         }
         // init expression mapping
-        Map<SlotReference, SlotReference> viewToQuerySlotMapping = 
slotMapping.inverse().toSlotReferenceMap();
+        Map<SlotReference, SlotReference> viewToQuerySlotMapping = 
queryToViewSlotMapping.inverse()
+                .toSlotReferenceMap();
         Map<Expression, Expression> queryShuttledExprToExprMap =
                 queryStructInfo.getShuttledHashConjunctsToConjunctsMap();
         Map<Expression, Expression> viewShuttledExprToExprMap =
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 3a8da2c752b..976eb9a5cfc 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
@@ -23,6 +23,7 @@ import org.apache.doris.catalog.PartitionInfo;
 import org.apache.doris.catalog.PartitionType;
 import org.apache.doris.catalog.TableIf;
 import org.apache.doris.mtmv.BaseTableInfo;
+import org.apache.doris.nereids.CascadesContext;
 import org.apache.doris.nereids.trees.expressions.Expression;
 import org.apache.doris.nereids.trees.expressions.NamedExpression;
 import org.apache.doris.nereids.trees.expressions.Slot;
@@ -124,6 +125,23 @@ public class MaterializedViewUtils {
         return analyzedPlan.accept(TableQueryOperatorChecker.INSTANCE, null);
     }
 
+    /**
+     * Extract struct info from plan, support to get struct info from logical 
plan or plan in group.
+     */
+    public static List<StructInfo> extractStructInfo(Plan plan, 
CascadesContext cascadesContext) {
+        if (plan.getGroupExpression().isPresent() && 
!plan.getGroupExpression().get().getOwnerGroup().getStructInfos()
+                .isEmpty()) {
+            return 
plan.getGroupExpression().get().getOwnerGroup().getStructInfos();
+        } else {
+            // build struct info and add them to current group
+            List<StructInfo> structInfos = StructInfo.of(plan);
+            if (plan.getGroupExpression().isPresent()) {
+                
plan.getGroupExpression().get().getOwnerGroup().addStructInfo(structInfos);
+            }
+            return structInfos;
+        }
+    }
+
     private static final class TableQueryOperatorChecker extends 
DefaultPlanVisitor<Boolean, Void> {
         public static final TableQueryOperatorChecker INSTANCE = new 
TableQueryOperatorChecker();
 
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 39a89c80c29..93bd1d314b1 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
@@ -17,15 +17,23 @@
 
 package org.apache.doris.nereids.rules.exploration.mv;
 
+import 
org.apache.doris.nereids.rules.exploration.mv.mapping.EquivalenceClassSetMapping;
+import org.apache.doris.nereids.rules.exploration.mv.mapping.SlotMapping;
+import org.apache.doris.nereids.trees.expressions.EqualTo;
 import org.apache.doris.nereids.trees.expressions.Expression;
+import org.apache.doris.nereids.trees.expressions.SlotReference;
 import org.apache.doris.nereids.trees.expressions.literal.BooleanLiteral;
 import org.apache.doris.nereids.util.ExpressionUtils;
 import org.apache.doris.nereids.util.Utils;
 
 import com.google.common.collect.ImmutableList;
+import com.google.common.collect.Sets;
 
 import java.util.HashSet;
+import java.util.Iterator;
 import java.util.List;
+import java.util.Map;
+import java.util.Objects;
 import java.util.Optional;
 import java.util.Set;
 import java.util.stream.Collectors;
@@ -72,6 +80,127 @@ public class Predicates {
         return predicatesSplit.getSplitPredicate();
     }
 
+    /**
+     * compensate equivalence predicates
+     */
+    public static Set<Expression> compensateEquivalence(StructInfo 
queryStructInfo,
+            StructInfo viewStructInfo,
+            SlotMapping viewToQuerySlotMapping,
+            ComparisonResult comparisonResult) {
+        EquivalenceClass queryEquivalenceClass = 
queryStructInfo.getEquivalenceClass();
+        EquivalenceClass viewEquivalenceClass = 
viewStructInfo.getEquivalenceClass();
+        Map<SlotReference, SlotReference> viewToQuerySlotMap = 
viewToQuerySlotMapping.toSlotReferenceMap();
+        EquivalenceClass viewEquivalenceClassQueryBased = 
viewEquivalenceClass.permute(viewToQuerySlotMap);
+        if (viewEquivalenceClassQueryBased == null) {
+            return null;
+        }
+        final Set<Expression> equalCompensateConjunctions = new HashSet<>();
+        if (queryEquivalenceClass.isEmpty() && viewEquivalenceClass.isEmpty()) 
{
+            equalCompensateConjunctions.add(BooleanLiteral.TRUE);
+        }
+        if (queryEquivalenceClass.isEmpty()
+                && !viewEquivalenceClass.isEmpty()) {
+            return null;
+        }
+        EquivalenceClassSetMapping queryToViewEquivalenceMapping =
+                EquivalenceClassSetMapping.generate(queryEquivalenceClass, 
viewEquivalenceClassQueryBased);
+        // can not map all target equivalence class, can not compensate
+        if (queryToViewEquivalenceMapping.getEquivalenceClassSetMap().size()
+                < viewEquivalenceClass.getEquivalenceSetList().size()) {
+            return null;
+        }
+        // do equal compensate
+        Set<Set<SlotReference>> mappedQueryEquivalenceSet =
+                
queryToViewEquivalenceMapping.getEquivalenceClassSetMap().keySet();
+        queryEquivalenceClass.getEquivalenceSetList().forEach(
+                queryEquivalenceSet -> {
+                    // compensate the equivalence in query but not in view
+                    if 
(!mappedQueryEquivalenceSet.contains(queryEquivalenceSet)) {
+                        Iterator<SlotReference> iterator = 
queryEquivalenceSet.iterator();
+                        SlotReference first = iterator.next();
+                        while (iterator.hasNext()) {
+                            Expression equals = new EqualTo(first, 
iterator.next());
+                            equalCompensateConjunctions.add(equals);
+                        }
+                    } else {
+                        // compensate the equivalence both in query and view, 
but query has more equivalence
+                        Set<SlotReference> viewEquivalenceSet =
+                                
queryToViewEquivalenceMapping.getEquivalenceClassSetMap().get(queryEquivalenceSet);
+                        Set<SlotReference> copiedQueryEquivalenceSet = new 
HashSet<>(queryEquivalenceSet);
+                        
copiedQueryEquivalenceSet.removeAll(viewEquivalenceSet);
+                        SlotReference first = 
viewEquivalenceSet.iterator().next();
+                        for (SlotReference slotReference : 
copiedQueryEquivalenceSet) {
+                            Expression equals = new EqualTo(first, 
slotReference);
+                            equalCompensateConjunctions.add(equals);
+                        }
+                    }
+                }
+        );
+        return equalCompensateConjunctions;
+    }
+
+    /**
+     * compensate range predicates
+     */
+    public static Set<Expression> compensateRangePredicate(StructInfo 
queryStructInfo,
+            StructInfo viewStructInfo,
+            SlotMapping viewToQuerySlotMapping,
+            ComparisonResult comparisonResult) {
+        // TODO Range predicates compensate, simplify implementation currently.
+        SplitPredicate querySplitPredicate = 
queryStructInfo.getSplitPredicate();
+        SplitPredicate viewSplitPredicate = viewStructInfo.getSplitPredicate();
+
+        Expression queryRangePredicate = 
querySplitPredicate.getRangePredicate();
+        Expression viewRangePredicate = viewSplitPredicate.getRangePredicate();
+        Expression viewRangePredicateQueryBased =
+                ExpressionUtils.replace(viewRangePredicate, 
viewToQuerySlotMapping.toSlotReferenceMap());
+
+        Set<Expression> queryRangeSet =
+                
Sets.newHashSet(ExpressionUtils.extractConjunction(queryRangePredicate));
+        Set<Expression> viewRangeQueryBasedSet =
+                
Sets.newHashSet(ExpressionUtils.extractConjunction(viewRangePredicateQueryBased));
+        // remove unnecessary literal BooleanLiteral.TRUE
+        queryRangeSet.remove(BooleanLiteral.TRUE);
+        viewRangeQueryBasedSet.remove(BooleanLiteral.TRUE);
+        // query residual predicate can not contain all view residual 
predicate when view have residual predicate,
+        // bail out
+        if (!queryRangeSet.containsAll(viewRangeQueryBasedSet)) {
+            return null;
+        }
+        queryRangeSet.removeAll(viewRangeQueryBasedSet);
+        return queryRangeSet;
+    }
+
+    /**
+     * compensate residual predicates
+     */
+    public static Set<Expression> compensateResidualPredicate(StructInfo 
queryStructInfo,
+            StructInfo viewStructInfo,
+            SlotMapping viewToQuerySlotMapping,
+            ComparisonResult comparisonResult) {
+        // TODO Residual predicates compensate, simplify implementation 
currently.
+        SplitPredicate querySplitPredicate = 
queryStructInfo.getSplitPredicate();
+        SplitPredicate viewSplitPredicate = viewStructInfo.getSplitPredicate();
+        Expression queryResidualPredicate = 
querySplitPredicate.getResidualPredicate();
+        Expression viewResidualPredicate = 
viewSplitPredicate.getResidualPredicate();
+        Expression viewResidualPredicateQueryBased =
+                ExpressionUtils.replace(viewResidualPredicate, 
viewToQuerySlotMapping.toSlotReferenceMap());
+        Set<Expression> queryResidualSet =
+                
Sets.newHashSet(ExpressionUtils.extractConjunction(queryResidualPredicate));
+        Set<Expression> viewResidualQueryBasedSet =
+                
Sets.newHashSet(ExpressionUtils.extractConjunction(viewResidualPredicateQueryBased));
+        // remove unnecessary literal BooleanLiteral.TRUE
+        queryResidualSet.remove(BooleanLiteral.TRUE);
+        viewResidualQueryBasedSet.remove(BooleanLiteral.TRUE);
+        // query residual predicate can not contain all view residual 
predicate when view have residual predicate,
+        // bail out
+        if (!queryResidualSet.containsAll(viewResidualQueryBasedSet)) {
+            return null;
+        }
+        queryResidualSet.removeAll(viewResidualQueryBasedSet);
+        return queryResidualSet;
+    }
+
     @Override
     public String toString() {
         return Utils.toSqlString("Predicates", "pulledUpPredicates", 
pulledUpPredicates);
@@ -81,9 +210,11 @@ public class Predicates {
      * The split different representation for predicate expression, such as 
equal, range and residual predicate.
      */
     public static final class SplitPredicate {
-        private Optional<Expression> equalPredicate;
-        private Optional<Expression> rangePredicate;
-        private Optional<Expression> residualPredicate;
+        public static final SplitPredicate INVALID_INSTANCE =
+                SplitPredicate.of(null, null, null);
+        private final Optional<Expression> equalPredicate;
+        private final Optional<Expression> rangePredicate;
+        private final Optional<Expression> residualPredicate;
 
         public SplitPredicate(Expression equalPredicate, Expression 
rangePredicate, Expression residualPredicate) {
             this.equalPredicate = Optional.ofNullable(equalPredicate);
@@ -103,10 +234,6 @@ public class Predicates {
             return residualPredicate.orElse(BooleanLiteral.TRUE);
         }
 
-        public static SplitPredicate invalid() {
-            return new SplitPredicate(null, null, null);
-        }
-
         /**
          * SplitPredicate construct
          */
@@ -117,27 +244,23 @@ public class Predicates {
         }
 
         /**
-         * isEmpty
+         * Check the predicates are invalid or not. If any of the predicates 
is null, it is invalid.
          */
-        public boolean isEmpty() {
-            return !equalPredicate.isPresent()
-                    && !rangePredicate.isPresent()
-                    && !residualPredicate.isPresent();
+        public boolean isInvalid() {
+            return Objects.equals(this, INVALID_INSTANCE);
         }
 
         public List<Expression> toList() {
-            return ImmutableList.of(equalPredicate.orElse(BooleanLiteral.TRUE),
-                    rangePredicate.orElse(BooleanLiteral.TRUE),
-                    residualPredicate.orElse(BooleanLiteral.TRUE));
+            return ImmutableList.of(getEqualPredicate(), getRangePredicate(), 
getResidualPredicate());
         }
 
         /**
          * Check the predicates in SplitPredicate is whether all true or not
          */
         public boolean isAlwaysTrue() {
-            Expression equalExpr = equalPredicate.orElse(BooleanLiteral.TRUE);
-            Expression rangeExpr = rangePredicate.orElse(BooleanLiteral.TRUE);
-            Expression residualExpr = 
residualPredicate.orElse(BooleanLiteral.TRUE);
+            Expression equalExpr = getEqualPredicate();
+            Expression rangeExpr = getRangePredicate();
+            Expression residualExpr = getResidualPredicate();
             return equalExpr instanceof BooleanLiteral
                     && rangeExpr instanceof BooleanLiteral
                     && residualExpr instanceof BooleanLiteral
@@ -146,6 +269,25 @@ public class Predicates {
                     && ((BooleanLiteral) residualExpr).getValue();
         }
 
+        @Override
+        public boolean equals(Object o) {
+            if (this == o) {
+                return true;
+            }
+            if (o == null || getClass() != o.getClass()) {
+                return false;
+            }
+            SplitPredicate that = (SplitPredicate) o;
+            return Objects.equals(equalPredicate, that.equalPredicate)
+                    && Objects.equals(rangePredicate, that.rangePredicate)
+                    && Objects.equals(residualPredicate, 
that.residualPredicate);
+        }
+
+        @Override
+        public int hashCode() {
+            return Objects.hash(equalPredicate, rangePredicate, 
residualPredicate);
+        }
+
         @Override
         public String toString() {
             return Utils.toSqlString("SplitPredicate",
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 0ac3085faeb..d79153bdc8e 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/StructInfo.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/StructInfo.java
@@ -23,6 +23,7 @@ import org.apache.doris.nereids.memo.Group;
 import org.apache.doris.nereids.memo.GroupExpression;
 import org.apache.doris.nereids.rules.exploration.mv.Predicates.SplitPredicate;
 import org.apache.doris.nereids.trees.expressions.EqualTo;
+import org.apache.doris.nereids.trees.expressions.ExprId;
 import org.apache.doris.nereids.trees.expressions.Expression;
 import org.apache.doris.nereids.trees.expressions.SlotReference;
 import org.apache.doris.nereids.trees.expressions.literal.Literal;
@@ -40,8 +41,10 @@ import 
org.apache.doris.nereids.trees.plans.logical.LogicalJoin;
 import org.apache.doris.nereids.trees.plans.logical.LogicalProject;
 import org.apache.doris.nereids.trees.plans.logical.LogicalRepeat;
 import org.apache.doris.nereids.trees.plans.visitor.DefaultPlanVisitor;
+import org.apache.doris.nereids.trees.plans.visitor.ExpressionLineageReplacer;
 import org.apache.doris.nereids.util.ExpressionUtils;
 
+import com.google.common.collect.ImmutableSet;
 import com.google.common.collect.Lists;
 import com.google.common.collect.Sets;
 
@@ -83,8 +86,11 @@ public class StructInfo {
     // split predicates is shuttled
     private SplitPredicate splitPredicate;
     private EquivalenceClass equivalenceClass;
-    // this is for LogicalCompatibilityContext later
+    // Key is the expression shuttled and the value is the origin expression
+    // this is for building LogicalCompatibilityContext later.
     private final Map<Expression, Expression> 
shuttledHashConjunctsToConjunctsMap = new HashMap<>();
+    // Record the exprId and the corresponding expr map, this is used by 
expression shuttled
+    private final Map<ExprId, Expression> namedExprIdAndExprMapping = new 
HashMap<>();
 
     private StructInfo(Plan originalPlan, @Nullable Plan topPlan, @Nullable 
Plan bottomPlan, HyperGraph hyperGraph) {
         this.originalPlan = originalPlan;
@@ -117,12 +123,22 @@ public class StructInfo {
         // Collect expression from join condition in hyper graph
         this.hyperGraph.getJoinEdges().forEach(edge -> {
             List<Expression> hashJoinConjuncts = edge.getHashJoinConjuncts();
+            // shuttle expression in edge for the build of 
LogicalCompatibilityContext later.
+            // Record the exprId to expr map in the processing to strut info
+            // TODO get exprId to expr map when complex project is ready in 
join dege
             hashJoinConjuncts.forEach(conjunctExpr -> {
-                // shuttle expression in edge for LogicalCompatibilityContext 
later
-                shuttledHashConjunctsToConjunctsMap.put(
-                        ExpressionUtils.shuttleExpressionWithLineage(
-                                Lists.newArrayList(conjunctExpr), 
edge.getJoin()).get(0),
-                        conjunctExpr);
+                ExpressionLineageReplacer.ExpressionReplaceContext 
replaceContext =
+                        new ExpressionLineageReplacer.ExpressionReplaceContext(
+                                Lists.newArrayList(conjunctExpr),
+                                ImmutableSet.of(),
+                                ImmutableSet.of());
+                this.topPlan.accept(ExpressionLineageReplacer.INSTANCE, 
replaceContext);
+                // Replace expressions by expression map
+                List<Expression> replacedExpressions = 
replaceContext.getReplacedExpressions();
+                
shuttledHashConjunctsToConjunctsMap.put(replacedExpressions.get(0), 
conjunctExpr);
+                // Record this, will be used in top level expression shuttle 
later, see the method
+                // ExpressionLineageReplacer#visitGroupPlan
+                
this.namedExprIdAndExprMapping.putAll(replaceContext.getExprIdExpressionMap());
             });
             List<Expression> otherJoinConjuncts = edge.getOtherJoinConjuncts();
             if (!otherJoinConjuncts.isEmpty()) {
@@ -267,6 +283,10 @@ public class StructInfo {
         return originalPlanId;
     }
 
+    public Map<ExprId, Expression> getNamedExprIdAndExprMapping() {
+        return namedExprIdAndExprMapping;
+    }
+
     /**
      * Judge the source graph logical is whether the same as target
      * For inner join should judge only the join tables,
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/ExpressionLineageReplacer.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/ExpressionLineageReplacer.java
index 2a4b81a5828..b7cc0dff80b 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/ExpressionLineageReplacer.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/ExpressionLineageReplacer.java
@@ -18,6 +18,8 @@
 package org.apache.doris.nereids.trees.plans.visitor;
 
 import org.apache.doris.catalog.TableIf.TableType;
+import org.apache.doris.nereids.memo.Group;
+import org.apache.doris.nereids.rules.exploration.mv.StructInfo;
 import org.apache.doris.nereids.trees.expressions.Alias;
 import org.apache.doris.nereids.trees.expressions.ExprId;
 import org.apache.doris.nereids.trees.expressions.Expression;
@@ -25,6 +27,7 @@ import 
org.apache.doris.nereids.trees.expressions.NamedExpression;
 import org.apache.doris.nereids.trees.expressions.Slot;
 import 
org.apache.doris.nereids.trees.expressions.visitor.DefaultExpressionRewriter;
 import 
org.apache.doris.nereids.trees.expressions.visitor.DefaultExpressionVisitor;
+import org.apache.doris.nereids.trees.plans.GroupPlan;
 import org.apache.doris.nereids.trees.plans.Plan;
 import 
org.apache.doris.nereids.trees.plans.visitor.ExpressionLineageReplacer.ExpressionReplaceContext;
 
@@ -56,6 +59,22 @@ public class ExpressionLineageReplacer extends 
DefaultPlanVisitor<Expression, Ex
         return super.visit(plan, context);
     }
 
+    @Override
+    public Expression visitGroupPlan(GroupPlan groupPlan, 
ExpressionReplaceContext context) {
+        Group group = groupPlan.getGroup();
+        if (group == null) {
+            return visit(groupPlan, context);
+        }
+        List<StructInfo> structInfos = group.getStructInfos();
+        if (structInfos.isEmpty()) {
+            return visit(groupPlan, context);
+        }
+        // TODO only support group has one struct info, will support more 
struct info later
+        StructInfo structInfo = structInfos.get(0);
+        
context.getExprIdExpressionMap().putAll(structInfo.getNamedExprIdAndExprMapping());
+        return visit(groupPlan, context);
+    }
+
     /**
      * Replace the expression with lineage according the exprIdExpressionMap
      */
@@ -93,7 +112,7 @@ public class ExpressionLineageReplacer extends 
DefaultPlanVisitor<Expression, Ex
     }
 
     /**
-     * The Collector for target named expressions in the whole plan, and will 
be used to
+     * The Collector for named expressions in the whole plan, and will be used 
to
      * replace the target expression later
      * TODO Collect named expression by targetTypes, tableIdentifiers
      */
@@ -128,7 +147,9 @@ public class ExpressionLineageReplacer extends 
DefaultPlanVisitor<Expression, Ex
         private Map<ExprId, Expression> exprIdExpressionMap;
         private List<Expression> replacedExpressions;
 
-        /**ExpressionReplaceContext*/
+        /**
+         * ExpressionReplaceContext
+         */
         public ExpressionReplaceContext(List<Expression> targetExpressions,
                 Set<TableType> targetTypes,
                 Set<String> tableIdentifiers) {
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/CompareOuterJoinTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/CompareOuterJoinTest.java
index 8914f4ac80b..f73d8a2923e 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/CompareOuterJoinTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/CompareOuterJoinTest.java
@@ -19,10 +19,10 @@ package org.apache.doris.nereids.jobs.joinorder.hypergraph;
 
 import org.apache.doris.nereids.CascadesContext;
 import org.apache.doris.nereids.rules.RuleSet;
-import 
org.apache.doris.nereids.rules.exploration.mv.AbstractMaterializedViewRule;
 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.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;
@@ -194,9 +194,9 @@ class CompareOuterJoinTest extends SqlTestBase {
     }
 
     LogicalCompatibilityContext constructContext(Plan p1, Plan p2) {
-        StructInfo st1 = AbstractMaterializedViewRule.extractStructInfo(p1,
+        StructInfo st1 = MaterializedViewUtils.extractStructInfo(p1,
                 null).get(0);
-        StructInfo st2 = AbstractMaterializedViewRule.extractStructInfo(p2,
+        StructInfo st2 = MaterializedViewUtils.extractStructInfo(p2,
                 null).get(0);
         RelationMapping rm = RelationMapping.generate(st1.getRelations(), 
st2.getRelations()).get(0);
         SlotMapping sm = SlotMapping.generate(rm);
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/InferJoinTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/InferJoinTest.java
index 33100c1180b..3f1ce3fc345 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/InferJoinTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/InferJoinTest.java
@@ -19,10 +19,10 @@ package org.apache.doris.nereids.jobs.joinorder.hypergraph;
 
 import org.apache.doris.nereids.CascadesContext;
 import org.apache.doris.nereids.rules.RuleSet;
-import 
org.apache.doris.nereids.rules.exploration.mv.AbstractMaterializedViewRule;
 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.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;
@@ -162,9 +162,9 @@ class InferJoinTest extends SqlTestBase {
     }
 
     LogicalCompatibilityContext constructContext(Plan p1, Plan p2) {
-        StructInfo st1 = AbstractMaterializedViewRule.extractStructInfo(p1,
+        StructInfo st1 = MaterializedViewUtils.extractStructInfo(p1,
                 null).get(0);
-        StructInfo st2 = AbstractMaterializedViewRule.extractStructInfo(p2,
+        StructInfo st2 = MaterializedViewUtils.extractStructInfo(p2,
                 null).get(0);
         RelationMapping rm = RelationMapping.generate(st1.getRelations(), 
st2.getRelations()).get(0);
         SlotMapping sm = SlotMapping.generate(rm);
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/InferPredicateTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/InferPredicateTest.java
index bd18f31a7c0..5ea49a51f5c 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/InferPredicateTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/InferPredicateTest.java
@@ -19,10 +19,10 @@ package org.apache.doris.nereids.jobs.joinorder.hypergraph;
 
 import org.apache.doris.nereids.CascadesContext;
 import org.apache.doris.nereids.rules.RuleSet;
-import 
org.apache.doris.nereids.rules.exploration.mv.AbstractMaterializedViewRule;
 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.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;
@@ -61,9 +61,9 @@ class InferPredicateTest extends SqlTestBase {
     }
 
     LogicalCompatibilityContext constructContext(Plan p1, Plan p2) {
-        StructInfo st1 = AbstractMaterializedViewRule.extractStructInfo(p1,
+        StructInfo st1 = MaterializedViewUtils.extractStructInfo(p1,
                 null).get(0);
-        StructInfo st2 = AbstractMaterializedViewRule.extractStructInfo(p2,
+        StructInfo st2 = MaterializedViewUtils.extractStructInfo(p2,
                 null).get(0);
         RelationMapping rm = RelationMapping.generate(st1.getRelations(), 
st2.getRelations()).get(0);
         SlotMapping sm = SlotMapping.generate(rm);
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/PullupExpressionTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/PullupExpressionTest.java
index 5489f06379c..44b60753947 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/PullupExpressionTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/jobs/joinorder/hypergraph/PullupExpressionTest.java
@@ -19,10 +19,10 @@ package org.apache.doris.nereids.jobs.joinorder.hypergraph;
 
 import org.apache.doris.nereids.CascadesContext;
 import org.apache.doris.nereids.rules.RuleSet;
-import 
org.apache.doris.nereids.rules.exploration.mv.AbstractMaterializedViewRule;
 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.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;
@@ -140,9 +140,9 @@ class PullupExpressionTest extends SqlTestBase {
     }
 
     LogicalCompatibilityContext constructContext(Plan p1, Plan p2) {
-        StructInfo st1 = AbstractMaterializedViewRule.extractStructInfo(p1,
+        StructInfo st1 = MaterializedViewUtils.extractStructInfo(p1,
                 null).get(0);
-        StructInfo st2 = AbstractMaterializedViewRule.extractStructInfo(p2,
+        StructInfo st2 = MaterializedViewUtils.extractStructInfo(p2,
                 null).get(0);
         RelationMapping rm = RelationMapping.generate(st1.getRelations(), 
st2.getRelations()).get(0);
         SlotMapping sm = SlotMapping.generate(rm);
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphComparatorTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphComparatorTest.java
index 77b7fd67294..b89934d4577 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphComparatorTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/mv/HyperGraphComparatorTest.java
@@ -161,9 +161,9 @@ class HyperGraphComparatorTest extends SqlTestBase {
     }
 
     LogicalCompatibilityContext constructContext(Plan p1, Plan p2) {
-        StructInfo st1 = AbstractMaterializedViewRule.extractStructInfo(p1,
+        StructInfo st1 = MaterializedViewUtils.extractStructInfo(p1,
                 null).get(0);
-        StructInfo st2 = AbstractMaterializedViewRule.extractStructInfo(p2,
+        StructInfo st2 = MaterializedViewUtils.extractStructInfo(p2,
                 null).get(0);
         RelationMapping rm = RelationMapping.generate(st1.getRelations(), 
st2.getRelations()).get(0);
         SlotMapping sm = SlotMapping.generate(rm);
diff --git 
a/regression-test/data/nereids_rules_p0/mv/agg_with_roll_up/aggregate_with_roll_up.out
 
b/regression-test/data/nereids_rules_p0/mv/agg_with_roll_up/aggregate_with_roll_up.out
index f20fc747473..10c593bd5f0 100644
--- 
a/regression-test/data/nereids_rules_p0/mv/agg_with_roll_up/aggregate_with_roll_up.out
+++ 
b/regression-test/data/nereids_rules_p0/mv/agg_with_roll_up/aggregate_with_roll_up.out
@@ -65,6 +65,14 @@
 2      3       2023-12-12      57.40
 2      4       2023-12-10      46.00
 
+-- !query19_1_before --
+2      3       2023-12-08      20.00   10.50   9.50    0       2
+2      3       2023-12-12      57.40   56.20   1.20    0       2
+
+-- !query19_1_after --
+2      3       2023-12-08      20.00   10.50   9.50    0       2
+2      3       2023-12-12      57.40   56.20   1.20    0       2
+
 -- !query20_0_before --
 2023-12-08     3       2023-12-08      20.00   10.50   9.50    2       0
 2023-12-09     3       2023-12-09      11.50   11.50   11.50   1       0
@@ -117,6 +125,14 @@
 3      3       2023-12-11      43.20   43.20   43.20   1       0
 4      3       2023-12-09      11.50   11.50   11.50   1       0
 
+-- !query22_1_before --
+2      3       2023-12-08      20.00   10.50   9.50    2       0
+2      3       2023-12-12      57.40   56.20   1.20    2       0
+
+-- !query22_1_after --
+2      3       2023-12-08      20.00   10.50   9.50    2       0
+2      3       2023-12-12      57.40   56.20   1.20    2       0
+
 -- !query23_0_before --
 2      3       2023-12-08      20.00   10.50   9.50    2       0
 
@@ -219,3 +235,23 @@
 2023-12-11     2       43.20   43.20   43.20   1       0       0
 2023-12-12     2       57.40   56.20   1.20    2       0       0
 
+-- !query28_0_before --
+2023-12-08     20.00
+2023-12-09     11.50
+2023-12-10     46.00
+2023-12-11     43.20
+2023-12-12     57.40
+
+-- !query28_0_after --
+2023-12-08     20.00
+2023-12-09     11.50
+2023-12-10     46.00
+2023-12-11     43.20
+2023-12-12     57.40
+
+-- !query29_0_before --
+8
+
+-- !query29_0_after --
+8
+
diff --git 
a/regression-test/data/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.out
 
b/regression-test/data/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.out
index 8bf8d7d4f9f..cc6e4730fdf 100644
--- 
a/regression-test/data/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.out
+++ 
b/regression-test/data/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.out
@@ -163,9 +163,37 @@
 4      2       43.20
 6      2       57.40
 
+-- !query19_2_before --
+2023-12-08     2       3       3       20.00   10.50   9.50    2
+2023-12-09     4       3       3       11.50   11.50   11.50   1
+2023-12-10     2       4       4       46.00   33.50   12.50   2
+2023-12-11     3       3       3       43.20   43.20   43.20   1
+2023-12-12     2       3       3       57.40   56.20   1.20    2
+
+-- !query19_2_after --
+2023-12-08     2       3       3       20.00   10.50   9.50    2
+2023-12-09     4       3       3       11.50   11.50   11.50   1
+2023-12-10     2       4       4       46.00   33.50   12.50   2
+2023-12-11     3       3       3       43.20   43.20   43.20   1
+2023-12-12     2       3       3       57.40   56.20   1.20    2
+
 -- !query20_0_before --
 0      0       0       0       0       0       0       0       0       0       
0       0
 
 -- !query20_0_after --
 0      0       0       0       0       0       0       0       0       0       
0       0
 
+-- !query20_1_before --
+56.00  8
+
+-- !query20_1_after --
+56.00  8
+
+-- !query20_2_before --
+b      a       39.00   6
+d      c       17.00   2
+
+-- !query20_2_after --
+b      a       39.00   6
+d      c       17.00   2
+
diff --git a/regression-test/data/nereids_rules_p0/mv/join/inner/inner_join.out 
b/regression-test/data/nereids_rules_p0/mv/join/inner/inner_join.out
index 62d3523c670..1cf464023ee 100644
--- a/regression-test/data/nereids_rules_p0/mv/join/inner/inner_join.out
+++ b/regression-test/data/nereids_rules_p0/mv/join/inner/inner_join.out
@@ -147,6 +147,22 @@
 6
 6
 
+-- !query2_2_before --
+4
+4
+4
+4
+6
+6
+
+-- !query2_2_after --
+4
+4
+4
+4
+6
+6
+
 -- !query2_3_before --
 4
 4
diff --git 
a/regression-test/data/nereids_rules_p0/mv/join/left_outer/outer_join.out 
b/regression-test/data/nereids_rules_p0/mv/join/left_outer/outer_join.out
index 4d0e131c5ff..845ef3933dc 100644
--- a/regression-test/data/nereids_rules_p0/mv/join/left_outer/outer_join.out
+++ b/regression-test/data/nereids_rules_p0/mv/join/left_outer/outer_join.out
@@ -127,6 +127,38 @@
 6
 6
 
+-- !query2_2_before --
+4
+4
+4
+4
+6
+6
+
+-- !query2_2_after --
+4
+4
+4
+4
+6
+6
+
+-- !query2_3_before --
+4
+4
+4
+4
+6
+6
+
+-- !query2_3_after --
+4
+4
+4
+4
+6
+6
+
 -- !query3_0_before --
 4
 4
@@ -215,6 +247,14 @@
 6
 6
 
+-- !query5_1_before --
+2023-12-08     2023-12-08      2       3
+2023-12-08     2023-12-08      2       3
+
+-- !query5_1_after --
+2023-12-08     2023-12-08      2       3
+2023-12-08     2023-12-08      2       3
+
 -- !query6_0_before --
 2      3       2023-12-08
 2      3       2023-12-08
@@ -237,6 +277,10 @@
 -- !query7_0_after --
 3      3       2023-12-11
 
+-- !query7_1_before --
+
+-- !query7_1_after --
+
 -- !query8_0_before --
 1      0       8       0       10.0000 10.50   9.50
 2      0       2       0       11.5000 11.50   11.50
diff --git 
a/regression-test/suites/nereids_rules_p0/mv/agg_with_roll_up/aggregate_with_roll_up.groovy
 
b/regression-test/suites/nereids_rules_p0/mv/agg_with_roll_up/aggregate_with_roll_up.groovy
index 9cdca736fe5..cffb031cb17 100644
--- 
a/regression-test/suites/nereids_rules_p0/mv/agg_with_roll_up/aggregate_with_roll_up.groovy
+++ 
b/regression-test/suites/nereids_rules_p0/mv/agg_with_roll_up/aggregate_with_roll_up.groovy
@@ -533,11 +533,10 @@ suite("aggregate_with_roll_up") {
             "o_orderdate, " +
             "l_partkey, " +
             "l_suppkey"
-//    // Should pass but not, tmp
-//    order_qt_query19_1_before "${query19_1}"
-//    check_rewrite(mv19_1, query19_1, "mv19_1")
-//    order_qt_query19_1_after "${query19_1}"
-//    sql """ DROP MATERIALIZED VIEW IF EXISTS mv19_1"""
+    order_qt_query19_1_before "${query19_1}"
+    check_rewrite(mv19_1, query19_1, "mv19_1")
+    order_qt_query19_1_after "${query19_1}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv19_1"""
 
 
     // filter inside + right + use not roll up dimension
@@ -708,11 +707,10 @@ suite("aggregate_with_roll_up") {
             "o_orderdate, " +
             "l_partkey, " +
             "l_suppkey"
-    // Should pass but not, tmp
-//    order_qt_query22_1_before "${query22_1}"
-//    check_rewrite(mv22_1, query22_1, "mv22_1")
-//    order_qt_query22_1_after "${query22_1}"
-//    sql """ DROP MATERIALIZED VIEW IF EXISTS mv22_0"""
+    order_qt_query22_1_before "${query22_1}"
+    check_rewrite(mv22_1, query22_1, "mv22_1")
+    order_qt_query22_1_after "${query22_1}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv22_1"""
 
 
     // filter outside + right + use not roll up dimension
@@ -1017,9 +1015,79 @@ suite("aggregate_with_roll_up") {
             group by
             o_orderdate,
             o_shippriority;
-    """
+            """
     order_qt_query27_0_before "${query27_0}"
     check_not_match(mv27_0, query27_0, "mv27_0")
     order_qt_query27_0_after "${query27_0}"
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv27_0"""
+
+    // with cte
+    // left join + aggregate
+    def mv28_0 = """
+           select l_shipdate, o_orderdate, l_partkey, l_suppkey,
+            sum(ifnull(o_totalprice, 0)) as sum_total,
+            max(o_totalprice) as max_total,
+            min(o_totalprice) as min_total,
+            count(*) as count_all,
+            bitmap_union(to_bitmap(case when o_shippriority > 1 and o_orderkey 
IN (1, 2, 3) then o_custkey else null end)) as  cnt_1,
+            bitmap_union(to_bitmap(case when o_shippriority > 2 and o_orderkey 
IN (3, 4, 5) then o_custkey else null end)) as cnt_2
+            from lineitem
+            left join orders on l_orderkey = o_orderkey and l_shipdate = 
o_orderdate
+            group by
+            l_shipdate,
+            o_orderdate,
+            l_partkey,
+            l_suppkey;
+    """
+    def query28_0 = """
+        with cte_view_1 as (
+            select l_shipdate, o_orderdate, l_partkey, l_suppkey,
+            ifnull(o_totalprice, 0) as price_with_no_null
+            from lineitem
+            left join orders on l_orderkey = o_orderkey and l_shipdate = 
o_orderdate
+        )
+        select
+          l_shipdate,
+          sum(price_with_no_null)
+        from
+          cte_view_1 cte_view
+        group by
+          l_shipdate
+        order by
+          l_shipdate
+        limit 10;
+    """
+    order_qt_query28_0_before "${query28_0}"
+    check_rewrite(mv28_0, query28_0, "mv28_0")
+    order_qt_query28_0_after "${query28_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv28_0"""
+
+
+    // scalar aggregate
+    def mv29_0 = """
+            select count(*)
+            from lineitem
+            left join orders on l_orderkey = o_orderkey and l_shipdate = 
o_orderdate;
+    """
+    def query29_0 = """
+        select *
+        from
+        (
+           with cte_view_1 as (
+            select l_shipdate, o_orderdate, l_partkey, l_suppkey,
+            ifnull(o_totalprice, 0) as price_with_no_null
+            from lineitem
+            left join orders on l_orderkey = o_orderkey and l_shipdate = 
o_orderdate
+            )    
+          select 
+            count(1) count_all
+          from
+            cte_view_1 cte_view
+          limit 1
+        ) as t;
+    """
+    order_qt_query29_0_before "${query29_0}"
+    check_rewrite(mv29_0, query29_0, "mv29_0")
+    order_qt_query29_0_after "${query29_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv29_0"""
 }
diff --git 
a/regression-test/suites/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.groovy
 
b/regression-test/suites/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.groovy
index be56ebe05f6..b254fe6ac30 100644
--- 
a/regression-test/suites/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.groovy
+++ 
b/regression-test/suites/nereids_rules_p0/mv/agg_without_roll_up/aggregate_without_roll_up.groovy
@@ -172,8 +172,8 @@ suite("aggregate_without_roll_up") {
         }
     }
 
-//    // single table
-//    // with filter
+    // single table
+    // with filter
     def mv1_0 = "select o_shippriority, o_comment, " +
             "sum(o_totalprice) as sum_total, " +
             "max(o_totalprice) as max_total, " +
@@ -726,6 +726,52 @@ suite("aggregate_without_roll_up") {
     order_qt_query19_1_after "${query19_1}"
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv19_1"""
 
+    // with duplicated group by column
+    def mv19_2 = """
+            select
+            o_orderdate as o_orderdate_1,
+            l_partkey as l_partkey_1,
+            l_suppkey as l_suppkey_1,
+            l_suppkey as l_suppkey_2,
+            l_partkey as l_partkey_2,
+            o_orderdate as o_orderdate_2,
+            sum(o_totalprice),
+            max(o_totalprice) as max_total,
+            min(o_totalprice) as min_total,
+            count(*) as count_all
+            from lineitem
+            left join orders on l_orderkey = o_orderkey and l_shipdate = 
o_orderdate
+            group by
+            o_orderdate,
+            l_partkey,
+            l_suppkey,
+            l_suppkey,
+            l_partkey,
+            o_orderdate;
+    """
+    def query19_2 = """
+            select
+            o_orderdate as o_orderdate_1,
+            l_partkey as l_partkey_1,
+            l_suppkey as l_suppkey_1,
+            l_suppkey as l_suppkey_2,
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*)
+            from lineitem
+            left join orders on l_orderkey = o_orderkey and l_shipdate = 
o_orderdate
+            group by
+            o_orderdate,
+            l_suppkey,
+            l_partkey,
+            l_suppkey;
+    """
+    order_qt_query19_2_before "${query19_2}"
+    check_rewrite(mv19_2, query19_2, "mv19_2")
+    order_qt_query19_2_after "${query19_2}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv19_2"""
+
     // without group, scalar aggregate
     def mv20_0 = "select count(distinct case when O_SHIPPRIORITY > 1 and 
O_ORDERKEY IN (1, 3) then O_ORDERSTATUS else null end) as filter_cnt_1, " +
             "count(distinct case when O_SHIPPRIORITY > 2 and O_ORDERKEY IN (2) 
then O_ORDERSTATUS else null end) as filter_cnt_2, " +
@@ -773,4 +819,61 @@ suite("aggregate_without_roll_up") {
     check_rewrite(mv20_0, query20_0, "mv20_0")
     order_qt_query20_0_after "${query20_0}"
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv20_0"""
+
+    // mv is not scalar aggregate bug query is, should not rewrite
+    def mv20_1 = """
+            select
+            l_shipmode,
+            l_shipinstruct,
+            sum(l_extendedprice),
+            count(*)
+            from lineitem
+            left join
+            orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY
+            group by 
+            l_shipmode,
+            l_shipinstruct;
+    """
+    def query20_1 =
+            """
+            select
+            sum(l_extendedprice),
+            count(*)
+            from lineitem
+            left join
+            orders
+            on lineitem.L_ORDERKEY = orders.O_ORDERKEY
+    """
+    order_qt_query20_1_before "${query20_1}"
+    check_not_match(mv20_1, query20_1, "mv20_1")
+    order_qt_query20_1_after "${query20_1}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv20_1"""
+
+    // mv is scalar aggregate bug query not, should not rewrite
+    def mv20_2 = """
+            select
+            sum(l_extendedprice),
+            count(*)
+            from lineitem
+            left join
+            orders
+            on lineitem.L_ORDERKEY = orders.O_ORDERKEY
+    """
+    def query20_2 = """
+            select
+            l_shipmode,
+            l_shipinstruct,
+            sum(l_extendedprice),
+            count(*)
+            from lineitem
+            left join
+            orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY
+            group by
+            l_shipmode,
+            l_shipinstruct;
+    """
+    order_qt_query20_2_before "${query20_2}"
+    check_not_match(mv20_2, query20_2, "mv20_2")
+    order_qt_query20_2_after "${query20_2}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv20_2"""
 }
diff --git 
a/regression-test/suites/nereids_rules_p0/mv/join/inner/inner_join.groovy 
b/regression-test/suites/nereids_rules_p0/mv/join/inner/inner_join.groovy
index 401dbd279a1..ed50e19c605 100644
--- a/regression-test/suites/nereids_rules_p0/mv/join/inner/inner_join.groovy
+++ b/regression-test/suites/nereids_rules_p0/mv/join/inner/inner_join.groovy
@@ -215,16 +215,20 @@ suite("inner_join") {
     order_qt_query1_3_after "${query1_3}"
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_3"""
 
-    def mv1_4 = "select  lineitem.L_LINENUMBER, orders.O_CUSTKEY, 
partsupp.PS_AVAILQTY " +
-            "from lineitem " +
-            "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " +
-            "inner join partsupp on lineitem.L_PARTKEY = partsupp.PS_PARTKEY " 
+
-            "and lineitem.L_SUPPKEY = partsupp.PS_SUPPKEY"
-    def query1_4 = "select  lineitem.L_LINENUMBER " +
-            "from lineitem " +
-            "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " +
-            "inner join partsupp on lineitem.L_PARTKEY = partsupp.PS_PARTKEY " 
+
-            "and lineitem.L_SUPPKEY = partsupp.PS_SUPPKEY"
+    def mv1_4 = """
+        select  lineitem.L_LINENUMBER, orders.O_CUSTKEY, partsupp.PS_AVAILQTY
+        from lineitem
+        inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY
+        inner join partsupp on lineitem.L_PARTKEY = partsupp.PS_PARTKEY
+        and lineitem.L_SUPPKEY = partsupp.PS_SUPPKEY;
+    """
+    def query1_4 = """
+        select  lineitem.L_LINENUMBER
+        from lineitem
+        inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY
+        inner join partsupp on lineitem.L_PARTKEY = partsupp.PS_PARTKEY
+        and lineitem.L_SUPPKEY = partsupp.PS_SUPPKEY;
+    """
     order_qt_query1_4_before "${query1_4}"
     check_rewrite(mv1_4, query1_4, "mv1_4")
     order_qt_query1_4_after "${query1_4}"
@@ -273,18 +277,17 @@ suite("inner_join") {
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_1"""
 
 
-    def mv2_2 = "select  t1.L_LINENUMBER, orders.O_CUSTKEY " +
+    def mv2_2 = "select  t1.L_LINENUMBER, orders.O_CUSTKEY, l_suppkey " +
             "from (select * from lineitem where L_LINENUMBER > 1) t1 " +
             "inner join orders on t1.L_ORDERKEY = orders.O_ORDERKEY "
     def query2_2 = "select lineitem.L_LINENUMBER " +
             "from lineitem " +
             "inner join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " +
             "where lineitem.L_LINENUMBER > 1 and l_suppkey = 3"
-    // Should success but not, because mv contains the part filter of mv, tmp
-//    order_qt_query2_2_before "${query2_2}"
-//    check_rewrite(mv2_2, query2_2, "mv2_2")
-//    order_qt_query2_2_after "${query2_2}"
-//    sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_2"""
+    order_qt_query2_2_before "${query2_2}"
+    check_rewrite(mv2_2, query2_2, "mv2_2")
+    order_qt_query2_2_after "${query2_2}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_2"""
 
 
     def mv2_3 = "select  lineitem.L_LINENUMBER, orders.O_CUSTKEY, 
partsupp.PS_AVAILQTY, l_suppkey " +
diff --git 
a/regression-test/suites/nereids_rules_p0/mv/join/left_outer/outer_join.groovy 
b/regression-test/suites/nereids_rules_p0/mv/join/left_outer/outer_join.groovy
index a407b4cd472..83b2ca726b7 100644
--- 
a/regression-test/suites/nereids_rules_p0/mv/join/left_outer/outer_join.groovy
+++ 
b/regression-test/suites/nereids_rules_p0/mv/join/left_outer/outer_join.groovy
@@ -244,18 +244,38 @@ suite("outer_join") {
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_1"""
 
 
-    def mv2_2 = "select  t1.L_LINENUMBER, orders.O_CUSTKEY " +
-            "from (select * from lineitem where L_LINENUMBER > 1) t1 " +
-            "left join orders on t1.L_ORDERKEY = orders.O_ORDERKEY "
-    def query2_2 = "select lineitem.L_LINENUMBER " +
-            "from lineitem " +
-            "left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY " +
-            "where lineitem.L_LINENUMBER > 1 and l_suppkey = 3"
-    // Should success but not, tmp
-//    order_qt_query2_2_before "${query2_2}"
-//    check_rewrite(mv2_2, query2_2, "mv2_2")
-//    order_qt_query2_2_after "${query2_2}"
-//    sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_2"""
+    def mv2_2 ="""
+            select  t1.L_LINENUMBER, orders.O_CUSTKEY
+            from (select * from lineitem where L_LINENUMBER > 1) t1
+            left join orders on t1.L_ORDERKEY = orders.O_ORDERKEY;
+    """
+    def query2_2 = """
+            select lineitem.L_LINENUMBER
+            from lineitem
+            left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY
+            where lineitem.L_LINENUMBER > 1 and l_suppkey = 3;
+    """
+    order_qt_query2_2_before "${query2_2}"
+    check_not_match(mv2_2, query2_2, "mv2_2")
+    order_qt_query2_2_after "${query2_2}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_2"""
+
+
+    def mv2_3 ="""
+            select  t1.L_LINENUMBER, orders.O_CUSTKEY, l_suppkey
+            from (select * from lineitem where L_LINENUMBER > 1) t1
+            left join orders on t1.L_ORDERKEY = orders.O_ORDERKEY;
+    """
+    def query2_3 = """
+            select lineitem.L_LINENUMBER
+            from lineitem
+            left join orders on lineitem.L_ORDERKEY = orders.O_ORDERKEY
+            where lineitem.L_LINENUMBER > 1 and l_suppkey = 3;
+    """
+    order_qt_query2_3_before "${query2_3}"
+    check_rewrite(mv2_3, query2_3, "mv2_3")
+    order_qt_query2_3_after "${query2_3}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_3"""
 
 
     // filter outside + right
@@ -338,6 +358,25 @@ suite("outer_join") {
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv5_0"""
 
 
+    def mv5_1 = """
+        select l_shipdate, o_orderdate, l_partkey, l_suppkey
+        from (select * from lineitem where l_shipdate = '2023-12-08' ) t1
+        left join orders
+        on t1.l_orderkey = orders.o_orderkey
+    """
+    def query5_1 = """
+        select l_shipdate, o_orderdate, l_partkey, l_suppkey
+        from lineitem
+        left join orders
+        on lineitem.l_orderkey = orders.o_orderkey
+        where o_orderdate = '2023-12-08'
+    """
+    order_qt_query5_1_before "${query5_1}"
+    check_not_match(mv5_1, query5_1, "mv5_1")
+    order_qt_query5_1_after "${query5_1}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv5_1"""
+
+
     // filter inside + right
     def mv6_0 = "select l_shipdate, o_orderdate, l_partkey, l_suppkey " +
             "from lineitem " +
@@ -370,6 +409,25 @@ suite("outer_join") {
     sql """ DROP MATERIALIZED VIEW IF EXISTS mv7_0"""
 
 
+    def mv7_1 = """
+        select l_shipdate, o_orderdate, l_partkey, l_suppkey
+        from lineitem
+        left join orders
+        on lineitem.l_orderkey = orders.o_orderkey
+        where l_shipdate = '2023-12-08' and o_orderdate = '2023-12-08';
+    """
+    def query7_1 = """
+        select l_shipdate, o_orderdate, l_partkey, l_suppkey
+        from (select * from lineitem where l_shipdate = '2023-10-17' ) t1
+        left join orders
+        on t1.l_orderkey = orders.o_orderkey;
+    """
+    order_qt_query7_1_before "${query7_1}"
+    check_not_match(mv7_1, query7_1, "mv7_1")
+    order_qt_query7_1_after "${query7_1}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv7_1"""
+
+
     // self join test
     def mv8_0 = """
     select


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

Reply via email to