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 2022/04/28 13:30:43 UTC

[GitHub] [incubator-doris] Kikyou1997 opened a new pull request, #9303: [feature](planner) Support MetaScan for global dict

Kikyou1997 opened a new pull request, #9303:
URL: https://github.com/apache/incubator-doris/pull/9303

   # Proposed changes
   
   Issue Number: #9242
   
   ## Problem Summary:
   
   This PR is a part of code of supports for global dict.
   
   To support global dict, we need to have someway to let BE gathering and merge all the local dict to a global one, so when everytime the DictManager initial builds or updates a column dict,it will send a SQL like this: 
   
   ```sql
   SELECT dict(d_date) FROM date [META];
   ```
   As above sample shows, we added a new hint `META` and a new special aggregate function named `dict`.
   
   In analyze phase, we will check if there is a hint named `META`, if there is, this stmt will be marked as a meta scan query.  For such stmt, we will generate MetaScanNode in Planner instead of OlapScanNode, and the column referenced by SlotRef in the function parameters will be add to MetaScanNode's `slotIdToDictId` field with Map type which be will generate global dict for, the value represents the dict_id which is always -1 for now it wil be used for the incremental update of column dict in BE in the future. 
   
   ## Checklist(Required)
   
   1. Does it affect the original behavior: (Yes)
   2. Has unit tests been added: (No)
   3. Has document been added or modified: (No Need)
   4. Does it need to update dependencies: (No)
   5. Are there any changes that cannot be rolled back: (No)
   
   ## Further comments
   
   If this is a relatively large or complex change, kick off the discussion at [dev@doris.apache.org](mailto:dev@doris.apache.org) by explaining why you chose the solution you did and what alternatives you considered, etc...
   


-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#issuecomment-1113951634

   > please fix conflicts heart
   
   done


-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] morrySnow commented on pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
morrySnow commented on PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#issuecomment-1113999836

   fe-core compile failed in p0 regression, please fix it ~


-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] morrySnow commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
morrySnow commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861608424


##########
fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java:
##########
@@ -1040,6 +1048,21 @@ private void analyzeAggregation(Analyzer analyzer) throws AnalysisException {
         aggExprs.clear();
         TreeNode.collect(substitutedAggs, Expr.isAggregatePredicate(), aggExprs);
 
+        if (metaQuery) {
+            long globalDictFuncCount = aggExprs.stream().filter(FunctionCallExpr::isGlobalDict).count();
+            if (globalDictFuncCount == 0) {
+                // simply ignore the hint
+                metaQuery = false;

Review Comment:
   OlapMetaScanNode still be created even if we turned off this flag, correct?



##########
fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java:
##########
@@ -1040,6 +1048,21 @@ private void analyzeAggregation(Analyzer analyzer) throws AnalysisException {
         aggExprs.clear();
         TreeNode.collect(substitutedAggs, Expr.isAggregatePredicate(), aggExprs);
 
+        if (metaQuery) {

Review Comment:
   should we remove meta hint, and turn ScanNode to OlapMetaScanNode automaticaly when we found some function need to do meta query?



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#issuecomment-1113510628

   > please add unit test for it ❤️
   
   Already committed in the newest patch


-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861137188


##########
fe/fe-core/src/main/java/org/apache/doris/analysis/AggregateInfo.java:
##########
@@ -341,6 +341,12 @@ public AggregateInfo getMergeAggInfo() {
 
     public boolean isMerge() { return aggPhase_.isMerge(); }
     public boolean isDistinctAgg() { return secondPhaseDistinctAggInfo_ != null; }
+
+    public boolean isMetaScan() {
+        return aggregateExprs_.size() > 0 &&
+            aggregateExprs_.get(0).getFn().getFunctionName().getFunction().equals(FunctionSet.DICT);

Review Comment:
   add check logic in the analyze phase of SelectStmt, if user do so, FE will throw a AnalysisException



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861141153


##########
fe/fe-core/src/main/java/org/apache/doris/planner/SingleNodePlanner.java:
##########
@@ -1159,6 +1159,10 @@ private PlanNode createAggregationPlan(SelectStmt selectStmt, Analyzer analyzer,
                     aggInfo.getSecondPhaseDistinctAggInfo());
             newRoot.init(analyzer);
             Preconditions.checkState(newRoot.hasValidStats());
+        } else if (aggInfo.isMetaScan()) {
+            Preconditions.checkState(root instanceof MetaScanNode);
+            MetaScanNode metaScanNode = (MetaScanNode) root;
+            metaScanNode.pullDictSlots(aggInfo);

Review Comment:
   I think it's  not necessary to do check here currently,  have done this in the analyze phase



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861144982


##########
fe/fe-core/src/main/java/org/apache/doris/planner/SingleNodePlanner.java:
##########
@@ -1683,6 +1687,11 @@ private PlanNode createScanNode(Analyzer analyzer, TableRef tblRef, SelectStmt s
 
         switch (tblRef.getTable().getType()) {
             case OLAP:
+                if (tblRef.isMetaScan()) {
+                    MetaScanNode metaScanNode = new MetaScanNode(ctx_.getNextNodeId(), tblRef.getDesc());

Review Comment:
   done



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] morrySnow commented on pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
morrySnow commented on PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#issuecomment-1113508279

   please add unit test for it ❤️


-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861712490


##########
gensrc/thrift/PlanNodes.thrift:
##########
@@ -52,6 +52,7 @@ enum TPlanNodeType {
   EXCEPT_NODE,
   ODBC_SCAN_NODE,
   TABLE_FUNCTION_NODE,
+  META_GATHER_NODE

Review Comment:
   removed



##########
fe/fe-core/src/test/java/org/apache/doris/analysis/CreateMaterializedViewStmtTest.java:
##########
@@ -99,7 +99,7 @@ public void testCountDistinct(@Injectable SlotRef slotRef, @Injectable Arithmeti
         FunctionParams functionParams = new FunctionParams(true, fnChildren);
         FunctionCallExpr functionCallExpr = new FunctionCallExpr(FunctionSet.COUNT, functionParams);
         functionCallExpr.setFn(AggregateFunction.createBuiltin(FunctionSet.COUNT,
-                new ArrayList<>(), Type.BIGINT, Type.BIGINT, false, true, true));
+                new ArrayList<>(), Type.BIGINT, Type.BIGINT, false, true, false));

Review Comment:
   reverted



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861137881


##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";
+
+    private Map<Integer, Integer> slotIdToDictId = new HashMap<>();
+
+    private final OlapTable olapTable;
+
+    public MetaScanNode(PlanNodeId id,
+                        TupleDescriptor tupleDesc) {
+        super(id, tupleDesc, NODE_NAME);
+        this.olapTable = (OlapTable) tupleDesc.getTable();
+    }
+
+    @Override
+    public List<TScanRangeLocations> getScanRangeLocations(long maxScanRangeLength) {
+        List<TScanRangeLocations> resultList = Lists.newArrayList();
+        Collection<Partition> partitionCollection = olapTable.getPartitions();
+        for (Partition partition : partitionCollection) {
+            long visibleVersion = partition.getVisibleVersion();
+            String visibleVersionStr = String.valueOf(visibleVersion);
+            MaterializedIndex index = partition.getBaseIndex();
+            List<Tablet> tablets = index.getTablets();
+            for (Tablet tablet : tablets) {
+                long tabletId = tablet.getId();
+                TScanRangeLocations scanRangeLocations = new TScanRangeLocations();
+                TPaloScanRange paloRange = new TPaloScanRange();
+                paloRange.setDbName("");
+                paloRange.setSchemaHash("");
+                paloRange.setVersion(visibleVersionStr);
+                paloRange.setVersionHash("");
+                paloRange.setTabletId(tabletId);
+
+                // random shuffle List && only collect one copy
+                List<Replica> replicas = tablet.getQueryableReplicas(visibleVersion);
+                if (replicas.isEmpty()) {
+                    LOG.error("no queryable replica found in tablet {}. visible version {}",
+                        tabletId, visibleVersion);
+                    if (LOG.isDebugEnabled()) {
+                        for (Replica replica : tablet.getReplicas()) {
+                            LOG.debug("tablet {}, replica: {}", tabletId, replica.toString());
+                        }
+                    }
+                    throw new RuntimeException("Failed to get scan range, no queryable replica found in tablet: " + tabletId);
+                }
+
+                Collections.shuffle(replicas);
+                boolean tabletIsNull = true;
+                List<String> errs = Lists.newArrayList();
+                for (Replica replica : replicas) {
+                    Backend backend = Catalog.getCurrentSystemInfo().getBackend(replica.getBackendId());
+                    if (backend == null || !backend.isAlive()) {
+                        LOG.debug("backend {} not exists or is not alive for replica {}",
+                            replica.getBackendId(), replica.getId());
+                        errs.add(replica.getId() + "'s backend " + replica.getBackendId() + " does not exist or not alive");
+                        continue;
+                    }
+                    String ip = backend.getHost();
+                    int port = backend.getBePort();
+                    TScanRangeLocation scanRangeLocation = new TScanRangeLocation(new TNetworkAddress(ip, port));
+                    scanRangeLocation.setBackendId(replica.getBackendId());
+                    scanRangeLocations.addToLocations(scanRangeLocation);
+                    paloRange.addToHosts(new TNetworkAddress(ip, port));
+                    tabletIsNull = false;
+                }
+                if (tabletIsNull) {
+                    throw new RuntimeException(tabletId + " have no queryable replicas. err: " + Joiner.on(", ").join(errs));
+                }
+                TScanRange scanRange = new TScanRange();
+                scanRange.setPaloScanRange(paloRange);
+                scanRangeLocations.setScanRange(scanRange);
+
+                resultList.add(scanRangeLocations);
+            }
+        }
+        return resultList;
+    }
+
+    public void pullDictSlots(AggregateInfo aggInfo) {
+        Preconditions.checkState(aggInfo.getGroupingExprs().isEmpty());
+        List<FunctionCallExpr> funcExprList = aggInfo.getAggregateExprs();
+        for (FunctionCallExpr funcExpr : funcExprList) {
+            FunctionParams funcParams = funcExpr.getFnParams();
+            for (Expr expr : funcParams.exprs()) {
+                if (expr instanceof SlotRef && expr.isAnalyzed()) {

Review Comment:
   removed it



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861139250


##########
fe/fe-core/src/main/java/org/apache/doris/planner/SingleNodePlanner.java:
##########
@@ -1159,6 +1159,10 @@ private PlanNode createAggregationPlan(SelectStmt selectStmt, Analyzer analyzer,
                     aggInfo.getSecondPhaseDistinctAggInfo());
             newRoot.init(analyzer);
             Preconditions.checkState(newRoot.hasValidStats());
+        } else if (aggInfo.isMetaScan()) {

Review Comment:
   If the precondition is false, it would throw a illegalstatementexception here



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861136029


##########
fe/fe-core/src/main/java/org/apache/doris/analysis/TableRef.java:
##########
@@ -422,9 +424,11 @@ protected void analyzeHints() throws AnalysisException {
         }
         // Currently only 'PREAGGOPEN' is supported

Review Comment:
   done



##########
fe/fe-core/src/main/java/org/apache/doris/catalog/FunctionSet.java:
##########
@@ -1310,6 +1310,8 @@ public void addBuiltinBothScalaAndVectorized(Function fn) {
 
     public static final String COUNT = "count";
     public static final String WINDOW_FUNNEL = "window_funnel";
+
+    public static final String DICT = "dict";

Review Comment:
   done



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] morrySnow commented on pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
morrySnow commented on PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#issuecomment-1113530023

   please fix conflicts ❤️


-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] yiguolei closed pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
yiguolei closed pull request #9303: [feature](planner) Support  MetaScan for global dict
URL: https://github.com/apache/incubator-doris/pull/9303


-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] yiguolei commented on pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
yiguolei commented on PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#issuecomment-1152050238

   duplicate with https://github.com/apache/incubator-doris/pull/9842


-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861712831


##########
fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java:
##########
@@ -1040,6 +1048,21 @@ private void analyzeAggregation(Analyzer analyzer) throws AnalysisException {
         aggExprs.clear();
         TreeNode.collect(substitutedAggs, Expr.isAggregatePredicate(), aggExprs);
 
+        if (metaQuery) {
+            long globalDictFuncCount = aggExprs.stream().filter(FunctionCallExpr::isGlobalDict).count();
+            if (globalDictFuncCount == 0) {
+                // simply ignore the hint
+                metaQuery = false;

Review Comment:
   No.it wouldn't be created in that case



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] morrySnow commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
morrySnow commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861602888


##########
gensrc/thrift/PlanNodes.thrift:
##########
@@ -52,6 +52,7 @@ enum TPlanNodeType {
   EXCEPT_NODE,
   ODBC_SCAN_NODE,
   TABLE_FUNCTION_NODE,
+  META_GATHER_NODE

Review Comment:
   what's this?



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861137665


##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";
+
+    private Map<Integer, Integer> slotIdToDictId = new HashMap<>();
+
+    private final OlapTable olapTable;
+
+    public MetaScanNode(PlanNodeId id,
+                        TupleDescriptor tupleDesc) {
+        super(id, tupleDesc, NODE_NAME);
+        this.olapTable = (OlapTable) tupleDesc.getTable();
+    }
+
+    @Override
+    public List<TScanRangeLocations> getScanRangeLocations(long maxScanRangeLength) {
+        List<TScanRangeLocations> resultList = Lists.newArrayList();
+        Collection<Partition> partitionCollection = olapTable.getPartitions();
+        for (Partition partition : partitionCollection) {
+            long visibleVersion = partition.getVisibleVersion();
+            String visibleVersionStr = String.valueOf(visibleVersion);
+            MaterializedIndex index = partition.getBaseIndex();
+            List<Tablet> tablets = index.getTablets();
+            for (Tablet tablet : tablets) {
+                long tabletId = tablet.getId();
+                TScanRangeLocations scanRangeLocations = new TScanRangeLocations();
+                TPaloScanRange paloRange = new TPaloScanRange();
+                paloRange.setDbName("");
+                paloRange.setSchemaHash("");
+                paloRange.setVersion(visibleVersionStr);
+                paloRange.setVersionHash("");
+                paloRange.setTabletId(tabletId);
+
+                // random shuffle List && only collect one copy
+                List<Replica> replicas = tablet.getQueryableReplicas(visibleVersion);
+                if (replicas.isEmpty()) {
+                    LOG.error("no queryable replica found in tablet {}. visible version {}",
+                        tabletId, visibleVersion);
+                    if (LOG.isDebugEnabled()) {
+                        for (Replica replica : tablet.getReplicas()) {
+                            LOG.debug("tablet {}, replica: {}", tabletId, replica.toString());
+                        }
+                    }
+                    throw new RuntimeException("Failed to get scan range, no queryable replica found in tablet: " + tabletId);
+                }
+
+                Collections.shuffle(replicas);
+                boolean tabletIsNull = true;
+                List<String> errs = Lists.newArrayList();
+                for (Replica replica : replicas) {
+                    Backend backend = Catalog.getCurrentSystemInfo().getBackend(replica.getBackendId());
+                    if (backend == null || !backend.isAlive()) {
+                        LOG.debug("backend {} not exists or is not alive for replica {}",
+                            replica.getBackendId(), replica.getId());
+                        errs.add(replica.getId() + "'s backend " + replica.getBackendId() + " does not exist or not alive");
+                        continue;
+                    }
+                    String ip = backend.getHost();
+                    int port = backend.getBePort();
+                    TScanRangeLocation scanRangeLocation = new TScanRangeLocation(new TNetworkAddress(ip, port));
+                    scanRangeLocation.setBackendId(replica.getBackendId());
+                    scanRangeLocations.addToLocations(scanRangeLocation);
+                    paloRange.addToHosts(new TNetworkAddress(ip, port));
+                    tabletIsNull = false;
+                }
+                if (tabletIsNull) {
+                    throw new RuntimeException(tabletId + " have no queryable replicas. err: " + Joiner.on(", ").join(errs));
+                }
+                TScanRange scanRange = new TScanRange();
+                scanRange.setPaloScanRange(paloRange);
+                scanRangeLocations.setScanRange(scanRange);
+
+                resultList.add(scanRangeLocations);
+            }
+        }
+        return resultList;
+    }
+
+    public void pullDictSlots(AggregateInfo aggInfo) {
+        Preconditions.checkState(aggInfo.getGroupingExprs().isEmpty());

Review Comment:
   yes, we do!



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861137383


##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.planner;
+

Review Comment:
   done



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] Kikyou1997 commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
Kikyou1997 commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861141153


##########
fe/fe-core/src/main/java/org/apache/doris/planner/SingleNodePlanner.java:
##########
@@ -1159,6 +1159,10 @@ private PlanNode createAggregationPlan(SelectStmt selectStmt, Analyzer analyzer,
                     aggInfo.getSecondPhaseDistinctAggInfo());
             newRoot.init(analyzer);
             Preconditions.checkState(newRoot.hasValidStats());
+        } else if (aggInfo.isMetaScan()) {
+            Preconditions.checkState(root instanceof MetaScanNode);
+            MetaScanNode metaScanNode = (MetaScanNode) root;
+            metaScanNode.pullDictSlots(aggInfo);

Review Comment:
   I think we not actually need check here currently,  have done this in the analyze phase



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] morrySnow commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
morrySnow commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r860916127


##########
fe/fe-core/src/main/java/org/apache/doris/catalog/FunctionSet.java:
##########
@@ -1310,6 +1310,8 @@ public void addBuiltinBothScalaAndVectorized(Function fn) {
 
     public static final String COUNT = "count";
     public static final String WINDOW_FUNNEL = "window_funnel";
+
+    public static final String DICT = "dict";

Review Comment:
   maybe `global_dict` is better?



##########
fe/fe-core/src/main/java/org/apache/doris/catalog/FunctionSet.java:
##########
@@ -1373,6 +1375,11 @@ private void initAggregateBuiltins() {
                 "",
                 true, false, true, true));
 
+        addBuiltin(AggregateFunction.createBuiltin(FunctionSet.DICT,
+                Lists.newArrayList(Type.VARCHAR), Type.VARCHAR, Type.VARCHAR,
+                true, false, false, false

Review Comment:
   the last arg should be true ?



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";
+
+    private Map<Integer, Integer> slotIdToDictId = new HashMap<>();

Review Comment:
   final ?



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";
+
+    private Map<Integer, Integer> slotIdToDictId = new HashMap<>();
+
+    private final OlapTable olapTable;
+
+    public MetaScanNode(PlanNodeId id,
+                        TupleDescriptor tupleDesc) {
+        super(id, tupleDesc, NODE_NAME);
+        this.olapTable = (OlapTable) tupleDesc.getTable();
+    }
+
+    @Override
+    public List<TScanRangeLocations> getScanRangeLocations(long maxScanRangeLength) {
+        List<TScanRangeLocations> resultList = Lists.newArrayList();
+        Collection<Partition> partitionCollection = olapTable.getPartitions();
+        for (Partition partition : partitionCollection) {
+            long visibleVersion = partition.getVisibleVersion();
+            String visibleVersionStr = String.valueOf(visibleVersion);
+            MaterializedIndex index = partition.getBaseIndex();
+            List<Tablet> tablets = index.getTablets();
+            for (Tablet tablet : tablets) {
+                long tabletId = tablet.getId();
+                TScanRangeLocations scanRangeLocations = new TScanRangeLocations();
+                TPaloScanRange paloRange = new TPaloScanRange();
+                paloRange.setDbName("");
+                paloRange.setSchemaHash("");
+                paloRange.setVersion(visibleVersionStr);
+                paloRange.setVersionHash("");
+                paloRange.setTabletId(tabletId);
+
+                // random shuffle List && only collect one copy
+                List<Replica> replicas = tablet.getQueryableReplicas(visibleVersion);
+                if (replicas.isEmpty()) {
+                    LOG.error("no queryable replica found in tablet {}. visible version {}",
+                        tabletId, visibleVersion);
+                    if (LOG.isDebugEnabled()) {
+                        for (Replica replica : tablet.getReplicas()) {
+                            LOG.debug("tablet {}, replica: {}", tabletId, replica.toString());
+                        }
+                    }
+                    throw new RuntimeException("Failed to get scan range, no queryable replica found in tablet: " + tabletId);
+                }
+
+                Collections.shuffle(replicas);
+                boolean tabletIsNull = true;
+                List<String> errs = Lists.newArrayList();
+                for (Replica replica : replicas) {
+                    Backend backend = Catalog.getCurrentSystemInfo().getBackend(replica.getBackendId());
+                    if (backend == null || !backend.isAlive()) {
+                        LOG.debug("backend {} not exists or is not alive for replica {}",
+                            replica.getBackendId(), replica.getId());
+                        errs.add(replica.getId() + "'s backend " + replica.getBackendId() + " does not exist or not alive");
+                        continue;
+                    }
+                    String ip = backend.getHost();
+                    int port = backend.getBePort();
+                    TScanRangeLocation scanRangeLocation = new TScanRangeLocation(new TNetworkAddress(ip, port));
+                    scanRangeLocation.setBackendId(replica.getBackendId());
+                    scanRangeLocations.addToLocations(scanRangeLocation);
+                    paloRange.addToHosts(new TNetworkAddress(ip, port));
+                    tabletIsNull = false;
+                }
+                if (tabletIsNull) {
+                    throw new RuntimeException(tabletId + " have no queryable replicas. err: " + Joiner.on(", ").join(errs));
+                }
+                TScanRange scanRange = new TScanRange();
+                scanRange.setPaloScanRange(paloRange);
+                scanRangeLocations.setScanRange(scanRange);
+
+                resultList.add(scanRangeLocations);
+            }
+        }
+        return resultList;
+    }
+
+    public void pullDictSlots(AggregateInfo aggInfo) {
+        Preconditions.checkState(aggInfo.getGroupingExprs().isEmpty());
+        List<FunctionCallExpr> funcExprList = aggInfo.getAggregateExprs();
+        for (FunctionCallExpr funcExpr : funcExprList) {
+            FunctionParams funcParams = funcExpr.getFnParams();
+            for (Expr expr : funcParams.exprs()) {
+                if (expr instanceof SlotRef && expr.isAnalyzed()) {

Review Comment:
   when thie function called, expr should be analyzed, so we need to check `expr.isAnalyzed` ?



##########
fe/fe-core/src/main/java/org/apache/doris/analysis/AggregateInfo.java:
##########
@@ -341,6 +341,12 @@ public AggregateInfo getMergeAggInfo() {
 
     public boolean isMerge() { return aggPhase_.isMerge(); }
     public boolean isDistinctAgg() { return secondPhaseDistinctAggInfo_ != null; }
+
+    public boolean isMetaScan() {
+        return aggregateExprs_.size() > 0 &&
+            aggregateExprs_.get(0).getFn().getFunctionName().getFunction().equals(FunctionSet.DICT);

Review Comment:
   what will happen if we use meta scan agg with normal agg?



##########
fe/fe-core/src/main/java/org/apache/doris/planner/SingleNodePlanner.java:
##########
@@ -1159,6 +1159,10 @@ private PlanNode createAggregationPlan(SelectStmt selectStmt, Analyzer analyzer,
                     aggInfo.getSecondPhaseDistinctAggInfo());
             newRoot.init(analyzer);
             Preconditions.checkState(newRoot.hasValidStats());
+        } else if (aggInfo.isMetaScan()) {

Review Comment:
   we need to throw exception here



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.planner;
+

Review Comment:
   import order should be:
   
   doris
   
   third party
   
   java



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";
+
+    private Map<Integer, Integer> slotIdToDictId = new HashMap<>();
+
+    private final OlapTable olapTable;
+
+    public MetaScanNode(PlanNodeId id,
+                        TupleDescriptor tupleDesc) {
+        super(id, tupleDesc, NODE_NAME);
+        this.olapTable = (OlapTable) tupleDesc.getTable();
+    }
+
+    @Override
+    public List<TScanRangeLocations> getScanRangeLocations(long maxScanRangeLength) {

Review Comment:
   add some comment to explain why are parameters not used



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";

Review Comment:
   could MetaScanNode do meta scan on non-olap table ?



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";
+
+    private Map<Integer, Integer> slotIdToDictId = new HashMap<>();
+
+    private final OlapTable olapTable;
+
+    public MetaScanNode(PlanNodeId id,
+                        TupleDescriptor tupleDesc) {
+        super(id, tupleDesc, NODE_NAME);
+        this.olapTable = (OlapTable) tupleDesc.getTable();
+    }
+
+    @Override
+    public List<TScanRangeLocations> getScanRangeLocations(long maxScanRangeLength) {
+        List<TScanRangeLocations> resultList = Lists.newArrayList();
+        Collection<Partition> partitionCollection = olapTable.getPartitions();
+        for (Partition partition : partitionCollection) {
+            long visibleVersion = partition.getVisibleVersion();
+            String visibleVersionStr = String.valueOf(visibleVersion);
+            MaterializedIndex index = partition.getBaseIndex();
+            List<Tablet> tablets = index.getTablets();
+            for (Tablet tablet : tablets) {
+                long tabletId = tablet.getId();
+                TScanRangeLocations scanRangeLocations = new TScanRangeLocations();
+                TPaloScanRange paloRange = new TPaloScanRange();
+                paloRange.setDbName("");
+                paloRange.setSchemaHash("");
+                paloRange.setVersion(visibleVersionStr);
+                paloRange.setVersionHash("");
+                paloRange.setTabletId(tabletId);
+
+                // random shuffle List && only collect one copy
+                List<Replica> replicas = tablet.getQueryableReplicas(visibleVersion);
+                if (replicas.isEmpty()) {
+                    LOG.error("no queryable replica found in tablet {}. visible version {}",
+                        tabletId, visibleVersion);
+                    if (LOG.isDebugEnabled()) {
+                        for (Replica replica : tablet.getReplicas()) {
+                            LOG.debug("tablet {}, replica: {}", tabletId, replica.toString());
+                        }
+                    }
+                    throw new RuntimeException("Failed to get scan range, no queryable replica found in tablet: " + tabletId);
+                }
+
+                Collections.shuffle(replicas);
+                boolean tabletIsNull = true;
+                List<String> errs = Lists.newArrayList();
+                for (Replica replica : replicas) {
+                    Backend backend = Catalog.getCurrentSystemInfo().getBackend(replica.getBackendId());
+                    if (backend == null || !backend.isAlive()) {
+                        LOG.debug("backend {} not exists or is not alive for replica {}",
+                            replica.getBackendId(), replica.getId());
+                        errs.add(replica.getId() + "'s backend " + replica.getBackendId() + " does not exist or not alive");
+                        continue;
+                    }
+                    String ip = backend.getHost();
+                    int port = backend.getBePort();
+                    TScanRangeLocation scanRangeLocation = new TScanRangeLocation(new TNetworkAddress(ip, port));
+                    scanRangeLocation.setBackendId(replica.getBackendId());
+                    scanRangeLocations.addToLocations(scanRangeLocation);
+                    paloRange.addToHosts(new TNetworkAddress(ip, port));
+                    tabletIsNull = false;
+                }
+                if (tabletIsNull) {
+                    throw new RuntimeException(tabletId + " have no queryable replicas. err: " + Joiner.on(", ").join(errs));
+                }
+                TScanRange scanRange = new TScanRange();
+                scanRange.setPaloScanRange(paloRange);
+                scanRangeLocations.setScanRange(scanRange);
+
+                resultList.add(scanRangeLocations);
+            }
+        }
+        return resultList;
+    }
+
+    public void pullDictSlots(AggregateInfo aggInfo) {
+        Preconditions.checkState(aggInfo.getGroupingExprs().isEmpty());

Review Comment:
   do we verify it and throw exception before this check?



##########
fe/fe-core/src/main/java/org/apache/doris/analysis/TableRef.java:
##########
@@ -422,9 +424,11 @@ protected void analyzeHints() throws AnalysisException {
         }
         // Currently only 'PREAGGOPEN' is supported

Review Comment:
   should we modify this comment?



##########
fe/fe-core/src/main/java/org/apache/doris/planner/SingleNodePlanner.java:
##########
@@ -1159,6 +1159,10 @@ private PlanNode createAggregationPlan(SelectStmt selectStmt, Analyzer analyzer,
                     aggInfo.getSecondPhaseDistinctAggInfo());
             newRoot.init(analyzer);
             Preconditions.checkState(newRoot.hasValidStats());
+        } else if (aggInfo.isMetaScan()) {
+            Preconditions.checkState(root instanceof MetaScanNode);
+            MetaScanNode metaScanNode = (MetaScanNode) root;
+            metaScanNode.pullDictSlots(aggInfo);

Review Comment:
   we need to check agg function name and then do `pullDictSlots`



##########
fe/fe-core/src/main/java/org/apache/doris/planner/SingleNodePlanner.java:
##########
@@ -1683,6 +1687,11 @@ private PlanNode createScanNode(Analyzer analyzer, TableRef tblRef, SelectStmt s
 
         switch (tblRef.getTable().getType()) {
             case OLAP:
+                if (tblRef.isMetaScan()) {
+                    MetaScanNode metaScanNode = new MetaScanNode(ctx_.getNextNodeId(), tblRef.getDesc());

Review Comment:
   if we only could do meta scan on olap table, should we named this node to `OlapMeataScanNode` ?



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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


[GitHub] [incubator-doris] morrySnow commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

Posted by GitBox <gi...@apache.org>.
morrySnow commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r861604227


##########
fe/fe-core/src/test/java/org/apache/doris/analysis/CreateMaterializedViewStmtTest.java:
##########
@@ -99,7 +99,7 @@ public void testCountDistinct(@Injectable SlotRef slotRef, @Injectable Arithmeti
         FunctionParams functionParams = new FunctionParams(true, fnChildren);
         FunctionCallExpr functionCallExpr = new FunctionCallExpr(FunctionSet.COUNT, functionParams);
         functionCallExpr.setFn(AggregateFunction.createBuiltin(FunctionSet.COUNT,
-                new ArrayList<>(), Type.BIGINT, Type.BIGINT, false, true, true));
+                new ArrayList<>(), Type.BIGINT, Type.BIGINT, false, true, false));

Review Comment:
   why change that?



-- 
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.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

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