Skip to content

Commit

Permalink
[feature](nereids) Multi table materialized view init
Browse files Browse the repository at this point in the history
  • Loading branch information
seawinde committed Nov 15, 2023
1 parent 2c6d225 commit 11912b9
Show file tree
Hide file tree
Showing 22 changed files with 1,246 additions and 0 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -44,6 +44,7 @@
import org.apache.doris.nereids.rules.RuleFactory;
import org.apache.doris.nereids.rules.RuleSet;
import org.apache.doris.nereids.rules.analysis.BindRelation.CustomTableResolver;
import org.apache.doris.nereids.rules.exploration.mv.MaterializationContext;
import org.apache.doris.nereids.trees.expressions.CTEId;
import org.apache.doris.nereids.trees.expressions.Expression;
import org.apache.doris.nereids.trees.expressions.NamedExpression;
Expand Down Expand Up @@ -112,6 +113,8 @@ public class CascadesContext implements ScheduleContext {
private final Optional<CTEId> currentTree;
private final Optional<CascadesContext> parent;

private List<MaterializationContext> materializationContexts;

/**
* Constructor of OptimizerContext.
*
Expand All @@ -133,6 +136,7 @@ private CascadesContext(Optional<CascadesContext> parent, Optional<CTEId> curren
this.currentJobContext = new JobContext(this, requireProperties, Double.MAX_VALUE);
this.subqueryExprIsAnalyzed = new HashMap<>();
this.runtimeFilterContext = new RuntimeFilterContext(getConnectContext().getSessionVariable());
this.materializationContexts = new ArrayList<>();
}

/**
Expand Down Expand Up @@ -309,6 +313,14 @@ public void setOuterScope(@Nullable Scope outerScope) {
this.outerScope = Optional.ofNullable(outerScope);
}

public List<MaterializationContext> getMaterializationContexts() {
return materializationContexts;
}

public void addMaterializationContext(MaterializationContext materializationContext) {
this.materializationContexts.add(materializationContext);
}

/**
* getAndCacheSessionVariable
*/
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -259,6 +259,10 @@ private void initCascadesContext(LogicalPlan plan, PhysicalProperties requirePro
if (statementContext.getConnectContext().getTables() != null) {
cascadesContext.setTables(statementContext.getConnectContext().getTables());
}
if (statementContext.getConnectContext().getSessionVariable().enableMaterializedViewRewrite) {
// TODO Pre handle materialized view to materializationContext and
// call cascadesContext.addMaterializationContext() to add it
}
}

private void analyze() {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -46,6 +46,9 @@ public void execute() {
countJobExecutionTimesOfGroupExpressions(groupExpression);
List<Rule> implementationRules = getRuleSet().getImplementationRules();
List<Rule> explorationRules = getExplorationRules();
if (context.getCascadesContext().getConnectContext().getSessionVariable().isEnableMaterializedViewRewrite()) {
explorationRules.addAll(getRuleSet().getMaterializedViewRules());
}

for (Rule rule : explorationRules) {
if (rule.isInvalid(disableRules, groupExpression)) {
Expand Down
11 changes: 11 additions & 0 deletions fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Group.java
Original file line number Diff line number Diff line change
Expand Up @@ -21,6 +21,7 @@
import org.apache.doris.nereids.cost.Cost;
import org.apache.doris.nereids.properties.LogicalProperties;
import org.apache.doris.nereids.properties.PhysicalProperties;
import org.apache.doris.nereids.rules.exploration.mv.StructInfo;
import org.apache.doris.nereids.trees.expressions.literal.Literal;
import org.apache.doris.nereids.trees.plans.JoinType;
import org.apache.doris.nereids.trees.plans.Plan;
Expand Down Expand Up @@ -74,6 +75,8 @@ public class Group {

private int chosenGroupExpressionId = -1;

private Optional<StructInfo> structInfo = Optional.empty();

/**
* Constructor for Group.
*
Expand Down Expand Up @@ -532,4 +535,12 @@ public String treeString() {

return TreeStringUtils.treeString(this, toString, getChildren, getExtraPlans, displayExtraPlan);
}

public Optional<StructInfo> getStructInfo() {
return structInfo;
}

public void setStructInfo(StructInfo structInfo) {
this.structInfo = Optional.ofNullable(structInfo);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -40,6 +40,7 @@
import org.apache.doris.nereids.rules.exploration.join.PushdownProjectThroughSemiJoin;
import org.apache.doris.nereids.rules.exploration.join.SemiJoinSemiJoinTranspose;
import org.apache.doris.nereids.rules.exploration.join.SemiJoinSemiJoinTransposeProject;
import org.apache.doris.nereids.rules.exploration.mv.MaterializedViewProjectJoinRule;
import org.apache.doris.nereids.rules.implementation.AggregateStrategies;
import org.apache.doris.nereids.rules.implementation.LogicalAssertNumRowsToPhysicalAssertNumRows;
import org.apache.doris.nereids.rules.implementation.LogicalCTEAnchorToPhysicalCTEAnchor;
Expand Down Expand Up @@ -220,6 +221,10 @@ public class RuleSet {
.add(JoinCommute.BUSHY.build())
.build();

public static final List<Rule> MATERIALIZED_VIEW_RULES = planRuleFactories()
.add(MaterializedViewProjectJoinRule.INSTANCE)
.build();

public List<Rule> getDPHypReorderRules() {
return DPHYP_REORDER_RULES;
}
Expand All @@ -240,6 +245,10 @@ public List<Rule> getImplementationRules() {
return IMPLEMENTATION_RULES;
}

public List<Rule> getMaterializedViewRules() {
return MATERIALIZED_VIEW_RULES;
}

public static RuleFactories planRuleFactories() {
return new RuleFactories();
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -225,6 +225,24 @@ public enum RuleType {
MATERIALIZED_INDEX_PROJECT_SCAN(RuleTypeClass.REWRITE),
MATERIALIZED_INDEX_PROJECT_FILTER_SCAN(RuleTypeClass.REWRITE),
MATERIALIZED_INDEX_FILTER_PROJECT_SCAN(RuleTypeClass.REWRITE),

MATERIALIZED_VIEW_PROJECT_JOIN(RuleTypeClass.REWRITE),
MATERIALIZED_VIEW_FILTER_JOIN(RuleTypeClass.REWRITE),
MATERIALIZED_VIEW_PROJECT_FILTER_JOIN(RuleTypeClass.REWRITE),
MATERIALIZED_VIEW_FILTER_PROJECT_JOIN(RuleTypeClass.REWRITE),
MATERIALIZED_VIEW_ONLY_JOIN(RuleTypeClass.REWRITE),

MATERIALIZED_VIEW_PROJECT_AGGREGATE(RuleTypeClass.REWRITE),
MATERIALIZED_VIEW_FILTER_AGGREGATE(RuleTypeClass.REWRITE),
MATERIALIZED_VIEW_PROJECT_FILTER_AGGREGATE(RuleTypeClass.REWRITE),
MATERIALIZED_VIEW_FILTER_PROJECT_AGGREGATE(RuleTypeClass.REWRITE),
MATERIALIZED_VIEW_ONLY_AGGREGATE(RuleTypeClass.REWRITE),

MATERIALIZED_VIEW_FILTER_SCAN(RuleTypeClass.REWRITE),
MATERIALIZED_VIEW_PROJECT_SCAN(RuleTypeClass.REWRITE),
MATERIALIZED_VIEW_FILTER_PROJECT_SCAN(RuleTypeClass.REWRITE),
MATERIALIZED_VIEW_PROJECT_FILTER_SCAN(RuleTypeClass.REWRITE),

OLAP_SCAN_PARTITION_PRUNE(RuleTypeClass.REWRITE),
FILE_SCAN_PARTITION_PRUNE(RuleTypeClass.REWRITE),
PUSH_CONJUNCTS_INTO_JDBC_SCAN(RuleTypeClass.REWRITE),
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,25 @@
// 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.nereids.rules.exploration.mv;

/**
* AbstractMaterializedViewAggregateRule
* This is responsible for common aggregate rewriting
* */
public abstract class AbstractMaterializedViewAggregateRule extends AbstractMaterializedViewRule {
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,62 @@
// 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.nereids.rules.exploration.mv;

import org.apache.doris.nereids.trees.expressions.NamedExpression;
import org.apache.doris.nereids.trees.plans.Plan;
import org.apache.doris.nereids.trees.plans.logical.LogicalProject;

import java.util.List;

/**
* AbstractMaterializedViewJoinRule
* This is responsible for common join rewriting
*/
public abstract class AbstractMaterializedViewJoinRule extends AbstractMaterializedViewRule {

@Override
protected Plan rewriteQueryByView(MatchMode matchMode,
StructInfo queryStructInfo,
StructInfo viewStructInfo,
RelationMapping queryToViewTableMappings,
Plan tempRewritedPlan) {

// Rewrite top projects, represent the query projects by view
List<NamedExpression> expressions = rewriteExpression(
queryStructInfo.getExpressions(),
queryStructInfo,
viewStructInfo,
queryToViewTableMappings,
tempRewritedPlan
);
// Can not rewrite, bail out
if (expressions == null) {
return null;
}
return new LogicalProject<>(expressions, tempRewritedPlan);
}

// Check join is whether valid or not. Support join's input can not contain aggregate
// Only support project, filter, join, logical relation node and
// join condition should be slot reference equals currently
@Override
protected boolean isPatternSupport(StructInfo structInfo) {
// TODO Should get struct info from hyper graph and check
return false;
}
}
Loading

0 comments on commit 11912b9

Please sign in to comment.