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 2022/04/11 06:03:48 UTC

[incubator-doris] 07/09: [fix](join) Fix error bucket num get in bucket shuffle join in dynamic partition (#8891)

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

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

commit 4f0172d89c673789297748a05a94538ff976605c
Author: HappenLee <ha...@hotmail.com>
AuthorDate: Sat Apr 9 19:11:44 2022 +0800

    [fix](join) Fix error bucket num get in bucket shuffle join in dynamic partition (#8891)
---
 be/src/service/CMakeLists.txt                      |  2 +-
 build.sh                                           |  3 --
 .../org/apache/doris/planner/OlapScanNode.java     |  2 ++
 .../main/java/org/apache/doris/qe/Coordinator.java |  4 ++-
 .../java/org/apache/doris/qe/CoordinatorTest.java  |  2 ++
 .../data/join/sql/bucket_shuffle_join.out          |  5 ++++
 .../apache/doris/regression/util/SuiteInfo.groovy  | 33 ----------------------
 .../suites/join/ddl/test_bucket_shuffle_join.sql   | 16 +++++++++++
 .../suites/join/sql/bucket_shuffle_join.sql        |  1 +
 9 files changed, 30 insertions(+), 38 deletions(-)

diff --git a/be/src/service/CMakeLists.txt b/be/src/service/CMakeLists.txt
index fecc5b4153..fb07d2cca7 100644
--- a/be/src/service/CMakeLists.txt
+++ b/be/src/service/CMakeLists.txt
@@ -44,7 +44,7 @@ if (${MAKE_TEST} STREQUAL "OFF")
     install(DIRECTORY DESTINATION ${OUTPUT_DIR}/lib/)
     install(TARGETS palo_be DESTINATION ${OUTPUT_DIR}/lib/)
 
-    if (${STRIP_DEBUG_INFO} STREQUAL "ON")
+    if ("${STRIP_DEBUG_INFO}" STREQUAL "ON")
         add_custom_command(TARGET palo_be POST_BUILD
             COMMAND ${CMAKE_OBJCOPY} --only-keep-debug $<TARGET_FILE:palo_be> $<TARGET_FILE:palo_be>.dbg
             COMMAND ${CMAKE_STRIP} --strip-debug --strip-unneeded $<TARGET_FILE:palo_be>
diff --git a/build.sh b/build.sh
index 16c7b792b0..f5be3971fb 100755
--- a/build.sh
+++ b/build.sh
@@ -293,9 +293,6 @@ if [ ${BUILD_FE} -eq 1 -o ${BUILD_SPARK_DPP} -eq 1 ]; then
 fi
 
 function build_ui() {
-    # check NPM env here, not in env.sh.
-    # Because UI should be considered a non-essential component at runtime.
-    # Only when the compilation is required, check the relevant compilation environment.
     NPM=npm
     if ! ${NPM} --version; then
         echo "Error: npm is not found"
diff --git a/fe/fe-core/src/main/java/org/apache/doris/planner/OlapScanNode.java b/fe/fe-core/src/main/java/org/apache/doris/planner/OlapScanNode.java
index 6310a7aaf8..0ba2e7a693 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/planner/OlapScanNode.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/planner/OlapScanNode.java
@@ -172,6 +172,8 @@ public class OlapScanNode extends ScanNode {
         setCanTurnOnPreAggr(false);
     }
 
+    public long getTotalTabletsNum() { return totalTabletsNum; }
+
     public boolean getForceOpenPreAgg() {
         return forceOpenPreAgg;
     }
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 7e03688b37..502b3bfa39 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
@@ -1786,7 +1786,9 @@ public class Coordinator {
         private void computeScanRangeAssignmentByBucket(
                 final OlapScanNode scanNode, ImmutableMap<Long, Backend> idToBackend, Map<TNetworkAddress, Long> addressToBackendID) throws Exception {
             if (!fragmentIdToSeqToAddressMap.containsKey(scanNode.getFragmentId())) {
-                fragmentIdToBucketNumMap.put(scanNode.getFragmentId(), scanNode.getOlapTable().getDefaultDistributionInfo().getBucketNum());
+                // The bucket shuffle join only hit when the partition is one. so the totalTabletsNum is all tablet of
+                // one hit partition. can be the right bucket num in bucket shuffle join
+                fragmentIdToBucketNumMap.put(scanNode.getFragmentId(), (int)scanNode.getTotalTabletsNum());
                 fragmentIdToSeqToAddressMap.put(scanNode.getFragmentId(), new HashedMap());
                 fragmentIdBucketSeqToScanRangeMap.put(scanNode.getFragmentId(), new BucketSeqToScanRange());
                 fragmentIdToBuckendIdBucketCountMap.put(scanNode.getFragmentId(), new HashMap<>());
diff --git a/fe/fe-core/src/test/java/org/apache/doris/qe/CoordinatorTest.java b/fe/fe-core/src/test/java/org/apache/doris/qe/CoordinatorTest.java
index fb86df6e33..e355f6c247 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/qe/CoordinatorTest.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/qe/CoordinatorTest.java
@@ -235,6 +235,7 @@ public class CoordinatorTest extends Coordinator {
         }
 
         Deencapsulation.setField(olapScanNode, "bucketSeq2locations", bucketseq2localtion);
+        Deencapsulation.setField(olapScanNode, "totalTabletsNum", 66);
         olapScanNode.setFragment(new PlanFragment(planFragmentId, olapScanNode,
                 new DataPartition(TPartitionType.UNPARTITIONED)));
 
@@ -357,6 +358,7 @@ public class CoordinatorTest extends Coordinator {
         }
 
         Deencapsulation.setField(olapScanNode, "bucketSeq2locations", bucketseq2localtion);
+        Deencapsulation.setField(olapScanNode, "totalTabletsNum", 66);
         olapScanNode.setFragment(new PlanFragment(planFragmentId, olapScanNode,
                 new DataPartition(TPartitionType.UNPARTITIONED)));
 
diff --git a/regression-test/data/join/sql/bucket_shuffle_join.out b/regression-test/data/join/sql/bucket_shuffle_join.out
new file mode 100644
index 0000000000..87f57761ba
--- /dev/null
+++ b/regression-test/data/join/sql/bucket_shuffle_join.out
@@ -0,0 +1,5 @@
+-- This file is automatically generated. You should know what you did if you want to edit this
+-- !bucket_shuffle_join --
+1	2021-12-01T00:00
+2	2021-12-01T00:00
+
diff --git a/regression-test/framework/src/main/groovy/org/apache/doris/regression/util/SuiteInfo.groovy b/regression-test/framework/src/main/groovy/org/apache/doris/regression/util/SuiteInfo.groovy
deleted file mode 100644
index 589d5b882c..0000000000
--- a/regression-test/framework/src/main/groovy/org/apache/doris/regression/util/SuiteInfo.groovy
+++ /dev/null
@@ -1,33 +0,0 @@
-// 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.regression.util
-
-import groovy.transform.CompileStatic
-
-@CompileStatic
-class SuiteInfo {
-    File file
-    String group
-    String suiteName
-
-    SuiteInfo(File file, String group, String suiteName) {
-        this.file = file
-        this.group = group
-        this.suiteName = suiteName
-    }
-}
diff --git a/regression-test/suites/join/ddl/test_bucket_shuffle_join.sql b/regression-test/suites/join/ddl/test_bucket_shuffle_join.sql
new file mode 100644
index 0000000000..7d461bf86a
--- /dev/null
+++ b/regression-test/suites/join/ddl/test_bucket_shuffle_join.sql
@@ -0,0 +1,16 @@
+CREATE TABLE `test_bucket_shuffle_join` (
+  `id` int(11) NOT NULL COMMENT "",
+  `rectime` datetime NOT NULL COMMENT ""
+) ENGINE=OLAP
+UNIQUE KEY(`id`, `rectime`)
+COMMENT "olap"
+PARTITION BY RANGE(`rectime`)
+(
+PARTITION p202111 VALUES [('2021-11-01 00:00:00'), ('2021-12-01 00:00:00')))
+DISTRIBUTED BY HASH(`id`) BUCKETS 10
+PROPERTIES (
+"replication_allocation" = "tag.location.default: 1",
+"in_memory" = "false",
+"storage_format" = "V2"
+)
+
diff --git a/regression-test/suites/join/sql/bucket_shuffle_join.sql b/regression-test/suites/join/sql/bucket_shuffle_join.sql
new file mode 100644
index 0000000000..807613e2e4
--- /dev/null
+++ b/regression-test/suites/join/sql/bucket_shuffle_join.sql
@@ -0,0 +1 @@
+select * from test_bucket_shuffle_join where rectime="2021-12-01 00:00:00" and id in (select k1 from test_join where k1 in (1,2))


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