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

lihaopeng 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 1514b5ab5c [Feature](Materialized-View) support advanced 
Materialized-View (#15212)
1514b5ab5c is described below

commit 1514b5ab5c8b05ff9ccb83c04f0e6be056e46f76
Author: Pxl <pxl...@qq.com>
AuthorDate: Mon Jan 9 09:53:11 2023 +0800

    [Feature](Materialized-View) support advanced Materialized-View (#15212)
---
 be/src/olap/schema_change.cpp                      |  42 +++--
 be/src/vec/exec/scan/new_olap_scanner.cpp          |   5 +
 .../doris/alter/MaterializedViewHandler.java       |  26 ++-
 .../java/org/apache/doris/alter/RollupJobV2.java   |  29 ++-
 .../java/org/apache/doris/analysis/Analyzer.java   |   2 +
 .../doris/analysis/CreateMaterializedViewStmt.java |  82 +++++----
 .../main/java/org/apache/doris/analysis/Expr.java  |  28 +++
 .../doris/analysis/MVColumnBitmapUnionPattern.java |   6 +-
 .../doris/analysis/MVColumnHLLUnionPattern.java    |   2 +-
 .../org/apache/doris/analysis/MVColumnItem.java    |  78 +++++++--
 .../java/org/apache/doris/analysis/SelectList.java |   9 +
 .../java/org/apache/doris/analysis/SelectStmt.java |  33 ++++
 .../java/org/apache/doris/analysis/SlotRef.java    |   4 +-
 .../main/java/org/apache/doris/catalog/Column.java |   9 +-
 .../doris/catalog/MaterializedIndexMeta.java       |  10 +-
 .../java/org/apache/doris/catalog/OlapTable.java   |   2 +-
 .../java/org/apache/doris/catalog/ScalarType.java  |  12 ++
 .../mv/AbstractSelectMaterializedIndexRule.java    |  11 +-
 .../mv/SelectMaterializedIndexWithAggregate.java   |   9 +-
 .../doris/planner/MaterializedViewSelector.java    | 195 ++++++++++++++++-----
 .../org/apache/doris/planner/OlapScanNode.java     |  11 +-
 .../doris/rewrite/mvrewrite/CountFieldToSum.java   |   2 +-
 .../doris/rewrite/mvrewrite/ExprToSlotRefRule.java | 125 +++++++++++++
 .../rewrite/mvrewrite/HLLHashToSlotRefRule.java    |   5 +-
 .../doris/rewrite/mvrewrite/SlotRefEqualRule.java  |   3 +-
 .../rewrite/mvrewrite/ToBitmapToSlotRefRule.java   |   4 +-
 .../doris/alter/MaterializedViewHandlerTest.java   |  74 ++++----
 .../analysis/CreateMaterializedViewStmtTest.java   |  99 +++++------
 .../java/org/apache/doris/analysis/ExprTest.java   |   4 -
 .../analysis/MVColumnBitmapUnionPatternTest.java   |   5 +
 .../analysis/MVColumnHLLUnionPatternTest.java      |   5 +
 .../planner/MaterializedViewFunctionTest.java      |   2 +-
 .../planner/MaterializedViewSelectorTest.java      |  67 +++++--
 .../apache/doris/statistics/MVStatisticsTest.java  |   7 -
 .../java/org/apache/doris/utframe/DorisAssert.java |   2 +-
 .../data/rollup/test_materialized_view_hll.out     | Bin 451 -> 450 bytes
 .../test_materialized_view_hll_with_light_sc.out   | Bin 466 -> 465 bytes
 .../data/rollup_p0/test_materialized_view.out      | Bin 1399 -> 1398 bytes
 .../test_agg_keys_schema_change.groovy             |   1 +
 39 files changed, 741 insertions(+), 269 deletions(-)

diff --git a/be/src/olap/schema_change.cpp b/be/src/olap/schema_change.cpp
index ab27b00289..3fed9c3c01 100644
--- a/be/src/olap/schema_change.cpp
+++ b/be/src/olap/schema_change.cpp
@@ -17,6 +17,7 @@
 
 #include "olap/schema_change.h"
 
+#include "common/config.h"
 #include "common/status.h"
 #include "gutil/integral_types.h"
 #include "olap/merger.h"
@@ -268,7 +269,8 @@ Status RowBlockChanger::change_block(vectorized::Block* 
ref_block,
                     << ", expect=" << row_size
                     << ", real=" << 
ref_block->get_by_position(result_column_id).column->size();
 
-            if (ctx->root()->node_type() == TExprNodeType::CAST_EXPR) {
+            if (ctx->root()->node_type() == TExprNodeType::CAST_EXPR ||
+                ctx->root()->node_type() == TExprNodeType::SLOT_REF) {
                 RETURN_IF_ERROR(
                         
_check_cast_valid(ref_block->get_by_position(ref_idx).column,
                                           
ref_block->get_by_position(result_column_id).column));
@@ -304,13 +306,6 @@ Status RowBlockChanger::change_block(vectorized::Block* 
ref_block,
                 auto* ref_nullable_col = 
assert_cast<vectorized::ColumnNullable*>(
                         std::move(*ref_col.column).mutate().get());
 
-                const auto* null_map = 
ref_nullable_col->get_null_map_column().get_data().data();
-
-                for (size_t i = 0; i < row_size; i++) {
-                    if (null_map[i]) {
-                        return Status::DataQualityError("is_null of data is 
changed!");
-                    }
-                }
                 ref_nullable_col->swap_nested_column(new_col.column);
             }
         } else {
@@ -318,7 +313,6 @@ Status RowBlockChanger::change_block(vectorized::Block* 
ref_block,
                     ref_block->get_by_position(it.first).column);
         }
     }
-
     return Status::OK();
 }
 
@@ -327,7 +321,19 @@ Status 
RowBlockChanger::_check_cast_valid(vectorized::ColumnPtr ref_column,
                                           vectorized::ColumnPtr new_column) 
const {
     if (ref_column->is_nullable() != new_column->is_nullable()) {
         if (ref_column->is_nullable()) {
-            return Status::DataQualityError("Can not change nullable column to 
not nullable");
+            auto* ref_null_map =
+                    
vectorized::check_and_get_column<vectorized::ColumnNullable>(ref_column)
+                            ->get_null_map_column()
+                            .get_data()
+                            .data();
+
+            bool is_changed = false;
+            for (size_t i = 0; i < ref_column->size(); i++) {
+                is_changed |= ref_null_map[i];
+            }
+            if (is_changed) {
+                return Status::DataQualityError("Null data is changed to not 
nullable");
+            }
         } else {
             auto* new_null_map =
                     
vectorized::check_and_get_column<vectorized::ColumnNullable>(new_column)
@@ -340,7 +346,7 @@ Status 
RowBlockChanger::_check_cast_valid(vectorized::ColumnPtr ref_column,
                 is_changed |= new_null_map[i];
             }
             if (is_changed) {
-                return Status::DataQualityError("is_null of data is changed!");
+                return Status::DataQualityError("Some data is changed to 
null");
             }
         }
     }
@@ -829,25 +835,17 @@ Status 
SchemaChangeHandler::_do_process_alter_tablet_v2(const TAlterTabletReqV2&
                     mv_param.origin_column_name = item.origin_column_name;
                 }
 
-                /*
-                * TODO(lhy)
-                * Building the materialized view function for schema_change 
here based on defineExpr.
-                * This is a trick because the current storage layer does not 
support expression evaluation.
-                * We can refactor this part of the code until the uniform 
expression evaluates the logic.
-                * count distinct materialized view will set mv_expr with 
to_bitmap or hll_hash.
-                * count materialized view will set mv_expr with count.
-                */
                 if (item.__isset.mv_expr) {
                     if (item.mv_expr.nodes[0].node_type == 
TExprNodeType::FUNCTION_CALL) {
                         mv_param.mv_expr = 
item.mv_expr.nodes[0].fn.name.function_name;
-                        if (!_supported_functions.count(mv_param.mv_expr)) {
+                        if (!config::enable_vectorized_alter_table &&
+                            !_supported_functions.count(mv_param.mv_expr)) {
                             return Status::NotSupported("Unknow materialized 
view expr " +
                                                         mv_param.mv_expr);
                         }
                     } else if (item.mv_expr.nodes[0].node_type == 
TExprNodeType::CASE_EXPR) {
                         mv_param.mv_expr = "count_field";
                     }
-
                     mv_param.expr = std::make_shared<TExpr>(item.mv_expr);
                 }
                 sc_params.materialized_params_map.insert(
@@ -1060,7 +1058,7 @@ Status 
SchemaChangeHandler::_convert_historical_rowsets(const SchemaChangeParams
             !res) {
             LOG(WARNING) << "failed to process the version."
                          << " version=" << rs_reader->version().first << "-"
-                         << rs_reader->version().second;
+                         << rs_reader->version().second << ", " << 
res.to_string();
             new_tablet->data_dir()->remove_pending_ids(ROWSET_ID_PREFIX +
                                                        
rowset_writer->rowset_id().to_string());
             return process_alter_exit();
diff --git a/be/src/vec/exec/scan/new_olap_scanner.cpp 
b/be/src/vec/exec/scan/new_olap_scanner.cpp
index 869809a7b7..4a52744c98 100644
--- a/be/src/vec/exec/scan/new_olap_scanner.cpp
+++ b/be/src/vec/exec/scan/new_olap_scanner.cpp
@@ -310,6 +310,11 @@ Status NewOlapScanner::_init_return_columns() {
                                 ? 
_tablet_schema->field_index(slot->col_unique_id())
                                 : 
_tablet_schema->field_index(slot->col_name());
 
+        if (index < 0) {
+            const std::string MATERIALIZED_VIEW_NAME_PREFIX = "mv_";
+            index = _tablet_schema->field_index(MATERIALIZED_VIEW_NAME_PREFIX 
+ slot->col_name());
+        }
+
         if (index < 0) {
             std::stringstream ss;
             ss << "field name is invalid. field=" << slot->col_name();
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/alter/MaterializedViewHandler.java 
b/fe/fe-core/src/main/java/org/apache/doris/alter/MaterializedViewHandler.java
index 5ea1b2a087..22a554083c 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/alter/MaterializedViewHandler.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/alter/MaterializedViewHandler.java
@@ -460,14 +460,20 @@ public class MaterializedViewHandler extends AlterHandler 
{
                         + " or unique table must has grouping columns");
             }
             for (MVColumnItem mvColumnItem : mvColumnItemList) {
-                String mvColumnName = mvColumnItem.getName();
+                if (mvColumnItem.getBaseColumnNames().size() != 1) {
+                    throw new DdlException(
+                            "mvColumnItem.getBaseColumnNames().size() != 1, 
mvColumnItem.getBaseColumnNames().size() = "
+                                    + 
mvColumnItem.getBaseColumnNames().size());
+                }
+
+                String mvColumnName = 
mvColumnItem.getBaseColumnNames().iterator().next();
                 Column baseColumn = olapTable.getColumn(mvColumnName);
                 if (mvColumnItem.isKey()) {
                     ++numOfKeys;
                 }
                 if (baseColumn == null) {
                     throw new DdlException("The mv column of agg or uniq table 
cannot be transformed "
-                            + "from original column[" + 
mvColumnItem.getBaseColumnName() + "]");
+                            + "from original column[" + String.join(",", 
mvColumnItem.getBaseColumnNames()) + "]");
                 }
                 Preconditions.checkNotNull(baseColumn, "Column[" + 
mvColumnName + "] does not exist");
                 AggregateType baseAggregationType = 
baseColumn.getAggregationType();
@@ -491,12 +497,20 @@ public class MaterializedViewHandler extends AlterHandler 
{
         } else {
             Set<String> partitionOrDistributedColumnName = 
olapTable.getPartitionColumnNames();
             
partitionOrDistributedColumnName.addAll(olapTable.getDistributionColumnNames());
+
             for (MVColumnItem mvColumnItem : mvColumnItemList) {
-                if 
(partitionOrDistributedColumnName.contains(mvColumnItem.getBaseColumnName().toLowerCase())
-                        && mvColumnItem.getAggregationType() != null) {
-                    throw new DdlException("The partition and distributed 
columns " + mvColumnItem.getBaseColumnName()
-                            + " must be key column in mv");
+                Set<String> names = mvColumnItem.getBaseColumnNames();
+                if (names == null) {
+                    throw new DdlException("Base columns is null");
                 }
+                for (String str : names) {
+                    if (partitionOrDistributedColumnName.contains(str)
+                            && mvColumnItem.getAggregationType() != null) {
+                        throw new DdlException("The partition and distributed 
columns " + str
+                                + " must be key column in mv");
+                    }
+                }
+
                 newMVColumns.add(mvColumnItem.toMVColumn(olapTable));
             }
         }
diff --git a/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java 
b/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java
index 3ca0ad2e0e..c600c15f6d 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java
@@ -80,9 +80,11 @@ import java.io.DataOutput;
 import java.io.IOException;
 import java.io.StringReader;
 import java.util.ArrayList;
+import java.util.HashSet;
 import java.util.List;
 import java.util.Map;
 import java.util.Map.Entry;
+import java.util.Set;
 import java.util.concurrent.TimeUnit;
 
 /**
@@ -377,7 +379,23 @@ public class RollupJobV2 extends AlterJobV2 implements 
GsonPostProcessable {
                     DescriptorTable descTable = new DescriptorTable();
                     TupleDescriptor destTupleDesc = 
descTable.createTupleDescriptor();
                     Map<String, SlotDescriptor> descMap = Maps.newHashMap();
-                    for (Column column : tbl.getFullSchema()) {
+
+                    List<Column> rollupColumns = new ArrayList<Column>();
+                    Set<String> columnNames = new HashSet<String>();
+                    for (Column column : tbl.getBaseSchema()) {
+                        rollupColumns.add(column);
+                        columnNames.add(column.getName());
+                    }
+
+                    for (Column column : rollupSchema) {
+                        if (columnNames.contains(column.getName())) {
+                            continue;
+                        }
+                        rollupColumns.add(column);
+                    }
+
+                    for (Column column : rollupColumns) {
+
                         SlotDescriptor destSlotDesc = 
descTable.addSlotDescriptor(destTupleDesc);
                         destSlotDesc.setIsMaterialized(true);
                         destSlotDesc.setColumn(column);
@@ -386,14 +404,19 @@ public class RollupJobV2 extends AlterJobV2 implements 
GsonPostProcessable {
                         descMap.put(column.getName(), destSlotDesc);
                     }
 
-                    for (Column column : tbl.getFullSchema()) {
+                    for (Column column : rollupColumns) {
                         if (column.getDefineExpr() != null) {
                             defineExprs.put(column.getName(), 
column.getDefineExpr());
 
                             List<SlotRef> slots = new ArrayList<>();
                             column.getDefineExpr().collect(SlotRef.class, 
slots);
                             Preconditions.checkArgument(slots.size() == 1);
-                            
slots.get(0).setDesc(descMap.get(slots.get(0).getColumnName()));
+                            SlotDescriptor slotDesc = 
descMap.get(slots.get(0).getColumnName());
+                            if (slotDesc == null) {
+                                slotDesc = descMap.get(column.getName());
+                            }
+                            Preconditions.checkArgument(slotDesc != null);
+                            slots.get(0).setDesc(slotDesc);
                         }
                     }
 
diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/Analyzer.java 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/Analyzer.java
index b3e21d6c38..c85b7a0eec 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/analysis/Analyzer.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/Analyzer.java
@@ -65,6 +65,7 @@ import 
org.apache.doris.rewrite.RoundLiteralInBinaryPredicatesRule;
 import org.apache.doris.rewrite.mvrewrite.CountDistinctToBitmap;
 import org.apache.doris.rewrite.mvrewrite.CountDistinctToBitmapOrHLLRule;
 import org.apache.doris.rewrite.mvrewrite.CountFieldToSum;
+import org.apache.doris.rewrite.mvrewrite.ExprToSlotRefRule;
 import org.apache.doris.rewrite.mvrewrite.HLLHashToSlotRefRule;
 import org.apache.doris.rewrite.mvrewrite.NDVToHll;
 import org.apache.doris.rewrite.mvrewrite.ToBitmapToSlotRefRule;
@@ -422,6 +423,7 @@ public class Analyzer {
             exprRewriter = new ExprRewriter(rules, onceRules);
             // init mv rewriter
             List<ExprRewriteRule> mvRewriteRules = Lists.newArrayList();
+            mvRewriteRules.add(ExprToSlotRefRule.INSTANCE);
             mvRewriteRules.add(ToBitmapToSlotRefRule.INSTANCE);
             mvRewriteRules.add(CountDistinctToBitmapOrHLLRule.INSTANCE);
             mvRewriteRules.add(CountDistinctToBitmap.INSTANCE);
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/analysis/CreateMaterializedViewStmt.java
 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/CreateMaterializedViewStmt.java
index 515e18aab1..7dbc0601b8 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/analysis/CreateMaterializedViewStmt.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/CreateMaterializedViewStmt.java
@@ -21,6 +21,7 @@ import org.apache.doris.catalog.AggregateType;
 import org.apache.doris.catalog.Column;
 import org.apache.doris.catalog.FunctionSet;
 import org.apache.doris.catalog.KeysType;
+import org.apache.doris.catalog.MaterializedIndexMeta;
 import org.apache.doris.catalog.PrimitiveType;
 import org.apache.doris.catalog.Type;
 import org.apache.doris.common.AnalysisException;
@@ -181,27 +182,18 @@ public class CreateMaterializedViewStmt extends DdlStmt {
         for (int i = 0; i < selectList.getItems().size(); i++) {
             SelectListItem selectListItem = selectList.getItems().get(i);
             Expr selectListItemExpr = selectListItem.getExpr();
-            if (!(selectListItemExpr instanceof SlotRef) && 
!(selectListItemExpr instanceof FunctionCallExpr)) {
+            if (!(selectListItemExpr instanceof SlotRef) && 
!(selectListItemExpr instanceof FunctionCallExpr)
+                    && !(selectListItemExpr instanceof ArithmeticExpr)) {
                 throw new AnalysisException("The materialized view only 
support the single column or function expr. "
                         + "Error column: " + selectListItemExpr.toSql());
             }
-            if (selectListItemExpr instanceof SlotRef) {
-                if (meetAggregate) {
-                    throw new AnalysisException("The aggregate column should 
be after the single column");
-                }
-                SlotRef slotRef = (SlotRef) selectListItemExpr;
-                // check duplicate column
-                String columnName = slotRef.getColumnName().toLowerCase();
-                if (!mvColumnNameSet.add(columnName)) {
-                    
ErrorReport.reportAnalysisException(ErrorCode.ERR_DUP_FIELDNAME, columnName);
-                }
-                MVColumnItem mvColumnItem = new MVColumnItem(columnName, 
slotRef.getType());
-                mvColumnItemList.add(mvColumnItem);
-            } else if (selectListItemExpr instanceof FunctionCallExpr) {
-                // Function must match pattern.
+
+            if (selectListItemExpr instanceof FunctionCallExpr
+                    && ((FunctionCallExpr) 
selectListItemExpr).isAggregateFunction()) {
                 FunctionCallExpr functionCallExpr = (FunctionCallExpr) 
selectListItemExpr;
                 String functionName = 
functionCallExpr.getFnName().getFunction();
-                // current version not support count(distinct) function in 
creating materialized view
+                // current version not support count(distinct) function in 
creating materialized
+                // view
                 if (!isReplay) {
                     MVColumnPattern mvColumnPattern = 
FN_NAME_TO_PATTERN.get(functionName.toLowerCase());
                     if (mvColumnPattern == null) {
@@ -228,8 +220,12 @@ public class CreateMaterializedViewStmt extends DdlStmt {
                 meetAggregate = true;
                 // build mv column item
                 mvColumnItemList.add(buildMVColumnItem(analyzer, 
functionCallExpr));
-                // TODO(ml): support REPLACE, REPLACE_IF_NOT_NULL, 
bitmap_union, hll_union only for aggregate table
-                // TODO(ml): support different type of column, int -> 
bigint(sum)
+            } else {
+                if (meetAggregate) {
+                    throw new AnalysisException("The aggregate column should 
be after the single column");
+                }
+                MVColumnItem mvColumnItem = new 
MVColumnItem(selectListItemExpr);
+                mvColumnItemList.add(mvColumnItem);
             }
         }
         // TODO(ml): only value columns of materialized view, such as select 
sum(v1) from table
@@ -270,9 +266,16 @@ public class CreateMaterializedViewStmt extends DdlStmt {
             }
             MVColumnItem mvColumnItem = mvColumnItemList.get(i);
             SlotRef slotRef = (SlotRef) orderByElement;
-            if 
(!mvColumnItem.getName().equalsIgnoreCase(slotRef.getColumnName())) {
+            if (mvColumnItem.getName() == null) {
+                throw new AnalysisException("mvColumnItem.getName() is null");
+            }
+            if (slotRef.getColumnName() == null) {
+                throw new AnalysisException("slotRef.getColumnName() is null");
+            }
+            if (!MaterializedIndexMeta.matchColumnName(mvColumnItem.getName(), 
slotRef.getColumnName())) {
                 throw new AnalysisException("The order of columns in order by 
clause must be same as "
-                        + "the order of columns in select list");
+                        + "the order of columns in select list, " + 
mvColumnItem.getName() + " vs "
+                        + slotRef.getColumnName());
             }
             Preconditions.checkState(mvColumnItem.getAggregationType() == 
null);
             mvColumnItem.setIsKey(true);
@@ -361,16 +364,18 @@ public class CreateMaterializedViewStmt extends DdlStmt {
         SlotRef baseColumnRef = slots.get(0);
         String baseColumnName = baseColumnRef.getColumnName().toLowerCase();
         Column baseColumn = baseColumnRef.getColumn();
-        Preconditions.checkNotNull(baseColumn);
+        if (baseColumn == null) {
+            throw new AnalysisException("baseColumn is null");
+        }
         Type baseType = baseColumn.getOriginType();
         Expr functionChild0 = functionCallExpr.getChild(0);
         String mvColumnName;
         AggregateType mvAggregateType;
-        Expr defineExpr = null;
+        Expr defineExpr = baseColumnRef;
         Type type;
         switch (functionName.toLowerCase()) {
             case "sum":
-                mvColumnName = baseColumnName;
+                mvColumnName = mvColumnBuilder(baseColumnName);
                 mvAggregateType = 
AggregateType.valueOf(functionName.toUpperCase());
                 PrimitiveType baseColumnType = 
baseColumnRef.getType().getPrimitiveType();
                 if (baseColumnType == PrimitiveType.TINYINT || baseColumnType 
== PrimitiveType.SMALLINT
@@ -381,17 +386,21 @@ public class CreateMaterializedViewStmt extends DdlStmt {
                 } else {
                     type = baseType;
                 }
+                if (type != baseType) {
+                    defineExpr = new CastExpr(type, baseColumnRef);
+                    defineExpr.analyze(analyzer);
+                }
                 break;
             case "min":
             case "max":
-                mvColumnName = baseColumnName;
+                mvColumnName = mvColumnBuilder(baseColumnName);
                 mvAggregateType = 
AggregateType.valueOf(functionName.toUpperCase());
                 type = baseType;
                 break;
             case FunctionSet.BITMAP_UNION:
                 // Compatible aggregation models
                 if (baseColumnRef.getType().getPrimitiveType() == 
PrimitiveType.BITMAP) {
-                    mvColumnName = baseColumnName;
+                    mvColumnName = mvColumnBuilder(baseColumnName);
                 } else {
                     mvColumnName = mvColumnBuilder(functionName, 
baseColumnName);
                     defineExpr = functionChild0;
@@ -402,7 +411,7 @@ public class CreateMaterializedViewStmt extends DdlStmt {
             case FunctionSet.HLL_UNION:
                 // Compatible aggregation models
                 if (baseColumnRef.getType().getPrimitiveType() == 
PrimitiveType.HLL) {
-                    mvColumnName = baseColumnName;
+                    mvColumnName = mvColumnBuilder(baseColumnName);
                 } else {
                     mvColumnName = mvColumnBuilder(functionName, 
baseColumnName);
                     defineExpr = functionChild0;
@@ -430,11 +439,9 @@ public class CreateMaterializedViewStmt extends DdlStmt {
         SelectList selectList = selectStmt.getSelectList();
         for (SelectListItem selectListItem : selectList.getItems()) {
             Expr selectListItemExpr = selectListItem.getExpr();
-            if (selectListItemExpr instanceof SlotRef) {
-                SlotRef slotRef = (SlotRef) selectListItemExpr;
-                result.put(slotRef.getColumnName(), null);
-            } else if (selectListItemExpr instanceof FunctionCallExpr) {
+            if (selectListItemExpr instanceof FunctionCallExpr) {
                 FunctionCallExpr functionCallExpr = (FunctionCallExpr) 
selectListItemExpr;
+
                 List<SlotRef> slots = new ArrayList<>();
                 functionCallExpr.collect(SlotRef.class, slots);
                 Preconditions.checkArgument(slots.size() == 1);
@@ -452,8 +459,8 @@ public class CreateMaterializedViewStmt extends DdlStmt {
                             CastExpr castExpr = new CastExpr(new 
TypeDef(Type.VARCHAR), baseSlotRef);
                             List<Expr> params = Lists.newArrayList();
                             params.add(castExpr);
-                            FunctionCallExpr defineExpr =
-                                    new 
FunctionCallExpr(FunctionSet.TO_BITMAP_WITH_CHECK, params);
+                            FunctionCallExpr defineExpr = new 
FunctionCallExpr(FunctionSet.TO_BITMAP_WITH_CHECK,
+                                    params);
                             result.put(mvColumnBuilder(functionName, 
baseColumnName), defineExpr);
                         } else {
                             result.put(baseColumnName, null);
@@ -473,14 +480,15 @@ public class CreateMaterializedViewStmt extends DdlStmt {
                     case FunctionSet.COUNT:
                         Expr defineExpr = new CaseExpr(null, 
Lists.newArrayList(
                                 new CaseWhenClause(new 
IsNullPredicate(slots.get(0), false),
-                                        new IntLiteral(0, Type.BIGINT))), new 
IntLiteral(1, Type.BIGINT));
+                                        new IntLiteral(0, Type.BIGINT))),
+                                new IntLiteral(1, Type.BIGINT));
                         result.put(mvColumnBuilder(functionName, 
baseColumnName), defineExpr);
                         break;
                     default:
-                        throw new AnalysisException("Unsupported function:" + 
functionName);
+                        result.put(mvColumnBuilder(functionCallExpr.toSql()), 
functionCallExpr);
                 }
             } else {
-                throw new AnalysisException("Unsupported select item:" + 
selectListItem.toSql());
+                result.put(mvColumnBuilder(selectListItemExpr.toSql()), 
selectListItemExpr);
             }
         }
         return result;
@@ -511,6 +519,10 @@ public class CreateMaterializedViewStmt extends DdlStmt {
                 .append(sourceColumnName).toString();
     }
 
+    public static String mvColumnBuilder(String name) {
+        return new 
StringBuilder().append(MATERIALIZED_VIEW_NAME_PREFIX).append(name).toString();
+    }
+
     @Override
     public String toSql() {
         return null;
diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/Expr.java 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/Expr.java
index 9c29c7ffee..35bd80e9a3 100755
--- a/fe/fe-core/src/main/java/org/apache/doris/analysis/Expr.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/Expr.java
@@ -1227,6 +1227,12 @@ public abstract class Expr extends TreeNode<Expr> 
implements ParseNode, Cloneabl
         return this;
     }
 
+    public Map<Long, Set<String>> getTableIdToColumnNames() {
+        Map<Long, Set<String>> tableIdToColumnNames = new HashMap<Long, 
Set<String>>();
+        getTableIdToColumnNames(tableIdToColumnNames);
+        return tableIdToColumnNames;
+    }
+
     public void getTableIdToColumnNames(Map<Long, Set<String>> 
tableIdToColumnNames) {
         Preconditions.checkState(tableIdToColumnNames != null);
         for (Expr child : children) {
@@ -2099,6 +2105,28 @@ public abstract class Expr extends TreeNode<Expr> 
implements ParseNode, Cloneabl
         }
     }
 
+    public boolean matchExprs(List<Expr> exprs) {
+        for (Expr expr : exprs) {
+            if (expr == null) {
+                continue;
+            }
+            if (expr.toSql().equals(toSql())) {
+                return true;
+            }
+        }
+
+        if (getChildren().isEmpty()) {
+            return false;
+        }
+
+        for (Expr expr : getChildren()) {
+            if (!expr.matchExprs(exprs)) {
+                return false;
+            }
+        }
+        return true;
+    }
+
     protected Type[] getActualArgTypes(Type[] originType) {
         return Arrays.stream(originType).map(
                 (Type type) -> {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnBitmapUnionPattern.java
 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnBitmapUnionPattern.java
index 25159fb252..a6a06e4729 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnBitmapUnionPattern.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnBitmapUnionPattern.java
@@ -19,6 +19,7 @@ package org.apache.doris.analysis;
 
 import org.apache.doris.catalog.FunctionSet;
 import org.apache.doris.catalog.PrimitiveType;
+import org.apache.doris.catalog.Type;
 
 public class MVColumnBitmapUnionPattern implements MVColumnPattern {
 
@@ -44,14 +45,13 @@ public class MVColumnBitmapUnionPattern implements 
MVColumnPattern {
             }
         } else if (fnExpr.getChild(0) instanceof FunctionCallExpr) {
             FunctionCallExpr child0FnExpr = (FunctionCallExpr) 
fnExpr.getChild(0);
-            if 
(!child0FnExpr.getFnName().getFunction().equalsIgnoreCase(FunctionSet.TO_BITMAP)
-                    && 
!child0FnExpr.getFnName().getFunction().equalsIgnoreCase(FunctionSet.TO_BITMAP_WITH_CHECK))
 {
+            if (!child0FnExpr.getType().equals(Type.BITMAP)) {
                 return false;
             }
             SlotRef slotRef = child0FnExpr.getChild(0).unwrapSlotRef();
             if (slotRef == null) {
                 return false;
-            } else if (slotRef.getType().isIntegerType()) {
+            } else if (slotRef.getType().isIntegerType() || 
slotRef.getType().isStringType()) {
                 return true;
             }
             return false;
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnHLLUnionPattern.java
 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnHLLUnionPattern.java
index 75d59a7b1f..1ea2e12153 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnHLLUnionPattern.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnHLLUnionPattern.java
@@ -44,7 +44,7 @@ public class MVColumnHLLUnionPattern implements 
MVColumnPattern {
             }
         } else if (fnExpr.getChild(0) instanceof FunctionCallExpr) {
             FunctionCallExpr child0FnExpr = (FunctionCallExpr) 
fnExpr.getChild(0);
-            if 
(!child0FnExpr.getFnName().getFunction().equalsIgnoreCase(FunctionSet.HLL_HASH))
 {
+            if (!child0FnExpr.getType().equals(Type.HLL)) {
                 return false;
             }
             SlotRef slotRef = child0FnExpr.getChild(0).unwrapSlotRef();
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnItem.java 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnItem.java
index b87a245c03..e2a1210432 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnItem.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/MVColumnItem.java
@@ -17,13 +17,19 @@
 
 package org.apache.doris.analysis;
 
-import org.apache.doris.analysis.ColumnDef.DefaultValue;
 import org.apache.doris.catalog.AggregateType;
 import org.apache.doris.catalog.Column;
+import org.apache.doris.catalog.MaterializedIndexMeta;
 import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.ScalarType;
 import org.apache.doris.catalog.Type;
+import org.apache.doris.common.AnalysisException;
 import org.apache.doris.common.DdlException;
 
+import java.util.HashSet;
+import java.util.Map;
+import java.util.Set;
+
 /**
  * This is a result of semantic analysis for AddMaterializedViewClause.
  * It is used to construct real mv column in MaterializedViewHandler.
@@ -38,7 +44,7 @@ public class MVColumnItem {
     private AggregateType aggregationType;
     private boolean isAggregationTypeImplicit;
     private Expr defineExpr;
-    private String baseColumnName;
+    private Set<String> baseColumnNames;
     private String baseTableName;
 
     public MVColumnItem(String name, Type type, AggregateType aggregateType, 
boolean isAggregationTypeImplicit,
@@ -53,14 +59,48 @@ public class MVColumnItem {
         this.aggregationType = aggregateType;
         this.isAggregationTypeImplicit = isAggregationTypeImplicit;
         this.defineExpr = defineExpr;
-        this.baseColumnName = baseColumnName;
+        baseColumnNames = new HashSet<>();
+        baseColumnNames.add(baseColumnName);
         this.baseTableName = baseTableName;
     }
 
     public MVColumnItem(String name, Type type) {
         this.name = name;
         this.type = type;
-        this.baseColumnName = name;
+        baseColumnNames = new HashSet<>();
+        baseColumnNames.add(name);
+    }
+
+    public MVColumnItem(Expr defineExpr) throws AnalysisException {
+        if (defineExpr instanceof SlotRef) {
+            this.name = defineExpr.toSql();
+        } else {
+            this.name = 
CreateMaterializedViewStmt.mvColumnBuilder(defineExpr.toSql());
+        }
+
+        if (this.name == null) {
+            throw new AnalysisException("defineExpr.toSql() is null");
+        }
+
+        this.name = MaterializedIndexMeta.normalizeName(this.name);
+
+        this.defineExpr = defineExpr;
+
+        this.type = defineExpr.getType();
+        if (this.type instanceof ScalarType && this.type.isStringType()) {
+            ((ScalarType) this.type).setMaxLength();
+        }
+
+        Map<Long, Set<String>> tableIdToColumnNames = 
defineExpr.getTableIdToColumnNames();
+
+        if (defineExpr instanceof SlotRef) {
+            baseColumnNames = new HashSet<>();
+            baseColumnNames.add(this.name);
+        } else if (tableIdToColumnNames.size() == 1) {
+            for (Map.Entry<Long, Set<String>> entry : 
tableIdToColumnNames.entrySet()) {
+                baseColumnNames = entry.getValue();
+            }
+        }
     }
 
     public String getName() {
@@ -104,8 +144,8 @@ public class MVColumnItem {
         this.defineExpr = defineExpr;
     }
 
-    public String getBaseColumnName() {
-        return baseColumnName;
+    public Set<String> getBaseColumnNames() {
+        return baseColumnNames;
     }
 
     public String getBaseTableName() {
@@ -113,20 +153,34 @@ public class MVColumnItem {
     }
 
     public Column toMVColumn(OlapTable olapTable) throws DdlException {
+        Column baseColumn = olapTable.getBaseColumn(name);
         Column result;
-        if (defineExpr != null) {
-            result = new Column(name, type, isKey, aggregationType, 
DefaultValue.ZERO, "");
-            result.setDefineExpr(defineExpr);
-        } else {
-            Column baseColumn = olapTable.getBaseColumn(baseColumnName);
+        if (baseColumn == null && defineExpr == null) {
+            // Some mtmv column have name diffrent with base column
+            baseColumn = 
olapTable.getBaseColumn(baseColumnNames.iterator().next());
+        }
+        if (baseColumn != null) {
             result = new Column(baseColumn);
+            if (result.getType() == null) {
+                throw new DdlException("base column's type is null");
+            }
             result.setName(name);
             result.setIsKey(isKey);
-            // If the mv column type is inconsistent with the base column 
type, the daily test will core.
+            // If the mv column type is inconsistent with the base column 
type, the daily
+            // test will core.
             // So, I comment this line firstly.
             // result.setType(type);
             result.setAggregationType(aggregationType, 
isAggregationTypeImplicit);
+        } else {
+            if (type == null) {
+                throw new DdlException("MVColumnItem type is null");
+            }
+            result = new Column(name, type, isKey, aggregationType, null, "");
+            if (defineExpr != null) {
+                result.setIsAllowNull(defineExpr.isNullable());
+            }
         }
+        result.setDefineExpr(defineExpr);
         return result;
     }
 
diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectList.java 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectList.java
index e7da518af8..59297ca1eb 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectList.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectList.java
@@ -26,6 +26,7 @@ import org.apache.doris.rewrite.ExprRewriter;
 import com.google.common.base.Predicates;
 import com.google.common.collect.Lists;
 
+import java.util.ArrayList;
 import java.util.List;
 import java.util.Map;
 
@@ -57,6 +58,14 @@ public class SelectList {
         isExcept = other.isExcept;
     }
 
+    public List<Expr> getExprs() {
+        List<Expr> exprs = new ArrayList<Expr>();
+        for (SelectListItem item : items) {
+            exprs.add(item.getExpr());
+        }
+        return exprs;
+    }
+
     public SelectList() {
         items = Lists.newArrayList();
         this.isDistinct = false;
diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java
index eac2cd4c26..acf57f28b2 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java
@@ -197,6 +197,32 @@ public class SelectStmt extends QueryStmt {
         groupingInfo = null;
     }
 
+    public List<Expr> getAllExprs() {
+        List<Expr> exprs = new ArrayList<Expr>();
+        if (originSelectList != null) {
+            exprs.addAll(originSelectList.getExprs());
+        }
+        if (havingClause != null) {
+            exprs.add(havingClause);
+        }
+        if (havingPred != null) {
+            exprs.add(havingPred);
+        }
+        if (havingClauseAfterAnaylzed != null) {
+            exprs.add(havingClauseAfterAnaylzed);
+        }
+        return exprs;
+    }
+
+    public boolean haveStar() {
+        for (SelectListItem item : selectList.getItems()) {
+            if (item.isStar()) {
+                return true;
+            }
+        }
+        return false;
+    }
+
     @Override
     public void resetSelectList() {
         if (originSelectList != null) {
@@ -1152,6 +1178,10 @@ public class SelectStmt extends QueryStmt {
                 }
             }
 
+            for (int i = 0; i < groupingExprs.size(); i++) {
+                groupingExprs.set(i, 
rewriteQueryExprByMvColumnExpr(groupingExprs.get(i), analyzer));
+            }
+
             if (groupingInfo != null) {
                 groupingInfo.genOutputTupleDescAndSMap(analyzer, 
groupingExprs, aggExprs);
                 // must do it before copying for createAggInfo()
@@ -1365,6 +1395,9 @@ public class SelectStmt extends QueryStmt {
             ArrayList<FunctionCallExpr> aggExprs,
             Analyzer analyzer)
             throws AnalysisException {
+        for (int i = 0; i < aggExprs.size(); i++) {
+            aggExprs.set(i, (FunctionCallExpr) 
rewriteQueryExprByMvColumnExpr(aggExprs.get(i), analyzer));
+        }
         if (selectList.isDistinct()) {
             // Create aggInfo for SELECT DISTINCT ... stmt:
             // - all select list items turn into grouping exprs
diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/SlotRef.java 
b/fe/fe-core/src/main/java/org/apache/doris/analysis/SlotRef.java
index 3a93298067..44197b3b5b 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/analysis/SlotRef.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/SlotRef.java
@@ -402,10 +402,10 @@ public class SlotRef extends Expr {
 
     @Override
     public void getTableIdToColumnNames(Map<Long, Set<String>> 
tableIdToColumnNames) {
-        Preconditions.checkState(desc != null);
-        if (!desc.isMaterialized()) {
+        if (desc == null) {
             return;
         }
+
         if (col == null) {
             for (Expr expr : desc.getSourceExprs()) {
                 expr.getTableIdToColumnNames(tableIdToColumnNames);
diff --git a/fe/fe-core/src/main/java/org/apache/doris/catalog/Column.java 
b/fe/fe-core/src/main/java/org/apache/doris/catalog/Column.java
index cd7cde3cbc..ae96365332 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/catalog/Column.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/catalog/Column.java
@@ -18,6 +18,7 @@
 package org.apache.doris.catalog;
 
 import org.apache.doris.alter.SchemaChangeHandler;
+import org.apache.doris.analysis.CreateMaterializedViewStmt;
 import org.apache.doris.analysis.DefaultValueExprDef;
 import org.apache.doris.analysis.Expr;
 import org.apache.doris.analysis.IndexDef;
@@ -205,11 +206,15 @@ public class Column implements Writable, 
GsonPostProcessable {
         return this.name;
     }
 
+    public String getNameWithoutMvPrefix() {
+        return 
this.getNameWithoutPrefix(CreateMaterializedViewStmt.MATERIALIZED_VIEW_NAME_PREFIX);
+    }
+
     public String getDisplayName() {
         if (defineExpr == null) {
-            return name;
+            return getNameWithoutMvPrefix();
         } else {
-            return defineExpr.toSql();
+            return MaterializedIndexMeta.normalizeName(defineExpr.toSql());
         }
     }
 
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/catalog/MaterializedIndexMeta.java 
b/fe/fe-core/src/main/java/org/apache/doris/catalog/MaterializedIndexMeta.java
index 20e40ff11f..77cfa9b088 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/catalog/MaterializedIndexMeta.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/catalog/MaterializedIndexMeta.java
@@ -147,9 +147,17 @@ public class MaterializedIndexMeta implements Writable, 
GsonPostProcessable {
         }
     }
 
+    public static String normalizeName(String name) {
+        return name.replace("`", "");
+    }
+
+    public static boolean matchColumnName(String lhs, String rhs) {
+        return normalizeName(lhs).equalsIgnoreCase(normalizeName(rhs));
+    }
+
     public Column getColumnByName(String columnName) {
         for (Column column : schema) {
-            if (column.getName().equalsIgnoreCase(columnName)) {
+            if (matchColumnName(column.getName(), columnName)) {
                 return column;
             }
         }
diff --git a/fe/fe-core/src/main/java/org/apache/doris/catalog/OlapTable.java 
b/fe/fe-core/src/main/java/org/apache/doris/catalog/OlapTable.java
index db628618d6..b9caa3aec1 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/catalog/OlapTable.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/catalog/OlapTable.java
@@ -403,7 +403,7 @@ public class OlapTable extends Table {
     public Column getVisibleColumn(String columnName) {
         for (MaterializedIndexMeta meta : getVisibleIndexIdToMeta().values()) {
             for (Column column : meta.getSchema()) {
-                if (column.getName().equalsIgnoreCase(columnName)) {
+                if (MaterializedIndexMeta.matchColumnName(column.getName(), 
columnName)) {
                     return column;
                 }
             }
diff --git a/fe/fe-core/src/main/java/org/apache/doris/catalog/ScalarType.java 
b/fe/fe-core/src/main/java/org/apache/doris/catalog/ScalarType.java
index ab1be2c4c9..466f94c63f 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/catalog/ScalarType.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/catalog/ScalarType.java
@@ -708,6 +708,18 @@ public class ScalarType extends Type {
         this.len = len;
     }
 
+    public void setMaxLength() {
+        if (type == PrimitiveType.CHAR) {
+            this.len = MAX_CHAR_LENGTH;
+        }
+        if (type == PrimitiveType.VARCHAR) {
+            this.len = MAX_VARCHAR_LENGTH;
+        }
+        if (type == PrimitiveType.STRING) {
+            this.len = MAX_STRING_LENGTH;
+        }
+    }
+
     public boolean isLengthSet() {
         return getPrimitiveType() == PrimitiveType.HLL || len > 0 || 
!Strings.isNullOrEmpty(lenStr);
     }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/mv/AbstractSelectMaterializedIndexRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/mv/AbstractSelectMaterializedIndexRule.java
index 8d26927a89..48215db57d 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/mv/AbstractSelectMaterializedIndexRule.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/mv/AbstractSelectMaterializedIndexRule.java
@@ -79,9 +79,14 @@ public abstract class AbstractSelectMaterializedIndexRule {
                 .map(slot -> exprIdToName.get(slot.getExprId()))
                 .collect(Collectors.toSet());
 
-        return table.getSchemaByIndexId(index.getId(), true).stream()
-                .map(Column::getName)
-                .collect(Collectors.toSet())
+        Set<String> nameMap = table.getSchemaByIndexId(index.getId(), 
true).stream()
+                .map(Column::getNameWithoutMvPrefix)
+                .collect(Collectors.toSet());
+
+        table.getSchemaByIndexId(index.getId(), true).stream()
+                .forEach(column -> nameMap.add(column.getName()));
+
+        return nameMap
                 .containsAll(requiredColumnNames);
     }
 
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/mv/SelectMaterializedIndexWithAggregate.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/mv/SelectMaterializedIndexWithAggregate.java
index a193b63901..07a67fb631 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/mv/SelectMaterializedIndexWithAggregate.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/mv/SelectMaterializedIndexWithAggregate.java
@@ -700,7 +700,11 @@ public class SelectMaterializedIndexWithAggregate extends 
AbstractSelectMaterial
 
         @Override
         public PreAggStatus visitBitmapUnionCount(BitmapUnionCount 
bitmapUnionCount, CheckContext context) {
-            Optional<Slot> slotOpt = 
ExpressionUtils.extractSlotOrCastOnSlot(bitmapUnionCount.child());
+            Expression expr = bitmapUnionCount.child();
+            if (expr instanceof ToBitmap) {
+                expr = expr.child(0);
+            }
+            Optional<Slot> slotOpt = 
ExpressionUtils.extractSlotOrCastOnSlot(expr);
             if (slotOpt.isPresent() && 
context.exprIdToValueColumn.containsKey(slotOpt.get().getExprId())) {
                 return PreAggStatus.on();
             } else {
@@ -769,8 +773,7 @@ public class SelectMaterializedIndexWithAggregate extends 
AbstractSelectMaterial
                     .stream()
                     .collect(Collectors.groupingBy(
                             Column::isKey,
-                            Collectors.toMap(Column::getName, 
Function.identity())
-                    ));
+                            Collectors.toMap(Column::getNameWithoutMvPrefix, 
Function.identity())));
             Map<String, Column> keyNameToColumn = 
nameToColumnGroupingByIsKey.get(true);
             Map<String, Column> valueNameToColumn = 
nameToColumnGroupingByIsKey.getOrDefault(false, ImmutableMap.of());
             Map<String, ExprId> nameToExprId = Stream.concat(
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/planner/MaterializedViewSelector.java
 
b/fe/fe-core/src/main/java/org/apache/doris/planner/MaterializedViewSelector.java
index a3e7f2b0f8..e611ea5b71 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/planner/MaterializedViewSelector.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/planner/MaterializedViewSelector.java
@@ -18,6 +18,7 @@
 package org.apache.doris.planner;
 
 import org.apache.doris.analysis.Analyzer;
+import org.apache.doris.analysis.CreateMaterializedViewStmt;
 import org.apache.doris.analysis.Expr;
 import org.apache.doris.analysis.FunctionCallExpr;
 import org.apache.doris.analysis.SelectStmt;
@@ -43,6 +44,7 @@ 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;
 import java.util.Iterator;
 import java.util.List;
@@ -268,10 +270,12 @@ public class MaterializedViewSelector {
         return selectedIndexId;
     }
 
-    // Step2: check all columns in compensating predicates are available in 
the view output
-    private void checkCompensatingPredicates(Set<String> columnsInPredicates, 
Map<Long, MaterializedIndexMeta>
-            candidateIndexIdToMeta) {
-        // When the query statement does not contain any columns in 
predicates, all candidate index can pass this check
+    // Step2: check all columns in compensating predicates are available in 
the view
+    // output
+    private void checkCompensatingPredicates(Set<String> columnsInPredicates,
+            Map<Long, MaterializedIndexMeta> candidateIndexIdToMeta) throws 
AnalysisException {
+        // When the query statement does not contain any columns in 
predicates, all
+        // candidate index can pass this check
         if (columnsInPredicates == null) {
             return;
         }
@@ -279,14 +283,42 @@ public class MaterializedViewSelector {
         while (iterator.hasNext()) {
             Map.Entry<Long, MaterializedIndexMeta> entry = iterator.next();
             Set<String> indexNonAggregatedColumnNames = new 
TreeSet<>(String.CASE_INSENSITIVE_ORDER);
+            List<Column> indexColumn = Lists.newArrayList();
             entry.getValue().getSchema().stream().filter(column -> 
!column.isAggregated())
-                    .forEach(column -> 
indexNonAggregatedColumnNames.add(column.getName()));
-            if 
(!indexNonAggregatedColumnNames.containsAll(columnsInPredicates)) {
+                    .forEach(column -> indexColumn.add(column));
+
+            indexColumn.forEach(column -> indexNonAggregatedColumnNames
+                    
.add(MaterializedIndexMeta.normalizeName(column.getName())));
+            List<Expr> indexExprs = new ArrayList<Expr>();
+            indexColumn
+                    .forEach(column -> indexExprs.add(column.getDefineExpr()));
+
+            Set<String> indexColumnNames = new 
TreeSet<>(String.CASE_INSENSITIVE_ORDER);
+            indexColumn
+                    .forEach(column -> 
indexColumnNames.add(MaterializedIndexMeta.normalizeName(column.getName())));
+
+            List<Expr> predicateExprs = Lists.newArrayList();
+            if (selectStmt.getWhereClause() != null) {
+                predicateExprs.add(selectStmt.getWhereClause());
+            }
+
+            for (TableRef tableRef : selectStmt.getTableRefs()) {
+                if (tableRef.getOnClause() == null) {
+                    continue;
+                }
+                predicateExprs.add(tableRef.getOnClause());
+            }
+
+            if 
(indexNonAggregatedColumnNames.containsAll(columnsInPredicates)) {
+                continue;
+            }
+
+            if (selectStmt.haveStar() || !matchAllExpr(predicateExprs, 
indexColumnNames, indexExprs)) {
                 iterator.remove();
             }
         }
         LOG.debug("Those mv pass the test of compensating predicates:"
-                          + 
Joiner.on(",").join(candidateIndexIdToMeta.keySet()));
+                + Joiner.on(",").join(candidateIndexIdToMeta.keySet()));
     }
 
     /**
@@ -299,42 +331,48 @@ public class MaterializedViewSelector {
      *
      * @param columnsInGrouping
      * @param candidateIndexIdToMeta
+     * @throws AnalysisException
      */
 
-    // Step3: group by list in query is the subset of group by list in view or 
view contains no aggregation
-    private void checkGrouping(OlapTable table, Set<String> columnsInGrouping, 
Map<Long, MaterializedIndexMeta>
-            candidateIndexIdToMeta) {
+    // Step3: group by list in query is the subset of group by list in view or 
view
+    // contains no aggregation
+    private void checkGrouping(OlapTable table, Set<String> columnsInGrouping,
+            Map<Long, MaterializedIndexMeta> candidateIndexIdToMeta) throws 
AnalysisException {
         Iterator<Map.Entry<Long, MaterializedIndexMeta>> iterator = 
candidateIndexIdToMeta.entrySet().iterator();
         while (iterator.hasNext()) {
             Map.Entry<Long, MaterializedIndexMeta> entry = iterator.next();
             Set<String> indexNonAggregatedColumnNames = new 
TreeSet<>(String.CASE_INSENSITIVE_ORDER);
             MaterializedIndexMeta candidateIndexMeta = entry.getValue();
-            List<Column> candidateIndexSchema = candidateIndexMeta.getSchema();
-            candidateIndexSchema.stream().filter(column -> 
!column.isAggregated())
-                    .forEach(column -> 
indexNonAggregatedColumnNames.add(column.getName()));
+            List<Column> candidateIndexSchema = Lists.newArrayList();
+            entry.getValue().getSchema().stream().filter(column -> 
!column.isAggregated())
+                    .forEach(column -> candidateIndexSchema.add(column));
+            candidateIndexSchema
+                    .forEach(column -> indexNonAggregatedColumnNames
+                            
.add(MaterializedIndexMeta.normalizeName(column.getName())));
             /*
-            If there is no aggregated column in duplicate index, the index 
will be SPJ.
-            For example:
-                duplicate table (k1, k2, v1)
-                duplicate mv index (k1, v1)
-            When the candidate index is SPJ type, it passes the verification 
directly
-
-            If there is no aggregated column in aggregate index, the index 
will be deduplicate index.
-            For example:
-                duplicate table (k1, k2, v1 sum)
-                aggregate mv index (k1, k2)
-            This kind of index is SPJG which same as select k1, k2 from 
aggregate_table group by k1, k2.
-            It also need to check the grouping column using following steps.
-
-            ISSUE-3016, MaterializedViewFunctionTest: testDeduplicateQueryInAgg
+             * If there is no aggregated column in duplicate index, the index 
will be SPJ.
+             * For example:
+             * duplicate table (k1, k2, v1)
+             * duplicate mv index (k1, v1)
+             * When the candidate index is SPJ type, it passes the 
verification directly
+             * If there is no aggregated column in aggregate index, the index 
will be
+             * deduplicate index.
+             * For example:
+             * duplicate table (k1, k2, v1 sum)
+             * aggregate mv index (k1, k2)
+             * This kind of index is SPJG which same as select k1, k2 from 
aggregate_table
+             * group by k1, k2.
+             * It also need to check the grouping column using following steps.
+             * ISSUE-3016, MaterializedViewFunctionTest: 
testDeduplicateQueryInAgg
              */
             boolean noNeedAggregation = candidateIndexMeta.getKeysType() == 
KeysType.DUP_KEYS
                     || (candidateIndexMeta.getKeysType() == 
KeysType.UNIQUE_KEYS
-                    && 
table.getTableProperty().getEnableUniqueKeyMergeOnWrite());
+                            && 
table.getTableProperty().getEnableUniqueKeyMergeOnWrite());
             if (indexNonAggregatedColumnNames.size() == 
candidateIndexSchema.size() && noNeedAggregation) {
                 continue;
             }
-            // When the query is SPJ type but the candidate index is SPJG 
type, it will not pass directly.
+            // When the query is SPJ type but the candidate index is SPJG 
type, it will not
+            // pass directly.
             if (isSPJQuery || disableSPJGView) {
                 iterator.remove();
                 continue;
@@ -344,13 +382,27 @@ public class MaterializedViewSelector {
             if (columnsInGrouping == null) {
                 continue;
             }
+
+            Set<String> indexColumnNames = new 
TreeSet<>(String.CASE_INSENSITIVE_ORDER);
+            candidateIndexSchema
+                    .forEach(column -> 
indexColumnNames.add(MaterializedIndexMeta.normalizeName(column.getName())));
+
+            List<Expr> indexExprs = new ArrayList<Expr>();
+            candidateIndexSchema.forEach(column -> 
indexExprs.add(column.getDefineExpr()));
+
+            List<Expr> groupingExprs = 
selectStmt.getAggInfo().getGroupingExprs();
+
             // The grouping columns in query must be subset of the grouping 
columns in view
-            if (!indexNonAggregatedColumnNames.containsAll(columnsInGrouping)) 
{
+            if (indexNonAggregatedColumnNames.containsAll(columnsInGrouping)) {
+                continue;
+            }
+
+            if (selectStmt.haveStar() || !matchAllExpr(groupingExprs, 
indexColumnNames, indexExprs)) {
                 iterator.remove();
             }
         }
         LOG.debug("Those mv pass the test of grouping:"
-                          + 
Joiner.on(",").join(candidateIndexIdToMeta.keySet()));
+                + Joiner.on(",").join(candidateIndexIdToMeta.keySet()));
     }
 
     // Step4: aggregation functions are available in the view output
@@ -364,24 +416,48 @@ public class MaterializedViewSelector {
             // When the candidate index is SPJ type, it passes the 
verification directly
             boolean noNeedAggregation = candidateIndexMeta.getKeysType() == 
KeysType.DUP_KEYS
                     || (candidateIndexMeta.getKeysType() == 
KeysType.UNIQUE_KEYS
-                    && 
table.getTableProperty().getEnableUniqueKeyMergeOnWrite());
-            if (indexAggColumnExpsList.size() == 0 && noNeedAggregation) {
+                            && 
table.getTableProperty().getEnableUniqueKeyMergeOnWrite());
+            if (!indexAggColumnExpsList.isEmpty() && selectStmt != null && 
selectStmt.getAggInfo() != null
+                    && selectStmt.getAggInfo().getSecondPhaseDistinctAggInfo() 
!= null) {
+
+                List<FunctionCallExpr> distinctExprs = 
selectStmt.getAggInfo().getSecondPhaseDistinctAggInfo()
+                        .getAggregateExprs();
+                boolean match = false;
+                for (Expr distinctExpr : distinctExprs) {
+                    for (Expr indexExpr : indexAggColumnExpsList) {
+                        if (distinctExpr.toSql() == indexExpr.toSql()) {
+                            match = true;
+                        }
+                    }
+                }
+                if (match) {
+                    iterator.remove();
+                    continue;
+                }
+            }
+
+            if (indexAggColumnExpsList.isEmpty() && noNeedAggregation) {
                 continue;
             }
-            // When the query is SPJ type but the candidate index is SPJG 
type, it will not pass directly.
+
+            // When the query is SPJ type but the candidate index is SPJG 
type, it will not
+            // pass directly.
             if (isSPJQuery || disableSPJGView) {
                 iterator.remove();
                 continue;
             }
             // The query is SPJG. The candidate index is SPJG too.
-            /* Situation1: The query is deduplicate SPJG when 
aggregatedColumnsInQueryOutput is null.
+            /*
+             * Situation1: The query is deduplicate SPJG when 
aggregatedColumnsInQueryOutput
+             * is null.
              * For example: select a , b from table group by a, b
              * The aggregation function check should be pass directly when MV 
is SPJG.
              */
             if (aggregatedColumnsInQueryOutput == null) {
                 continue;
             }
-            // The aggregated columns in query output must be subset of the 
aggregated columns in view
+            // The aggregated columns in query output must be subset of the 
aggregated
+            // columns in view
             if (!aggFunctionsMatchAggColumns(aggregatedColumnsInQueryOutput, 
indexAggColumnExpsList)) {
                 iterator.remove();
             }
@@ -390,25 +466,60 @@ public class MaterializedViewSelector {
                           + 
Joiner.on(",").join(candidateIndexIdToMeta.keySet()));
     }
 
-    // Step5: columns required to compute output expr are available in the 
view output
+    private boolean matchAllExpr(List<Expr> exprs, Set<String> 
indexColumnNames, List<Expr> indexExprs)
+            throws AnalysisException {
+        if (exprs.isEmpty()) {
+            return false;
+        }
+
+        for (Expr expr : exprs) {
+            if (expr == null) {
+                throw new AnalysisException("match expr input null");
+            }
+            String raw = MaterializedIndexMeta.normalizeName(expr.toSql());
+            String withPrefix = 
CreateMaterializedViewStmt.mvColumnBuilder(raw);
+            if (indexColumnNames.contains(raw) || 
indexColumnNames.contains(withPrefix)) {
+                continue;
+            }
+            if (!expr.matchExprs(indexExprs)) {
+                return false;
+            }
+
+        }
+        return true;
+    }
+
+    // Step5: columns required to compute output expr are available in the view
+    // output
     private void checkOutputColumns(Set<String> columnNamesInQueryOutput,
-            Map<Long, MaterializedIndexMeta> candidateIndexIdToMeta) {
+            Map<Long, MaterializedIndexMeta> candidateIndexIdToMeta) throws 
AnalysisException {
         if (columnNamesInQueryOutput == null) {
             return;
         }
         Iterator<Map.Entry<Long, MaterializedIndexMeta>> iterator = 
candidateIndexIdToMeta.entrySet().iterator();
         while (iterator.hasNext()) {
             Map.Entry<Long, MaterializedIndexMeta> entry = iterator.next();
-            Set<String> indexColumnNames = new 
TreeSet<>(String.CASE_INSENSITIVE_ORDER);
             List<Column> candidateIndexSchema = entry.getValue().getSchema();
-            candidateIndexSchema.forEach(column -> 
indexColumnNames.add(column.getName()));
+
+            Set<String> indexColumnNames = new 
TreeSet<>(String.CASE_INSENSITIVE_ORDER);
+            candidateIndexSchema
+                    .forEach(column -> 
indexColumnNames.add(MaterializedIndexMeta.normalizeName(column.getName())));
+
+            List<Expr> indexExprs = new ArrayList<Expr>();
+            candidateIndexSchema.forEach(column -> 
indexExprs.add(column.getDefineExpr()));
+
+            List<Expr> exprs = selectStmt.getAllExprs();
+
             // The columns in query output must be subset of the columns in 
SPJ view
-            if (!indexColumnNames.containsAll(columnNamesInQueryOutput)) {
+            if (indexColumnNames.containsAll(columnNamesInQueryOutput)) {
+                continue;
+            }
+            if (selectStmt.haveStar() || !matchAllExpr(exprs, 
indexColumnNames, indexExprs)) {
                 iterator.remove();
             }
         }
         LOG.debug("Those mv pass the test of output columns:"
-                          + 
Joiner.on(",").join(candidateIndexIdToMeta.keySet()));
+                + Joiner.on(",").join(candidateIndexIdToMeta.keySet()));
     }
 
     private void compensateCandidateIndex(Map<Long, MaterializedIndexMeta> 
candidateIndexIdToMeta, Map<Long,
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/planner/OlapScanNode.java 
b/fe/fe-core/src/main/java/org/apache/doris/planner/OlapScanNode.java
index 750ec7d956..d1624c51a3 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/planner/OlapScanNode.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/planner/OlapScanNode.java
@@ -21,6 +21,7 @@ import org.apache.doris.analysis.Analyzer;
 import org.apache.doris.analysis.BaseTableRef;
 import org.apache.doris.analysis.BinaryPredicate;
 import org.apache.doris.analysis.CastExpr;
+import org.apache.doris.analysis.CreateMaterializedViewStmt;
 import org.apache.doris.analysis.Expr;
 import org.apache.doris.analysis.InPredicate;
 import org.apache.doris.analysis.IntLiteral;
@@ -392,7 +393,7 @@ public class OlapScanNode extends ScanNode {
      * changed to bigint in here.
      * Currently, only `SUM` aggregate type could match this changed.
      */
-    private void updateColumnType() {
+    private void updateColumnType() throws UserException {
         if (selectedIndexId == olapTable.getBaseIndexId()) {
             return;
         }
@@ -404,7 +405,13 @@ public class OlapScanNode extends ScanNode {
             Column baseColumn = slotDescriptor.getColumn();
             Preconditions.checkNotNull(baseColumn);
             Column mvColumn = meta.getColumnByName(baseColumn.getName());
-            Preconditions.checkNotNull(mvColumn);
+            if (mvColumn == null) {
+                mvColumn = 
meta.getColumnByName(CreateMaterializedViewStmt.mvColumnBuilder(baseColumn.getName()));
+            }
+            if (mvColumn == null) {
+                throw new UserException("Do not found mvColumn " + 
baseColumn.getName());
+            }
+
             if (mvColumn.getType() != baseColumn.getType()) {
                 slotDescriptor.setColumn(mvColumn);
             }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/CountFieldToSum.java
 
b/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/CountFieldToSum.java
index ee57c6317a..7a26494179 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/CountFieldToSum.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/CountFieldToSum.java
@@ -70,7 +70,7 @@ public class CountFieldToSum implements ExprRewriteRule {
         OlapTable olapTable = (OlapTable) table;
 
         // check column
-        String queryColumnName = column.getName();
+        String queryColumnName = column.getNameWithoutMvPrefix();
         String mvColumnName = 
CreateMaterializedViewStmt.mvColumnBuilder(FunctionSet.COUNT, queryColumnName);
         Column mvColumn = olapTable.getVisibleColumn(mvColumnName);
         if (mvColumn == null) {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/ExprToSlotRefRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/ExprToSlotRefRule.java
new file mode 100644
index 0000000000..5f9859dec2
--- /dev/null
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/ExprToSlotRefRule.java
@@ -0,0 +1,125 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+package org.apache.doris.rewrite.mvrewrite;
+
+import org.apache.doris.analysis.Analyzer;
+import org.apache.doris.analysis.ArithmeticExpr;
+import org.apache.doris.analysis.CreateMaterializedViewStmt;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TableName;
+import org.apache.doris.catalog.Column;
+import org.apache.doris.catalog.MaterializedIndexMeta;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.TableIf;
+import org.apache.doris.common.AnalysisException;
+import org.apache.doris.rewrite.ExprRewriteRule;
+import org.apache.doris.rewrite.ExprRewriter;
+
+import com.google.common.base.Preconditions;
+
+import java.util.ArrayList;
+import java.util.List;
+
+/**
+ * Rewrite expr to mv_expr
+ */
+public class ExprToSlotRefRule implements ExprRewriteRule {
+
+    public static final ExprRewriteRule INSTANCE = new ExprToSlotRefRule();
+
+    @Override
+    public Expr apply(Expr expr, Analyzer analyzer, ExprRewriter.ClauseType 
clauseType) throws AnalysisException {
+        // todo: support more pattern
+        if (expr instanceof ArithmeticExpr || expr instanceof 
FunctionCallExpr) {
+            return matchExpr(expr, analyzer, clauseType);
+        } else if (expr instanceof SlotRef) {
+            return matchSlotRef((SlotRef) expr, analyzer, clauseType);
+        } else {
+            return expr;
+        }
+    }
+
+    private Expr matchExpr(Expr expr, Analyzer analyzer, 
ExprRewriter.ClauseType clauseType)
+            throws AnalysisException {
+        List<SlotRef> slots = new ArrayList<>();
+        expr.collect(SlotRef.class, slots);
+
+        if (slots.size() != 1) {
+            return expr;
+        }
+
+        SlotRef queryColumnSlotRef = slots.get(0);
+
+        Column column = queryColumnSlotRef.getColumn();
+        TableIf table = queryColumnSlotRef.getTable();
+        if (column == null || table == null || !(table instanceof OlapTable)) {
+            return expr;
+        }
+        OlapTable olapTable = (OlapTable) table;
+
+        Column mvColumn = olapTable.getVisibleColumn(expr.toSql());
+        if (mvColumn == null) {
+            mvColumn = olapTable.getVisibleColumn(
+                    
MaterializedIndexMeta.normalizeName(CreateMaterializedViewStmt.mvColumnBuilder(expr.toSql())));
+        }
+
+        if (mvColumn == null) {
+            mvColumn = 
olapTable.getVisibleColumn(CreateMaterializedViewStmt.mvColumnBuilder(expr.toSql()));
+        }
+
+        if (mvColumn == null) {
+            return expr;
+        }
+
+        return rewriteExpr(queryColumnSlotRef, mvColumn, analyzer);
+    }
+
+    private Expr matchSlotRef(SlotRef slot, Analyzer analyzer, 
ExprRewriter.ClauseType clauseType)
+            throws AnalysisException {
+        Column column = slot.getColumn();
+        TableIf table = slot.getTable();
+        if (column == null || table == null || !(table instanceof OlapTable)) {
+            return slot;
+        }
+        OlapTable olapTable = (OlapTable) table;
+
+        Column mvColumn = 
olapTable.getVisibleColumn(CreateMaterializedViewStmt.mvColumnBuilder(slot.toSql()));
+        if (mvColumn == null) {
+            mvColumn = olapTable.getVisibleColumn(
+                    
MaterializedIndexMeta.normalizeName(CreateMaterializedViewStmt.mvColumnBuilder(slot.toSql())));
+        }
+
+        if (mvColumn == null) {
+            return slot;
+        }
+
+        return rewriteExpr(slot, mvColumn, analyzer);
+    }
+
+    private Expr rewriteExpr(SlotRef queryColumnSlotRef, Column mvColumn, 
Analyzer analyzer) {
+        Preconditions.checkNotNull(mvColumn);
+        Preconditions.checkNotNull(queryColumnSlotRef);
+        TableName tableName = queryColumnSlotRef.getTableName();
+        Preconditions.checkNotNull(tableName);
+        SlotRef mvSlotRef = new SlotRef(tableName, mvColumn.getName());
+        mvSlotRef.analyzeNoThrow(analyzer);
+        return mvSlotRef;
+    }
+}
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/HLLHashToSlotRefRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/HLLHashToSlotRefRule.java
index 34ce7a60e7..685789d2f6 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/HLLHashToSlotRefRule.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/HLLHashToSlotRefRule.java
@@ -29,6 +29,7 @@ import org.apache.doris.catalog.Column;
 import org.apache.doris.catalog.FunctionSet;
 import org.apache.doris.catalog.OlapTable;
 import org.apache.doris.catalog.TableIf;
+import org.apache.doris.catalog.Type;
 import org.apache.doris.common.AnalysisException;
 import org.apache.doris.rewrite.ExprRewriteRule;
 import org.apache.doris.rewrite.ExprRewriter;
@@ -67,7 +68,7 @@ public class HLLHashToSlotRefRule implements ExprRewriteRule {
             return expr;
         }
         FunctionCallExpr child0FnExpr = (FunctionCallExpr) fnExpr.getChild(0);
-        if 
(!child0FnExpr.getFnName().getFunction().equalsIgnoreCase(FunctionSet.HLL_HASH))
 {
+        if (!child0FnExpr.getType().equals(Type.HLL)) {
             return expr;
         }
         if (child0FnExpr.getChild(0) instanceof SlotRef) {
@@ -98,7 +99,7 @@ public class HLLHashToSlotRefRule implements ExprRewriteRule {
         }
 
         // equal expr
-        return rewriteExpr(fnNameString, queryColumnSlotRef, mvColumn, 
analyzer);
+        return rewriteExpr(fnExpr.getFnName().getFunction(), 
queryColumnSlotRef, mvColumn, analyzer);
     }
 
     private Expr rewriteExpr(String fnName, SlotRef queryColumnSlotRef, Column 
mvColumn, Analyzer analyzer) {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/SlotRefEqualRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/SlotRefEqualRule.java
index 76b59046f9..d70d4e6a15 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/SlotRefEqualRule.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/SlotRefEqualRule.java
@@ -31,7 +31,8 @@ public class SlotRefEqualRule implements MVExprEqualRule {
         }
         SlotRef querySlotRef = (SlotRef) queryExpr;
         SlotRef mvColumnSlotRef = (SlotRef) mvColumnExpr;
-        if 
(querySlotRef.getColumnName().equalsIgnoreCase(mvColumnSlotRef.getColumnName()))
 {
+        if (querySlotRef.getColumnName() != null
+                && 
querySlotRef.getColumnName().equalsIgnoreCase(mvColumnSlotRef.getColumnName())) 
{
             return true;
         }
         return false;
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/ToBitmapToSlotRefRule.java
 
b/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/ToBitmapToSlotRefRule.java
index a486c6185f..767acd4965 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/ToBitmapToSlotRefRule.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/rewrite/mvrewrite/ToBitmapToSlotRefRule.java
@@ -29,6 +29,7 @@ import org.apache.doris.catalog.Column;
 import org.apache.doris.catalog.FunctionSet;
 import org.apache.doris.catalog.OlapTable;
 import org.apache.doris.catalog.TableIf;
+import org.apache.doris.catalog.Type;
 import org.apache.doris.common.AnalysisException;
 import org.apache.doris.rewrite.ExprRewriteRule;
 import org.apache.doris.rewrite.ExprRewriter;
@@ -65,8 +66,7 @@ public class ToBitmapToSlotRefRule implements ExprRewriteRule 
{
             return expr;
         }
         FunctionCallExpr child0FnExpr = (FunctionCallExpr) fnExpr.getChild(0);
-        if 
(!child0FnExpr.getFnName().getFunction().equalsIgnoreCase("to_bitmap")
-                && 
!child0FnExpr.getFnName().getFunction().equalsIgnoreCase("to_bitmap_with_check"))
 {
+        if (!child0FnExpr.getType().equals(Type.BITMAP)) {
             return expr;
         }
         if (child0FnExpr.getChild(0) instanceof SlotRef) {
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/alter/MaterializedViewHandlerTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/alter/MaterializedViewHandlerTest.java
index 9e81e48588..07af1b5e73 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/alter/MaterializedViewHandlerTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/alter/MaterializedViewHandlerTest.java
@@ -19,6 +19,7 @@ package org.apache.doris.alter;
 
 import org.apache.doris.analysis.CreateMaterializedViewStmt;
 import org.apache.doris.analysis.MVColumnItem;
+import org.apache.doris.analysis.SlotRef;
 import org.apache.doris.catalog.AggregateType;
 import org.apache.doris.catalog.Column;
 import org.apache.doris.catalog.Database;
@@ -27,6 +28,7 @@ import org.apache.doris.catalog.MaterializedIndex;
 import org.apache.doris.catalog.OlapTable;
 import org.apache.doris.catalog.Partition;
 import org.apache.doris.catalog.Type;
+import org.apache.doris.common.AnalysisException;
 import org.apache.doris.common.DdlException;
 import org.apache.doris.common.jmockit.Deencapsulation;
 
@@ -173,42 +175,6 @@ public class MaterializedViewHandlerTest {
         }
     }
 
-    @Test
-    public void testInvalidAggregateType(@Injectable 
CreateMaterializedViewStmt createMaterializedViewStmt,
-                                         @Injectable OlapTable olapTable) {
-        final String mvName = "mv1";
-        final String columnName = "mv_k1";
-        Column baseColumn = new Column(columnName, Type.INT, false, 
AggregateType.SUM, "", "");
-        MVColumnItem mvColumnItem = new MVColumnItem(columnName, Type.BIGINT);
-        mvColumnItem.setIsKey(true);
-        mvColumnItem.setAggregationType(null, false);
-        new Expectations() {
-            {
-                olapTable.hasMaterializedIndex(mvName);
-                result = false;
-                createMaterializedViewStmt.getMVName();
-                result = mvName;
-                createMaterializedViewStmt.getMVColumnItemList();
-                result = Lists.newArrayList(mvColumnItem);
-                createMaterializedViewStmt.getMVKeysType();
-                result = KeysType.AGG_KEYS;
-                olapTable.getColumn(columnName);
-                result = baseColumn;
-                olapTable.getKeysType();
-                result = KeysType.AGG_KEYS;
-            }
-        };
-        MaterializedViewHandler materializedViewHandler = new 
MaterializedViewHandler();
-        try {
-            Deencapsulation.invoke(materializedViewHandler, 
"checkAndPrepareMaterializedView",
-                    createMaterializedViewStmt, olapTable);
-            Assert.fail();
-        } catch (Exception e) {
-            System.out.print(e.getMessage());
-        }
-    }
-
-
     @Test
     public void testInvalidKeysType(@Injectable CreateMaterializedViewStmt 
createMaterializedViewStmt,
                                     @Injectable OlapTable olapTable) {
@@ -236,20 +202,29 @@ public class MaterializedViewHandlerTest {
                                    @Injectable OlapTable olapTable) {
         final String mvName = "mv1";
         final String columnName1 = "k1";
-        Column baseColumn1 = new Column(columnName1, Type.VARCHAR, false, 
AggregateType.NONE, "", "");
-        MVColumnItem mvColumnItem = new MVColumnItem(columnName1, 
Type.VARCHAR);
+        SlotRef slot = new SlotRef(Type.VARCHAR, false);
+        slot.setCol(columnName1);
+        slot.setLabel(columnName1);
+
+        MVColumnItem mvColumnItem = null;
+        try {
+            mvColumnItem = new MVColumnItem(slot);
+        } catch (AnalysisException e) {
+            Assert.fail(e.getMessage());
+        }
         mvColumnItem.setIsKey(true);
         mvColumnItem.setAggregationType(null, false);
+        List<MVColumnItem> list = Lists.newArrayList(mvColumnItem);
         new Expectations() {
             {
+                olapTable.getBaseColumn(columnName1);
+                result = null;
                 olapTable.hasMaterializedIndex(mvName);
                 result = false;
                 createMaterializedViewStmt.getMVName();
                 result = mvName;
                 createMaterializedViewStmt.getMVColumnItemList();
-                result = Lists.newArrayList(mvColumnItem);
-                olapTable.getBaseColumn(columnName1);
-                result = baseColumn1;
+                result = list;
                 olapTable.getKeysType();
                 result = KeysType.DUP_KEYS;
             }
@@ -267,6 +242,7 @@ public class MaterializedViewHandlerTest {
             Assert.assertEquals(false, 
newMVColumn.isAggregationTypeImplicit());
             Assert.assertEquals(Type.VARCHAR, newMVColumn.getType());
         } catch (Exception e) {
+            e.printStackTrace();
             Assert.fail(e.getMessage());
         }
     }
@@ -276,9 +252,21 @@ public class MaterializedViewHandlerTest {
                                            @Injectable OlapTable olapTable) 
throws DdlException {
         final String mvName = "mv1";
         final String columnName1 = "k1";
-        MVColumnItem mvColumnItem = new MVColumnItem(columnName1, 
Type.VARCHAR);
+
+        SlotRef slot = new SlotRef(Type.VARCHAR, false);
+        slot.setCol(columnName1);
+        slot.setLabel(columnName1);
+
+        MVColumnItem mvColumnItem = null;
+        try {
+            mvColumnItem = new MVColumnItem(slot);
+        } catch (AnalysisException e) {
+            Assert.fail(e.getMessage());
+        }
+
         mvColumnItem.setIsKey(false);
         mvColumnItem.setAggregationType(AggregateType.SUM, false);
+        List<MVColumnItem> list = Lists.newArrayList(mvColumnItem);
         Set<String> partitionColumnNames = Sets.newHashSet();
         partitionColumnNames.add(columnName1);
         new Expectations() {
@@ -288,7 +276,7 @@ public class MaterializedViewHandlerTest {
                 createMaterializedViewStmt.getMVName();
                 result = mvName;
                 createMaterializedViewStmt.getMVColumnItemList();
-                result = Lists.newArrayList(mvColumnItem);
+                result = list;
                 olapTable.getKeysType();
                 result = KeysType.DUP_KEYS;
                 olapTable.getPartitionColumnNames();
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/analysis/CreateMaterializedViewStmtTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/analysis/CreateMaterializedViewStmtTest.java
index 6513170ba2..0379275f7e 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/analysis/CreateMaterializedViewStmtTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/analysis/CreateMaterializedViewStmtTest.java
@@ -112,7 +112,7 @@ public class CreateMaterializedViewStmtTest {
                 result = selectList;
                 arithmeticExpr.toString();
                 result = "a+b";
-                slotRef.getColumnName();
+                slotRef.toSql();
                 result = "k1";
                 selectStmt.getWhereClause();
                 minTimes = 0;
@@ -195,7 +195,7 @@ public class CreateMaterializedViewStmtTest {
                 result = Lists.newArrayList(tableRef1, tableRef2);
                 selectStmt.getSelectList();
                 result = selectList;
-                slotRef.getColumnName();
+                slotRef.toSql();
                 result = "k1";
             }
         };
@@ -228,7 +228,7 @@ public class CreateMaterializedViewStmtTest {
                 result = Lists.newArrayList(tableRef);
                 selectStmt.getWhereClause();
                 result = whereClause;
-                slotRef.getColumnName();
+                slotRef.toSql();
                 result = "k1";
             }
         };
@@ -264,7 +264,7 @@ public class CreateMaterializedViewStmtTest {
                 result = null;
                 selectStmt.getHavingPred();
                 result = havingClause;
-                slotRef.getColumnName();
+                slotRef.toSql();
                 result = "k1";
             }
         };
@@ -306,17 +306,20 @@ public class CreateMaterializedViewStmtTest {
                 result = null;
                 selectStmt.getOrderByElements();
                 result = orderByElementList;
-                slotRef1.getColumnName();
+                slotRef1.toSql();
                 result = "k1";
-                slotRef2.getColumnName();
+                slotRef2.toSql();
                 result = "k2";
+                slotRef2.getColumnName();
+                result = "k1";
             }
         };
         CreateMaterializedViewStmt createMaterializedViewStmt = new 
CreateMaterializedViewStmt("test", selectStmt, null);
         try {
             createMaterializedViewStmt.analyze(analyzer);
             Assert.fail();
-        } catch (UserException e) {
+        } catch (Exception e) {
+            e.printStackTrace();
             System.out.print(e.getMessage());
         }
     }
@@ -358,12 +361,12 @@ public class CreateMaterializedViewStmtTest {
                 result = null;
                 selectStmt.getOrderByElements();
                 result = orderByElementList;
-                slotRef1.getColumnName();
-                result = "k1";
                 slotDescriptor.getColumn();
                 result = column2;
                 column2.getOriginType();
                 result = Type.INT;
+                slotRef1.toSql();
+                result = "k1";
             }
         };
         CreateMaterializedViewStmt createMaterializedViewStmt = new 
CreateMaterializedViewStmt("test", selectStmt, null);
@@ -401,7 +404,7 @@ public class CreateMaterializedViewStmtTest {
         try {
             createMaterializedViewStmt.analyze(analyzer);
             Assert.fail();
-        } catch (UserException e) {
+        } catch (Exception e) {
             System.out.print(e.getMessage());
         }
     }
@@ -435,12 +438,12 @@ public class CreateMaterializedViewStmtTest {
                 selectStmt.analyze(analyzer);
                 selectStmt.getSelectList();
                 result = selectList;
-                slotRef1.getColumnName();
-                result = "k1";
                 slotDescriptor.getColumn();
                 result = column2;
                 column2.getOriginType();
                 result = Type.INT;
+                slotRef1.toSql();
+                result = "k1";
             }
         };
         CreateMaterializedViewStmt createMaterializedViewStmt = new 
CreateMaterializedViewStmt("test", selectStmt, null);
@@ -491,14 +494,14 @@ public class CreateMaterializedViewStmtTest {
                 result = null;
                 selectStmt.getOrderByElements();
                 result = orderByElementList;
-                slotRef1.getColumnName();
-                result = "k1";
-                slotRef2.getColumnName();
-                result = "non-k2";
                 slotDescriptor.getColumn();
                 result = column3;
                 column3.getOriginType();
                 result = Type.INT;
+                slotRef1.toSql();
+                result = "k1";
+                slotRef2.toSql();
+                result = "k2";
             }
         };
         CreateMaterializedViewStmt createMaterializedViewStmt = new 
CreateMaterializedViewStmt("test", selectStmt, null);
@@ -563,13 +566,13 @@ public class CreateMaterializedViewStmtTest {
                 selectStmt.getLimit();
                 result = -1;
                 selectStmt.analyze(analyzer);
-                slotRef1.getColumnName();
+                slotRef1.toSql();
                 result = columnName1;
-                slotRef2.getColumnName();
+                slotRef2.toSql();
                 result = columnName2;
-                slotRef3.getColumnName();
+                slotRef3.toSql();
                 result = columnName3;
-                slotRef4.getColumnName();
+                slotRef4.toSql();
                 result = columnName4;
                 functionChild0.getColumn();
                 result = column5;
@@ -608,7 +611,7 @@ public class CreateMaterializedViewStmtTest {
             MVColumnItem mvColumn4 = mvColumns.get(4);
             Assert.assertFalse(mvColumn4.isKey());
             Assert.assertFalse(mvColumn4.isAggregationTypeImplicit());
-            Assert.assertEquals(columnName5, mvColumn4.getName());
+            
Assert.assertEquals(CreateMaterializedViewStmt.mvColumnBuilder(columnName5), 
mvColumn4.getName());
             Assert.assertEquals(AggregateType.SUM, 
mvColumn4.getAggregationType());
         } catch (UserException e) {
             Assert.fail(e.getMessage());
@@ -653,13 +656,13 @@ public class CreateMaterializedViewStmtTest {
                 selectStmt.getLimit();
                 result = -1;
                 selectStmt.analyze(analyzer);
-                slotRef1.getColumnName();
+                slotRef1.toSql();
                 result = columnName1;
-                slotRef2.getColumnName();
+                slotRef2.toSql();
                 result = columnName2;
-                slotRef3.getColumnName();
+                slotRef3.toSql();
                 result = columnName3;
-                slotRef4.getColumnName();
+                slotRef4.toSql();
                 result = columnName4;
                 slotRef1.getType().getIndexSize();
                 result = 34;
@@ -753,13 +756,13 @@ public class CreateMaterializedViewStmtTest {
                 selectStmt.getLimit();
                 result = -1;
                 selectStmt.analyze(analyzer);
-                slotRef1.getColumnName();
+                slotRef1.toSql();
                 result = columnName1;
-                slotRef2.getColumnName();
+                slotRef2.toSql();
                 result = columnName2;
-                slotRef3.getColumnName();
+                slotRef3.toSql();
                 result = columnName3;
-                slotRef4.getColumnName();
+                slotRef4.toSql();
                 result = columnName4;
                 slotRef1.getType().getIndexSize();
                 result = 1;
@@ -851,13 +854,13 @@ public class CreateMaterializedViewStmtTest {
                 selectStmt.getLimit();
                 result = -1;
                 selectStmt.analyze(analyzer);
-                slotRef1.getColumnName();
+                slotRef1.toSql();
                 result = columnName1;
-                slotRef2.getColumnName();
+                slotRef2.toSql();
                 result = columnName2;
-                slotRef3.getColumnName();
+                slotRef3.toSql();
                 result = columnName3;
-                slotRef4.getColumnName();
+                slotRef4.toSql();
                 result = columnName4;
                 slotRef1.getType().getIndexSize();
                 result = 1;
@@ -918,8 +921,6 @@ public class CreateMaterializedViewStmtTest {
         SelectListItem selectListItem1 = new SelectListItem(slotRef1, null);
         selectList.addItem(selectListItem1);
 
-        final String columnName1 = "k1";
-
         new Expectations() {
             {
                 analyzer.getClusterName();
@@ -928,19 +929,7 @@ public class CreateMaterializedViewStmtTest {
                 result = null;
                 selectStmt.getSelectList();
                 result = selectList;
-                selectStmt.getTableRefs();
-                result = Lists.newArrayList(tableRef);
-                selectStmt.getWhereClause();
-                result = null;
-                selectStmt.getHavingPred();
-                result = null;
-                selectStmt.getOrderByElements();
-                result = null;
                 selectStmt.analyze(analyzer);
-                slotRef1.getColumnName();
-                result = columnName1;
-                slotRef1.getType().isFloatingPointType();
-                result = true;
                 selectStmt.getAggInfo(); // return null, so that the mv can be 
a duplicate mv
                 result = null;
             }
@@ -987,7 +976,7 @@ public class CreateMaterializedViewStmtTest {
                 selectStmt.getLimit();
                 result = -1;
                 selectStmt.analyze(analyzer);
-                slotRef1.getColumnName();
+                slotRef1.toSql();
                 result = columnName1;
                 slotRef1.getType().getPrimitiveType();
                 result = PrimitiveType.VARCHAR;
@@ -1060,6 +1049,10 @@ public class CreateMaterializedViewStmtTest {
                 selectStmt.getLimit();
                 result = -1;
                 selectStmt.analyze(analyzer);
+                slotRef1.toSql();
+                result = columnName1;
+                slotRef2.toSql();
+                result = columnName2;
                 slotRef1.getColumnName();
                 result = columnName1;
                 slotRef2.getColumnName();
@@ -1090,7 +1083,7 @@ public class CreateMaterializedViewStmtTest {
             MVColumnItem mvColumn2 = mvColumns.get(2);
             Assert.assertFalse(mvColumn2.isKey());
             Assert.assertFalse(mvColumn2.isAggregationTypeImplicit());
-            Assert.assertEquals(columnName3, mvColumn2.getName());
+            
Assert.assertEquals(CreateMaterializedViewStmt.mvColumnBuilder(columnName3), 
mvColumn2.getName());
             Assert.assertEquals(AggregateType.SUM, 
mvColumn2.getAggregationType());
             Assert.assertEquals(KeysType.AGG_KEYS, 
createMaterializedViewStmt.getMVKeysType());
         } catch (UserException e) {
@@ -1107,7 +1100,6 @@ public class CreateMaterializedViewStmtTest {
         SelectList selectList = new SelectList();
         SelectListItem selectListItem1 = new SelectListItem(slotRef1, null);
         selectList.addItem(selectListItem1);
-        final String columnName1 = "k1";
         new Expectations() {
             {
                 analyzer.getClusterName();
@@ -1121,12 +1113,12 @@ public class CreateMaterializedViewStmtTest {
                 result = Lists.newArrayList(tableRef);
                 selectStmt.getWhereClause();
                 result = null;
-                slotRef1.getColumnName();
-                result = columnName1;
                 selectStmt.getHavingPred();
                 result = null;
                 selectStmt.getLimit();
                 result = -1;
+                slotRef1.toSql();
+                result = "k1";
             }
         };
 
@@ -1137,7 +1129,8 @@ public class CreateMaterializedViewStmtTest {
             List<MVColumnItem> mvSchema = 
createMaterializedViewStmt.getMVColumnItemList();
             Assert.assertEquals(1, mvSchema.size());
             Assert.assertTrue(mvSchema.get(0).isKey());
-        } catch (UserException e) {
+        } catch (Exception e) {
+            e.printStackTrace();
             Assert.fail(e.getMessage());
         }
 
diff --git a/fe/fe-core/src/test/java/org/apache/doris/analysis/ExprTest.java 
b/fe/fe-core/src/test/java/org/apache/doris/analysis/ExprTest.java
index 68801ea18f..ad4a3c21d6 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/analysis/ExprTest.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/analysis/ExprTest.java
@@ -64,10 +64,6 @@ public class ExprTest {
         Deencapsulation.setField(tableBColumn1, "desc", slotDesc2);
         new Expectations() {
             {
-                slotDesc1.isMaterialized();
-                result = true;
-                slotDesc2.isMaterialized();
-                result = true;
                 slotDesc1.getColumn().getName();
                 result = "c1";
                 slotDesc2.getColumn().getName();
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/analysis/MVColumnBitmapUnionPatternTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/analysis/MVColumnBitmapUnionPatternTest.java
index 9113405d64..79b24cc1da 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/analysis/MVColumnBitmapUnionPatternTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/analysis/MVColumnBitmapUnionPatternTest.java
@@ -43,6 +43,7 @@ public class MVColumnBitmapUnionPatternTest {
         List<Expr> child0Params = Lists.newArrayList();
         child0Params.add(slotRef);
         FunctionCallExpr child0 = new FunctionCallExpr(FunctionSet.TO_BITMAP, 
child0Params);
+        child0.setType(Type.BITMAP);
         List<Expr> params = Lists.newArrayList();
         params.add(child0);
         FunctionCallExpr expr = new FunctionCallExpr(FunctionSet.BITMAP_UNION, 
params);
@@ -65,6 +66,7 @@ public class MVColumnBitmapUnionPatternTest {
         List<Expr> child0Params = Lists.newArrayList();
         child0Params.add(castExpr);
         FunctionCallExpr child0 = new FunctionCallExpr(FunctionSet.TO_BITMAP, 
child0Params);
+        child0.setType(Type.BITMAP);
         List<Expr> params = Lists.newArrayList();
         params.add(child0);
         FunctionCallExpr expr = new FunctionCallExpr(FunctionSet.BITMAP_UNION, 
params);
@@ -81,6 +83,7 @@ public class MVColumnBitmapUnionPatternTest {
         List<Expr> child0Params = Lists.newArrayList();
         child0Params.add(slotRef);
         FunctionCallExpr child0 = new 
FunctionCallExpr(FunctionSet.TO_BITMAP.toUpperCase(), child0Params);
+        child0.setType(Type.BITMAP);
         List<Expr> params = Lists.newArrayList();
         params.add(child0);
         FunctionCallExpr expr = new 
FunctionCallExpr(FunctionSet.BITMAP_UNION.toUpperCase(), params);
@@ -112,6 +115,7 @@ public class MVColumnBitmapUnionPatternTest {
         List<Expr> child0Params = Lists.newArrayList();
         child0Params.add(arithmeticExpr);
         FunctionCallExpr child0 = new FunctionCallExpr(FunctionSet.TO_BITMAP, 
child0Params);
+        child0.setType(Type.BITMAP);
         List<Expr> params = Lists.newArrayList();
         params.add(child0);
         FunctionCallExpr expr = new FunctionCallExpr(FunctionSet.BITMAP_UNION, 
params);
@@ -128,6 +132,7 @@ public class MVColumnBitmapUnionPatternTest {
         List<Expr> child0Params = Lists.newArrayList();
         child0Params.add(slotRef1);
         FunctionCallExpr child0 = new FunctionCallExpr(FunctionSet.TO_BITMAP, 
child0Params);
+        child0.setType(Type.BITMAP);
         List<Expr> params = Lists.newArrayList();
         params.add(child0);
         FunctionCallExpr expr = new FunctionCallExpr(FunctionSet.BITMAP_UNION, 
params);
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/analysis/MVColumnHLLUnionPatternTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/analysis/MVColumnHLLUnionPatternTest.java
index f08df6337f..071cc359c0 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/analysis/MVColumnHLLUnionPatternTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/analysis/MVColumnHLLUnionPatternTest.java
@@ -42,6 +42,7 @@ public class MVColumnHLLUnionPatternTest {
         List<Expr> child0Params = Lists.newArrayList();
         child0Params.add(slotRef);
         FunctionCallExpr child0 = new FunctionCallExpr(FunctionSet.HLL_HASH, 
child0Params);
+        child0.setType(Type.HLL);
         List<Expr> params = Lists.newArrayList();
         params.add(child0);
         FunctionCallExpr expr = new FunctionCallExpr(FunctionSet.HLL_UNION, 
params);
@@ -63,6 +64,7 @@ public class MVColumnHLLUnionPatternTest {
         List<Expr> child0Params = Lists.newArrayList();
         child0Params.add(castExpr);
         FunctionCallExpr child0 = new FunctionCallExpr(FunctionSet.HLL_HASH, 
child0Params);
+        child0.setType(Type.HLL);
         List<Expr> params = Lists.newArrayList();
         params.add(child0);
         FunctionCallExpr expr = new FunctionCallExpr(FunctionSet.HLL_UNION, 
params);
@@ -78,6 +80,7 @@ public class MVColumnHLLUnionPatternTest {
         List<Expr> child0Params = Lists.newArrayList();
         child0Params.add(slotRef);
         FunctionCallExpr child0 = new 
FunctionCallExpr(FunctionSet.HLL_HASH.toUpperCase(), child0Params);
+        child0.setType(Type.HLL);
         List<Expr> params = Lists.newArrayList();
         params.add(child0);
         FunctionCallExpr expr = new 
FunctionCallExpr(FunctionSet.HLL_UNION.toUpperCase(), params);
@@ -103,6 +106,7 @@ public class MVColumnHLLUnionPatternTest {
         List<Expr> child0Params = Lists.newArrayList();
         child0Params.add(intLiteral);
         FunctionCallExpr child0 = new FunctionCallExpr(FunctionSet.HLL_HASH, 
child0Params);
+        child0.setType(Type.HLL);
         List<Expr> params = Lists.newArrayList();
         params.add(child0);
         FunctionCallExpr expr = new FunctionCallExpr(FunctionSet.HLL_UNION, 
params);
@@ -119,6 +123,7 @@ public class MVColumnHLLUnionPatternTest {
         List<Expr> child0Params = Lists.newArrayList();
         child0Params.add(slotRef);
         FunctionCallExpr child0 = new FunctionCallExpr(FunctionSet.HLL_HASH, 
child0Params);
+        child0.setType(Type.HLL);
         List<Expr> params = Lists.newArrayList();
         params.add(child0);
         FunctionCallExpr expr = new FunctionCallExpr(FunctionSet.HLL_UNION, 
params);
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/planner/MaterializedViewFunctionTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/planner/MaterializedViewFunctionTest.java
index cf418a6830..74bd09956f 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/planner/MaterializedViewFunctionTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/planner/MaterializedViewFunctionTest.java
@@ -549,7 +549,7 @@ public class MaterializedViewFunctionTest {
     @Test
     public void testMultiMVMultiUsage() throws Exception {
         String createEmpsMVSql01 = "create materialized view emp_mv_01 as 
select deptno, empid, salary "
-                + "from " + EMPS_TABLE_NAME + " order by deptno;";
+                + "from " + EMPS_TABLE_NAME + ";";
         String createEmpsMVSql02 = "create materialized view emp_mv_02 as 
select deptno, sum(salary) "
                 + "from " + EMPS_TABLE_NAME + " group by deptno;";
         String query = "select * from (select deptno, empid from " + 
EMPS_TABLE_NAME + " where deptno>100) A join "
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/planner/MaterializedViewSelectorTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/planner/MaterializedViewSelectorTest.java
index 833eb7ad4d..4d1a5390bb 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/planner/MaterializedViewSelectorTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/planner/MaterializedViewSelectorTest.java
@@ -82,8 +82,6 @@ public class MaterializedViewSelectorTest {
                 result = aggregateInfo;
                 aggregateInfo.getGroupingExprs();
                 result = Lists.newArrayList(tableAColumn1);
-                tableAColumn1Desc.isMaterialized();
-                result = true;
                 tableAColumn1Desc.getColumn().getName();
                 result = "c1";
                 tableAColumn1Desc.getParent();
@@ -117,10 +115,6 @@ public class MaterializedViewSelectorTest {
                 tableB.getId();
                 result = 2;
 
-                tableAColumn2Desc.isMaterialized();
-                result = true;
-                tableBColumn1Desc.isMaterialized();
-                result = true;
                 tableAColumn2Desc.getColumn().getName();
                 result = "c2";
                 tableBColumn1Desc.getColumn().getName();
@@ -166,30 +160,39 @@ public class MaterializedViewSelectorTest {
             @Injectable MaterializedIndexMeta indexMeta1,
             @Injectable MaterializedIndexMeta indexMeta2,
             @Injectable MaterializedIndexMeta indexMeta3,
-            @Injectable MaterializedIndexMeta indexMeta4) {
+            @Injectable MaterializedIndexMeta indexMeta4, @Injectable SlotRef 
slotRef1, @Injectable SlotRef slotRef2) {
         Set<String> tableAColumnNames = Sets.newHashSet();
         tableAColumnNames.add("C1");
         Map<Long, MaterializedIndexMeta> candidateIndexIdToSchema = 
Maps.newHashMap();
         List<Column> index1Columns = Lists.newArrayList();
         Column index1Column1 = new Column("c1", Type.INT, true, null, true, 
"", "");
         index1Columns.add(index1Column1);
+        index1Column1.setDefineExpr(slotRef1);
         candidateIndexIdToSchema.put(new Long(1), indexMeta1);
         List<Column> index2Columns = Lists.newArrayList();
         Column index2Column1 = new Column("c1", Type.INT, false, 
AggregateType.NONE, true, "", "");
         index2Columns.add(index2Column1);
+        index2Column1.setDefineExpr(slotRef1);
         candidateIndexIdToSchema.put(new Long(2), indexMeta2);
         List<Column> index3Columns = Lists.newArrayList();
         Column index3Column1 = new Column("c1", Type.INT, false, 
AggregateType.SUM, true, "", "");
+        index3Column1.setDefineExpr(slotRef1);
         index3Columns.add(index3Column1);
         candidateIndexIdToSchema.put(new Long(3), indexMeta3);
         List<Column> index4Columns = Lists.newArrayList();
         Column index4Column2 = new Column("c2", Type.INT, true, null, true, 
"", "");
+        index4Column2.setDefineExpr(slotRef2);
         index4Columns.add(index4Column2);
         candidateIndexIdToSchema.put(new Long(4), indexMeta4);
+
+        List<Expr> whereList = Lists.newArrayList();
+        whereList.add(slotRef1);
         new Expectations() {
             {
                 selectStmt.getAggInfo();
                 result = null;
+                selectStmt.getWhereClause();
+                result = whereList;
                 indexMeta1.getSchema();
                 result = index1Columns;
                 indexMeta2.getSchema();
@@ -198,11 +201,20 @@ public class MaterializedViewSelectorTest {
                 result = index3Columns;
                 indexMeta4.getSchema();
                 result = index4Columns;
+                slotRef1.toSql();
+                result = "c1";
             }
         };
 
         MaterializedViewSelector selector = new 
MaterializedViewSelector(selectStmt, analyzer);
-        Deencapsulation.invoke(selector, "checkCompensatingPredicates", 
tableAColumnNames, candidateIndexIdToSchema);
+        try {
+            Deencapsulation.invoke(selector, "checkCompensatingPredicates", 
tableAColumnNames,
+                    candidateIndexIdToSchema);
+        } catch (Exception e) {
+            e.printStackTrace();
+            Assert.fail(e.getMessage());
+        }
+
         Assert.assertEquals(2, candidateIndexIdToSchema.size());
         Assert.assertTrue(candidateIndexIdToSchema.keySet().contains(new 
Long(1)));
         Assert.assertTrue(candidateIndexIdToSchema.keySet().contains(new 
Long(2)));
@@ -213,30 +225,40 @@ public class MaterializedViewSelectorTest {
             @Injectable OlapTable table,
             @Injectable MaterializedIndexMeta indexMeta1,
             @Injectable MaterializedIndexMeta indexMeta2,
-            @Injectable MaterializedIndexMeta indexMeta3) {
+            @Injectable MaterializedIndexMeta indexMeta3, @Injectable SlotRef 
slotRef1, @Injectable SlotRef slotRef2) {
         Set<String> tableAColumnNames = Sets.newHashSet();
         tableAColumnNames.add("C1");
         Map<Long, MaterializedIndexMeta> candidateIndexIdToSchema = 
Maps.newHashMap();
         List<Column> index1Columns = Lists.newArrayList();
         Column index1Column1 = new Column("c2", Type.INT, true, null, true, 
"", "");
+        index1Column1.setDefineExpr(slotRef2);
         index1Columns.add(index1Column1);
         candidateIndexIdToSchema.put(new Long(1), indexMeta1);
         List<Column> index2Columns = Lists.newArrayList();
         Column index2Column1 = new Column("c1", Type.INT, true, null, true, 
"", "");
+        index2Column1.setDefineExpr(slotRef1);
         index2Columns.add(index2Column1);
         Column index2Column2 = new Column("c2", Type.INT, false, 
AggregateType.SUM, true, "", "");
+        index2Column2.setDefineExpr(slotRef2);
         index2Columns.add(index2Column2);
         candidateIndexIdToSchema.put(new Long(2), indexMeta2);
         List<Column> index3Columns = Lists.newArrayList();
         Column index3Column1 = new Column("c2", Type.INT, true, null, true, 
"", "");
+        index3Column1.setDefineExpr(slotRef2);
         index3Columns.add(index3Column1);
         Column index3Column2 = new Column("c1", Type.INT, false, 
AggregateType.SUM, true, "", "");
+        index3Column1.setDefineExpr(slotRef1);
         index3Columns.add(index3Column2);
         candidateIndexIdToSchema.put(new Long(3), indexMeta3);
+        List<Expr> groupingList = Lists.newArrayList();
+        groupingList.add(slotRef1);
+        List<Expr> aggList = Lists.newArrayList();
         new Expectations() {
             {
-                selectStmt.getAggInfo();
-                result = null;
+                selectStmt.getAggInfo().getGroupingExprs();
+                result = groupingList;
+                selectStmt.getAggInfo().getAggregateExprs();
+                result = aggList;
                 indexMeta1.getSchema();
                 result = index1Columns;
                 indexMeta1.getKeysType();
@@ -245,12 +267,20 @@ public class MaterializedViewSelectorTest {
                 result = index2Columns;
                 indexMeta3.getSchema();
                 result = index3Columns;
+                slotRef1.toSql();
+                result = "c1";
             }
         };
 
-        MaterializedViewSelector selector = new 
MaterializedViewSelector(selectStmt, analyzer);
-        Deencapsulation.setField(selector, "isSPJQuery", false);
-        Deencapsulation.invoke(selector, "checkGrouping", table, 
tableAColumnNames, candidateIndexIdToSchema);
+        try {
+            MaterializedViewSelector selector = new 
MaterializedViewSelector(selectStmt, analyzer);
+            Deencapsulation.setField(selector, "isSPJQuery", false);
+            Deencapsulation.invoke(selector, "checkGrouping", table, 
tableAColumnNames, candidateIndexIdToSchema);
+        } catch (Exception e) {
+            e.printStackTrace();
+            Assert.fail(e.getMessage());
+        }
+
         Assert.assertEquals(2, candidateIndexIdToSchema.size());
         Assert.assertTrue(candidateIndexIdToSchema.keySet().contains(new 
Long(1)));
         Assert.assertTrue(candidateIndexIdToSchema.keySet().contains(new 
Long(2)));
@@ -301,8 +331,13 @@ public class MaterializedViewSelectorTest {
         Set<FunctionCallExpr> aggregatedColumnsInQueryOutput = 
Sets.newHashSet();
         aggregatedColumnsInQueryOutput.add(functionCallExpr);
         Deencapsulation.setField(selector, "isSPJQuery", false);
-        Deencapsulation.invoke(selector, "checkAggregationFunction", table, 
aggregatedColumnsInQueryOutput,
-                candidateIndexIdToSchema);
+        try {
+            Deencapsulation.invoke(selector, "checkAggregationFunction", 
table, aggregatedColumnsInQueryOutput,
+                    candidateIndexIdToSchema);
+        } catch (Exception e) {
+            e.printStackTrace();
+            Assert.fail(e.getMessage());
+        }
         Assert.assertEquals(2, candidateIndexIdToSchema.size());
         Assert.assertTrue(candidateIndexIdToSchema.keySet().contains(new 
Long(1)));
         Assert.assertTrue(candidateIndexIdToSchema.keySet().contains(new 
Long(3)));
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/statistics/MVStatisticsTest.java 
b/fe/fe-core/src/test/java/org/apache/doris/statistics/MVStatisticsTest.java
index 6040731a14..b65763c7f9 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/statistics/MVStatisticsTest.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/statistics/MVStatisticsTest.java
@@ -23,7 +23,6 @@ import org.apache.doris.statistics.util.StatisticsUtil;
 import org.apache.doris.system.SystemInfoService;
 import org.apache.doris.utframe.TestWithFeService;
 
-import mockit.Expectations;
 import mockit.Injectable;
 import mockit.Mock;
 import mockit.MockUp;
@@ -52,12 +51,6 @@ public class MVStatisticsTest extends TestWithFeService {
 
     @Test
     public void testCreate() throws Exception {
-        new Expectations() {
-            {
-                statisticsCache.refreshSync(anyLong, anyLong, anyString);
-                times = 5;
-            }
-        };
         new MockUp<StatisticsRepository>() {
         };
         new MockUp<StatisticsUtil>() {
diff --git a/fe/fe-core/src/test/java/org/apache/doris/utframe/DorisAssert.java 
b/fe/fe-core/src/test/java/org/apache/doris/utframe/DorisAssert.java
index 2b518dcb39..4b04d6c67a 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/utframe/DorisAssert.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/utframe/DorisAssert.java
@@ -173,7 +173,7 @@ public class DorisAssert {
         }
 
         public void explainContains(String... keywords) throws Exception {
-            
Assert.assertTrue(Stream.of(keywords).allMatch(explainQuery()::contains));
+            Assert.assertTrue(explainQuery(), 
Stream.of(keywords).allMatch(explainQuery()::contains));
         }
 
         public void explainContains(String keywords, int count) throws 
Exception {
diff --git a/regression-test/data/rollup/test_materialized_view_hll.out 
b/regression-test/data/rollup/test_materialized_view_hll.out
index 5d792bd68b..84c3e18727 100644
Binary files a/regression-test/data/rollup/test_materialized_view_hll.out and 
b/regression-test/data/rollup/test_materialized_view_hll.out differ
diff --git 
a/regression-test/data/rollup/test_materialized_view_hll_with_light_sc.out 
b/regression-test/data/rollup/test_materialized_view_hll_with_light_sc.out
index ad10392ab1..4b95174f3d 100644
Binary files 
a/regression-test/data/rollup/test_materialized_view_hll_with_light_sc.out and 
b/regression-test/data/rollup/test_materialized_view_hll_with_light_sc.out 
differ
diff --git a/regression-test/data/rollup_p0/test_materialized_view.out 
b/regression-test/data/rollup_p0/test_materialized_view.out
index c390165cf8..74bac182b1 100644
Binary files a/regression-test/data/rollup_p0/test_materialized_view.out and 
b/regression-test/data/rollup_p0/test_materialized_view.out differ
diff --git 
a/regression-test/suites/schema_change_p0/test_agg_keys_schema_change.groovy 
b/regression-test/suites/schema_change_p0/test_agg_keys_schema_change.groovy
index 5cf7fd59b5..f58ab5e37d 100644
--- a/regression-test/suites/schema_change_p0/test_agg_keys_schema_change.groovy
+++ b/regression-test/suites/schema_change_p0/test_agg_keys_schema_change.groovy
@@ -106,6 +106,7 @@ suite ("test_agg_keys_schema_change") {
             }
             Thread.sleep(100)
         }
+        Thread.sleep(1000)
 
         sql """ INSERT INTO ${tableName} 
(`user_id`,`date`,`city`,`age`,`sex`,`cost`,`max_dwell_time`,`min_dwell_time`, 
`hll_col`, `bitmap_col`)
                 VALUES


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

Reply via email to