Skip to content

Commit 3761b7d

Browse files
amaliujiahvanhovell
authored andcommitted
[SPARK-44574][SQL][CONNECT] Errors that moved into sq/api should also use AnalysisException
### What changes were proposed in this pull request? #42130 moved AnalysisException to sql/api. So we can use AnalysisException for those errors that were moved into sql/api but changed to SparkException. ### Why are the changes needed? Restore to previous behavior. ### Does this PR introduce _any_ user-facing change? No. This PR recovers the behaviors to the past which users should see AnalysisException upon many cases. ### How was this patch tested? Existing UT. Closes #42190 from amaliujia/convert_back_errors. Authored-by: Rui Wang <rui.wang@databricks.com> Signed-off-by: Herman van Hovell <herman@databricks.com>
1 parent 85a4d1e commit 3761b7d

File tree

10 files changed

+68
-76
lines changed

10 files changed

+68
-76
lines changed

sql/api/src/main/scala/org/apache/spark/sql/errors/DataTypeErrors.scala

Lines changed: 23 additions & 30 deletions
Original file line numberDiff line numberDiff line change
@@ -17,8 +17,10 @@
1717
package org.apache.spark.sql.errors
1818

1919
import org.apache.spark.{SparkArithmeticException, SparkException, SparkIllegalArgumentException, SparkNumberFormatException, SparkRuntimeException, SparkUnsupportedOperationException}
20+
import org.apache.spark.sql.AnalysisException
2021
import org.apache.spark.sql.catalyst.trees.{Origin, SQLQueryContext}
2122
import org.apache.spark.sql.catalyst.util.QuotingUtils
23+
import org.apache.spark.sql.catalyst.util.QuotingUtils.toSQLSchema
2224
import org.apache.spark.sql.types.{DataType, Decimal, StringType}
2325
import org.apache.spark.unsafe.types.UTF8String
2426

@@ -97,49 +99,45 @@ private[sql] object DataTypeErrors extends DataTypeErrorsBase {
9799
}
98100

99101
def schemaFailToParseError(schema: String, e: Throwable): Throwable = {
100-
new SparkException(
102+
new AnalysisException(
101103
errorClass = "INVALID_SCHEMA.PARSE_ERROR",
102104
messageParameters = Map(
103-
"inputSchema" -> QuotingUtils.toSQLSchema(schema),
105+
"inputSchema" -> toSQLSchema(schema),
104106
"reason" -> e.getMessage
105107
),
106-
cause = e)
108+
cause = Some(e))
107109
}
108110

109111
def invalidDayTimeIntervalType(startFieldName: String, endFieldName: String): Throwable = {
110-
new SparkException(
112+
new AnalysisException(
111113
errorClass = "_LEGACY_ERROR_TEMP_1224",
112114
messageParameters = Map(
113115
"startFieldName" -> startFieldName,
114-
"endFieldName" -> endFieldName),
115-
cause = null)
116+
"endFieldName" -> endFieldName))
116117
}
117118

118119
def invalidDayTimeField(field: Byte, supportedIds: Seq[String]): Throwable = {
119-
new SparkException(
120+
new AnalysisException(
120121
errorClass = "_LEGACY_ERROR_TEMP_1223",
121122
messageParameters = Map(
122123
"field" -> field.toString,
123-
"supportedIds" -> supportedIds.mkString(", ")),
124-
cause = null)
124+
"supportedIds" -> supportedIds.mkString(", ")))
125125
}
126126

127127
def invalidYearMonthField(field: Byte, supportedIds: Seq[String]): Throwable = {
128-
new SparkException(
128+
new AnalysisException(
129129
errorClass = "_LEGACY_ERROR_TEMP_1225",
130130
messageParameters = Map(
131131
"field" -> field.toString,
132-
"supportedIds" -> supportedIds.mkString(", ")),
133-
cause = null)
132+
"supportedIds" -> supportedIds.mkString(", ")))
134133
}
135134

136135
def decimalCannotGreaterThanPrecisionError(scale: Int, precision: Int): Throwable = {
137-
new SparkException(
136+
new AnalysisException(
138137
errorClass = "_LEGACY_ERROR_TEMP_1228",
139138
messageParameters = Map(
140139
"scale" -> scale.toString,
141-
"precision" -> precision.toString),
142-
cause = null)
140+
"precision" -> precision.toString))
143141
}
144142

145143
def negativeScaleNotAllowedError(scale: Int): Throwable = {
@@ -150,10 +148,9 @@ private[sql] object DataTypeErrors extends DataTypeErrorsBase {
150148
}
151149

152150
def attributeNameSyntaxError(name: String): Throwable = {
153-
new SparkException(
151+
new AnalysisException(
154152
errorClass = "_LEGACY_ERROR_TEMP_1049",
155-
messageParameters = Map("name" -> name),
156-
cause = null)
153+
messageParameters = Map("name" -> name))
157154
}
158155

159156
def cannotMergeIncompatibleDataTypesError(left: DataType, right: DataType): Throwable = {
@@ -182,13 +179,12 @@ private[sql] object DataTypeErrors extends DataTypeErrorsBase {
182179
}
183180

184181
def invalidFieldName(fieldName: Seq[String], path: Seq[String], context: Origin): Throwable = {
185-
new SparkException(
182+
new AnalysisException(
186183
errorClass = "INVALID_FIELD_NAME",
187184
messageParameters = Map(
188185
"fieldName" -> toSQLId(fieldName),
189186
"path" -> toSQLId(path)),
190-
cause = null,
191-
context = context.getQueryContext)
187+
origin = context)
192188
}
193189

194190
def unscaledValueTooLargeForPrecisionError(
@@ -241,13 +237,12 @@ private[sql] object DataTypeErrors extends DataTypeErrorsBase {
241237

242238
def ambiguousColumnOrFieldError(
243239
name: Seq[String], numMatches: Int, context: Origin): Throwable = {
244-
new SparkException(
240+
new AnalysisException(
245241
errorClass = "AMBIGUOUS_COLUMN_OR_FIELD",
246242
messageParameters = Map(
247243
"name" -> toSQLId(name),
248244
"n" -> numMatches.toString),
249-
cause = null,
250-
context = context.getQueryContext)
245+
origin = context)
251246
}
252247

253248
def castingCauseOverflowError(t: String, from: DataType, to: DataType): ArithmeticException = {
@@ -283,16 +278,14 @@ private[sql] object DataTypeErrors extends DataTypeErrorsBase {
283278
}
284279

285280
def charOrVarcharTypeAsStringUnsupportedError(): Throwable = {
286-
new SparkException(
281+
new AnalysisException(
287282
errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING",
288-
messageParameters = Map.empty,
289-
cause = null)
283+
messageParameters = Map.empty)
290284
}
291285

292286
def userSpecifiedSchemaUnsupportedError(operation: String): Throwable = {
293-
new SparkException(
287+
new AnalysisException(
294288
errorClass = "_LEGACY_ERROR_TEMP_1189",
295-
messageParameters = Map("operation" -> operation),
296-
cause = null)
289+
messageParameters = Map("operation" -> operation))
297290
}
298291
}

sql/catalyst/src/main/scala/org/apache/spark/sql/errors/QueryCompilationErrors.scala

Lines changed: 1 addition & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -1921,12 +1921,7 @@ private[sql] object QueryCompilationErrors extends QueryErrorsBase with Compilat
19211921

19221922
def ambiguousColumnOrFieldError(
19231923
name: Seq[String], numMatches: Int, context: Origin): Throwable = {
1924-
new AnalysisException(
1925-
errorClass = "AMBIGUOUS_COLUMN_OR_FIELD",
1926-
messageParameters = Map(
1927-
"name" -> toSQLId(name),
1928-
"n" -> numMatches.toString),
1929-
origin = context)
1924+
DataTypeErrors.ambiguousColumnOrFieldError(name, numMatches, context)
19301925
}
19311926

19321927
def ambiguousReferenceError(name: String, ambiguousReferences: Seq[Attribute]): Throwable = {

sql/catalyst/src/test/scala/org/apache/spark/sql/types/StructTypeSuite.scala

Lines changed: 13 additions & 13 deletions
Original file line numberDiff line numberDiff line change
@@ -324,7 +324,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
324324
check(Seq("s1", "S12"), Some(Seq("s1") -> StructField("s12", IntegerType)))
325325
caseSensitiveCheck(Seq("s1", "S12"), None)
326326
check(Seq("S1.non_exist"), None)
327-
var e = intercept[SparkException] {
327+
var e = intercept[AnalysisException] {
328328
check(Seq("S1", "S12", "S123"), None)
329329
}
330330
checkError(
@@ -335,17 +335,17 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
335335
"path" -> "`s1`.`s12`"))
336336

337337
// ambiguous name
338-
var e2 = intercept[SparkException] {
338+
e = intercept[AnalysisException] {
339339
check(Seq("S2", "x"), None)
340340
}
341341
checkError(
342-
exception = e2,
342+
exception = e,
343343
errorClass = "AMBIGUOUS_COLUMN_OR_FIELD",
344344
parameters = Map("name" -> "`S2`.`x`", "n" -> "2"))
345345
caseSensitiveCheck(Seq("s2", "x"), Some(Seq("s2") -> StructField("x", IntegerType)))
346346

347347
// simple map type
348-
e = intercept[SparkException] {
348+
e = intercept[AnalysisException] {
349349
check(Seq("m1", "key"), None)
350350
}
351351
checkError(
@@ -356,7 +356,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
356356
"path" -> "`m1`"))
357357
checkCollection(Seq("m1", "key"), Some(Seq("m1") -> StructField("key", IntegerType, false)))
358358
checkCollection(Seq("M1", "value"), Some(Seq("m1") -> StructField("value", IntegerType)))
359-
e = intercept[SparkException] {
359+
e = intercept[AnalysisException] {
360360
checkCollection(Seq("M1", "key", "name"), None)
361361
}
362362
checkError(
@@ -365,7 +365,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
365365
parameters = Map(
366366
"fieldName" -> "`M1`.`key`.`name`",
367367
"path" -> "`m1`.`key`"))
368-
e = intercept[SparkException] {
368+
e = intercept[AnalysisException] {
369369
checkCollection(Seq("M1", "value", "name"), None)
370370
}
371371
checkError(
@@ -382,7 +382,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
382382
checkCollection(Seq("M2", "value", "b"),
383383
Some(Seq("m2", "value") -> StructField("b", IntegerType)))
384384
checkCollection(Seq("M2", "value", "non_exist"), None)
385-
e = intercept[SparkException] {
385+
e = intercept[AnalysisException] {
386386
checkCollection(Seq("m2", "key", "A", "name"), None)
387387
}
388388
checkError(
@@ -391,7 +391,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
391391
parameters = Map(
392392
"fieldName" -> "`m2`.`key`.`A`.`name`",
393393
"path" -> "`m2`.`key`.`a`"))
394-
e = intercept[SparkException] {
394+
e = intercept[AnalysisException] {
395395
checkCollection(Seq("M2", "value", "b", "name"), None)
396396
}
397397
checkError(
@@ -401,7 +401,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
401401
"fieldName" -> "`M2`.`value`.`b`.`name`",
402402
"path" -> "`m2`.`value`.`b`"))
403403
// simple array type
404-
e = intercept[SparkException] {
404+
e = intercept[AnalysisException] {
405405
check(Seq("A1", "element"), None)
406406
}
407407
checkError(
@@ -411,7 +411,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
411411
"fieldName" -> "`A1`.`element`",
412412
"path" -> "`a1`"))
413413
checkCollection(Seq("A1", "element"), Some(Seq("a1") -> StructField("element", IntegerType)))
414-
e = intercept[SparkException] {
414+
e = intercept[AnalysisException] {
415415
checkCollection(Seq("A1", "element", "name"), None)
416416
}
417417
checkError(
@@ -425,7 +425,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
425425
checkCollection(Seq("A2", "element", "C"),
426426
Some(Seq("a2", "element") -> StructField("c", IntegerType)))
427427
checkCollection(Seq("A2", "element", "non_exist"), None)
428-
e = intercept[SparkException] {
428+
e = intercept[AnalysisException] {
429429
checkCollection(Seq("a2", "element", "C", "name"), None)
430430
}
431431
checkError(
@@ -439,7 +439,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
439439
checkCollection(Seq("M3", "value", "value", "MA"),
440440
Some(Seq("m3", "value", "value") -> StructField("ma", IntegerType)))
441441
checkCollection(Seq("M3", "value", "value", "non_exist"), None)
442-
e = intercept[SparkException] {
442+
e = intercept[AnalysisException] {
443443
checkCollection(Seq("M3", "value", "value", "MA", "name"), None)
444444
}
445445
checkError(
@@ -453,7 +453,7 @@ class StructTypeSuite extends SparkFunSuite with SQLHelper {
453453
checkCollection(Seq("A3", "element", "element", "D"),
454454
Some(Seq("a3", "element", "element") -> StructField("d", IntegerType)))
455455
checkCollection(Seq("A3", "element", "element", "non_exist"), None)
456-
e = intercept[SparkException] {
456+
e = intercept[AnalysisException] {
457457
checkCollection(Seq("A3", "element", "element", "D", "name"), None)
458458
}
459459
checkError(

sql/core/src/test/scala/org/apache/spark/sql/CharVarcharTestSuite.scala

Lines changed: 16 additions & 12 deletions
Original file line numberDiff line numberDiff line change
@@ -794,11 +794,15 @@ class BasicCharVarcharTestSuite extends QueryTest with SharedSparkSession {
794794

795795
test("invalidate char/varchar in functions") {
796796
checkError(
797-
exception = intercept[SparkException] {
797+
exception = intercept[AnalysisException] {
798798
sql("""SELECT from_json('{"a": "str"}', 'a CHAR(5)')""")
799799
},
800800
errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING",
801-
parameters = Map.empty
801+
parameters = Map.empty,
802+
context = ExpectedContext(
803+
fragment = "from_json('{\"a\": \"str\"}', 'a CHAR(5)')",
804+
start = 7,
805+
stop = 44)
802806
)
803807
withSQLConf((SQLConf.LEGACY_CHAR_VARCHAR_AS_STRING.key, "true")) {
804808
val df = sql("""SELECT from_json('{"a": "str"}', 'a CHAR(5)')""")
@@ -812,19 +816,19 @@ class BasicCharVarcharTestSuite extends QueryTest with SharedSparkSession {
812816
val df = spark.range(10).map(_.toString).toDF()
813817
val schema = new StructType().add("id", CharType(5))
814818
checkError(
815-
exception = intercept[SparkException] {
819+
exception = intercept[AnalysisException] {
816820
spark.createDataFrame(df.collectAsList(), schema)
817821
},
818822
errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING"
819823
)
820824
checkError(
821-
exception = intercept[SparkException] {
825+
exception = intercept[AnalysisException] {
822826
spark.createDataFrame(df.rdd, schema)
823827
},
824828
errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING"
825829
)
826830
checkError(
827-
exception = intercept[SparkException] {
831+
exception = intercept[AnalysisException] {
828832
spark.createDataFrame(df.toJavaRDD, schema)
829833
},
830834
errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING"
@@ -838,12 +842,12 @@ class BasicCharVarcharTestSuite extends QueryTest with SharedSparkSession {
838842

839843
test("invalidate char/varchar in spark.read.schema") {
840844
checkError(
841-
exception = intercept[SparkException] {
845+
exception = intercept[AnalysisException] {
842846
spark.read.schema(new StructType().add("id", CharType(5)))
843847
},
844848
errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING")
845849
checkError(
846-
exception = intercept[SparkException] {
850+
exception = intercept[AnalysisException] {
847851
spark.read.schema("id char(5)")
848852
},
849853
errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING"
@@ -880,13 +884,13 @@ class BasicCharVarcharTestSuite extends QueryTest with SharedSparkSession {
880884

881885
test("invalidate char/varchar in udf's result type") {
882886
checkError(
883-
exception = intercept[SparkException] {
887+
exception = intercept[AnalysisException] {
884888
spark.udf.register("testchar", () => "B", VarcharType(1))
885889
},
886890
errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING"
887891
)
888892
checkError(
889-
exception = intercept[SparkException] {
893+
exception = intercept[AnalysisException] {
890894
spark.udf.register("testchar2", (x: String) => x, VarcharType(1))
891895
},
892896
errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING"
@@ -905,13 +909,13 @@ class BasicCharVarcharTestSuite extends QueryTest with SharedSparkSession {
905909

906910
test("invalidate char/varchar in spark.readStream.schema") {
907911
checkError(
908-
exception = intercept[SparkException] {
912+
exception = intercept[AnalysisException] {
909913
spark.readStream.schema(new StructType().add("id", CharType(5)))
910914
},
911915
errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING"
912916
)
913917
checkError(
914-
exception = intercept[SparkException] {
918+
exception = intercept[AnalysisException] {
915919
spark.readStream.schema("id char(5)")
916920
},
917921
errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING"
@@ -934,7 +938,7 @@ class BasicCharVarcharTestSuite extends QueryTest with SharedSparkSession {
934938
sql("CREATE TABLE t(c char(10), v varchar(255)) USING parquet")
935939
sql("INSERT INTO t VALUES('spark', 'awesome')")
936940
val df = sql("SELECT * FROM t")
937-
checkError(exception = intercept[SparkException] {
941+
checkError(exception = intercept[AnalysisException] {
938942
df.to(newSchema)
939943
}, errorClass = "UNSUPPORTED_CHAR_OR_VARCHAR_AS_STRING", parameters = Map.empty)
940944
withSQLConf((SQLConf.LEGACY_CHAR_VARCHAR_AS_STRING.key, "true")) {

sql/core/src/test/scala/org/apache/spark/sql/DataFrameSuite.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -1721,7 +1721,7 @@ class DataFrameSuite extends QueryTest
17211721

17221722
def checkSyntaxError(name: String): Unit = {
17231723
checkError(
1724-
exception = intercept[SparkException] {
1724+
exception = intercept[org.apache.spark.sql.AnalysisException] {
17251725
df(name)
17261726
},
17271727
errorClass = "_LEGACY_ERROR_TEMP_1049",

sql/core/src/test/scala/org/apache/spark/sql/JsonFunctionsSuite.scala

Lines changed: 3 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -1150,7 +1150,7 @@ class JsonFunctionsSuite extends QueryTest with SharedSparkSession {
11501150
val invalidJsonSchema = """{"fields": [{"a":123}], "type": "struct"}"""
11511151
val invalidJsonSchemaReason = "Failed to convert the JSON string '{\"a\":123}' to a field."
11521152
checkError(
1153-
exception = intercept[SparkException] {
1153+
exception = intercept[AnalysisException] {
11541154
df.select(from_json($"json", invalidJsonSchema, Map.empty[String, String])).collect()
11551155
},
11561156
errorClass = "INVALID_SCHEMA.PARSE_ERROR",
@@ -1165,7 +1165,7 @@ class JsonFunctionsSuite extends QueryTest with SharedSparkSession {
11651165
"was expecting (JSON String, Number, Array, Object or token 'null', 'true' or 'false')\n " +
11661166
"at [Source: (String)\"MAP<INT, cow>\"; line: 1, column: 4]"
11671167
checkError(
1168-
exception = intercept[SparkException] {
1168+
exception = intercept[AnalysisException] {
11691169
df.select(from_json($"json", invalidDataType, Map.empty[String, String])).collect()
11701170
},
11711171
errorClass = "INVALID_SCHEMA.PARSE_ERROR",
@@ -1180,7 +1180,7 @@ class JsonFunctionsSuite extends QueryTest with SharedSparkSession {
11801180
"was expecting (JSON String, Number, Array, Object or token 'null', 'true' or 'false')\n" +
11811181
" at [Source: (String)\"x INT, a cow\"; line: 1, column: 2]"
11821182
checkError(
1183-
exception = intercept[SparkException] {
1183+
exception = intercept[AnalysisException] {
11841184
df.select(from_json($"json", invalidTableSchema, Map.empty[String, String])).collect()
11851185
},
11861186
errorClass = "INVALID_SCHEMA.PARSE_ERROR",

0 commit comments

Comments
 (0)