You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by GitBox <gi...@apache.org> on 2020/07/06 08:27:23 UTC

[GitHub] [incubator-doris] morningman commented on a change in pull request #4014: Add framework of mv selector

morningman commented on a change in pull request #4014:
URL: https://github.com/apache/incubator-doris/pull/4014#discussion_r450045369



##########
File path: fe/src/main/java/org/apache/doris/planner/SingleNodePlanner.java
##########
@@ -773,19 +776,26 @@ public void selectMaterializedView(QueryStmt queryStmt, Analyzer analyzer) throw
                 if (olapScanNode.getSelectedPartitionIds().size() == 0 && !FeConstants.runningUnitTest) {
                     continue;
                 }
-                MaterializedViewSelector.BestIndexInfo bestIndexInfo = materializedViewSelector.selectBestMV(
-                        olapScanNode);
-                olapScanNode.updateScanRangeInfoByNewMVSelector(bestIndexInfo.getBestIndexId(), bestIndexInfo.isPreAggregation(),
-                        bestIndexInfo.getReasonOfDisable());
+                MaterializedViewSelector.BestIndexInfo bestIndexInfo = materializedViewSelector.selectBestMV(olapScanNode);
+                if (bestIndexInfo == null) {
+                    selectFailed |= true;
+                    TupleId tupleId = olapScanNode.getTupleId();
+                    selectStmt.updateDisableTuplesMVRewriter(tupleId);
+                    LOG.info("MV rewriter of tuple [] will be disable", tupleId);

Review comment:
       ```suggestion
                       LOG.debug("MV rewriter of tuple [] will be disable", tupleId);
   ```

##########
File path: fe/src/main/java/org/apache/doris/analysis/CreateMaterializedViewStmt.java
##########
@@ -50,7 +53,21 @@
  * [PROPERTIES ("key" = "value")]
  */
 public class CreateMaterializedViewStmt extends DdlStmt {
-    public static final String MATERIALIZED_VIEW_NAME_PRFIX = "__doris_materialized_view_";
+    public static final String MATERIALIZED_VIEW_NAME_PREFIX = "mv_";
+    public static final Map<String, MVColumnPatten> fnNameToPatten;

Review comment:
       ```suggestion
       public static final Map<String, MVColumnPatten> FN_NAME_TO_PATTERN;
   ```

##########
File path: fe/src/main/java/org/apache/doris/qe/StmtExecutor.java
##########
@@ -440,58 +439,10 @@ public void analyze(TQueryOptions tQueryOptions) throws AnalysisException, UserE
 
             lock(dbs);
             try {
-                parsedStmt.analyze(analyzer);
-                if (parsedStmt instanceof QueryStmt || parsedStmt instanceof InsertStmt) {
-                    boolean isExplain = parsedStmt.isExplain();
-                    // Apply expr and subquery rewrites.
-                    boolean reAnalyze = false;
-
-                    ExprRewriter rewriter = analyzer.getExprRewriter();
-                    rewriter.reset();
-                    parsedStmt.rewriteExprs(rewriter);
-                    reAnalyze = rewriter.changed();
-                    if (analyzer.containSubquery()) {
-                        parsedStmt = StmtRewriter.rewrite(analyzer, parsedStmt);
-                        reAnalyze = true;
-                    }
-                    if (reAnalyze) {
-                        // The rewrites should have no user-visible effect. Remember the original result
-                        // types and column labels to restore them after the rewritten stmt has been
-                        // reset() and re-analyzed.
-                        List<Type> origResultTypes = Lists.newArrayList();
-                        for (Expr e: parsedStmt.getResultExprs()) {
-                            origResultTypes.add(e.getType());
-                        }
-                        List<String> origColLabels =
-                                Lists.newArrayList(parsedStmt.getColLabels());
-
-                        // Re-analyze the stmt with a new analyzer.
-                        analyzer = new Analyzer(context.getCatalog(), context);
-
-                        // query re-analyze
-                        parsedStmt.reset();
-                        parsedStmt.analyze(analyzer);
-
-                        // Restore the original result types and column labels.
-                        parsedStmt.castResultExprs(origResultTypes);
-                        parsedStmt.setColLabels(origColLabels);
-                        if (LOG.isTraceEnabled()) {
-                            LOG.trace("rewrittenStmt: " + parsedStmt.toSql());
-                        }
-                        if (isExplain) parsedStmt.setIsExplain(isExplain);
-                    }
-                }
-
-                // create plan
-                planner = new Planner();
-                if (parsedStmt instanceof QueryStmt || parsedStmt instanceof InsertStmt) {
-                    planner.plan(parsedStmt, analyzer, tQueryOptions);
-                } else {
-                    planner.plan(((CreateTableAsSelectStmt) parsedStmt).getInsertStmt(),
-                            analyzer, new TQueryOptions());
-                }
-                // TODO(zc):
-                // Preconditions.checkState(!analyzer.hasUnassignedConjuncts());
+                analyzeAndGenerateQueryPlan(tQueryOptions);
+            } catch (MVSelectFailedException e) {
+                resetAnalyzerAndStmt();
+                analyzeAndGenerateQueryPlan(tQueryOptions);

Review comment:
       Add comment to explain why retrying.

##########
File path: fe/src/main/java/org/apache/doris/analysis/QueryStmt.java
##########
@@ -113,6 +113,40 @@
     // represent the "INTO OUTFILE" clause
     protected OutFileClause outFileClause;
 
+    /**
+     * If the query stmt belongs to CreateMaterializedViewStmt,
+     * such as
+     * `CREATE MATERIALIZED VIEW mv AS SELECT bitmap_union(to_bitmap(k1)) from table`
+     * query stmt will not be rewrite by MVRewriter.
+     * The `bitmap_union(to_bitmap(k1))` is the definition of the mv column rather then a expr.
+     * So `forbiddenMVRewrite` will be set to true to protect the definition of the mv column from being overwritten.
+     * <p>
+     * In other query case, `forbiddenMVRewrite` is always false.
+     */
+    private boolean forbiddenMVRewrite = false;
+
+    /**
+     * If the tuple id in `disableMVRewriteTupleIds`, the expr which belongs to this tuple will not be MVRewritten.
+     * Initially this set is an empty set.
+     * When the scan node is unable to match any index in selecting the materialized view,
+     *   the tuple is added to this set.
+     * The query will be re-executed, and this tuple will not be mv rewritten.
+     * For example:
+     * TableA: (k1 int, k2 int, k3 int)
+     * MV: (k1 int, mv_bitmap_union_k2 bitmap bitmap_union)
+     * Query: select k3, bitmap_union(to_bitmap(k2)) from TableA
+     * First analyze: MV rewriter enable and this set is empty
+     *     select k3, bitmap_union(mv_bitmap_union_k2) from TableA
+     * SingleNodePlanner: could not select any index for TableA
+     *     Add table to disableMVRewriteTupleIds.
+     * `disableMVRewriteTupleIds` = {TableA}
+     * Re-executed:
+     * Second analyze: MV rewrite disable in table and use origin stmt.
+     *     select k3, bitmap_union(to_bitmap(k2)) from TableA
+     * SingleNodePlanner: base index selected

Review comment:
       Are you sure after when re-execute the stmt, the tuple id is same as first execution?

##########
File path: fe/src/main/java/org/apache/doris/rewrite/mvrewrite/CountFieldToSum.java
##########
@@ -0,0 +1,94 @@
+// 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.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.OlapTable;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.common.AnalysisException;
+import org.apache.doris.rewrite.ExprRewriteRule;
+
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+
+import java.util.List;
+
+/**
+ * Rewrite count(k1) to sum(mv_count_k1) when MV Column exists.
+ * For example:
+ * Table: (k1 int ,k2 varchar)
+ * MV: (k1 int, mv_count_k2 bigint sum)
+ *       mv_count_k1 = case when k2 is null then 0 else 1
+ * Query: select k1, count(k2) from table group by k1
+ * Rewritten query: select k1, sum(mv_count_k2) from table group by k1
+ */
+public class CountFieldToSum implements ExprRewriteRule {
+    public static final ExprRewriteRule INSTANCE = new CountFieldToSum();
+
+    @Override
+    public Expr apply(Expr expr, Analyzer analyzer) throws AnalysisException {
+        // meet condition
+        if (!(expr instanceof FunctionCallExpr)) {
+            return expr;
+        }
+        FunctionCallExpr fnExpr = (FunctionCallExpr) expr;
+        if (!fnExpr.getFnName().getFunction().equalsIgnoreCase("count")) {
+            return expr;
+        }
+        if (fnExpr.getChildren().size() != 1 || !(fnExpr.getChild(0) instanceof SlotRef)) {
+            return expr;
+        }
+        SlotRef fnChild0 = (SlotRef) fnExpr.getChild(0);
+        Column column = fnChild0.getColumn();
+        Table table = fnChild0.getTable();
+        if (column == null || table == null || !(table instanceof OlapTable)) {
+            return expr;
+        }
+        OlapTable olapTable = (OlapTable) table;
+
+        // check column
+        String queryColumnName = column.getName();
+        String mvColumnName = CreateMaterializedViewStmt.mvColumnBuilder("count", queryColumnName);
+        Column mvColumn = olapTable.getVisibleColumn(mvColumnName);
+        if (mvColumn == null) {
+            return expr;
+        }
+
+        // rewrite expr
+        return rewriteExpr(fnChild0, mvColumn, analyzer);
+    }
+
+    public Expr rewriteExpr(SlotRef queryColumnSlotRef, Column mvColumn, Analyzer analyzer) {

Review comment:
       private?




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



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