This is an automated email from the ASF dual-hosted git repository.
englefly 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 812eed41b6a [improvement](mtmv) Add id to statistics map in statement
context for cost estimation later (#35436)
812eed41b6a is described below
commit 812eed41b6a326a3ce6ed5ab4ba00a8d94741154
Author: seawinde <[email protected]>
AuthorDate: Tue May 28 11:11:56 2024 +0800
[improvement](mtmv) Add id to statistics map in statement context for cost
estimation later (#35436)
Add id to statistics map in statement context for cost estimation later
this helps to improve the probability to use materialized view when
query a single table with aggregate and many filter
---
.../main/java/org/apache/doris/mtmv/MTMVCache.java | 14 ++-
.../org/apache/doris/nereids/StatementContext.java | 25 +++++
.../mv/AbstractMaterializedViewAggregateRule.java | 6 +-
.../mv/AbstractMaterializedViewJoinRule.java | 4 +-
.../mv/AbstractMaterializedViewRule.java | 17 ++-
.../mv/AsyncMaterializationContext.java | 39 +++++--
.../exploration/mv/MaterializationContext.java | 117 ++++++++++++---------
.../exploration/mv/MaterializedViewScanRule.java | 4 +-
.../IdStatisticsMapTest.java} | 22 ++--
.../nereids/{memo => mv}/MvTableIdIsLongTest.java | 2 +-
10 files changed, 171 insertions(+), 79 deletions(-)
diff --git a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVCache.java
b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVCache.java
index 8bd87e2e149..c65125de2f2 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVCache.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVCache.java
@@ -34,6 +34,7 @@ import
org.apache.doris.nereids.trees.plans.logical.LogicalResultSink;
import org.apache.doris.nereids.trees.plans.visitor.DefaultPlanRewriter;
import org.apache.doris.qe.ConnectContext;
import org.apache.doris.qe.OriginStatement;
+import org.apache.doris.statistics.Statistics;
import com.google.common.collect.ImmutableList;
@@ -47,10 +48,12 @@ public class MTMVCache {
private final Plan logicalPlan;
// The original plan of mv def sql
private final Plan originalPlan;
+ private final Statistics statistics;
- public MTMVCache(Plan logicalPlan, Plan originalPlan) {
+ public MTMVCache(Plan logicalPlan, Plan originalPlan, Statistics
statistics) {
this.logicalPlan = logicalPlan;
this.originalPlan = originalPlan;
+ this.statistics = statistics;
}
public Plan getLogicalPlan() {
@@ -61,6 +64,10 @@ public class MTMVCache {
return originalPlan;
}
+ public Statistics getStatistics() {
+ return statistics;
+ }
+
public static MTMVCache from(MTMV mtmv, ConnectContext connectContext) {
LogicalPlan unboundMvPlan = new
NereidsParser().parseSingle(mtmv.getQuerySql());
StatementContext mvSqlStatementContext = new
StatementContext(connectContext,
@@ -71,7 +78,8 @@ public class MTMVCache {
}
// Can not convert to table sink, because use the same column from
different table when self join
// the out slot is wrong
- Plan originPlan = planner.plan(unboundMvPlan, PhysicalProperties.ANY,
ExplainLevel.REWRITTEN_PLAN);
+ planner.plan(unboundMvPlan, PhysicalProperties.ANY,
ExplainLevel.ALL_PLAN);
+ Plan originPlan = planner.getCascadesContext().getRewritePlan();
// Eliminate result sink because sink operator is useless in query
rewrite by materialized view
// and the top sort can also be removed
Plan mvPlan = originPlan.accept(new DefaultPlanRewriter<Object>() {
@@ -88,6 +96,6 @@ public class MTMVCache {
ImmutableList.of(Rewriter.custom(RuleType.ELIMINATE_SORT,
EliminateSort::new))).execute();
return childContext.getRewritePlan();
}, mvPlan, originPlan);
- return new MTMVCache(mvPlan, originPlan);
+ return new MTMVCache(mvPlan, originPlan,
planner.getCascadesContext().getMemo().getRoot().getStatistics());
}
}
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java
index c2240e536d3..0d658b9e74f 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/StatementContext.java
@@ -21,6 +21,7 @@ import org.apache.doris.analysis.StatementBase;
import org.apache.doris.catalog.Column;
import org.apache.doris.catalog.TableIf;
import org.apache.doris.catalog.constraint.TableIdentifier;
+import org.apache.doris.common.Id;
import org.apache.doris.common.IdGenerator;
import org.apache.doris.common.Pair;
import org.apache.doris.nereids.hint.Hint;
@@ -42,7 +43,9 @@ import org.apache.doris.qe.ConnectContext;
import org.apache.doris.qe.OriginStatement;
import org.apache.doris.qe.SessionVariable;
import org.apache.doris.qe.cache.CacheAnalyzer;
+import org.apache.doris.statistics.Statistics;
+import com.google.common.annotations.VisibleForTesting;
import com.google.common.base.Stopwatch;
import com.google.common.base.Supplier;
import com.google.common.base.Suppliers;
@@ -149,6 +152,10 @@ public class StatementContext implements Closeable {
// Record table id mapping, the key is the hash code of union catalogId,
databaseId, tableId
// the value is the auto-increment id in the cascades context
private final Map<TableIdentifier, TableId> tableIdMapping = new
LinkedHashMap<>();
+ // Record the materialization statistics by id which is used for cost
estimation.
+ // Maybe return null, which means the id according statistics should calc
normally rather than getting
+ // form this map
+ private final Map<RelationId, Statistics> relationIdToStatisticsMap = new
LinkedHashMap<>();
public StatementContext() {
this(ConnectContext.get(), null, 0);
@@ -415,6 +422,24 @@ public class StatementContext implements Closeable {
indexInSqlToString.put(pair, replacement);
}
+ public void addStatistics(Id id, Statistics statistics) {
+ if (id instanceof RelationId) {
+ this.relationIdToStatisticsMap.put((RelationId) id, statistics);
+ }
+ }
+
+ public Optional<Statistics> getStatistics(Id id) {
+ if (id instanceof RelationId) {
+ return
Optional.ofNullable(this.relationIdToStatisticsMap.get((RelationId) id));
+ }
+ return Optional.empty();
+ }
+
+ @VisibleForTesting
+ public Map<RelationId, Statistics> getRelationIdToStatisticsMap() {
+ return relationIdToStatisticsMap;
+ }
+
/** addTableReadLock */
public synchronized void addTableReadLock(TableIf tableIf) {
if (!tableIf.needReadLockWhenPlan()) {
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
index b37b04d8022..1e013498a12 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewAggregateRule.java
@@ -97,7 +97,7 @@ public abstract class AbstractMaterializedViewAggregateRule
extends AbstractMate
viewStructInfo)) {
List<Expression> rewrittenQueryExpressions =
rewriteExpression(queryTopPlan.getOutput(),
queryTopPlan,
- materializationContext.getMvExprToMvScanExprMapping(),
+ materializationContext.getExprToScanExprMapping(),
viewToQuerySlotMapping,
true,
queryStructInfo.getTableBitSet());
@@ -121,7 +121,7 @@ public abstract class AbstractMaterializedViewAggregateRule
extends AbstractMate
() -> String.format("expressionToWrite = %s,\n
mvExprToMvScanExprMapping = %s,\n"
+ "viewToQuerySlotMapping = %s",
queryTopPlan.getOutput(),
-
materializationContext.getMvExprToMvScanExprMapping(),
+ materializationContext.getExprToScanExprMapping(),
viewToQuerySlotMapping));
}
// if view is scalar aggregate but query is not. Or if query is scalar
aggregate but view is not
@@ -150,7 +150,7 @@ public abstract class AbstractMaterializedViewAggregateRule
extends AbstractMate
List<? extends Expression> queryExpressions = queryTopPlan.getOutput();
// permute the mv expr mapping to query based
Map<Expression, Expression> mvExprToMvScanExprQueryBased =
-
materializationContext.getMvExprToMvScanExprMapping().keyPermute(viewToQuerySlotMapping)
+
materializationContext.getExprToScanExprMapping().keyPermute(viewToQuerySlotMapping)
.flattenMap().get(0);
for (Expression topExpression : queryExpressions) {
// if agg function, try to roll up and rewrite
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java
index 3b20cefbba8..4f95c248eca 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewJoinRule.java
@@ -46,7 +46,7 @@ public abstract class AbstractMaterializedViewJoinRule
extends AbstractMateriali
List<Expression> expressionsRewritten = rewriteExpression(
queryStructInfo.getExpressions(),
queryStructInfo.getTopPlan(),
- materializationContext.getMvExprToMvScanExprMapping(),
+ materializationContext.getExprToScanExprMapping(),
targetToSourceMapping,
true,
queryStructInfo.getTableBitSet()
@@ -57,7 +57,7 @@ public abstract class AbstractMaterializedViewJoinRule
extends AbstractMateriali
"Rewrite expressions by view in join fail",
() -> String.format("expressionToRewritten is %s,\n
mvExprToMvScanExprMapping is %s,\n"
+ "targetToSourceMapping = %s",
queryStructInfo.getExpressions(),
-
materializationContext.getMvExprToMvScanExprMapping(),
+ materializationContext.getExprToScanExprMapping(),
targetToSourceMapping));
return null;
}
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
index 8442d2485c7..ea057e8bfcc 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AbstractMaterializedViewRule.java
@@ -24,6 +24,7 @@ import org.apache.doris.catalog.PartitionInfo;
import org.apache.doris.catalog.PartitionItem;
import org.apache.doris.catalog.PartitionType;
import org.apache.doris.catalog.TableIf;
+import org.apache.doris.common.Id;
import org.apache.doris.common.Pair;
import org.apache.doris.mtmv.BaseTableInfo;
import org.apache.doris.mtmv.MTMVPartitionInfo;
@@ -59,6 +60,7 @@ import
org.apache.doris.nereids.trees.plans.logical.LogicalProject;
import org.apache.doris.nereids.trees.plans.logical.LogicalUnion;
import org.apache.doris.nereids.util.ExpressionUtils;
import org.apache.doris.nereids.util.TypeUtils;
+import org.apache.doris.statistics.Statistics;
import com.google.common.collect.ImmutableList;
import com.google.common.collect.ImmutableMultimap;
@@ -77,6 +79,7 @@ import java.util.LinkedHashMap;
import java.util.List;
import java.util.Map;
import java.util.Objects;
+import java.util.Optional;
import java.util.Set;
import java.util.stream.Collectors;
@@ -226,21 +229,21 @@ public abstract class AbstractMaterializedViewRule
implements ExplorationRuleFac
continue;
}
Plan rewrittenPlan;
- Plan mvScan = materializationContext.getMvScanPlan();
+ Plan mvScan = materializationContext.getScanPlan();
Plan queryPlan = queryStructInfo.getTopPlan();
if (compensatePredicates.isAlwaysTrue()) {
rewrittenPlan = mvScan;
} else {
// Try to rewrite compensate predicates by using mv scan
List<Expression> rewriteCompensatePredicates =
rewriteExpression(compensatePredicates.toList(),
- queryPlan,
materializationContext.getMvExprToMvScanExprMapping(),
+ queryPlan,
materializationContext.getExprToScanExprMapping(),
viewToQuerySlotMapping, true,
queryStructInfo.getTableBitSet());
if (rewriteCompensatePredicates.isEmpty()) {
materializationContext.recordFailReason(queryStructInfo,
"Rewrite compensate predicate by view fail",
() -> String.format("compensatePredicates = %s,\n
mvExprToMvScanExprMapping = %s,\n"
+ "viewToQuerySlotMapping = %s",
- compensatePredicates,
materializationContext.getMvExprToMvScanExprMapping(),
+ compensatePredicates,
materializationContext.getExprToScanExprMapping(),
viewToQuerySlotMapping));
continue;
}
@@ -334,9 +337,15 @@ public abstract class AbstractMaterializedViewRule
implements ExplorationRuleFac
continue;
}
recordIfRewritten(queryStructInfo.getOriginalPlan(),
materializationContext);
+ Optional<Pair<Id, Statistics>> materializationPlanStatistics =
+ materializationContext.getPlanStatistics(cascadesContext);
+ if (materializationPlanStatistics.isPresent() &&
materializationPlanStatistics.get().key() != null) {
+ cascadesContext.getStatementContext().addStatistics(
+ materializationPlanStatistics.get().key(),
materializationPlanStatistics.get().value());
+ }
rewriteResults.add(rewrittenPlan);
// if rewrite successfully, try to regenerate mv scan because it
maybe used again
- materializationContext.tryReGenerateMvScanPlan(cascadesContext);
+ materializationContext.tryReGenerateScanPlan(cascadesContext);
}
return rewriteResults;
}
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AsyncMaterializationContext.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AsyncMaterializationContext.java
index 1c0d854dd90..f1c4372f104 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AsyncMaterializationContext.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/AsyncMaterializationContext.java
@@ -19,14 +19,20 @@ package org.apache.doris.nereids.rules.exploration.mv;
import org.apache.doris.catalog.MTMV;
import org.apache.doris.catalog.Table;
+import org.apache.doris.common.AnalysisException;
+import org.apache.doris.common.Id;
import org.apache.doris.common.Pair;
+import org.apache.doris.mtmv.MTMVCache;
import org.apache.doris.nereids.CascadesContext;
import org.apache.doris.nereids.rules.exploration.mv.mapping.ExpressionMapping;
import org.apache.doris.nereids.trees.plans.ObjectId;
import org.apache.doris.nereids.trees.plans.Plan;
+import org.apache.doris.nereids.trees.plans.RelationId;
import org.apache.doris.nereids.trees.plans.algebra.Relation;
+import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan;
import org.apache.doris.nereids.trees.plans.physical.PhysicalCatalogRelation;
import org.apache.doris.nereids.util.Utils;
+import org.apache.doris.statistics.Statistics;
import com.google.common.collect.Multimap;
import org.apache.logging.log4j.LogManager;
@@ -35,6 +41,7 @@ import org.apache.logging.log4j.Logger;
import java.util.Collection;
import java.util.List;
import java.util.Map;
+import java.util.Optional;
/**
* Async context for query rewrite by materialized view
@@ -58,7 +65,7 @@ public class AsyncMaterializationContext extends
MaterializationContext {
}
@Override
- Plan doGenerateMvPlan(CascadesContext cascadesContext) {
+ Plan doGenerateScanPlan(CascadesContext cascadesContext) {
return MaterializedViewUtils.generateMvScanPlan(this.mtmv,
cascadesContext);
}
@@ -85,6 +92,24 @@ public class AsyncMaterializationContext extends
MaterializationContext {
"failReason", failReasonBuilder.toString());
}
+ @Override
+ public Optional<Pair<Id, Statistics>> getPlanStatistics(CascadesContext
cascadesContext) {
+ MTMVCache mtmvCache;
+ try {
+ mtmvCache =
mtmv.getOrGenerateCache(cascadesContext.getConnectContext());
+ } catch (AnalysisException e) {
+ LOG.warn(String.format("get mv plan statistics fail,
materialization qualifier is %s",
+ getMaterializationQualifier()), e);
+ return Optional.empty();
+ }
+ RelationId relationId = null;
+ List<Object> scanObjs = this.getPlan().collectFirst(plan -> plan
instanceof LogicalOlapScan);
+ if (scanObjs != null && !scanObjs.isEmpty()) {
+ relationId = ((LogicalOlapScan) scanObjs.get(0)).getRelationId();
+ }
+ return Optional.of(Pair.of(relationId, mtmvCache.getStatistics()));
+ }
+
@Override
boolean isFinalChosen(Relation relation) {
if (!(relation instanceof PhysicalCatalogRelation)) {
@@ -93,8 +118,8 @@ public class AsyncMaterializationContext extends
MaterializationContext {
return ((PhysicalCatalogRelation) relation).getTable() instanceof MTMV;
}
- public Plan getMvScanPlan() {
- return mvScanPlan;
+ public Plan getScanPlan() {
+ return scanPlan;
}
public List<Table> getBaseTables() {
@@ -105,16 +130,16 @@ public class AsyncMaterializationContext extends
MaterializationContext {
return baseViews;
}
- public ExpressionMapping getMvExprToMvScanExprMapping() {
- return mvExprToMvScanExprMapping;
+ public ExpressionMapping getExprToScanExprMapping() {
+ return exprToScanExprMapping;
}
public boolean isAvailable() {
return available;
}
- public Plan getMvPlan() {
- return mvPlan;
+ public Plan getPlan() {
+ return plan;
}
public Multimap<ObjectId, Pair<String, String>> getFailReason() {
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java
index 13e8c4456c0..d67390476eb 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializationContext.java
@@ -19,6 +19,7 @@ package org.apache.doris.nereids.rules.exploration.mv;
import org.apache.doris.analysis.StatementBase;
import org.apache.doris.catalog.Table;
+import org.apache.doris.common.Id;
import org.apache.doris.common.Pair;
import org.apache.doris.nereids.CascadesContext;
import org.apache.doris.nereids.memo.GroupId;
@@ -34,6 +35,7 @@ import
org.apache.doris.nereids.trees.plans.physical.PhysicalPlan;
import org.apache.doris.nereids.trees.plans.physical.PhysicalRelation;
import org.apache.doris.nereids.trees.plans.visitor.DefaultPlanVisitor;
import org.apache.doris.nereids.util.ExpressionUtils;
+import org.apache.doris.statistics.Statistics;
import com.google.common.collect.HashMultimap;
import com.google.common.collect.ImmutableSet;
@@ -47,6 +49,7 @@ import java.util.HashSet;
import java.util.List;
import java.util.Map;
import java.util.Objects;
+import java.util.Optional;
import java.util.Set;
import java.util.function.Supplier;
import java.util.stream.Collectors;
@@ -56,69 +59,75 @@ import java.util.stream.Collectors;
*/
public abstract class MaterializationContext {
private static final Logger LOG =
LogManager.getLogger(MaterializationContext.class);
- public final Map<RelationMapping, SlotMapping> queryToMvSlotMappingCache =
new HashMap<>();
+ public final Map<RelationMapping, SlotMapping>
queryToMaterializationSlotMappingCache = new HashMap<>();
protected List<Table> baseTables;
protected List<Table> baseViews;
- // The plan of mv def sql
- protected Plan mvPlan;
- // The original plan of mv def sql
- protected Plan originalMvPlan;
+ // The plan of materialization def sql
+ protected Plan plan;
+ // The original plan of materialization sql
+ protected Plan originalPlan;
// Should regenerate when materialization is already rewritten
successfully because one query may hit repeatedly
// make sure output is different in multi using
- protected Plan mvScanPlan;
- // The mvPlan output shuttled expression, this is used by generate field
mvExprToMvScanExprMapping
- protected List<? extends Expression> mvPlanOutputShuttledExpressions;
- // Generated mapping from mv plan out shuttled expr to mv scan plan out
slot mapping, this is used for later used
- protected ExpressionMapping mvExprToMvScanExprMapping;
+ protected Plan scanPlan;
+ // The materialization plan output shuttled expression, this is used by
generate field
+ // exprToScanExprMapping
+ protected List<? extends Expression> planOutputShuttledExpressions;
+ // Generated mapping from materialization plan out shuttled expr to
materialization scan plan out slot mapping,
+ // this is used for later used
+ protected ExpressionMapping exprToScanExprMapping;
// This mark the materialization context is available or not,
// will not be used in query transparent rewritten if false
protected boolean available = true;
- // Mark the mv plan in the context is already rewritten successfully or not
+ // Mark the materialization plan in the context is already rewritten
successfully or not
protected boolean success = false;
// Mark enable record failure detail info or not, because record failure
detail info is performance-depleting
protected final boolean enableRecordFailureDetail;
- // The mv plan struct info
+ // The materialization plan struct info
protected final StructInfo structInfo;
- // Group id set that are rewritten unsuccessfully by this mv for reducing
rewrite times
+ // Group id set that are rewritten unsuccessfully by this materialization
for reducing rewrite times
protected final Set<GroupId> matchedFailGroups = new HashSet<>();
- // Group id set that are rewritten successfully by this mv for reducing
rewrite times
+ // Group id set that are rewritten successfully by this materialization
for reducing rewrite times
protected final Set<GroupId> matchedSuccessGroups = new HashSet<>();
- // Record the reason, if rewrite by mv fail. The failReason should be
empty if success.
+ // Record the reason, if rewrite by materialization fail. The failReason
should be empty if success.
// The key is the query belonged group expression objectId, the value is
the fail reasons because
// for one materialization query may be multi when nested materialized
view.
protected final Multimap<ObjectId, Pair<String, String>> failReason =
HashMultimap.create();
/**
- * MaterializationContext, this contains necessary info for query
rewriting by mv
+ * MaterializationContext, this contains necessary info for query
rewriting by materialization
*/
- public MaterializationContext(Plan mvPlan, Plan originalMvPlan, Plan
mvScanPlan, CascadesContext cascadesContext) {
- this.mvPlan = mvPlan;
- this.originalMvPlan = originalMvPlan;
- this.mvScanPlan = mvScanPlan;
+ public MaterializationContext(Plan plan, Plan originalPlan,
+ Plan scanPlan, CascadesContext cascadesContext) {
+ this.plan = plan;
+ this.originalPlan = originalPlan;
+ this.scanPlan = scanPlan;
StatementBase parsedStatement =
cascadesContext.getStatementContext().getParsedStatement();
this.enableRecordFailureDetail = parsedStatement != null &&
parsedStatement.isExplain()
&& ExplainLevel.MEMO_PLAN ==
parsedStatement.getExplainOptions().getExplainLevel();
- this.mvPlanOutputShuttledExpressions =
ExpressionUtils.shuttleExpressionWithLineage(
- originalMvPlan.getOutput(),
- originalMvPlan,
+ this.planOutputShuttledExpressions =
ExpressionUtils.shuttleExpressionWithLineage(
+ originalPlan.getOutput(),
+ originalPlan,
new BitSet());
- // mv output expression shuttle, this will be used to expression
rewrite
- this.mvExprToMvScanExprMapping =
ExpressionMapping.generate(this.mvPlanOutputShuttledExpressions,
- this.mvScanPlan.getOutput());
- // Construct mv struct info, catch exception which may cause planner
roll back
+ // materialization output expression shuttle, this will be used to
expression rewrite
+ this.exprToScanExprMapping = ExpressionMapping.generate(
+ this.planOutputShuttledExpressions,
+ this.scanPlan.getOutput());
+ // Construct materialization struct info, catch exception which may
cause planner roll back
List<StructInfo> viewStructInfos;
try {
- viewStructInfos = MaterializedViewUtils.extractStructInfo(mvPlan,
cascadesContext, new BitSet());
+ viewStructInfos = MaterializedViewUtils.extractStructInfo(plan,
cascadesContext, new BitSet());
if (viewStructInfos.size() > 1) {
// view struct info should only have one, log error and use
the first struct info
- LOG.warn(String.format("view strut info is more than one, mv
scan plan is %s, mv plan is %s",
- mvScanPlan.treeString(), mvPlan.treeString()));
+ LOG.warn(String.format("view strut info is more than one,
materialization scan plan is %s, "
+ + "materialization plan is %s",
+ scanPlan.treeString(), plan.treeString()));
}
} catch (Exception exception) {
- LOG.warn(String.format("construct mv struct info fail, mv scan
plan is %s, mv plan is %s",
- mvScanPlan.treeString(), mvPlan.treeString()), exception);
+ LOG.warn(String.format("construct materialization struct info
fail, materialization scan plan is %s, "
+ + "materialization plan is %s",
+ scanPlan.treeString(), plan.treeString()), exception);
this.available = false;
this.structInfo = null;
return;
@@ -141,32 +150,33 @@ public abstract class MaterializationContext {
/**
* Try to generate scan plan for materialization
* if MaterializationContext is already rewritten successfully, then
should generate new scan plan in later
- * query rewrite, because one plan may hit the materialized view
repeatedly and the mv scan output
+ * query rewrite, because one plan may hit the materialized view
repeatedly and the materialization scan output
* should be different.
* This method should be called when query rewrite successfully
*/
- public void tryReGenerateMvScanPlan(CascadesContext cascadesContext) {
- this.mvScanPlan = doGenerateMvPlan(cascadesContext);
- // mv output expression shuttle, this will be used to expression
rewrite
- this.mvExprToMvScanExprMapping =
ExpressionMapping.generate(this.mvPlanOutputShuttledExpressions,
- this.mvScanPlan.getOutput());
+ public void tryReGenerateScanPlan(CascadesContext cascadesContext) {
+ this.scanPlan = doGenerateScanPlan(cascadesContext);
+ // materialization output expression shuttle, this will be used to
expression rewrite
+ this.exprToScanExprMapping = ExpressionMapping.generate(
+ this.planOutputShuttledExpressions,
+ this.scanPlan.getOutput());
}
public void addSlotMappingToCache(RelationMapping relationMapping,
SlotMapping slotMapping) {
- queryToMvSlotMappingCache.put(relationMapping, slotMapping);
+ queryToMaterializationSlotMappingCache.put(relationMapping,
slotMapping);
}
public SlotMapping getSlotMappingFromCache(RelationMapping
relationMapping) {
- return queryToMvSlotMappingCache.get(relationMapping);
+ return queryToMaterializationSlotMappingCache.get(relationMapping);
}
/**
* Try to generate scan plan for materialization
* if MaterializationContext is already rewritten successfully, then
should generate new scan plan in later
- * query rewrite, because one plan may hit the materialized view
repeatedly and the mv scan output
+ * query rewrite, because one plan may hit the materialized view
repeatedly and the materialization scan output
* should be different
*/
- abstract Plan doGenerateMvPlan(CascadesContext cascadesContext);
+ abstract Plan doGenerateScanPlan(CascadesContext cascadesContext);
/**
* Get materialization unique qualifier which identify it
@@ -178,21 +188,28 @@ public abstract class MaterializationContext {
*/
abstract String getStringInfo();
+ /**
+ * Get materialization plan statistics, the key is the identifier of
statistics
+ * the value is Statistics.
+ * the statistics is used by cost estimation when the materialization is
used
+ */
+ abstract Optional<Pair<Id, Statistics>> getPlanStatistics(CascadesContext
cascadesContext);
+
/**
* Calc the relation is chosen finally or not
*/
abstract boolean isFinalChosen(Relation relation);
- public Plan getMvPlan() {
- return mvPlan;
+ public Plan getPlan() {
+ return plan;
}
- public Plan getOriginalMvPlan() {
- return originalMvPlan;
+ public Plan getOriginalPlan() {
+ return originalPlan;
}
- public Plan getMvScanPlan() {
- return mvScanPlan;
+ public Plan getScanPlan() {
+ return scanPlan;
}
public List<Table> getBaseTables() {
@@ -203,8 +220,8 @@ public abstract class MaterializationContext {
return baseViews;
}
- public ExpressionMapping getMvExprToMvScanExprMapping() {
- return mvExprToMvScanExprMapping;
+ public ExpressionMapping getExprToScanExprMapping() {
+ return exprToScanExprMapping;
}
public boolean isAvailable() {
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewScanRule.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewScanRule.java
index d6d7817d35f..82e7944a81e 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewScanRule.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/mv/MaterializedViewScanRule.java
@@ -47,7 +47,7 @@ public abstract class MaterializedViewScanRule extends
AbstractMaterializedViewR
List<Expression> expressionsRewritten = rewriteExpression(
queryStructInfo.getExpressions(),
queryStructInfo.getTopPlan(),
- materializationContext.getMvExprToMvScanExprMapping(),
+ materializationContext.getExprToScanExprMapping(),
targetToSourceMapping,
true,
queryStructInfo.getTableBitSet()
@@ -58,7 +58,7 @@ public abstract class MaterializedViewScanRule extends
AbstractMaterializedViewR
"Rewrite expressions by view in scan fail",
() -> String.format("expressionToRewritten is %s,\n
mvExprToMvScanExprMapping is %s,\n"
+ "targetToSourceMapping = %s",
queryStructInfo.getExpressions(),
-
materializationContext.getMvExprToMvScanExprMapping(),
+ materializationContext.getExprToScanExprMapping(),
targetToSourceMapping));
return null;
}
diff --git
a/fe/fe-core/src/test/java/org/apache/doris/nereids/memo/MvTableIdIsLongTest.java
b/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/IdStatisticsMapTest.java
similarity index 74%
copy from
fe/fe-core/src/test/java/org/apache/doris/nereids/memo/MvTableIdIsLongTest.java
copy to
fe/fe-core/src/test/java/org/apache/doris/nereids/mv/IdStatisticsMapTest.java
index 75aa7718c00..6660457b884 100644
---
a/fe/fe-core/src/test/java/org/apache/doris/nereids/memo/MvTableIdIsLongTest.java
+++
b/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/IdStatisticsMapTest.java
@@ -15,15 +15,17 @@
// specific language governing permissions and limitations
// under the License.
-package org.apache.doris.nereids.memo;
+package org.apache.doris.nereids.mv;
import org.apache.doris.catalog.MTMV;
import org.apache.doris.mtmv.MTMVRelationManager;
import org.apache.doris.nereids.CascadesContext;
import org.apache.doris.nereids.sqltest.SqlTestBase;
+import org.apache.doris.nereids.trees.plans.RelationId;
import org.apache.doris.nereids.util.PlanChecker;
import org.apache.doris.qe.ConnectContext;
import org.apache.doris.qe.SessionVariable;
+import org.apache.doris.statistics.Statistics;
import mockit.Mock;
import mockit.MockUp;
@@ -31,14 +33,16 @@ import org.junit.jupiter.api.Assertions;
import org.junit.jupiter.api.Test;
import java.util.BitSet;
+import java.util.Map;
+import java.util.Optional;
/**
- * Test mv rewrite when base table id is lager then integer
+ * Test idStatisticsMap in StatementContext is valid
*/
-public class MvTableIdIsLongTest extends SqlTestBase {
+public class IdStatisticsMapTest extends SqlTestBase {
@Test
- void testMvRewriteWhenBaseTableIdIsLong() throws Exception {
+ void testIdStatisticsIsExistWhenRewriteByMv() throws Exception {
connectContext.getSessionVariable().setDisableNereidsRules("PRUNE_EMPTY_PARTITION");
BitSet disableNereidsRules =
connectContext.getSessionVariable().getDisableNereidsRules();
new MockUp<SessionVariable>() {
@@ -55,7 +59,7 @@ public class MvTableIdIsLongTest extends SqlTestBase {
};
connectContext.getSessionVariable().enableMaterializedViewRewrite =
true;
connectContext.getSessionVariable().enableMaterializedViewNestRewrite
= true;
- createMvByNereids("create materialized view mv1 BUILD IMMEDIATE
REFRESH COMPLETE ON MANUAL\n"
+ createMvByNereids("create materialized view mv100 BUILD IMMEDIATE
REFRESH COMPLETE ON MANUAL\n"
+ " DISTRIBUTED BY RANDOM BUCKETS 1\n"
+ " PROPERTIES ('replication_num' = '1') \n"
+ " as select T1.id from T1 inner join T2 "
@@ -71,7 +75,11 @@ public class MvTableIdIsLongTest extends SqlTestBase {
.rewrite()
.optimize()
.printlnBestPlanTree();
- Assertions.assertTrue(c1.getMemo().toString().contains("mv1"));
- dropMvByNereids("drop materialized view mv1");
+ Map<RelationId, Statistics> idStatisticsMap =
c1.getStatementContext().getRelationIdToStatisticsMap();
+ Assertions.assertFalse(idStatisticsMap.isEmpty());
+ RelationId relationId = idStatisticsMap.keySet().iterator().next();
+ Optional<Statistics> statistics =
c1.getStatementContext().getStatistics(relationId);
+ Assertions.assertTrue(statistics.isPresent());
+ dropMvByNereids("drop materialized view mv100");
}
}
diff --git
a/fe/fe-core/src/test/java/org/apache/doris/nereids/memo/MvTableIdIsLongTest.java
b/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/MvTableIdIsLongTest.java
similarity index 98%
rename from
fe/fe-core/src/test/java/org/apache/doris/nereids/memo/MvTableIdIsLongTest.java
rename to
fe/fe-core/src/test/java/org/apache/doris/nereids/mv/MvTableIdIsLongTest.java
index 75aa7718c00..fd3887d3cfd 100644
---
a/fe/fe-core/src/test/java/org/apache/doris/nereids/memo/MvTableIdIsLongTest.java
+++
b/fe/fe-core/src/test/java/org/apache/doris/nereids/mv/MvTableIdIsLongTest.java
@@ -15,7 +15,7 @@
// specific language governing permissions and limitations
// under the License.
-package org.apache.doris.nereids.memo;
+package org.apache.doris.nereids.mv;
import org.apache.doris.catalog.MTMV;
import org.apache.doris.mtmv.MTMVRelationManager;
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]