@@ -2278,10 +2278,18 @@ index d083cac48ff..3c11bcde807 100644
22782278 import testImplicits._
22792279
22802280diff --git a/sql/core/src/test/scala/org/apache/spark/sql/sources/BucketedReadSuite.scala b/sql/core/src/test/scala/org/apache/spark/sql/sources/BucketedReadSuite.scala
2281- index 746f289c393..0c99d028163 100644
2281+ index 746f289c393..a90106a1463 100644
22822282--- a/sql/core/src/test/scala/org/apache/spark/sql/sources/BucketedReadSuite.scala
22832283+++ b/sql/core/src/test/scala/org/apache/spark/sql/sources/BucketedReadSuite.scala
2284- @@ -25,10 +25,11 @@ import org.apache.spark.sql.catalyst.expressions
2284+ @@ -19,16 +19,19 @@ package org.apache.spark.sql.sources
2285+
2286+ import scala.util.Random
2287+
2288+ + import org.apache.comet.CometConf
2289+ +
2290+ import org.apache.spark.sql._
2291+ import org.apache.spark.sql.catalyst.catalog.BucketSpec
2292+ import org.apache.spark.sql.catalyst.expressions
22852293 import org.apache.spark.sql.catalyst.expressions._
22862294 import org.apache.spark.sql.catalyst.plans.physical.HashPartitioning
22872295 import org.apache.spark.sql.catalyst.types.DataTypeUtils
@@ -2295,7 +2303,7 @@ index 746f289c393..0c99d028163 100644
22952303 import org.apache.spark.sql.execution.joins.SortMergeJoinExec
22962304 import org.apache.spark.sql.functions._
22972305 import org.apache.spark.sql.internal.SQLConf
2298- @@ -102,12 +103 ,20 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
2306+ @@ -102,12 +105 ,20 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
22992307 }
23002308 }
23012309
@@ -2318,7 +2326,7 @@ index 746f289c393..0c99d028163 100644
23182326 // To verify if the bucket pruning works, this function checks two conditions:
23192327 // 1) Check if the pruned buckets (before filtering) are empty.
23202328 // 2) Verify the final result is the same as the expected one
2321- @@ -156,7 +165 ,8 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
2329+ @@ -156,7 +167 ,8 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
23222330 val planWithoutBucketedScan = bucketedDataFrame.filter(filterCondition)
23232331 .queryExecution.executedPlan
23242332 val fileScan = getFileScan(planWithoutBucketedScan)
@@ -2328,7 +2336,7 @@ index 746f289c393..0c99d028163 100644
23282336
23292337 val bucketColumnType = bucketedDataFrame.schema.apply(bucketColumnIndex).dataType
23302338 val rowsWithInvalidBuckets = fileScan.execute().filter(row => {
2331- @@ -452,28 +462 ,49 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
2339+ @@ -452,28 +464 ,49 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
23322340 val joinOperator = if (joined.sqlContext.conf.adaptiveExecutionEnabled) {
23332341 val executedPlan =
23342342 joined.queryExecution.executedPlan.asInstanceOf[AdaptiveSparkPlanExec].executedPlan
@@ -2386,7 +2394,7 @@ index 746f289c393..0c99d028163 100644
23862394 s"expected sort in the right child to be $sortRight but found\n${joinOperator.right}")
23872395
23882396 // check the output partitioning
2389- @@ -836,11 +867 ,11 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
2397+ @@ -836,11 +869 ,11 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
23902398 df1.write.format("parquet").bucketBy(8, "i").saveAsTable("bucketed_table")
23912399
23922400 val scanDF = spark.table("bucketed_table").select("j")
@@ -2400,7 +2408,40 @@ index 746f289c393..0c99d028163 100644
24002408 checkAnswer(aggDF, df1.groupBy("j").agg(max("k")))
24012409 }
24022410 }
2403- @@ -1029,15 +1060,21 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
2411+ @@ -895,7 +928,10 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
2412+ }
2413+
2414+ test("SPARK-29655 Read bucketed tables obeys spark.sql.shuffle.partitions") {
2415+ + // Range partitioning uses random samples, so per-partition comparisons do not always yield
2416+ + // the same results. Disable Comet native range partitioning.
2417+ withSQLConf(
2418+ + CometConf.COMET_EXEC_SHUFFLE_WITH_RANGE_PARTITIONING_ENABLED.key -> "false",
2419+ SQLConf.SHUFFLE_PARTITIONS.key -> "5",
2420+ SQLConf.COALESCE_PARTITIONS_INITIAL_PARTITION_NUM.key -> "7") {
2421+ val bucketSpec = Some(BucketSpec(6, Seq("i", "j"), Nil))
2422+ @@ -914,7 +950,10 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
2423+ }
2424+
2425+ test("SPARK-32767 Bucket join should work if SHUFFLE_PARTITIONS larger than bucket number") {
2426+ + // Range partitioning uses random samples, so per-partition comparisons do not always yield
2427+ + // the same results. Disable Comet native range partitioning.
2428+ withSQLConf(
2429+ + CometConf.COMET_EXEC_SHUFFLE_WITH_RANGE_PARTITIONING_ENABLED.key -> "false",
2430+ SQLConf.SHUFFLE_PARTITIONS.key -> "9",
2431+ SQLConf.COALESCE_PARTITIONS_INITIAL_PARTITION_NUM.key -> "10") {
2432+
2433+ @@ -944,7 +983,10 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
2434+ }
2435+
2436+ test("bucket coalescing eliminates shuffle") {
2437+ + // Range partitioning uses random samples, so per-partition comparisons do not always yield
2438+ + // the same results. Disable Comet native range partitioning.
2439+ withSQLConf(
2440+ + CometConf.COMET_EXEC_SHUFFLE_WITH_RANGE_PARTITIONING_ENABLED.key -> "false",
2441+ SQLConf.COALESCE_BUCKETS_IN_JOIN_ENABLED.key -> "true",
2442+ SQLConf.ADAPTIVE_EXECUTION_ENABLED.key -> "false") {
2443+ // The side with bucketedTableTestSpec1 will be coalesced to have 4 output partitions.
2444+ @@ -1029,15 +1071,21 @@ abstract class BucketedReadSuite extends QueryTest with SQLTestUtils with Adapti
24042445 Seq(true, false).foreach { aqeEnabled =>
24052446 withSQLConf(SQLConf.ADAPTIVE_EXECUTION_ENABLED.key -> aqeEnabled.toString) {
24062447 val plan = sql(query).queryExecution.executedPlan
@@ -2830,6 +2871,34 @@ index 549431ef4f4..e48f1730da6 100644
28302871 withTempDir { dir =>
28312872 withSQLConf(
28322873 "parquet.crypto.factory.class" ->
2874+ diff --git a/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/SQLQuerySuite.scala b/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/SQLQuerySuite.scala
2875+ index 6160c3e5f6c..0956d7d9edc 100644
2876+ --- a/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/SQLQuerySuite.scala
2877+ +++ b/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/SQLQuerySuite.scala
2878+ @@ -24,6 +24,7 @@ import java.sql.{Date, Timestamp}
2879+ import java.util.{Locale, Set}
2880+
2881+ import com.google.common.io.Files
2882+ + import org.apache.comet.CometConf
2883+ import org.apache.hadoop.fs.{FileSystem, Path}
2884+
2885+ import org.apache.spark.{SparkException, TestUtils}
2886+ @@ -838,8 +839,13 @@ abstract class SQLQuerySuiteBase extends QueryTest with SQLTestUtils with TestHi
2887+ }
2888+
2889+ test("SPARK-2554 SumDistinct partial aggregation") {
2890+ - checkAnswer(sql("SELECT sum( distinct key) FROM src group by key order by key"),
2891+ - sql("SELECT distinct key FROM src order by key").collect().toSeq)
2892+ + // Range partitioning uses random samples, so per-partition comparisons do not always yield
2893+ + // the same results. Disable Comet native range partitioning.
2894+ + withSQLConf(CometConf.COMET_EXEC_SHUFFLE_WITH_RANGE_PARTITIONING_ENABLED.key -> "false")
2895+ + {
2896+ + checkAnswer(sql("SELECT sum( distinct key) FROM src group by key order by key"),
2897+ + sql("SELECT distinct key FROM src order by key").collect().toSeq)
2898+ + }
2899+ }
2900+
2901+ test("SPARK-4963 DataFrame sample on mutable row return wrong result") {
28332902diff --git a/sql/hive/src/test/scala/org/apache/spark/sql/hive/test/TestHive.scala b/sql/hive/src/test/scala/org/apache/spark/sql/hive/test/TestHive.scala
28342903index 1d646f40b3e..7f2cdb8f061 100644
28352904--- a/sql/hive/src/test/scala/org/apache/spark/sql/hive/test/TestHive.scala
0 commit comments