You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by mo...@apache.org on 2021/11/08 13:58:27 UTC

[incubator-doris] 09/11: Union stmt support 'OutFileClause' (#7026)

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

morningman pushed a commit to branch branch-0.15
in repository https://gitbox.apache.org/repos/asf/incubator-doris.git

commit 1d4116233458d29913ea148b27061379f2141ccd
Author: EmmyMiao87 <52...@qq.com>
AuthorDate: Sat Nov 6 13:28:52 2021 +0800

    Union stmt support 'OutFileClause' (#7026)
    
    The union(set operation) stmt also need to analyze 'OutFileClause'.
    
    Whether the fragment is colocate only needs to check the plan node belonging to this fragment.
---
 .../org/apache/doris/analysis/OutFileClause.java   |   8 +-
 .../java/org/apache/doris/analysis/SelectStmt.java |   2 +-
 .../apache/doris/analysis/SetOperationStmt.java    |   2 +
 .../org/apache/doris/journal/JournalEntity.java    |   1 -
 .../main/java/org/apache/doris/qe/Coordinator.java |  15 +--
 .../doris/analysis/OutFileClauseFunctionTest.java  | 101 +++++++++++++++++++++
 .../org/apache/doris/planner/ColocatePlanTest.java |  14 +++
 7 files changed, 121 insertions(+), 22 deletions(-)

diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/OutFileClause.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/OutFileClause.java
index b2268ab..05f1569 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/analysis/OutFileClause.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/OutFileClause.java
@@ -158,7 +158,7 @@ public class OutFileClause {
         return schema;
     }
 
-    private void analyze(Analyzer analyzer) throws UserException {
+    public void analyze(Analyzer analyzer, List<Expr> resultExprs) throws UserException {
         if (isAnalyzed) {
             // If the query stmt is rewritten, the whole stmt will be analyzed again.
             // But some of fields in this OutfileClause has been changed,
@@ -187,13 +187,9 @@ public class OutFileClause {
             throw new AnalysisException("Must specify BROKER properties in OUTFILE clause");
         }
         isAnalyzed = true;
-    }
-
-    public void analyze(Analyzer analyzer, SelectStmt stmt) throws UserException {
-        analyze(analyzer);
 
         if (isParquetFormat()) {
-            analyzeForParquetFormat(stmt.getResultExprs());
+            analyzeForParquetFormat(resultExprs);
         }
     }
 
diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java
index 0f2167e..6a0630b 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/SelectStmt.java
@@ -537,7 +537,7 @@ public class SelectStmt extends QueryStmt {
             }
         }
         if (hasOutFileClause()) {
-            outFileClause.analyze(analyzer, this);
+            outFileClause.analyze(analyzer, resultExprs);
         }
     }
 
diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/SetOperationStmt.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/SetOperationStmt.java
index f735315..0d43295 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/analysis/SetOperationStmt.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/SetOperationStmt.java
@@ -258,6 +258,8 @@ public class SetOperationStmt extends QueryStmt {
         setOpsResultExprs_ = Expr.cloneList(resultExprs);
         if (evaluateOrderBy) createSortTupleInfo(analyzer);
         baseTblResultExprs = resultExprs;
+
+        if (hasOutFileClause()) outFileClause.analyze(analyzer, resultExprs);
     }
 
     /**
diff --git a/fe/fe-core/src/main/java/org/apache/doris/journal/JournalEntity.java b/fe/fe-core/src/main/java/org/apache/doris/journal/JournalEntity.java
index 03036aa..d386ae0 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/journal/JournalEntity.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/journal/JournalEntity.java
@@ -48,7 +48,6 @@ import org.apache.doris.load.loadv2.LoadJob.LoadJobStateUpdateInfo;
 import org.apache.doris.load.loadv2.LoadJobFinalOperation;
 import org.apache.doris.load.routineload.RoutineLoadJob;
 import org.apache.doris.load.sync.SyncJob;
-import org.apache.doris.master.Checkpoint;
 import org.apache.doris.mysql.privilege.UserPropertyInfo;
 import org.apache.doris.persist.AlterRoutineLoadJobOperationLog;
 import org.apache.doris.persist.AlterViewInfo;
diff --git a/fe/fe-core/src/main/java/org/apache/doris/qe/Coordinator.java b/fe/fe-core/src/main/java/org/apache/doris/qe/Coordinator.java
index ea82f2a..a091d87 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/qe/Coordinator.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/qe/Coordinator.java
@@ -1222,23 +1222,10 @@ public class Coordinator {
         }
 
         if (planFragment.hasColocatePlanNode()) {
+            colocateFragmentIds.add(planFragment.getId().asInt());
             return true;
         }
 
-        if (node instanceof HashJoinNode) {
-            HashJoinNode joinNode = (HashJoinNode) node;
-            if (joinNode.isColocate()) {
-                colocateFragmentIds.add(joinNode.getFragmentId().asInt());
-                return true;
-            }
-        }
-
-        for (PlanNode childNode : node.getChildren()) {
-            if (childNode.getFragmentId().asInt() == planFragment.getId().asInt() && isColocateFragment(planFragment, childNode)) {
-                return true;
-            }
-        }
-
         return false;
     }
 
diff --git a/fe/fe-core/src/test/java/org/apache/doris/analysis/OutFileClauseFunctionTest.java b/fe/fe-core/src/test/java/org/apache/doris/analysis/OutFileClauseFunctionTest.java
new file mode 100644
index 0000000..d2bc45a
--- /dev/null
+++ b/fe/fe-core/src/test/java/org/apache/doris/analysis/OutFileClauseFunctionTest.java
@@ -0,0 +1,101 @@
+// 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.analysis;
+
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.common.Config;
+import org.apache.doris.common.FeConstants;
+import org.apache.doris.common.jmockit.Deencapsulation;
+import org.apache.doris.qe.ConnectContext;
+import org.apache.doris.qe.StmtExecutor;
+import org.apache.doris.thrift.TFileFormatType;
+import org.apache.doris.utframe.DorisAssert;
+import org.apache.doris.utframe.UtFrameUtils;
+
+import java.io.File;
+import java.util.UUID;
+
+import org.junit.AfterClass;
+import org.junit.Assert;
+import org.junit.BeforeClass;
+import org.junit.Test;
+
+public class OutFileClauseFunctionTest {
+
+    private static String baseDir = "fe";
+    private static String runningDir = baseDir + "/mocked/MaterializedViewFunctionTest/"
+            + UUID.randomUUID().toString() + "/";
+    private static ConnectContext ctx;
+
+    private static final String DB_NAME = "db1";
+
+    @BeforeClass
+    public static void beforeClass() throws Exception {
+        FeConstants.default_scheduler_interval_millisecond = 10;
+        FeConstants.runningUnitTest = true;
+        Config.enable_outfile_to_local = true;
+        ctx = UtFrameUtils.createDefaultCtx();
+        UtFrameUtils.createDorisCluster(runningDir);
+        CreateDbStmt createDbStmt = (CreateDbStmt) UtFrameUtils.parseAndAnalyzeStmt("create database db1;", ctx);
+        Catalog.getCurrentCatalog().createDb(createDbStmt);
+        String createTableSQL = "create table " + DB_NAME
+                + ".test  (k1 int, k2 varchar ) "
+                + "distributed by hash(k1) buckets 3 properties('replication_num' = '1');";
+        CreateTableStmt createTableStmt = (CreateTableStmt) UtFrameUtils.parseAndAnalyzeStmt(createTableSQL, ctx);
+        Catalog.getCurrentCatalog().createTable(createTableStmt);
+    }
+
+    @AfterClass
+    public static void tearDown() {
+        File file = new File(runningDir);
+        file.delete();
+    }
+
+    @Test
+    public void testSelectStmtOutFileClause() throws Exception {
+        String query1 = "select * from db1.test into outfile \"file:///" + runningDir + "/result_\";";
+        QueryStmt analyzedQueryStmt = (QueryStmt) UtFrameUtils.parseAndAnalyzeStmt(query1, ctx);
+        Assert.assertTrue(analyzedQueryStmt.hasOutFileClause());
+        OutFileClause outFileClause = analyzedQueryStmt.getOutFileClause();
+        boolean isOutFileClauseAnalyzed = Deencapsulation.getField(outFileClause, "isAnalyzed");
+        Assert.assertTrue(isOutFileClauseAnalyzed);
+        Assert.assertEquals(outFileClause.getFileFormatType(), TFileFormatType.FORMAT_CSV_PLAIN);
+    }
+
+    @Test
+    public void testSetOperationStmtOutFileClause() throws Exception {
+        String query1 = "select * from db1.test union select * from db1.test into outfile \"file:///" + runningDir + "/result_\";";
+        QueryStmt analyzedSetOperationStmt = (QueryStmt) UtFrameUtils.parseAndAnalyzeStmt(query1, ctx);
+        Assert.assertTrue(analyzedSetOperationStmt.hasOutFileClause());
+        OutFileClause outFileClause = analyzedSetOperationStmt.getOutFileClause();
+        boolean isOutFileClauseAnalyzed = Deencapsulation.getField(outFileClause, "isAnalyzed");
+        Assert.assertTrue(isOutFileClauseAnalyzed);
+        Assert.assertEquals(outFileClause.getFileFormatType(), TFileFormatType.FORMAT_CSV_PLAIN);
+    }
+
+    @Test
+    public void testParquetFormat() throws Exception {
+        String query1 = "select * from db1.test union select * from db1.test into outfile \"file:///" + runningDir + "/result_\" FORMAT AS PARQUET;";
+        QueryStmt analyzedSetOperationStmt = (QueryStmt) UtFrameUtils.parseAndAnalyzeStmt(query1, ctx);
+        Assert.assertTrue(analyzedSetOperationStmt.hasOutFileClause());
+        OutFileClause outFileClause = analyzedSetOperationStmt.getOutFileClause();
+        boolean isOutFileClauseAnalyzed = Deencapsulation.getField(outFileClause, "isAnalyzed");
+        Assert.assertTrue(isOutFileClauseAnalyzed);
+        Assert.assertEquals(outFileClause.getFileFormatType(), TFileFormatType.FORMAT_PARQUET);
+    }
+}
diff --git a/fe/fe-core/src/test/java/org/apache/doris/planner/ColocatePlanTest.java b/fe/fe-core/src/test/java/org/apache/doris/planner/ColocatePlanTest.java
index ac288dd..43d062a 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/planner/ColocatePlanTest.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/planner/ColocatePlanTest.java
@@ -172,4 +172,18 @@ public class ColocatePlanTest {
         Assert.assertEquals(instanceInfo.size(), 2);
     }
 
+    @Test
+    public void checkColocatePlanFragment() throws Exception {
+        String sql = "select a.k1 from db1.test_colocate a, db1.test_colocate b where a.k1=b.k1 and a.k2=b.k2 group by a.k1;";
+        StmtExecutor executor = UtFrameUtils.getSqlStmtExecutor(ctx, sql);
+        Planner planner = executor.planner();
+        Coordinator coordinator = Deencapsulation.getField(executor, "coord");
+        boolean isColocateFragment0 = Deencapsulation.invoke(coordinator, "isColocateFragment",
+                planner.getFragments().get(1), planner.getFragments().get(1).getPlanRoot());
+        Assert.assertFalse(isColocateFragment0);
+        boolean isColocateFragment1 = Deencapsulation.invoke(coordinator, "isColocateFragment",
+                planner.getFragments().get(2), planner.getFragments().get(2).getPlanRoot());
+        Assert.assertTrue(isColocateFragment1);
+    }
+
 }

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