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

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


The following commit(s) were added to refs/heads/branch-2.1 by this push:
     new fbc954e8be6 [feat](mtmv) Support grouping_sets rewrite when query 
rewrite by materialized view (#36056) (#37436)
fbc954e8be6 is described below

commit fbc954e8be687a1496a6e21f0bae833fea21640b
Author: seawinde <149132972+seawi...@users.noreply.github.com>
AuthorDate: Mon Jul 8 15:06:16 2024 +0800

    [feat](mtmv) Support grouping_sets rewrite when query rewrite by 
materialized view (#36056) (#37436)
    
    cherry pick from master
    pr: #36056
    commitId: 569c9772
---
 .../nereids/rules/analysis/NormalizeRepeat.java    |  45 +-
 .../mv/AbstractMaterializedViewAggregateRule.java  | 275 +++++----
 .../mv/AbstractMaterializedViewRule.java           |  20 +
 .../exploration/mv/MaterializedViewUtils.java      |   5 +-
 .../nereids/rules/exploration/mv/StructInfo.java   |  10 +-
 .../mv/grouping_sets/grouping_sets.out             | 225 ++++++++
 .../mv/grouping_sets/grouping_sets.groovy          | 616 +++++++++++++++++++++
 7 files changed, 1057 insertions(+), 139 deletions(-)

diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/NormalizeRepeat.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/NormalizeRepeat.java
index fd6cce8b4fb..6465b81da30 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/NormalizeRepeat.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/NormalizeRepeat.java
@@ -89,16 +89,23 @@ public class NormalizeRepeat extends OneAnalysisRuleFactory 
{
                     return new 
LogicalAggregate<>(repeat.getGroupByExpressions(),
                             repeat.getOutputExpressions(), repeat.child());
                 }
-                checkRepeatLegality(repeat);
-                repeat = removeDuplicateColumns(repeat);
-                // add virtual slot, LogicalAggregate and LogicalProject for 
normalize
-                LogicalAggregate<Plan> agg = normalizeRepeat(repeat);
-                return dealSlotAppearBothInAggFuncAndGroupingSets(agg);
+                return doNormalize(repeat);
             })
         );
     }
 
-    private LogicalRepeat<Plan> removeDuplicateColumns(LogicalRepeat<Plan> 
repeat) {
+    /**
+     * Normalize repeat, this can be used directly, if optimize the repeat
+     */
+    public static LogicalAggregate<Plan> doNormalize(LogicalRepeat<Plan> 
repeat) {
+        checkRepeatLegality(repeat);
+        repeat = removeDuplicateColumns(repeat);
+        // add virtual slot, LogicalAggregate and LogicalProject for normalize
+        LogicalAggregate<Plan> agg = normalizeRepeat(repeat);
+        return dealSlotAppearBothInAggFuncAndGroupingSets(agg);
+    }
+
+    private static LogicalRepeat<Plan> 
removeDuplicateColumns(LogicalRepeat<Plan> repeat) {
         List<List<Expression>> groupingSets = repeat.getGroupingSets();
         ImmutableList.Builder<List<Expression>> builder = 
ImmutableList.builder();
         for (List<Expression> sets : groupingSets) {
@@ -108,11 +115,11 @@ public class NormalizeRepeat extends 
OneAnalysisRuleFactory {
         return repeat.withGroupSets(builder.build());
     }
 
-    private void checkRepeatLegality(LogicalRepeat<Plan> repeat) {
+    private static void checkRepeatLegality(LogicalRepeat<Plan> repeat) {
         checkGroupingSetsSize(repeat);
     }
 
-    private void checkGroupingSetsSize(LogicalRepeat<Plan> repeat) {
+    private static void checkGroupingSetsSize(LogicalRepeat<Plan> repeat) {
         Set<Expression> flattenGroupingSetExpr = ImmutableSet.copyOf(
                 ExpressionUtils.flatExpressions(repeat.getGroupingSets()));
         if (flattenGroupingSetExpr.size() > 
LogicalRepeat.MAX_GROUPING_SETS_NUM) {
@@ -122,7 +129,7 @@ public class NormalizeRepeat extends OneAnalysisRuleFactory 
{
         }
     }
 
-    private LogicalAggregate<Plan> normalizeRepeat(LogicalRepeat<Plan> repeat) 
{
+    private static LogicalAggregate<Plan> normalizeRepeat(LogicalRepeat<Plan> 
repeat) {
         Set<Expression> needToSlotsGroupingExpr = 
collectNeedToSlotGroupingExpr(repeat);
         NormalizeToSlotContext groupingExprContext = buildContext(repeat, 
needToSlotsGroupingExpr);
         Map<Expression, NormalizeToSlotTriplet> groupingExprMap = 
groupingExprContext.getNormalizeToSlotMap();
@@ -198,14 +205,14 @@ public class NormalizeRepeat extends 
OneAnalysisRuleFactory {
                 Optional.of(normalizedRepeat), normalizedRepeat);
     }
 
-    private Set<Expression> collectNeedToSlotGroupingExpr(LogicalRepeat<Plan> 
repeat) {
+    private static Set<Expression> 
collectNeedToSlotGroupingExpr(LogicalRepeat<Plan> repeat) {
         // grouping sets should be pushed down, e.g. grouping sets((k + 1)),
         // we should push down the `k + 1` to the bottom plan
         return ImmutableSet.copyOf(
                 ExpressionUtils.flatExpressions(repeat.getGroupingSets()));
     }
 
-    private Set<Expression> 
collectNeedToSlotArgsOfGroupingScalarFuncAndAggFunc(LogicalRepeat<Plan> repeat) 
{
+    private static Set<Expression> 
collectNeedToSlotArgsOfGroupingScalarFuncAndAggFunc(LogicalRepeat<Plan> repeat) 
{
         Set<GroupingScalarFunction> groupingScalarFunctions = 
ExpressionUtils.collect(
                 repeat.getOutputExpressions(), 
GroupingScalarFunction.class::isInstance);
         ImmutableSet.Builder<Expression> argumentsSetBuilder = 
ImmutableSet.builder();
@@ -237,7 +244,7 @@ public class NormalizeRepeat extends OneAnalysisRuleFactory 
{
                 .build();
     }
 
-    private Plan pushDownProject(Set<NamedExpression> pushedExprs, Plan 
originBottomPlan) {
+    private static Plan pushDownProject(Set<NamedExpression> pushedExprs, Plan 
originBottomPlan) {
         if (!pushedExprs.equals(originBottomPlan.getOutputSet()) && 
!pushedExprs.isEmpty()) {
             return new LogicalProject<>(ImmutableList.copyOf(pushedExprs), 
originBottomPlan);
         }
@@ -245,7 +252,7 @@ public class NormalizeRepeat extends OneAnalysisRuleFactory 
{
     }
 
     /** buildContext */
-    public NormalizeToSlotContext buildContext(Repeat<? extends Plan> repeat,
+    public static NormalizeToSlotContext buildContext(Repeat<? extends Plan> 
repeat,
             Set<? extends Expression> sourceExpressions) {
         Set<Alias> aliases = 
ExpressionUtils.collect(repeat.getOutputExpressions(), Alias.class::isInstance);
         Map<Expression, Alias> existsAliasMap = Maps.newLinkedHashMap();
@@ -271,7 +278,7 @@ public class NormalizeRepeat extends OneAnalysisRuleFactory 
{
         return new NormalizeToSlotContext(normalizeToSlotMap);
     }
 
-    private Optional<NormalizeToSlotTriplet> 
toGroupingSetExpressionPushDownTriplet(
+    private static Optional<NormalizeToSlotTriplet> 
toGroupingSetExpressionPushDownTriplet(
             Expression expression, @Nullable Alias existsAlias) {
         NormalizeToSlotTriplet originTriplet = 
NormalizeToSlotTriplet.toTriplet(expression, existsAlias);
         SlotReference remainSlot = (SlotReference) originTriplet.remainExpr;
@@ -279,7 +286,8 @@ public class NormalizeRepeat extends OneAnalysisRuleFactory 
{
         return Optional.of(new NormalizeToSlotTriplet(expression, newSlot, 
originTriplet.pushedExpr));
     }
 
-    private Expression 
normalizeAggFuncChildrenAndGroupingScalarFunc(NormalizeToSlotContext context, 
Expression expr) {
+    private static Expression 
normalizeAggFuncChildrenAndGroupingScalarFunc(NormalizeToSlotContext context,
+            Expression expr) {
         if (expr instanceof AggregateFunction) {
             AggregateFunction function = (AggregateFunction) expr;
             List<Expression> normalizedRealExpressions = 
context.normalizeToUseSlotRef(function.getArguments());
@@ -296,7 +304,7 @@ public class NormalizeRepeat extends OneAnalysisRuleFactory 
{
         }
     }
 
-    private Set<Alias> getExistsAlias(LogicalRepeat<Plan> repeat,
+    private static Set<Alias> getExistsAlias(LogicalRepeat<Plan> repeat,
             Map<Expression, NormalizeToSlotTriplet> groupingExprMap) {
         Set<Alias> existsAlias = Sets.newHashSet();
         Set<Alias> aliases = 
ExpressionUtils.collect(repeat.getOutputExpressions(), Alias.class::isInstance);
@@ -323,7 +331,7 @@ public class NormalizeRepeat extends OneAnalysisRuleFactory 
{
      *   +--LogicalRepeat (groupingSets=[[a#0]], outputExpr=[a#0, a#3, 
GROUPING_ID#1]
      *      +--LogicalProject (projects =[a#0, a#0 as `a`#3])
      */
-    private LogicalAggregate<Plan> dealSlotAppearBothInAggFuncAndGroupingSets(
+    private static LogicalAggregate<Plan> 
dealSlotAppearBothInAggFuncAndGroupingSets(
             @NotNull LogicalAggregate<Plan> aggregate) {
         LogicalRepeat<Plan> repeat = (LogicalRepeat<Plan>) aggregate.child();
         Map<Slot, Alias> commonSlotToAliasMap = 
getCommonSlotToAliasMap(repeat, aggregate);
@@ -370,7 +378,8 @@ public class NormalizeRepeat extends OneAnalysisRuleFactory 
{
         return aggregate.withAggOutput(newOutputExpressions);
     }
 
-    private Map<Slot, Alias> getCommonSlotToAliasMap(LogicalRepeat<Plan> 
repeat, LogicalAggregate<Plan> aggregate) {
+    private static Map<Slot, Alias> 
getCommonSlotToAliasMap(LogicalRepeat<Plan> repeat,
+            LogicalAggregate<Plan> aggregate) {
         List<AggregateFunction> aggregateFunctions =
                 
CollectNonWindowedAggFuncs.collect(aggregate.getOutputExpressions());
         ImmutableSet.Builder<Slot> aggUsedSlotBuilder = ImmutableSet.builder();
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 6883247ce16..997792f0f4c 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
@@ -19,6 +19,7 @@ package org.apache.doris.nereids.rules.exploration.mv;
 
 import org.apache.doris.common.Pair;
 import org.apache.doris.nereids.CascadesContext;
+import org.apache.doris.nereids.rules.analysis.NormalizeRepeat;
 import 
org.apache.doris.nereids.rules.exploration.mv.StructInfo.PlanCheckContext;
 import 
org.apache.doris.nereids.rules.exploration.mv.StructInfo.PlanSplitContext;
 import org.apache.doris.nereids.rules.exploration.mv.mapping.SlotMapping;
@@ -32,13 +33,16 @@ import org.apache.doris.nereids.trees.expressions.ExprId;
 import org.apache.doris.nereids.trees.expressions.Expression;
 import org.apache.doris.nereids.trees.expressions.NamedExpression;
 import org.apache.doris.nereids.trees.expressions.Slot;
+import org.apache.doris.nereids.trees.expressions.VirtualSlotReference;
 import org.apache.doris.nereids.trees.expressions.functions.Function;
 import 
org.apache.doris.nereids.trees.expressions.functions.agg.AggregateFunction;
-import org.apache.doris.nereids.trees.expressions.functions.agg.RollUpTrait;
+import 
org.apache.doris.nereids.trees.expressions.functions.scalar.GroupingScalarFunction;
 import 
org.apache.doris.nereids.trees.expressions.visitor.DefaultExpressionRewriter;
 import org.apache.doris.nereids.trees.plans.Plan;
+import org.apache.doris.nereids.trees.plans.algebra.Repeat;
 import org.apache.doris.nereids.trees.plans.logical.LogicalAggregate;
 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.ExpressionLineageReplacer;
 import org.apache.doris.nereids.util.ExpressionUtils;
 
@@ -51,7 +55,9 @@ import java.util.BitSet;
 import java.util.HashSet;
 import java.util.List;
 import java.util.Map;
+import java.util.Optional;
 import java.util.Set;
+import java.util.function.Supplier;
 import java.util.stream.Collectors;
 
 /**
@@ -91,10 +97,16 @@ public abstract class AbstractMaterializedViewAggregateRule 
extends AbstractMate
                     () -> String.format("query plan = %s\n", 
queryStructInfo.getOriginalPlan().treeString()));
             return null;
         }
-        // Firstly,if group by expression between query and view is equals, 
try to rewrite expression directly
         Plan queryTopPlan = queryTopPlanAndAggPair.key();
-        if (isGroupByEquals(queryTopPlanAndAggPair, viewTopPlanAndAggPair, 
viewToQuerySlotMapping, queryStructInfo,
-                viewStructInfo, materializationContext)) {
+        LogicalAggregate<Plan> queryAggregate = queryTopPlanAndAggPair.value();
+        if (!checkCompatibility(queryStructInfo, queryAggregate, 
viewTopPlanAndAggPair.value(),
+                materializationContext)) {
+            return null;
+        }
+        boolean queryContainsGroupSets = 
queryAggregate.getSourceRepeat().isPresent();
+        // If group by expression between query and view is equals, try to 
rewrite expression directly
+        if (!queryContainsGroupSets && isGroupByEquals(queryTopPlanAndAggPair, 
viewTopPlanAndAggPair,
+                viewToQuerySlotMapping, queryStructInfo, viewStructInfo, 
materializationContext)) {
             List<Expression> rewrittenQueryExpressions = 
rewriteExpression(queryTopPlan.getOutput(),
                     queryTopPlan,
                     materializationContext.getShuttledExprToScanExprMapping(),
@@ -124,20 +136,6 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
                             
materializationContext.getShuttledExprToScanExprMapping(),
                             viewToQuerySlotMapping));
         }
-        // if view is scalar aggregate but query is not. Or if query is scalar 
aggregate but view is not
-        // Should not rewrite
-        List<Expression> queryGroupByExpressions = 
queryTopPlanAndAggPair.value().getGroupByExpressions();
-        List<Expression> viewGroupByExpressions = 
viewTopPlanAndAggPair.value().getGroupByExpressions();
-        if ((queryGroupByExpressions.isEmpty() && 
!viewGroupByExpressions.isEmpty())
-                || (!queryGroupByExpressions.isEmpty() && 
viewGroupByExpressions.isEmpty())) {
-            materializationContext.recordFailReason(queryStructInfo,
-                    "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
@@ -147,46 +145,31 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
         // try to rewrite, contains both roll up aggregate functions and 
aggregate group expression
         List<NamedExpression> finalOutputExpressions = new ArrayList<>();
         List<Expression> finalGroupExpressions = new ArrayList<>();
-        List<? extends Expression> queryExpressions = queryTopPlan.getOutput();
         // permute the mv expr mapping to query based
         Map<Expression, Expression> mvExprToMvScanExprQueryBased =
                 
materializationContext.getShuttledExprToScanExprMapping().keyPermute(viewToQuerySlotMapping)
                         .flattenMap().get(0);
-        for (Expression topExpression : queryExpressions) {
-            // if agg function, try to roll up and rewrite
+        for (Expression topExpression : queryTopPlan.getOutput()) {
             if (queryTopPlanFunctionSet.contains(topExpression)) {
-                Expression queryFunctionShuttled = 
ExpressionUtils.shuttleExpressionWithLineage(
-                        topExpression,
-                        queryTopPlan,
-                        queryStructInfo.getTableBitSet());
-                AggregateExpressionRewriteContext context = new 
AggregateExpressionRewriteContext(
-                        false, mvExprToMvScanExprQueryBased, queryTopPlan, 
queryStructInfo.getTableBitSet());
-                // queryFunctionShuttled maybe sum(column) + count(*), so need 
to use expression rewriter
-                Expression rollupedExpression = 
queryFunctionShuttled.accept(AGGREGATE_EXPRESSION_REWRITER,
-                        context);
-                if (!context.isValid()) {
-                    materializationContext.recordFailReason(queryStructInfo,
-                            "Query function roll up fail",
-                            () -> String.format("queryFunctionShuttled = %s,\n 
mvExprToMvScanExprQueryBased = %s",
-                                    queryFunctionShuttled, 
mvExprToMvScanExprQueryBased));
+                // if agg function, try to roll up and rewrite
+                Expression rollupedExpression = 
tryRewriteExpression(queryStructInfo, topExpression,
+                        mvExprToMvScanExprQueryBased, false, 
materializationContext,
+                        "Query function roll up fail",
+                        () -> String.format("queryExpression = %s,\n 
mvExprToMvScanExprQueryBased = %s",
+                                topExpression, mvExprToMvScanExprQueryBased));
+                if (rollupedExpression == null) {
                     return null;
                 }
                 finalOutputExpressions.add(new Alias(rollupedExpression));
             } else {
-                // if group by expression, try to rewrite group by expression
-                Expression queryGroupShuttledExpr = 
ExpressionUtils.shuttleExpressionWithLineage(
-                        topExpression, queryTopPlan, 
queryStructInfo.getTableBitSet());
-                AggregateExpressionRewriteContext context = new 
AggregateExpressionRewriteContext(true,
-                        mvExprToMvScanExprQueryBased, queryTopPlan, 
queryStructInfo.getTableBitSet());
-                // group by expression maybe group by a + b, so we need 
expression rewriter
-                Expression rewrittenGroupByExpression = 
queryGroupShuttledExpr.accept(AGGREGATE_EXPRESSION_REWRITER,
-                        context);
-                if (!context.isValid()) {
+                // if group by dimension, try to rewrite
+                Expression rewrittenGroupByExpression = 
tryRewriteExpression(queryStructInfo, topExpression,
+                        mvExprToMvScanExprQueryBased, true, 
materializationContext,
+                        "View dimensions doesn't not cover the query 
dimensions",
+                        () -> String.format("mvExprToMvScanExprQueryBased is 
%s,\n queryExpression is %s",
+                                mvExprToMvScanExprQueryBased, topExpression));
+                if (rewrittenGroupByExpression == null) {
                     // group expr can not rewrite by view
-                    materializationContext.recordFailReason(queryStructInfo,
-                            "View dimensions doesn't not cover the query 
dimensions",
-                            () -> String.format("mvExprToMvScanExprQueryBased 
is %s,\n queryGroupShuttledExpr is %s",
-                                    mvExprToMvScanExprQueryBased, 
queryGroupShuttledExpr));
                     return null;
                 }
                 NamedExpression groupByExpression = rewrittenGroupByExpression 
instanceof NamedExpression
@@ -195,26 +178,19 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
                 finalGroupExpressions.add(groupByExpression);
             }
         }
-        // add project to guarantee group by column ref is slot reference,
-        // this is necessary because physical createHash will need 
slotReference later
+        List<Expression> queryGroupByExpressions = 
queryAggregate.getGroupByExpressions();
+        // handle the scene that query top plan not use the group by in query 
bottom aggregate
         if (queryGroupByExpressions.size() != queryTopPlanGroupBySet.size()) {
             for (Expression expression : queryGroupByExpressions) {
                 if (queryTopPlanGroupBySet.contains(expression)) {
                     continue;
                 }
-                Expression queryGroupShuttledExpr = 
ExpressionUtils.shuttleExpressionWithLineage(
-                        expression, queryTopPlan, 
queryStructInfo.getTableBitSet());
-                AggregateExpressionRewriteContext context = new 
AggregateExpressionRewriteContext(true,
-                        mvExprToMvScanExprQueryBased, queryTopPlan, 
queryStructInfo.getTableBitSet());
-                // group by expression maybe group by a + b, so we need 
expression rewriter
-                Expression rewrittenGroupByExpression = 
queryGroupShuttledExpr.accept(AGGREGATE_EXPRESSION_REWRITER,
-                        context);
-                if (!context.isValid()) {
-                    // group expr can not rewrite by view
-                    materializationContext.recordFailReason(queryStructInfo,
-                            "View dimensions doesn't not cover the query 
dimensions in bottom agg ",
-                            () -> String.format("mvExprToMvScanExprQueryBased 
is %s,\n queryGroupShuttledExpr is %s",
-                                    mvExprToMvScanExprQueryBased, 
queryGroupShuttledExpr));
+                Expression rewrittenGroupByExpression = 
tryRewriteExpression(queryStructInfo, expression,
+                        mvExprToMvScanExprQueryBased, true, 
materializationContext,
+                        "View dimensions doesn't not cover the query 
dimensions in bottom agg ",
+                        () -> String.format("mvExprToMvScanExprQueryBased is 
%s,\n expression is %s",
+                                mvExprToMvScanExprQueryBased, expression));
+                if (rewrittenGroupByExpression == null) {
                     return null;
                 }
                 NamedExpression groupByExpression = rewrittenGroupByExpression 
instanceof NamedExpression
@@ -222,31 +198,90 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
                 finalGroupExpressions.add(groupByExpression);
             }
         }
-        List<Expression> copiedFinalGroupExpressions = new 
ArrayList<>(finalGroupExpressions);
-        List<NamedExpression> projectsUnderAggregate = 
copiedFinalGroupExpressions.stream()
-                .map(NamedExpression.class::cast)
-                .collect(Collectors.toList());
-        projectsUnderAggregate.addAll(tempRewritedPlan.getOutput());
-        LogicalProject<Plan> mvProject = new 
LogicalProject<>(projectsUnderAggregate, tempRewritedPlan);
-        // add agg rewrite
-        Map<ExprId, Slot> projectOutPutExprIdMap = 
mvProject.getOutput().stream()
-                .distinct()
-                .collect(Collectors.toMap(NamedExpression::getExprId, slot -> 
slot));
-        // make the expressions to re reference project output
-        finalGroupExpressions = finalGroupExpressions.stream()
-                .map(expr -> {
-                    ExprId exprId = ((NamedExpression) expr).getExprId();
-                    if (projectOutPutExprIdMap.containsKey(exprId)) {
-                        return projectOutPutExprIdMap.get(exprId);
+        if (queryContainsGroupSets) {
+            // construct group sets for repeat
+            List<List<Expression>> rewrittenGroupSetsExpressions = new 
ArrayList<>();
+            List<List<Expression>> groupingSets = 
queryAggregate.getSourceRepeat().get().getGroupingSets();
+            for (List<Expression> groupingSet : groupingSets) {
+                if (groupingSet.isEmpty()) {
+                    rewrittenGroupSetsExpressions.add(ImmutableList.of());
+                } else {
+                    List<Expression> rewrittenGroupSetExpressions = new 
ArrayList<>();
+                    for (Expression expression : groupingSet) {
+                        Expression rewrittenGroupByExpression = 
tryRewriteExpression(queryStructInfo, expression,
+                                mvExprToMvScanExprQueryBased, true, 
materializationContext,
+                                "View dimensions doesn't not cover the query 
group set dimensions",
+                                () -> 
String.format("mvExprToMvScanExprQueryBased is %s,\n queryExpression is %s",
+                                        mvExprToMvScanExprQueryBased, 
expression));
+                        if (rewrittenGroupByExpression == null) {
+                            return null;
+                        }
+                        
rewrittenGroupSetExpressions.add(rewrittenGroupByExpression);
                     }
-                    return (NamedExpression) expr;
-                })
-                .collect(Collectors.toList());
-        finalOutputExpressions = finalOutputExpressions.stream()
-                .map(expr -> 
projectOutPutExprIdMap.containsKey(expr.getExprId())
-                        ? projectOutPutExprIdMap.get(expr.getExprId()) : expr)
-                .collect(Collectors.toList());
-        return new LogicalAggregate(finalGroupExpressions, 
finalOutputExpressions, mvProject);
+                    
rewrittenGroupSetsExpressions.add(rewrittenGroupSetExpressions);
+                }
+            }
+            LogicalRepeat<Plan> repeat = new 
LogicalRepeat<>(rewrittenGroupSetsExpressions,
+                    finalOutputExpressions, tempRewritedPlan);
+            return NormalizeRepeat.doNormalize(repeat);
+        }
+        return new LogicalAggregate<>(finalGroupExpressions, 
finalOutputExpressions, tempRewritedPlan);
+    }
+
+    /**
+     * Try to rewrite query expression by view, contains both group by 
dimension and aggregate function
+     */
+    protected Expression tryRewriteExpression(StructInfo queryStructInfo, 
Expression queryExpression,
+            Map<Expression, Expression> mvShuttledExprToMvScanExprQueryBased, 
boolean isGroupBy,
+            MaterializationContext materializationContext, String 
summaryIfFail, Supplier<String> detailIfFail) {
+        Expression queryFunctionShuttled = 
ExpressionUtils.shuttleExpressionWithLineage(
+                queryExpression,
+                queryStructInfo.getTopPlan(),
+                queryStructInfo.getTableBitSet());
+        AggregateExpressionRewriteContext expressionRewriteContext = new 
AggregateExpressionRewriteContext(
+                isGroupBy, mvShuttledExprToMvScanExprQueryBased, 
queryStructInfo.getTopPlan(),
+                queryStructInfo.getTableBitSet());
+        Expression rewrittenExpression = 
queryFunctionShuttled.accept(AGGREGATE_EXPRESSION_REWRITER,
+                expressionRewriteContext);
+        if (!expressionRewriteContext.isValid()) {
+            materializationContext.recordFailReason(queryStructInfo, 
summaryIfFail, detailIfFail);
+            return null;
+        }
+        return rewrittenExpression;
+    }
+
+    /**
+     * Check query and view aggregate compatibility
+     */
+    private static boolean checkCompatibility(
+            StructInfo queryStructInfo,
+            LogicalAggregate<Plan> queryAggregate, LogicalAggregate<Plan> 
viewAggregate,
+            MaterializationContext materializationContext) {
+        // if view is scalar aggregate but query is not. Or if query is scalar 
aggregate but view is not
+        // Should not rewrite
+        List<Expression> queryGroupByExpressions = 
queryAggregate.getGroupByExpressions();
+        List<Expression> viewGroupByExpressions = 
viewAggregate.getGroupByExpressions();
+        if (!queryGroupByExpressions.isEmpty() && 
viewGroupByExpressions.isEmpty()) {
+            materializationContext.recordFailReason(queryStructInfo,
+                    "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",
+                            queryAggregate.treeString(),
+                            viewAggregate.treeString()));
+            return false;
+        }
+        boolean viewHasGroupSets = viewAggregate.getSourceRepeat()
+                .map(repeat -> repeat.getGroupingSets().size()).orElse(0) > 0;
+        // if both query and view has group sets, or query doesn't hava, mv 
have, not supported
+        if (viewHasGroupSets) {
+            materializationContext.recordFailReason(queryStructInfo,
+                    "both query and view have group sets, or query doesn't 
have but view have, not supported",
+                    () -> String.format("query aggregate = %s,\n view 
aggregate = %s,\n",
+                            queryAggregate.treeString(),
+                            viewAggregate.treeString()));
+            return false;
+        }
+        return true;
     }
 
     private boolean isGroupByEquals(Pair<Plan, LogicalAggregate<Plan>> 
queryTopPlanAndAggPair,
@@ -259,14 +294,18 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
         Plan viewTopPlan = viewTopPlanAndAggPair.key();
         LogicalAggregate<Plan> queryAggregate = queryTopPlanAndAggPair.value();
         LogicalAggregate<Plan> viewAggregate = viewTopPlanAndAggPair.value();
-        Set<? extends Expression> queryGroupShuttledExpression = new HashSet<>(
-                ExpressionUtils.shuttleExpressionWithLineage(
-                        queryAggregate.getGroupByExpressions(), queryTopPlan, 
queryStructInfo.getTableBitSet()));
-        Set<? extends Expression> viewGroupShuttledExpressionQueryBased = 
ExpressionUtils.shuttleExpressionWithLineage(
-                        viewAggregate.getGroupByExpressions(), viewTopPlan, 
viewStructInfo.getTableBitSet())
-                .stream()
-                .map(expr -> ExpressionUtils.replace(expr, 
viewToQuerySlotMapping.toSlotReferenceMap()))
-                .collect(Collectors.toSet());
+
+        Set<Expression> queryGroupShuttledExpression = new HashSet<>();
+        for (Expression queryExpression : 
ExpressionUtils.shuttleExpressionWithLineage(
+                queryAggregate.getGroupByExpressions(), queryTopPlan, 
queryStructInfo.getTableBitSet())) {
+            queryGroupShuttledExpression.add(queryExpression);
+        }
+        Set<Expression> viewGroupShuttledExpressionQueryBased = new 
HashSet<>();
+        for (Expression viewExpression : 
ExpressionUtils.shuttleExpressionWithLineage(
+                viewAggregate.getGroupByExpressions(), viewTopPlan, 
viewStructInfo.getTableBitSet())) {
+            viewGroupShuttledExpressionQueryBased.add(
+                    ExpressionUtils.replace(viewExpression, 
viewToQuerySlotMapping.toSlotReferenceMap()));
+        }
         return 
queryGroupShuttledExpression.equals(viewGroupShuttledExpressionQueryBased);
     }
 
@@ -305,23 +344,6 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
         return null;
     }
 
-    // Check the aggregate function can roll up or not, return true if could 
roll up
-    // if view aggregate function is distinct or is in the un supported rollup 
functions, it doesn't support
-    // roll up.
-    private static boolean canRollup(Expression rollupExpression) {
-        if (rollupExpression == null) {
-            return false;
-        }
-        if (rollupExpression instanceof Function && !(rollupExpression 
instanceof AggregateFunction)) {
-            return false;
-        }
-        if (rollupExpression instanceof AggregateFunction) {
-            AggregateFunction aggregateFunction = (AggregateFunction) 
rollupExpression;
-            return !aggregateFunction.isDistinct() && aggregateFunction 
instanceof RollUpTrait;
-        }
-        return true;
-    }
-
     private Pair<Set<? extends Expression>, Set<? extends Expression>> 
topPlanSplitToGroupAndFunction(
             Pair<Plan, LogicalAggregate<Plan>> topPlanAndAggPair, StructInfo 
queryStructInfo) {
         LogicalAggregate<Plan> bottomQueryAggregate = 
topPlanAndAggPair.value();
@@ -405,11 +427,40 @@ public abstract class 
AbstractMaterializedViewAggregateRule extends AbstractMate
             return rollupAggregateFunction;
         }
 
+        @Override
+        public Expression visitGroupingScalarFunction(GroupingScalarFunction 
groupingScalarFunction,
+                AggregateExpressionRewriteContext context) {
+            List<Expression> children = groupingScalarFunction.children();
+            List<Expression> rewrittenChildren = new ArrayList<>();
+            for (Expression child : children) {
+                Expression rewrittenChild = child.accept(this, context);
+                if (!context.isValid()) {
+                    return groupingScalarFunction;
+                }
+                rewrittenChildren.add(rewrittenChild);
+            }
+            return groupingScalarFunction.withChildren(rewrittenChildren);
+        }
+
         @Override
         public Expression visitSlot(Slot slot, 
AggregateExpressionRewriteContext rewriteContext) {
             if (!rewriteContext.isValid()) {
                 return slot;
             }
+            if (slot instanceof VirtualSlotReference) {
+                Optional<GroupingScalarFunction> originExpression = 
((VirtualSlotReference) slot).getOriginExpression();
+                if (!originExpression.isPresent()) {
+                    return Repeat.generateVirtualGroupingIdSlot();
+                } else {
+                    GroupingScalarFunction groupingScalarFunction = 
originExpression.get();
+                    groupingScalarFunction =
+                            (GroupingScalarFunction) 
groupingScalarFunction.accept(this, rewriteContext);
+                    if (!rewriteContext.isValid()) {
+                        return slot;
+                    }
+                    return 
Repeat.generateVirtualSlotByFunction(groupingScalarFunction);
+                }
+            }
             if 
(rewriteContext.getMvExprToMvScanExprQueryBasedMapping().containsKey(slot)) {
                 return 
rewriteContext.getMvExprToMvScanExprQueryBasedMapping().get(slot);
             }
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 32d42b420f7..b5377181126 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
@@ -43,6 +43,7 @@ import 
org.apache.doris.nereids.trees.expressions.NamedExpression;
 import org.apache.doris.nereids.trees.expressions.Not;
 import org.apache.doris.nereids.trees.expressions.Slot;
 import org.apache.doris.nereids.trees.expressions.SlotReference;
+import org.apache.doris.nereids.trees.expressions.VirtualSlotReference;
 import 
org.apache.doris.nereids.trees.expressions.functions.agg.AggregateFunction;
 import org.apache.doris.nereids.trees.expressions.functions.scalar.NonNullable;
 import org.apache.doris.nereids.trees.expressions.functions.scalar.Nullable;
@@ -306,7 +307,18 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                     }
                 }
             }
+            List<Slot> rewrittenPlanOutput = rewrittenPlan.getOutput();
             rewrittenPlan = normalizeExpressions(rewrittenPlan, queryPlan);
+            if (rewrittenPlan == null) {
+                // maybe virtual slot reference added automatically
+                materializationContext.recordFailReason(queryStructInfo,
+                        "RewrittenPlan output logical properties is different 
with target group",
+                        () -> String.format("materialized view rule 
normalizeExpressions, output size between "
+                                        + "origin and rewritten plan is 
different, rewritten output is %s, "
+                                        + "origin output is %s",
+                                rewrittenPlanOutput, queryPlan.getOutput()));
+                continue;
+            }
             if (!isOutputValid(queryPlan, rewrittenPlan)) {
                 LogicalProperties logicalProperties = 
rewrittenPlan.getLogicalProperties();
                 materializationContext.recordFailReason(queryStructInfo,
@@ -343,6 +355,9 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
 
     // Normalize expression such as nullable property and output slot id
     protected Plan normalizeExpressions(Plan rewrittenPlan, Plan originPlan) {
+        if (rewrittenPlan.getOutput().size() != originPlan.getOutput().size()) 
{
+            return null;
+        }
         // normalize nullable
         List<NamedExpression> normalizeProjects = new ArrayList<>();
         for (int i = 0; i < originPlan.getOutput().size(); i++) {
@@ -693,6 +708,11 @@ public abstract class AbstractMaterializedViewRule 
implements ExplorationRuleFac
                 && 
context.alreadyRewrite(plan.getGroupExpression().get().getOwnerGroup().getGroupId());
     }
 
+    protected boolean isEmptyVirtualSlot(Expression expression) {
+        return expression instanceof VirtualSlotReference
+                && ((VirtualSlotReference) 
expression).getRealExpressions().isEmpty();
+    }
+
     // check mv plan is valid or not, this can use cache for performance
     private boolean isMaterializationValid(CascadesContext cascadesContext, 
MaterializationContext context) {
         long materializationId = 
context.getMaterializationQualifier().hashCode();
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 5c6a666fdb4..b2baa5918c4 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
@@ -204,7 +204,7 @@ public class MaterializedViewUtils {
      * should be different
      */
     public static Plan generateMvScanPlan(MTMV materializedView, 
CascadesContext cascadesContext) {
-        LogicalOlapScan mvScan = new LogicalOlapScan(
+        return new LogicalOlapScan(
                 cascadesContext.getStatementContext().getNextRelationId(),
                 materializedView,
                 materializedView.getFullQualifiers(),
@@ -216,9 +216,6 @@ public class MaterializedViewUtils {
                 // this must be empty, or it will be used to sample
                 ImmutableList.of(),
                 Optional.empty());
-        List<NamedExpression> mvProjects = 
mvScan.getOutput().stream().map(NamedExpression.class::cast)
-                .collect(Collectors.toList());
-        return new LogicalProject<Plan>(mvProjects, mvScan);
     }
 
     /**
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 d23806339c6..4dcb7457119 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
@@ -54,6 +54,7 @@ import 
org.apache.doris.nereids.trees.plans.logical.LogicalJoin;
 import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan;
 import org.apache.doris.nereids.trees.plans.logical.LogicalPlan;
 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.logical.LogicalSort;
 import org.apache.doris.nereids.trees.plans.visitor.DefaultPlanRewriter;
 import org.apache.doris.nereids.trees.plans.visitor.DefaultPlanVisitor;
@@ -607,9 +608,6 @@ public class StructInfo {
                 checkContext.setContainsTopAggregate(true);
                 checkContext.plusTopAggregateNum();
             }
-            if (aggregate.getSourceRepeat().isPresent()) {
-                return false;
-            }
             return visit(aggregate, checkContext);
         }
 
@@ -627,7 +625,8 @@ public class StructInfo {
                     || plan instanceof Join
                     || plan instanceof LogicalSort
                     || plan instanceof LogicalAggregate
-                    || plan instanceof GroupPlan) {
+                    || plan instanceof GroupPlan
+                    || plan instanceof LogicalRepeat) {
                 return doVisit(plan, checkContext);
             }
             return false;
@@ -660,7 +659,8 @@ public class StructInfo {
             if (plan instanceof Filter
                     || plan instanceof Project
                     || plan instanceof CatalogRelation
-                    || plan instanceof GroupPlan) {
+                    || plan instanceof GroupPlan
+                    || plan instanceof LogicalRepeat) {
                 return doVisit(plan, checkContext);
             }
             return false;
diff --git 
a/regression-test/data/nereids_rules_p0/mv/grouping_sets/grouping_sets.out 
b/regression-test/data/nereids_rules_p0/mv/grouping_sets/grouping_sets.out
new file mode 100644
index 00000000000..b6e7a275984
--- /dev/null
+++ b/regression-test/data/nereids_rules_p0/mv/grouping_sets/grouping_sets.out
@@ -0,0 +1,225 @@
+-- This file is automatically generated. You should know what you did if you 
want to edit this
+-- !query1_0_before --
+\N     \N      \N      178.10  56.20   1.20    8       0
+\N     \N      a       77.50   33.50   9.50    5       0
+\N     \N      c       100.60  56.20   1.20    3       0
+o      \N      \N      178.10  56.20   1.20    8       0
+o      2023-12-08      \N      20.00   10.50   9.50    2       0
+o      2023-12-09      \N      11.50   11.50   11.50   1       0
+o      2023-12-10      \N      46.00   33.50   12.50   2       0
+o      2023-12-11      \N      43.20   43.20   43.20   1       0
+o      2023-12-12      \N      57.40   56.20   1.20    2       0
+
+-- !query1_0_after --
+\N     \N      \N      178.10  56.20   1.20    8       0
+\N     \N      a       77.50   33.50   9.50    5       0
+\N     \N      c       100.60  56.20   1.20    3       0
+o      \N      \N      178.10  56.20   1.20    8       0
+o      2023-12-08      \N      20.00   10.50   9.50    2       0
+o      2023-12-09      \N      11.50   11.50   11.50   1       0
+o      2023-12-10      \N      46.00   33.50   12.50   2       0
+o      2023-12-11      \N      43.20   43.20   43.20   1       0
+o      2023-12-12      \N      57.40   56.20   1.20    2       0
+
+-- !query2_0_before --
+\N     \N      \N      7       3       1       178.10  56.20   1.20    8       0
+\N     \N      a       6       3       1       77.50   33.50   9.50    5       0
+\N     \N      c       6       3       1       100.60  56.20   1.20    3       0
+o      \N      \N      3       1       1       178.10  56.20   1.20    8       0
+o      2023-12-08      \N      1       0       0       20.00   10.50   9.50    
2       0
+o      2023-12-09      \N      1       0       0       11.50   11.50   11.50   
1       0
+o      2023-12-10      \N      1       0       0       46.00   33.50   12.50   
2       0
+o      2023-12-11      \N      1       0       0       43.20   43.20   43.20   
1       0
+o      2023-12-12      \N      1       0       0       57.40   56.20   1.20    
2       0
+
+-- !query2_0_after --
+\N     \N      \N      7       3       1       178.10  56.20   1.20    8       0
+\N     \N      a       6       3       1       77.50   33.50   9.50    5       0
+\N     \N      c       6       3       1       100.60  56.20   1.20    3       0
+o      \N      \N      3       1       1       178.10  56.20   1.20    8       0
+o      2023-12-08      \N      1       0       0       20.00   10.50   9.50    
2       0
+o      2023-12-09      \N      1       0       0       11.50   11.50   11.50   
1       0
+o      2023-12-10      \N      1       0       0       46.00   33.50   12.50   
2       0
+o      2023-12-11      \N      1       0       0       43.20   43.20   43.20   
1       0
+o      2023-12-12      \N      1       0       0       57.40   56.20   1.20    
2       0
+
+-- !query3_0_before --
+\N     \N      \N      43.20   43.20   43.20   1       0
+\N     3       \N      43.20   43.20   43.20   1       0
+3      \N      2023-12-11      43.20   43.20   43.20   1       0
+3      3       \N      43.20   43.20   43.20   1       0
+
+-- !query3_0_after --
+\N     \N      \N      43.20   43.20   43.20   1       0
+\N     3       \N      43.20   43.20   43.20   1       0
+3      \N      2023-12-11      43.20   43.20   43.20   1       0
+3      3       \N      43.20   43.20   43.20   1       0
+
+-- !query4_0_before --
+\N     \N      \N      1       1       3       7       43.20   43.20   43.20   
1       0
+\N     3       \N      0       1       2       5       43.20   43.20   43.20   
1       0
+3      \N      2023-12-11      1       0       1       2       43.20   43.20   
43.20   1       0
+3      3       \N      0       1       0       1       43.20   43.20   43.20   
1       0
+
+-- !query4_0_after --
+\N     \N      \N      1       1       3       7       43.20   43.20   43.20   
1       0
+\N     3       \N      0       1       2       5       43.20   43.20   43.20   
1       0
+3      \N      2023-12-11      1       0       1       2       43.20   43.20   
43.20   1       0
+3      3       \N      0       1       0       1       43.20   43.20   43.20   
1       0
+
+-- !query5_0_before --
+\N     \N      178.10  56.20   1.20    8       0
+\N     a       77.50   33.50   9.50    5       0
+\N     c       100.60  56.20   1.20    3       0
+o      \N      178.10  56.20   1.20    8       0
+o      a       77.50   33.50   9.50    5       0
+o      c       100.60  56.20   1.20    3       0
+
+-- !query5_0_after --
+\N     \N      178.10  56.20   1.20    8       0
+\N     a       77.50   33.50   9.50    5       0
+\N     c       100.60  56.20   1.20    3       0
+o      \N      178.10  56.20   1.20    8       0
+o      a       77.50   33.50   9.50    5       0
+o      c       100.60  56.20   1.20    3       0
+
+-- !query6_0_before --
+\N     \N      3       1       1       178.10  56.20   1.20    8       0
+\N     a       2       1       1       77.50   33.50   9.50    5       0
+\N     c       2       1       1       100.60  56.20   1.20    3       0
+o      \N      1       0       0       178.10  56.20   1.20    8       0
+o      a       0       0       0       77.50   33.50   9.50    5       0
+o      c       0       0       0       100.60  56.20   1.20    3       0
+
+-- !query6_0_after --
+\N     \N      3       1       1       178.10  56.20   1.20    8       0
+\N     a       2       1       1       77.50   33.50   9.50    5       0
+\N     c       2       1       1       100.60  56.20   1.20    3       0
+o      \N      1       0       0       178.10  56.20   1.20    8       0
+o      a       0       0       0       77.50   33.50   9.50    5       0
+o      c       0       0       0       100.60  56.20   1.20    3       0
+
+-- !query7_0_before --
+\N     \N      \N      43.20   43.20   43.20   1       0
+\N     \N      2023-12-11      43.20   43.20   43.20   1       0
+\N     3       \N      43.20   43.20   43.20   1       0
+\N     3       2023-12-11      43.20   43.20   43.20   1       0
+3      \N      \N      43.20   43.20   43.20   1       0
+3      \N      2023-12-11      43.20   43.20   43.20   1       0
+3      3       \N      43.20   43.20   43.20   1       0
+3      3       2023-12-11      43.20   43.20   43.20   1       0
+
+-- !query7_0_after --
+\N     \N      \N      43.20   43.20   43.20   1       0
+\N     \N      2023-12-11      43.20   43.20   43.20   1       0
+\N     3       \N      43.20   43.20   43.20   1       0
+\N     3       2023-12-11      43.20   43.20   43.20   1       0
+3      \N      \N      43.20   43.20   43.20   1       0
+3      \N      2023-12-11      43.20   43.20   43.20   1       0
+3      3       \N      43.20   43.20   43.20   1       0
+3      3       2023-12-11      43.20   43.20   43.20   1       0
+
+-- !query8_0_before --
+\N     \N      \N      1       1       3       7       43.20   43.20   43.20   
1       0
+\N     \N      2023-12-11      1       0       3       6       43.20   43.20   
43.20   1       0
+\N     3       \N      0       1       2       5       43.20   43.20   43.20   
1       0
+\N     3       2023-12-11      0       0       2       4       43.20   43.20   
43.20   1       0
+3      \N      \N      1       1       1       3       43.20   43.20   43.20   
1       0
+3      \N      2023-12-11      1       0       1       2       43.20   43.20   
43.20   1       0
+3      3       \N      0       1       0       1       43.20   43.20   43.20   
1       0
+3      3       2023-12-11      0       0       0       0       43.20   43.20   
43.20   1       0
+
+-- !query8_0_after --
+\N     \N      \N      1       1       3       7       43.20   43.20   43.20   
1       0
+\N     \N      2023-12-11      1       0       3       6       43.20   43.20   
43.20   1       0
+\N     3       \N      0       1       2       5       43.20   43.20   43.20   
1       0
+\N     3       2023-12-11      0       0       2       4       43.20   43.20   
43.20   1       0
+3      \N      \N      1       1       1       3       43.20   43.20   43.20   
1       0
+3      \N      2023-12-11      1       0       1       2       43.20   43.20   
43.20   1       0
+3      3       \N      0       1       0       1       43.20   43.20   43.20   
1       0
+3      3       2023-12-11      0       0       0       0       43.20   43.20   
43.20   1       0
+
+-- !query9_0_before --
+\N     \N      178.10  56.20   1.20    8       0
+o      \N      178.10  56.20   1.20    8       0
+o      a       77.50   33.50   9.50    5       0
+o      c       100.60  56.20   1.20    3       0
+
+-- !query9_0_after --
+\N     \N      178.10  56.20   1.20    8       0
+o      \N      178.10  56.20   1.20    8       0
+o      a       77.50   33.50   9.50    5       0
+o      c       100.60  56.20   1.20    3       0
+
+-- !query10_0_before --
+\N     \N      3       1       1       178.10  56.20   1.20    8       0
+o      \N      1       0       0       178.10  56.20   1.20    8       0
+o      a       0       0       0       77.50   33.50   9.50    5       0
+o      c       0       0       0       100.60  56.20   1.20    3       0
+
+-- !query10_0_after --
+\N     \N      3       1       1       178.10  56.20   1.20    8       0
+o      \N      1       0       0       178.10  56.20   1.20    8       0
+o      a       0       0       0       77.50   33.50   9.50    5       0
+o      c       0       0       0       100.60  56.20   1.20    3       0
+
+-- !query11_0_before --
+\N     \N      \N      43.20   43.20   43.20   1       0
+3      \N      \N      43.20   43.20   43.20   1       0
+3      3       \N      43.20   43.20   43.20   1       0
+3      3       2023-12-11      43.20   43.20   43.20   1       0
+
+-- !query11_0_after --
+\N     \N      \N      43.20   43.20   43.20   1       0
+3      \N      \N      43.20   43.20   43.20   1       0
+3      3       \N      43.20   43.20   43.20   1       0
+3      3       2023-12-11      43.20   43.20   43.20   1       0
+
+-- !query12_0_before --
+\N     \N      \N      1       1       3       7       43.20   43.20   43.20   
1       0
+3      \N      \N      1       1       1       3       43.20   43.20   43.20   
1       0
+3      3       \N      0       1       0       1       43.20   43.20   43.20   
1       0
+3      3       2023-12-11      0       0       0       0       43.20   43.20   
43.20   1       0
+
+-- !query12_0_after --
+\N     \N      \N      1       1       3       7       43.20   43.20   43.20   
1       0
+3      \N      \N      1       1       1       3       43.20   43.20   43.20   
1       0
+3      3       \N      0       1       0       1       43.20   43.20   43.20   
1       0
+3      3       2023-12-11      0       0       0       0       43.20   43.20   
43.20   1       0
+
+-- !query13_0_before --
+\N     \N      \N      178.10  56.20   1.20    8       0
+\N     \N      a       77.50   33.50   9.50    5       0
+\N     \N      c       100.60  56.20   1.20    3       0
+o      \N      \N      178.10  56.20   1.20    8       0
+o      2023-12-08      \N      20.00   10.50   9.50    2       0
+o      2023-12-09      \N      11.50   11.50   11.50   1       0
+o      2023-12-10      \N      46.00   33.50   12.50   2       0
+o      2023-12-11      \N      43.20   43.20   43.20   1       0
+o      2023-12-12      \N      57.40   56.20   1.20    2       0
+
+-- !query13_0_after --
+\N     \N      \N      178.10  56.20   1.20    8       0
+\N     \N      a       77.50   33.50   9.50    5       0
+\N     \N      c       100.60  56.20   1.20    3       0
+o      \N      \N      178.10  56.20   1.20    8       0
+o      2023-12-08      \N      20.00   10.50   9.50    2       0
+o      2023-12-09      \N      11.50   11.50   11.50   1       0
+o      2023-12-10      \N      46.00   33.50   12.50   2       0
+o      2023-12-11      \N      43.20   43.20   43.20   1       0
+o      2023-12-12      \N      57.40   56.20   1.20    2       0
+
+-- !query14_0_before --
+o      2023-12-08      a       20.00   10.50   9.50    2       \N
+o      2023-12-09      a       11.50   11.50   11.50   1       \N
+o      2023-12-10      a       46.00   33.50   12.50   2       \N
+o      2023-12-11      c       43.20   43.20   43.20   1       \N
+o      2023-12-12      c       57.40   56.20   1.20    2       \N
+
+-- !query14_0_after --
+o      2023-12-08      a       20.00   10.50   9.50    2       \N
+o      2023-12-09      a       11.50   11.50   11.50   1       \N
+o      2023-12-10      a       46.00   33.50   12.50   2       \N
+o      2023-12-11      c       43.20   43.20   43.20   1       \N
+o      2023-12-12      c       57.40   56.20   1.20    2       \N
+
diff --git 
a/regression-test/suites/nereids_rules_p0/mv/grouping_sets/grouping_sets.groovy 
b/regression-test/suites/nereids_rules_p0/mv/grouping_sets/grouping_sets.groovy
new file mode 100644
index 00000000000..73c30527990
--- /dev/null
+++ 
b/regression-test/suites/nereids_rules_p0/mv/grouping_sets/grouping_sets.groovy
@@ -0,0 +1,616 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+suite("materialized_view_grouping_sets") {
+    String db = context.config.getDbNameByFile(context.file)
+    sql "use ${db}"
+    sql "set runtime_filter_mode=OFF";
+    sql "SET ignore_shape_nodes='PhysicalDistribute,PhysicalProject'"
+    sql "SET enable_fallback_to_original_planner=false"
+    sql "SET enable_materialized_view_rewrite=true"
+    sql "SET enable_nereids_planner=true"
+
+
+    sql """
+    drop table if exists orders
+    """
+
+    sql """
+    CREATE TABLE IF NOT EXISTS orders  (
+      o_orderkey       INTEGER NOT NULL,
+      o_custkey        INTEGER NOT NULL,
+      o_orderstatus    CHAR(1) NOT NULL,
+      o_totalprice     DECIMALV3(15,2) NOT NULL,
+      o_orderdate      DATE NOT NULL,
+      o_orderpriority  CHAR(15) NOT NULL,  
+      o_clerk          CHAR(15) NOT NULL, 
+      o_shippriority   INTEGER NOT NULL,
+      O_COMMENT        VARCHAR(79) NOT NULL
+    )
+    DUPLICATE KEY(o_orderkey, o_custkey)
+    PARTITION BY RANGE(o_orderdate) (
+    PARTITION `day_2` VALUES LESS THAN ('2023-12-9'),
+    PARTITION `day_3` VALUES LESS THAN ("2023-12-11"),
+    PARTITION `day_4` VALUES LESS THAN ("2023-12-30")
+    )
+    DISTRIBUTED BY HASH(o_orderkey) BUCKETS 3
+    PROPERTIES (
+      "replication_num" = "1"
+    );
+    """
+
+    sql """
+    drop table if exists lineitem
+    """
+
+    sql """
+    CREATE TABLE IF NOT EXISTS lineitem (
+      l_orderkey    INTEGER NOT NULL,
+      l_partkey     INTEGER NOT NULL,
+      l_suppkey     INTEGER NOT NULL,
+      l_linenumber  INTEGER NOT NULL,
+      l_quantity    DECIMALV3(15,2) NOT NULL,
+      l_extendedprice  DECIMALV3(15,2) NOT NULL,
+      l_discount    DECIMALV3(15,2) NOT NULL,
+      l_tax         DECIMALV3(15,2) NOT NULL,
+      l_returnflag  CHAR(1) NOT NULL,
+      l_linestatus  CHAR(1) NOT NULL,
+      l_shipdate    DATE NOT NULL,
+      l_commitdate  DATE NOT NULL,
+      l_receiptdate DATE NOT NULL,
+      l_shipinstruct CHAR(25) NOT NULL,
+      l_shipmode     CHAR(10) NOT NULL,
+      l_comment      VARCHAR(44) NOT NULL
+    )
+    DUPLICATE KEY(l_orderkey, l_partkey, l_suppkey, l_linenumber)
+    PARTITION BY RANGE(l_shipdate) (
+    PARTITION `day_1` VALUES LESS THAN ('2023-12-9'),
+    PARTITION `day_2` VALUES LESS THAN ("2023-12-11"),
+    PARTITION `day_3` VALUES LESS THAN ("2023-12-30"))
+    DISTRIBUTED BY HASH(l_orderkey) BUCKETS 3
+    PROPERTIES (
+      "replication_num" = "1"
+    )
+    """
+
+    sql """
+    drop table if exists partsupp
+    """
+
+    sql """
+    CREATE TABLE IF NOT EXISTS partsupp (
+      ps_partkey     INTEGER NOT NULL,
+      ps_suppkey     INTEGER NOT NULL,
+      ps_availqty    INTEGER NOT NULL,
+      ps_supplycost  DECIMALV3(15,2)  NOT NULL,
+      ps_comment     VARCHAR(199) NOT NULL 
+    )
+    DUPLICATE KEY(ps_partkey, ps_suppkey)
+    DISTRIBUTED BY HASH(ps_partkey) BUCKETS 3
+    PROPERTIES (
+      "replication_num" = "1"
+    )
+    """
+
+    sql """ insert into lineitem values
+    (1, 2, 3, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-08', '2023-12-09', 
'2023-12-10', 'a', 'b', 'yyyyyyyyy'),
+    (2, 4, 3, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-09', '2023-12-09', 
'2023-12-10', 'a', 'b', 'yyyyyyyyy'),
+    (3, 2, 4, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-10', '2023-12-09', 
'2023-12-10', 'a', 'b', 'yyyyyyyyy'),
+    (4, 3, 3, 4, 5.5, 6.5, 7.5, 8.5, 'o', 'k', '2023-12-11', '2023-12-09', 
'2023-12-10', 'a', 'b', 'yyyyyyyyy'),
+    (5, 2, 3, 6, 7.5, 8.5, 9.5, 10.5, 'k', 'o', '2023-12-12', '2023-12-12', 
'2023-12-13', 'c', 'd', 'xxxxxxxxx');
+    """
+
+    sql """
+    insert into orders values
+    (1, 1, 'o', 9.5, '2023-12-08', 'a', 'b', 1, 'yy'),
+    (1, 1, 'o', 10.5, '2023-12-08', 'a', 'b', 1, 'yy'),
+    (2, 1, 'o', 11.5, '2023-12-09', 'a', 'b', 1, 'yy'),
+    (3, 1, 'o', 12.5, '2023-12-10', 'a', 'b', 1, 'yy'),
+    (3, 1, 'o', 33.5, '2023-12-10', 'a', 'b', 1, 'yy'),
+    (4, 2, 'o', 43.2, '2023-12-11', 'c','d',2, 'mm'),
+    (5, 2, 'o', 56.2, '2023-12-12', 'c','d',2, 'mi'),
+    (5, 2, 'o', 1.2, '2023-12-12', 'c','d',2, 'mi');  
+    """
+
+    sql """
+    insert into partsupp values
+    (2, 3, 9, 10.01, 'supply1'),
+    (2, 3, 10, 11.01, 'supply2');
+    """
+
+    // query has group sets, and mv doesn't
+    // single table grouping sets without grouping scalar function
+    def mv1_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from orders
+            group by
+            o_orderstatus, o_orderdate, o_orderpriority;
+            """
+    def query1_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from orders
+            group by
+            GROUPING SETS ((o_orderstatus, o_orderdate), (o_orderpriority), 
(o_orderstatus), ());
+            """
+    order_qt_query1_0_before "${query1_0}"
+    check_mv_rewrite_success(db, mv1_0, query1_0, "mv1_0")
+    order_qt_query1_0_after "${query1_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv1_0"""
+
+    // single table grouping sets with grouping scalar function
+    def mv2_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from orders
+            group by
+            o_orderstatus, o_orderdate, o_orderpriority;
+            """
+    def query2_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            grouping_id(o_orderstatus, o_orderdate, o_orderpriority),
+            grouping_id(o_orderstatus, o_orderdate),
+            grouping(o_orderdate),
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from orders
+            group by
+            GROUPING SETS ((o_orderstatus, o_orderdate), (o_orderpriority), 
(o_orderstatus), ());
+            """
+    order_qt_query2_0_before "${query2_0}"
+    check_mv_rewrite_success(db, mv2_0, query2_0, "mv2_0")
+    order_qt_query2_0_after "${query2_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv2_0"""
+
+
+    // multi table grouping sets without grouping scalar function
+    def mv3_0 =
+            """
+            select l_shipdate, o_orderdate, l_partkey, l_suppkey,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from lineitem
+            left join orders on lineitem.l_orderkey = orders.o_orderkey and 
l_shipdate = o_orderdate
+            group by
+            l_shipdate,
+            o_orderdate,
+            l_partkey,
+            l_suppkey;
+            """
+    def query3_0 =
+            """
+            select t1.l_partkey, t1.l_suppkey, o_orderdate,
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from (select * from lineitem where l_shipdate = '2023-12-11') t1
+            left join orders on t1.l_orderkey = orders.o_orderkey and 
t1.l_shipdate = o_orderdate
+            group by
+            GROUPING SETS ((l_shipdate, o_orderdate, l_partkey), (l_partkey, 
l_suppkey), (l_suppkey), ());
+            """
+    order_qt_query3_0_before "${query3_0}"
+    check_mv_rewrite_success(db, mv3_0, query3_0, "mv3_0")
+    order_qt_query3_0_after "${query3_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv3_0"""
+
+
+    // multi table grouping sets with grouping scalar function
+    def mv4_0 =
+            """
+            select l_shipdate, o_orderdate, l_partkey, l_suppkey,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from lineitem
+            left join orders on lineitem.l_orderkey = orders.o_orderkey and 
l_shipdate = o_orderdate
+            group by
+            l_shipdate,
+            o_orderdate,
+            l_partkey,
+            l_suppkey;
+            """
+    def query4_0 =
+            """
+            select t1.l_partkey, t1.l_suppkey, o_orderdate,
+            grouping(t1.l_suppkey),
+            grouping(o_orderdate),
+            grouping_id(t1.l_partkey, t1.l_suppkey),
+            grouping_id(t1.l_partkey, t1.l_suppkey, o_orderdate),
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from (select * from lineitem where l_shipdate = '2023-12-11') t1
+            left join orders on t1.l_orderkey = orders.o_orderkey and 
t1.l_shipdate = o_orderdate
+            group by
+            GROUPING SETS ((l_shipdate, o_orderdate, l_partkey), (l_partkey, 
l_suppkey), (l_suppkey), ());
+            """
+    order_qt_query4_0_before "${query4_0}"
+    check_mv_rewrite_success(db, mv4_0, query4_0, "mv4_0")
+    order_qt_query4_0_after "${query4_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv4_0"""
+
+
+    // single table cube without grouping scalar function
+    def mv5_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from orders
+            group by
+            o_orderstatus, o_orderdate, o_orderpriority;
+            """
+    def query5_0 =
+            """
+            select o_orderstatus, o_orderpriority,
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from orders
+            group by
+            CUBE (o_orderstatus, o_orderpriority);
+            """
+    order_qt_query5_0_before "${query5_0}"
+    check_mv_rewrite_success(db, mv5_0, query5_0, "mv5_0")
+    order_qt_query5_0_after "${query5_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv5_0"""
+
+    // single table cube with grouping scalar function
+    def mv6_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from orders
+            group by
+            o_orderstatus, o_orderdate, o_orderpriority;
+            """
+    def query6_0 =
+            """
+            select o_orderstatus, o_orderpriority,
+            grouping_id(o_orderstatus, o_orderpriority),
+            grouping_id(o_orderstatus),
+            grouping(o_orderstatus),
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from orders
+            group by
+            CUBE (o_orderstatus, o_orderpriority);
+            """
+    order_qt_query6_0_before "${query6_0}"
+    check_mv_rewrite_success(db, mv6_0, query6_0, "mv6_0")
+    order_qt_query6_0_after "${query6_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv6_0"""
+
+    // multi table cube without grouping scalar function
+    def mv7_0 =
+            """
+            select l_shipdate, o_orderdate, l_partkey, l_suppkey,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from lineitem
+            left join orders on lineitem.l_orderkey = orders.o_orderkey and 
l_shipdate = o_orderdate
+            group by
+            l_shipdate,
+            o_orderdate,
+            l_partkey,
+            l_suppkey;
+            """
+    def query7_0 =
+            """
+            select t1.l_partkey, t1.l_suppkey, o_orderdate,
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from (select * from lineitem where l_shipdate = '2023-12-11') t1
+            left join orders on t1.l_orderkey = orders.o_orderkey and 
t1.l_shipdate = o_orderdate
+            group by
+            CUBE (t1.l_partkey, t1.l_suppkey, o_orderdate);
+            """
+    order_qt_query7_0_before "${query7_0}"
+    check_mv_rewrite_success(db, mv7_0, query7_0, "mv7_0")
+    order_qt_query7_0_after "${query7_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv7_0"""
+
+    // multi table cube with grouping scalar function
+    def mv8_0 =
+            """
+            select l_shipdate, o_orderdate, l_partkey, l_suppkey,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from lineitem
+            left join orders on lineitem.l_orderkey = orders.o_orderkey and 
l_shipdate = o_orderdate
+            group by
+            l_shipdate,
+            o_orderdate,
+            l_partkey,
+            l_suppkey;
+            """
+    def query8_0 =
+            """
+            select t1.l_partkey, t1.l_suppkey, o_orderdate,
+            grouping(t1.l_suppkey),
+            grouping(o_orderdate),
+            grouping_id(t1.l_partkey, t1.l_suppkey),
+            grouping_id(t1.l_partkey, t1.l_suppkey, o_orderdate),
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from (select * from lineitem where l_shipdate = '2023-12-11') t1
+            left join orders on t1.l_orderkey = orders.o_orderkey and 
t1.l_shipdate = o_orderdate
+            group by
+            CUBE (t1.l_partkey, t1.l_suppkey, o_orderdate);
+            """
+    order_qt_query8_0_before "${query8_0}"
+    check_mv_rewrite_success(db, mv8_0, query8_0, "mv8_0")
+    order_qt_query8_0_after "${query8_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv8_0"""
+
+
+
+    // single table rollup without grouping scalar function
+    def mv9_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from orders
+            group by
+            o_orderstatus, o_orderdate, o_orderpriority;
+            """
+    def query9_0 =
+            """
+            select o_orderstatus, o_orderpriority,
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from orders
+            group by
+            ROLLUP (o_orderstatus, o_orderpriority);
+            """
+    order_qt_query9_0_before "${query9_0}"
+    check_mv_rewrite_success(db, mv9_0, query9_0, "mv9_0")
+    order_qt_query9_0_after "${query9_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv9_0"""
+
+    // single table rollup with grouping scalar function
+    def mv10_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from orders
+            group by
+            o_orderstatus, o_orderdate, o_orderpriority;
+            """
+    def query10_0 =
+            """
+            select o_orderstatus, o_orderpriority,
+            grouping_id(o_orderstatus, o_orderpriority),
+            grouping_id(o_orderstatus),
+            grouping(o_orderstatus),
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from orders
+            group by
+            ROLLUP (o_orderstatus, o_orderpriority);
+            """
+    order_qt_query10_0_before "${query10_0}"
+    check_mv_rewrite_success(db, mv10_0, query10_0, "mv10_0")
+    order_qt_query10_0_after "${query10_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv10_0"""
+
+    // multi table rollup without grouping scalar function
+    def mv11_0 =
+            """
+            select l_shipdate, o_orderdate, l_partkey, l_suppkey,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from lineitem
+            left join orders on lineitem.l_orderkey = orders.o_orderkey and 
l_shipdate = o_orderdate
+            group by
+            l_shipdate,
+            o_orderdate,
+            l_partkey,
+            l_suppkey;
+            """
+    def query11_0 =
+            """
+            select t1.l_partkey, t1.l_suppkey, o_orderdate,
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from (select * from lineitem where l_shipdate = '2023-12-11') t1
+            left join orders on t1.l_orderkey = orders.o_orderkey and 
t1.l_shipdate = o_orderdate
+            group by
+            ROLLUP (t1.l_partkey, t1.l_suppkey, o_orderdate);
+            """
+    order_qt_query11_0_before "${query11_0}"
+    check_mv_rewrite_success(db, mv11_0, query11_0, "mv11_0")
+    order_qt_query11_0_after "${query11_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv11_0"""
+
+    // multi table rollup with grouping scalar function
+    def mv12_0 =
+            """
+            select l_shipdate, o_orderdate, l_partkey, l_suppkey,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from lineitem
+            left join orders on lineitem.l_orderkey = orders.o_orderkey and 
l_shipdate = o_orderdate
+            group by
+            l_shipdate,
+            o_orderdate,
+            l_partkey,
+            l_suppkey;
+            """
+    def query12_0 =
+            """
+            select t1.l_partkey, t1.l_suppkey, o_orderdate,
+            grouping(t1.l_suppkey),
+            grouping(o_orderdate),
+            grouping_id(t1.l_partkey, t1.l_suppkey),
+            grouping_id(t1.l_partkey, t1.l_suppkey, o_orderdate),
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from (select * from lineitem where l_shipdate = '2023-12-11') t1
+            left join orders on t1.l_orderkey = orders.o_orderkey and 
t1.l_shipdate = o_orderdate
+            group by
+            ROLLUP (t1.l_partkey, t1.l_suppkey, o_orderdate);
+            """
+    order_qt_query12_0_before "${query12_0}"
+    check_mv_rewrite_success(db, mv12_0, query12_0, "mv12_0")
+    order_qt_query12_0_after "${query12_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv12_0"""
+
+
+    // both query and mv has group sets
+    // not support
+    def mv13_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from orders
+            group by
+            GROUPING SETS ((o_orderstatus, o_orderdate), (o_orderpriority), 
(o_orderstatus), ());
+            """
+    def query13_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from orders
+            group by
+            GROUPING SETS ((o_orderstatus, o_orderdate), (o_orderpriority), 
(o_orderstatus), ());
+            """
+    order_qt_query13_0_before "${query13_0}"
+    check_mv_rewrite_fail(db, mv13_0, query13_0, "mv13_0")
+    order_qt_query13_0_after "${query13_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv13_0"""
+
+
+    // mv has group sets, and query doesn't
+    // not support
+    def mv14_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            sum(o_totalprice),
+            max(o_totalprice),
+            min(o_totalprice),
+            count(*),
+            count(distinct case when o_shippriority > 1 and o_orderkey IN (1, 
3) then o_custkey else null end)
+            from orders
+            group by
+            GROUPING SETS ((o_orderstatus, o_orderdate), (o_orderpriority), 
(o_orderstatus), ());
+            """
+    def query14_0 =
+            """
+            select o_orderstatus, o_orderdate, o_orderpriority,
+            sum(o_totalprice) 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, 3) then o_custkey else null end)) as bitmap_union_basic
+            from orders
+            group by
+            o_orderstatus, o_orderdate, o_orderpriority;
+            """
+    order_qt_query14_0_before "${query14_0}"
+    check_mv_rewrite_fail(db, mv14_0, query14_0, "mv14_0")
+    order_qt_query14_0_after "${query14_0}"
+    sql """ DROP MATERIALIZED VIEW IF EXISTS mv14_0"""
+}
+


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

Reply via email to