1
0

[HUDI-1869] Upgrading Spark3 To 3.1 (#3844)

Co-authored-by: pengzhiwei <pengzhiwei2015@icloud.com>
This commit is contained in:
Yann Byron
2021-11-03 09:25:12 +08:00
committed by GitHub
parent dee3a14aae
commit 1f17467f73
22 changed files with 315 additions and 57 deletions

View File

@@ -137,7 +137,8 @@ class TestDataSourceForBootstrap {
verifyIncrementalViewResult(commitInstantTime1, commitInstantTime2, isPartitioned = false, isHiveStylePartitioned = false)
}
@Test def testMetadataBootstrapCOWHiveStylePartitioned(): Unit = {
@Test
def testMetadataBootstrapCOWHiveStylePartitioned(): Unit = {
val timestamp = Instant.now.toEpochMilli
val jsc = JavaSparkContext.fromSparkContext(spark.sparkContext)

View File

@@ -46,8 +46,8 @@ class TestAlterTableDropPartition extends TestHoodieSqlBase {
// insert data
spark.sql(s"""insert into $tableName values (1, "z3", "v1", "2021-10-01"), (2, "l4", "v1", "2021-10-02")""")
checkException(s"alter table $tableName drop partition (dt='2021-10-01')")(
s"dt is not a valid partition column in table `default`.`${tableName}`.;")
checkExceptionContain(s"alter table $tableName drop partition (dt='2021-10-01')")(
s"dt is not a valid partition column in table `default`.`$tableName`.")
}
Seq(false, true).foreach { urlencode =>
@@ -115,12 +115,8 @@ class TestAlterTableDropPartition extends TestHoodieSqlBase {
spark.sql(s"""insert into $tableName values (1, "z3", "v1", "2021-10-01"), (2, "l4", "v1", "2021-10-02")""")
// specify duplicate partition columns
try {
spark.sql(s"alter table $tableName drop partition (dt='2021-10-01', dt='2021-10-02')")
} catch {
case NonFatal(e) =>
assert(e.getMessage.contains("Found duplicate keys 'dt'"))
}
checkExceptionContain(s"alter table $tableName drop partition (dt='2021-10-01', dt='2021-10-02')")(
"Found duplicate keys 'dt'")
// drop 2021-10-01 partition
spark.sql(s"alter table $tableName drop partition (dt='2021-10-01')")
@@ -164,8 +160,8 @@ class TestAlterTableDropPartition extends TestHoodieSqlBase {
|""".stripMargin)
// not specified all partition column
checkException(s"alter table $tableName drop partition (year='2021', month='10')")(
"All partition columns need to be specified for Hoodie's dropping partition;"
checkExceptionContain(s"alter table $tableName drop partition (year='2021', month='10')")(
"All partition columns need to be specified for Hoodie's dropping partition"
)
// drop 2021-10-01 partition
spark.sql(s"alter table $tableName drop partition (year='2021', month='10', day='01')")

View File

@@ -18,13 +18,15 @@
package org.apache.spark.sql.hudi
import java.io.File
import org.apache.log4j.Level
import org.apache.spark.sql.catalyst.util.DateTimeUtils
import org.apache.spark.sql.{Row, SparkSession}
import org.apache.spark.util.Utils
import org.scalactic.source
import org.scalatest.{BeforeAndAfterAll, FunSuite, Tag}
import java.util.TimeZone
class TestHoodieSqlBase extends FunSuite with BeforeAndAfterAll {
org.apache.log4j.Logger.getRootLogger.setLevel(Level.WARN)
@@ -34,6 +36,7 @@ class TestHoodieSqlBase extends FunSuite with BeforeAndAfterAll {
dir
}
TimeZone.setDefault(DateTimeUtils.getTimeZone("CTT"))
protected lazy val spark: SparkSession = SparkSession.builder()
.master("local[1]")
.appName("hoodie sql test")
@@ -43,6 +46,7 @@ class TestHoodieSqlBase extends FunSuite with BeforeAndAfterAll {
.config("hoodie.upsert.shuffle.parallelism", "4")
.config("hoodie.delete.shuffle.parallelism", "4")
.config("spark.sql.warehouse.dir", sparkWareHouse.getCanonicalPath)
.config("spark.sql.session.timeZone", "CTT")
.getOrCreate()
private var tableId = 0
@@ -92,6 +96,19 @@ class TestHoodieSqlBase extends FunSuite with BeforeAndAfterAll {
assertResult(true)(hasException)
}
protected def checkExceptionContain(sql: String)(errorMsg: String): Unit = {
var hasException = false
try {
spark.sql(sql)
} catch {
case e: Throwable =>
assertResult(true)(e.getMessage.contains(errorMsg))
hasException = true
}
assertResult(true)(hasException)
}
protected def removeQuotes(value: Any): Any = {
value match {
case s: String => s.stripPrefix("'").stripSuffix("'")

View File

@@ -353,19 +353,7 @@ class TestMergeIntoTable2 extends TestHoodieSqlBase {
|""".stripMargin
if (HoodieSqlUtils.isSpark3) {
checkException(mergeSql)(
"\nColumns aliases are not allowed in MERGE.(line 5, pos 5)\n\n" +
"== SQL ==\n\r\n" +
s" merge into $tableName\r\n" +
" using (\r\n" +
" select 1, 'a1', 10, 1000, '1'\r\n" +
" ) s0(id,name,price,ts,flag)\r\n" +
"-----^^^\n" +
s" on s0.id = $tableName.id\r\n" +
" when matched and flag = '1' then update set\r\n" +
" id = s0.id, name = s0.name, price = s0.price, ts = s0.ts\r\n" +
" when not matched and flag = '1' then insert *\r\n"
)
checkExceptionContain(mergeSql)("Columns aliases are not allowed in MERGE")
} else {
spark.sql(mergeSql)
checkAnswer(s"select id, name, price, ts from $tableName")(

View File

@@ -98,7 +98,7 @@ class TestPartialUpdateForMergeInto extends TestHoodieSqlBase {
| preCombineField = '_ts'
|)""".stripMargin)
checkException(
checkExceptionContain(
s"""
|merge into $tableName t0
|using ( select 1 as id, 'a1' as name, 12 as price) s0
@@ -106,7 +106,7 @@ class TestPartialUpdateForMergeInto extends TestHoodieSqlBase {
|when matched then update set price = s0.price
""".stripMargin)(
"Missing specify value for the preCombineField: _ts in merge-into update action. " +
"You should add '... update set _ts = xx....' to the when-matched clause.;")
"You should add '... update set _ts = xx....' to the when-matched clause.")
val tableName2 = generateTableName
spark.sql(
@@ -123,7 +123,7 @@ class TestPartialUpdateForMergeInto extends TestHoodieSqlBase {
| preCombineField = '_ts'
|)""".stripMargin)
checkException(
checkExceptionContain(
s"""
|merge into $tableName2 t0
|using ( select 1 as id, 'a1' as name, 12 as price, 1000 as ts) s0
@@ -132,6 +132,6 @@ class TestPartialUpdateForMergeInto extends TestHoodieSqlBase {
""".stripMargin)(
"Missing specify the value for target field: 'id' in merge into update action for MOR table. " +
"Currently we cannot support partial update for MOR, please complete all the target fields " +
"just like '...update set id = s0.id, name = s0.name ....';")
"just like '...update set id = s0.id, name = s0.name ....'")
}
}