迁移指南:SQL、数据集和 DataFrame
- 从 Spark SQL 3.5 升级到 4.0
- 从 Spark SQL 3.5.3 升级到 3.5.4
- 从 Spark SQL 3.5.1 升级到 3.5.2
- 从 Spark SQL 3.5.0 升级到 3.5.1
- 从 Spark SQL 3.4 升级到 3.5
- 从 Spark SQL 3.3 升级到 3.4
- 从 Spark SQL 3.2 升级到 3.3
- 从 Spark SQL 3.1 升级到 3.2
- 从 Spark SQL 3.0 升级到 3.1
- 从 Spark SQL 3.0.1 升级到 3.0.2
- 从 Spark SQL 3.0 升级到 3.0.1
- 从 Spark SQL 2.4 升级到 3.0
- 从 Spark SQL 2.4.7 升级到 2.4.8
- 从 Spark SQL 2.4.5 升级到 2.4.6
- 从 Spark SQL 2.4.4 升级到 2.4.5
- 从 Spark SQL 2.4.3 升级到 2.4.4
- 从 Spark SQL 2.4 升级到 2.4.1
- 从 Spark SQL 2.3 升级到 2.4
- 从 Spark SQL 2.2 升级到 2.3
- 从 Spark SQL 2.1 升级到 2.2
- 从 Spark SQL 2.0 升级到 2.1
- 从 Spark SQL 1.6 升级到 2.0
- 从 Spark SQL 1.5 升级到 1.6
- 从 Spark SQL 1.4 升级到 1.5
- 从 Spark SQL 1.3 升级到 1.4
- 从 Spark SQL 1.0-1.2 升级到 1.3
- 与 Apache Hive 的兼容性
从 Spark SQL 3.5 升级到 4.0
- 从 Spark 4.0 开始,
spark.sql.ansi.enabled
默认开启。要恢复以前的行为,请将spark.sql.ansi.enabled
设置为false
或将SPARK_ANSI_SQL_MODE
设置为false
。 - 从 Spark 4.0 开始,不带
USING
和STORED AS
的CREATE TABLE
语法将使用spark.sql.sources.default
的值作为表提供者,而不是Hive
。要恢复以前的行为,请将spark.sql.legacy.createHiveTableByDefault
设置为true
或将SPARK_SQL_LEGACY_CREATE_HIVE_TABLE
设置为true
。 - 从 Spark 4.0 开始,在 map 中插入元素时的默认行为更改为首先将键 -0.0 规范化为 0.0。受影响的 SQL 函数是
create_map
、map_from_arrays
、map_from_entries
和map_concat
。要恢复以前的行为,请将spark.sql.legacy.disableMapKeyNormalization
设置为true
。 - 从 Spark 4.0 开始,
spark.sql.maxSinglePartitionBytes
的默认值从Long.MaxValue
更改为128m
。要恢复以前的行为,请将spark.sql.maxSinglePartitionBytes
设置为9223372036854775807
(Long.MaxValue
)。 - 从 Spark 4.0 开始,任何对 SQL 表的读取都会考虑 SQL 配置
spark.sql.files.ignoreCorruptFiles
/spark.sql.files.ignoreMissingFiles
,而不是核心配置spark.files.ignoreCorruptFiles
/spark.files.ignoreMissingFiles
。 - 从 Spark 4.0 开始,当读取 SQL 表时遇到
org.apache.hadoop.security.AccessControlException
和org.apache.hadoop.hdfs.BlockMissingException
时,即使spark.sql.files.ignoreCorruptFiles
设置为true
,也会抛出异常并使任务失败。 - 从 Spark 4.0 开始,
spark.sql.hive.metastore
不再支持 Hive 2.0.0 之前的版本,因为它们需要 Spark 不再支持的 JDK 8。用户应迁移到更高版本。 - 从 Spark 4.0 开始,Spark 移除了
hive-llap-common
依赖项。要恢复以前的行为,请将hive-llap-common
jar 添加到类路径中。 - 从 Spark 4.0 开始,
spark.sql.parquet.compression.codec
不再支持编解码器名称lz4raw
,请改用lz4_raw
。 - 从 Spark 4.0 开始,在非 ANSI 模式下将时间戳转换为字节/短整型/整型时发生溢出,Spark 将返回 null 而不是包装值。
- 从 Spark 4.0 开始,
encode()
和decode()
函数仅支持以下字符集:“US-ASCII”、“ISO-8859-1”、“UTF-8”、“UTF-16BE”、“UTF-16LE”、“UTF-16”、“UTF-32”。要恢复函数接受 Spark 当前使用的 JDK 字符集的以前行为,请将spark.sql.legacy.javaCharsets
设置为true
。 - 从 Spark 4.0 开始,
encode()
和decode()
函数在处理不可映射字符时会抛出MALFORMED_CHARACTER_CODING
错误,而在 Spark 3.5 及更早版本中,这些字符将被替换为乱码。要恢复以前的行为,请将spark.sql.legacy.codingErrorAction
设置为true
。例如,如果您尝试使用“UTF-8”解码字符串值tést
/ [116, -23, 115, 116](以 latin1 编码),您会得到t�st
。 - 从 Spark 4.0 开始,带有前缀
spark.sql.legacy
的旧版日期时间重设基准 SQL 配置已被移除。要恢复以前的行为,请使用以下配置:spark.sql.parquet.int96RebaseModeInWrite
代替spark.sql.legacy.parquet.int96RebaseModeInWrite
spark.sql.parquet.datetimeRebaseModeInWrite
代替spark.sql.legacy.parquet.datetimeRebaseModeInWrite
spark.sql.parquet.int96RebaseModeInRead
代替spark.sql.legacy.parquet.int96RebaseModeInRead
spark.sql.avro.datetimeRebaseModeInWrite
代替spark.sql.legacy.avro.datetimeRebaseModeInWrite
spark.sql.avro.datetimeRebaseModeInRead
代替spark.sql.legacy.avro.datetimeRebaseModeInRead
- 从 Spark 4.0 开始,
spark.sql.orc.compression.codec
的默认值从snappy
更改为zstd
。要恢复以前的行为,请将spark.sql.orc.compression.codec
设置为snappy
。 - 从 Spark 4.0 开始,SQL 配置
spark.sql.legacy.allowZeroIndexInFormatString
已被弃用。请考虑将format_string
函数的strfmt
更改为使用 1-基于的索引。第一个参数必须通过1$
引用,第二个通过2$
引用,依此类推。 - 从 Spark 4.0 开始,Postgres JDBC 数据源将 JDBC 读取的 TIMESTAMP WITH TIME ZONE 作为 TimestampType 读取,无论 JDBC 读取选项
preferTimestampNTZ
的值如何,而在 3.5 及之前版本中,当preferTimestampNTZ=true
时,会读取为 TimestampNTZType。要恢复以前的行为,请将spark.sql.legacy.postgres.datetimeMapping.enabled
设置为true
。 - 从 Spark 4.0 开始,Postgres JDBC 数据源将 TimestampType 写入为 TIMESTAMP WITH TIME ZONE,而在 3.5 及之前版本中,它写入为 TIMESTAMP,也即 TIMESTAMP WITHOUT TIME ZONE。要恢复以前的行为,请将
spark.sql.legacy.postgres.datetimeMapping.enabled
设置为true
。 - 从 Spark 4.0 开始,MySQL JDBC 数据源将 TIMESTAMP 作为 TimestampType 读取,无论 JDBC 读取选项
preferTimestampNTZ
的值如何,而在 3.5 及之前版本中,当preferTimestampNTZ=true
时,会读取为 TimestampNTZType。要恢复以前的行为,请将spark.sql.legacy.mysql.timestampNTZMapping.enabled
设置为true
,MySQL DATETIME 不受影响。 - 从 Spark 4.0 开始,MySQL JDBC 数据源将 SMALLINT 读取为 ShortType,而在 Spark 3.5 及之前版本中,它被读取为 IntegerType。MEDIUMINT UNSIGNED 被读取为 IntegerType,而在 Spark 3.5 及之前版本中,它被读取为 LongType。要恢复以前的行为,您可以将列强制转换为旧类型。
- 从 Spark 4.0 开始,MySQL JDBC 数据源将 FLOAT 读取为 FloatType,而在 Spark 3.5 及之前版本中,它被读取为 DoubleType。要恢复以前的行为,您可以将列强制转换为旧类型。
- 从 Spark 4.0 开始,MySQL JDBC 数据源将 BIT(n > 1) 读取为 BinaryType,而在 Spark 3.5 及之前版本中,它被读取为 LongType。要恢复以前的行为,请将
spark.sql.legacy.mysql.bitArrayMapping.enabled
设置为true
。 - 从 Spark 4.0 开始,MySQL JDBC 数据源将 ShortType 写入为 SMALLINT,而在 Spark 3.5 及之前版本中,写入为 INTEGER。要恢复以前的行为,您可以在写入之前将列替换为 IntegerType。
- 从 Spark 4.0 开始,MySQL JDBC 数据源将 TimestampNTZType 写入为 MySQL DATETIME,因为它们都表示 TIMESTAMP WITHOUT TIME ZONE,而在 3.5 及之前版本中,它写入为 MySQL TIMESTAMP。要恢复以前的行为,请将
spark.sql.legacy.mysql.timestampNTZMapping.enabled
设置为true
。 - 从 Spark 4.0 开始,Oracle JDBC 数据源将 TimestampType 写入为 TIMESTAMP WITH LOCAL TIME ZONE,而在 Spark 3.5 及之前版本中,写入为 TIMESTAMP。要恢复以前的行为,请将
spark.sql.legacy.oracle.timestampMapping.enabled
设置为true
。 - 从 Spark 4.0 开始,MsSQL Server JDBC 数据源将 TINYINT 读取为 ShortType,而在 Spark 3.5 及之前版本中,它被读取为 IntegerType。要恢复以前的行为,请将
spark.sql.legacy.mssqlserver.numericMapping.enabled
设置为true
。 - 从 Spark 4.0 开始,MsSQL Server JDBC 数据源将 DATETIMEOFFSET 读取为 TimestampType,而在 Spark 3.5 及之前版本中,它被读取为 StringType。要恢复以前的行为,请将
spark.sql.legacy.mssqlserver.datetimeoffsetMapping.enabled
设置为true
。 - 从 Spark 4.0 开始,DB2 JDBC 数据源将 SMALLINT 读取为 ShortType,而在 Spark 3.5 及之前版本中,它被读取为 IntegerType。要恢复以前的行为,请将
spark.sql.legacy.db2.numericMapping.enabled
设置为true
。 - 从 Spark 4.0 开始,DB2 JDBC 数据源将 BooleanType 写入为 BOOLEAN,而在 Spark 3.5 及之前版本中,写入为 CHAR(1)。要恢复以前的行为,请将
spark.sql.legacy.db2.booleanMapping.enabled
设置为true
。 - 从 Spark 4.0 开始,
spark.sql.legacy.ctePrecedencePolicy
的默认值已从EXCEPTION
更改为CORRECTED
。内部 CTE 定义现在优先于外部定义,而不是抛出错误。 - 从 Spark 4.0 开始,
spark.sql.legacy.timeParserPolicy
的默认值已从EXCEPTION
更改为CORRECTED
。如果启用 ANSI 模式,将抛出CANNOT_PARSE_TIMESTAMP
,而不是抛出INCONSISTENT_BEHAVIOR_CROSS_VERSION
错误。如果禁用 ANSI 模式,将返回NULL
。详情请参见日期时间格式化和解析模式。 - 从 Spark 4.0 开始,一个错误被修复,该错误曾错误地允许在
!
不是前缀运算符时使用!
代替NOT
。现在,诸如expr ! IN (...)
、expr ! BETWEEN ...
或col ! NULL
的子句会抛出语法错误。要恢复以前的行为,请将spark.sql.legacy.bangEqualsNot
设置为true
。 - 从 Spark 4.0 开始,默认情况下,视图可以容忍查询中列类型的更改并进行强制转换以进行补偿。要恢复以前的行为(只允许向上转换),请将
spark.sql.legacy.viewSchemaCompensation
设置为false
。 - 从 Spark 4.0 开始,视图允许控制它们如何响应底层查询更改。默认情况下,视图可以容忍查询中列类型的更改并进行强制转换以进行补偿。要禁用此功能,请将
spark.sql.legacy.viewSchemaBindingMode
设置为false
。这也会从DESCRIBE EXTENDED
和SHOW CREATE TABLE
中移除该子句。 - 从 Spark 4.0 开始,存储分区连接功能标志
spark.sql.sources.v2.bucketing.pushPartValues.enabled
被设置为true
。要恢复以前的行为,请将spark.sql.sources.v2.bucketing.pushPartValues.enabled
设置为false
。 - 从 Spark 4.0 开始,当
language
参数不为NULL
且country
参数为NULL
时,sentences
函数使用Locale(language)
而不是Locale.US
。 - 从 Spark 4.0 开始,从文件源表读取时将正确遵守查询选项,例如分隔符。以前,第一个查询计划会被缓存,随后的选项更改将被忽略。要恢复以前的行为,请将
spark.sql.legacy.readFileSourceTableCacheIgnoreOptions
设置为true
。
从 Spark SQL 3.5.3 升级到 3.5.4
- 从 Spark 3.5.4 开始,当读取 SQL 表时遇到
org.apache.hadoop.security.AccessControlException
和org.apache.hadoop.hdfs.BlockMissingException
时,即使spark.sql.files.ignoreCorruptFiles
设置为true
,也会抛出异常并使任务失败。
从 Spark SQL 3.5.1 升级到 3.5.2
- 从 3.5.2 开始,MySQL JDBC 数据源将 TINYINT UNSIGNED 读取为 ShortType,而在 3.5.1 中,它被错误地读取为 ByteType。
从 Spark SQL 3.5.0 升级到 3.5.1
- 从 Spark 3.5.1 开始,MySQL JDBC 数据源将 TINYINT(n > 1) 和 TINYINT UNSIGNED 读取为 ByteType,而在 Spark 3.5.0 及更早版本中,它们被读取为 IntegerType。要恢复以前的行为,您可以将列强制转换为旧类型。
从 Spark SQL 3.4 升级到 3.5
- 从 Spark 3.5 开始,与 DS V2 下推相关的 JDBC 选项默认设置为
true
。这些选项包括:pushDownAggregate
、pushDownLimit
、pushDownOffset
和pushDownTableSample
。要恢复旧版行为,请将它们设置为false
。例如,将spark.sql.catalog.your_catalog_name.pushDownAggregate
设置为false
。 - 从 Spark 3.5 开始,Spark thrift server 在取消正在运行的语句时将中断任务。要恢复以前的行为,请将
spark.sql.thriftServer.interruptOnCancel
设置为false
。 - 从 Spark 3.5 开始,Row 的 json 和 prettyJson 方法已移至
ToJsonUtil
。 - 从 Spark 3.5 开始,
plan
字段已从AnalysisException
移至EnhancedAnalysisException
。 - 从 Spark 3.5 开始,
spark.sql.optimizer.canChangeCachedPlanOutputPartitioning
默认启用。要恢复以前的行为,请将spark.sql.optimizer.canChangeCachedPlanOutputPartitioning
设置为false
。 - 从 Spark 3.5 开始,
array_insert
函数对于负索引是 1-基于的。它将新元素插入到输入数组的末尾,索引为 -1。要恢复以前的行为,请将spark.sql.legacy.negativeIndexInArrayInsert
设置为true
。 - 从 Spark 3.5 开始,Avro 在读取 Interval 类型为 Date 或 Timestamp 类型,或者读取精度较低的 Decimal 类型时将抛出
AnalysisException
。要恢复旧版行为,请将spark.sql.legacy.avro.allowIncompatibleSchema
设置为true
。
从 Spark SQL 3.3 升级到 3.4
- 从 Spark 3.4 开始,带有显式列列表且列数少于目标表的 INSERT INTO 命令将自动为剩余列添加相应的默认值(对于缺少显式指定默认值的任何列,则为 NULL)。在 Spark 3.3 或更早版本中,这些命令会因报告提供的列数与目标表中的列数不匹配的错误而失败。请注意,禁用
spark.sql.defaultColumn.useNullsForMissingDefaultValues
将恢复以前的行为。 - 从 Spark 3.4 开始,Teradata 的 Number 或 Number(*) 将被视为 Decimal(38,18)。在 Spark 3.3 或更早版本中,Teradata 的 Number 或 Number(*) 将被视为 Decimal(38, 0),在这种情况下,小数部分将被移除。
- 从 Spark 3.4 开始,如果定义了数据库,v1 数据库、表、永久视图和函数标识符将包含“spark_catalog”作为目录名称,例如,表标识符将是:
spark_catalog.default.t
。要恢复旧版行为,请将spark.sql.legacy.v1IdentifierNoCatalog
设置为true
。 - 从 Spark 3.4 开始,当 ANSI SQL 模式(配置
spark.sql.ansi.enabled
)开启时,Spark SQL 在获取具有不存在键的 map 值时始终返回 NULL 结果。在 Spark 3.3 或更早版本中,将出现错误。 - 从 Spark 3.4 开始,SQL CLI
spark-sql
不再在AnalysisException
的错误消息前打印前缀Error in query:
。 - 从 Spark 3.4 开始,当
regex
参数为空时,split
函数将忽略尾随空字符串。 - 从 Spark 3.4 开始,
to_binary
函数对于格式错误的str
输入会抛出错误。请使用try_to_binary
来容忍格式错误的输入并返回 NULL。- 有效的 Base64 字符串应包含 Base64 字母表(A-Za-z0-9+/)中的符号,可选的填充字符(
=
)和可选的空白字符。空白字符在转换时会被跳过,除非它们前面有填充字符。如果存在填充字符,它应位于字符串末尾并遵循 RFC 4648 § 4 中描述的规则。 - 有效的十六进制字符串应只包含允许的符号 (0-9A-Fa-f)。
- 有效的
fmt
值是不区分大小写的hex
、base64
、utf-8
、utf8
。
- 有效的 Base64 字符串应包含 Base64 字母表(A-Za-z0-9+/)中的符号,可选的填充字符(
- 从 Spark 3.4 开始,当 Spark 创建分区但其中一些已存在时,它将只抛出
PartitionsAlreadyExistException
。在 Spark 3.3 或更早版本中,Spark 可以抛出PartitionsAlreadyExistException
或PartitionAlreadyExistsException
。 - 从 Spark 3.4 开始,Spark 将对 ALTER PARTITION 中的分区规范进行验证,以遵循
spark.sql.storeAssignmentPolicy
的行为,如果类型转换失败,这可能会导致异常,例如,如果列p
是 int 类型,则执行ALTER TABLE .. ADD PARTITION(p='a')
。要恢复旧版行为,请将spark.sql.legacy.skipTypeValidationOnAlterPartition
设置为true
。 - 从 Spark 3.4 开始,嵌套数据类型(数组、映射和结构)默认启用矢量化读取器。要恢复旧版行为,请将
spark.sql.orc.enableNestedColumnVectorizedReader
和spark.sql.parquet.enableNestedColumnVectorizedReader
设置为false
。 - 从 Spark 3.4 开始,CSV 数据源不再支持
BinaryType
。在 Spark 3.3 或更早版本中,用户可以在 CSV 数据源中写入二进制列,但 CSV 文件中的输出内容是无意义的Object.toString()
;同时,如果用户读取包含二进制列的 CSV 表,Spark 将抛出Unsupported type: binary
异常。 - 从 Spark 3.4 开始,默认启用 Bloom 过滤器连接。要恢复旧版行为,请将
spark.sql.optimizer.runtime.bloomFilter.enabled
设置为false
。 - 从 Spark 3.4 开始,当对外部 Parquet 文件进行 schema 推断时,带有注解
isAdjustedToUTC=false
的 INT64 时间戳将被推断为 TimestampNTZ 类型,而不是 Timestamp 类型。要恢复旧版行为,请将spark.sql.parquet.inferTimestampNTZ.enabled
设置为false
。 - 从 Spark 3.4 开始,当
spark.sql.legacy.allowNonEmptyLocationInCTAS
设置为true
时,CREATE TABLE AS SELECT ...
的行为从 OVERWRITE 更改为 APPEND。建议用户避免使用非空表位置的 CTAS。
从 Spark SQL 3.2 升级到 3.3
-
从 Spark 3.3 开始,Spark SQL 中的
histogram_numeric
函数返回一个结构体数组 (x, y) 作为输出类型,其中返回值的 'x' 字段的类型从聚合函数中使用的输入值传播。在 Spark 3.2 或更早版本中,'x' 始终具有 double 类型。另外,从 Spark 3.3 开始,可以使用配置spark.sql.legacy.histogramNumericPropagateInputType
恢复到以前的行为。 -
从 Spark 3.3 开始,Spark SQL 中的
DayTimeIntervalType
在ArrowWriter
和ArrowColumnVector
开发者 API 中映射到 Arrow 的Duration
类型。以前,DayTimeIntervalType
映射到 Arrow 的Interval
类型,这与 Spark SQL 映射的其他语言类型不匹配。例如,DayTimeIntervalType
在 Java 中映射到java.time.Duration
。 -
从 Spark 3.3 开始,
lpad
和rpad
函数已被重载以支持字节序列。当第一个参数是字节序列时,可选的填充模式也必须是字节序列,结果是 BINARY 值。在这种情况下,默认填充模式是零字节。要恢复始终返回字符串类型的旧版行为,请将spark.sql.legacy.lpadRpadAlwaysReturnString
设置为true
。 -
从 Spark 3.3 开始,当用户指定模式且模式包含非空字段时,Spark 会将非空模式转换为可空模式,适用于 API
DataFrameReader.schema(schema: StructType).json(jsonDataset: Dataset[String])
和DataFrameReader.schema(schema: StructType).csv(csvDataset: Dataset[String])
。要恢复遵守可空性的旧版行为,请将spark.sql.legacy.respectNullabilityInTextDatasetConversion
设置为true
。 -
从 Spark 3.3 开始,当未指定日期或时间戳模式时,Spark 使用
CAST
表达式方法将输入字符串转换为日期/时间戳。这些更改会影响 CSV/JSON 数据源和分区值的解析。在 Spark 3.2 或更早版本中,当未设置日期或时间戳模式时,Spark 使用默认模式:日期为yyyy-MM-dd
,时间戳为yyyy-MM-dd HH:mm:ss
。更改后,Spark 仍然识别模式以及日期模式
[+-]yyyy*
[+-]yyyy*-[m]m
[+-]yyyy*-[m]m-[d]d
[+-]yyyy*-[m]m-[d]d
[+-]yyyy*-[m]m-[d]d *
[+-]yyyy*-[m]m-[d]dT*
时间戳模式
[+-]yyyy*
[+-]yyyy*-[m]m
[+-]yyyy*-[m]m-[d]d
[+-]yyyy*-[m]m-[d]d
[+-]yyyy*-[m]m-[d]d [h]h:[m]m:[s]s.[ms][ms][ms][us][us][us][zone_id]
[+-]yyyy*-[m]m-[d]dT[h]h:[m]m:[s]s.[ms][ms][ms][us][us][us][zone_id]
[h]h:[m]m:[s]s.[ms][ms][ms][us][us][us][zone_id]
T[h]h:[m]m:[s]s.[ms][ms][ms][us][us][us][zone_id]
-
从 Spark 3.3 开始,
format_string(strfmt, obj, ...)
和printf(strfmt, obj, ...)
中的strfmt
不再支持使用0$
指定第一个参数,当使用参数索引指示参数在参数列表中的位置时,第一个参数应始终通过1$
引用。 -
从 Spark 3.3 开始,CSV 数据源中的 null 元素默认写入为空字符串。在 Spark 3.2 或更早版本中,null 写入为带引号的空字符串,即
""
。要恢复以前的行为,请将nullValue
设置为""
,或将配置spark.sql.legacy.nullValueWrittenAsQuotedEmptyStringCsv
设置为true
。 -
从 Spark 3.3 开始,如果函数不存在,DESCRIBE FUNCTION 会失败。在 Spark 3.2 或更早版本中,DESCRIBE FUNCTION 仍然可以运行并打印“Function: func_name not found”。
-
从 Spark 3.3 开始,表属性
external
成为保留字。如果您指定external
属性,某些命令将失败,例如CREATE TABLE ... TBLPROPERTIES
和ALTER TABLE ... SET TBLPROPERTIES
。在 Spark 3.2 及更早版本中,表属性external
会被静默忽略。您可以将spark.sql.legacy.notReserveProperties
设置为true
以恢复旧行为。 -
从 Spark 3.3 开始,如果函数名与内置函数名之一匹配且未限定,DROP FUNCTION 将失败。在 Spark 3.2 或更早版本中,即使函数名未限定且与内置函数名相同,DROP FUNCTION 仍可以删除持久化函数。
-
从 Spark 3.3 开始,当从定义为
FloatType
或DoubleType
的 JSON 属性读取值时,字符串"+Infinity"
、"+INF"
和"-INF"
现在可以解析为适当的值,除了已经支持的"Infinity"
和"-Infinity"
变体。此更改是为了提高与 Jackson 解析这些值的未加引号版本的兼容性。此外,现在会遵守allowNonNumericNumbers
选项,因此如果此选项被禁用,这些字符串将被视为无效。 -
从 Spark 3.3 开始,Spark 将尝试在
INSERT OVERWRITE DIRECTORY
中使用内置数据源写入器而不是 Hive serde。此行为仅在 Parquet 和 ORC 格式分别启用spark.sql.hive.convertMetastoreParquet
或spark.sql.hive.convertMetastoreOrc
时才有效。要恢复 Spark 3.3 之前的行为,您可以将spark.sql.hive.convertMetastoreInsertDir
设置为false
。 -
从 Spark 3.3 开始,round-like 函数的返回类型精度已修复。这可能导致 Spark 在使用早期版本创建的视图时抛出
CANNOT_UP_CAST_DATATYPE
错误类的AnalysisException
。在这种情况下,您需要使用 ALTER VIEW AS 或 CREATE OR REPLACE VIEW AS 重新创建视图,并使用更新的 Spark 版本。 -
从 Spark 3.3 开始,
unbase64
函数对于格式错误的str
输入会抛出错误。请使用try_to_binary(<str>, 'base64')
来容忍格式错误的输入并返回 NULL。在 Spark 3.2 及更早版本中,unbase64
函数对于格式错误的str
输入会返回尽力而为的结果。 -
从 Spark 3.3 开始,当读取非 Spark 生成的 Parquet 文件时,带有注解
isAdjustedToUTC = false
的 Parquet 时间戳列在 schema 推断期间被推断为 TIMESTAMP_NTZ 类型。在 Spark 3.2 及更早版本中,这些列被推断为 TIMESTAMP 类型。要恢复 Spark 3.3 之前的行为,您可以将spark.sql.parquet.inferTimestampNTZ.enabled
设置为false
。 -
从 Spark 3.3.1 和 3.2.3 开始,对于
SELECT ... GROUP BY a GROUPING SETS (b)
风格的 SQL 语句,grouping__id
返回的值与 Apache Spark 3.2.0、3.2.1、3.2.2 和 3.3.0 不同。它根据用户给定的 group-by 表达式加上分组集列进行计算。要恢复 3.3.1 和 3.2.3 之前的行为,您可以设置spark.sql.legacy.groupingIdWithAppendedUserGroupBy
。有关详细信息,请参阅 SPARK-40218 和 SPARK-40562。
从 Spark SQL 3.1 升级到 3.2
-
从 Spark 3.2 开始,ADD FILE/JAR/ARCHIVE 命令要求如果路径包含空格,则每个路径必须用
"
或'
括起来。 -
从 Spark 3.2 开始,所有支持的 JDBC 方言都将 ROWID 使用 StringType。在 Spark 3.1 或更早版本中,Oracle 方言使用 StringType,其他方言使用 LongType。
-
从 Spark 3.2 开始,具有纳秒精度时间戳类型(
INT64 (TIMESTAMP(NANOS, true))
)的 Parquet 文件无法读取。要恢复 Spark 3.2 之前的行为,可以将spark.sql.legacy.parquet.nanosAsLong
设置为true
。 -
在 Spark 3.2 中,PostgreSQL JDBC 方言对 MONEY 使用 StringType,并且由于 PostgreSQL 的 JDBC 驱动程序无法正确处理这些类型,MONEY[] 不受支持。在 Spark 3.1 或更早版本中,分别使用 DoubleType 和 DoubleType 的 ArrayType。
-
在 Spark 3.2 中,
spark.sql.adaptive.enabled
默认启用。要恢复 Spark 3.2 之前的行为,可以将spark.sql.adaptive.enabled
设置为false
。 - 在 Spark 3.2 中,
show()
操作中会转义以下元字符。在 Spark 3.1 或更早版本中,以下元字符会按原样输出。\n
(换行符)\r
(回车符)\t
(水平制表符)\f
(换页符)\b
(退格符)\u000B
(垂直制表符)\u0007
(响铃符)
-
在 Spark 3.2 中,对于 Hive 外部表,当目标分区已存在时,
ALTER TABLE .. RENAME TO PARTITION
会抛出PartitionAlreadyExistsException
而不是AnalysisException
。 -
在 Spark 3.2 中,对于无 serde 模式,脚本转换的默认 FIELD DELIMIT 是
\u0001
;当用户指定 serde 时,对于 Hive serde 模式,serde 属性field.delim
是\t
。在 Spark 3.1 或更早版本中,默认 FIELD DELIMIT 是\t
;当用户指定 serde 时,对于 Hive serde 模式,serde 属性field.delim
是\u0001
。 -
在 Spark 3.2 中,自动生成的
Cast
(例如由类型强制规则添加的)在生成列别名时将被删除。例如,sql("SELECT floor(1)").columns
将是FLOOR(1)
而不是FLOOR(CAST(1 AS DOUBLE))
。 -
在 Spark 3.2 中,
SHOW TABLES
的输出模式变为namespace: string, tableName: string, isTemporary: boolean
。在 Spark 3.1 或更早版本中,对于内置目录,namespace
字段名为database
,并且 v2 目录没有isTemporary
字段。要恢复带有内置目录的旧模式,可以将spark.sql.legacy.keepCommandOutputSchema
设置为true
。 -
在 Spark 3.2 中,
SHOW TABLE EXTENDED
的输出模式变为namespace: string, tableName: string, isTemporary: boolean, information: string
。在 Spark 3.1 或更早版本中,对于内置目录,namespace
字段名为database
,v2 目录没有变化。要恢复带有内置目录的旧模式,可以将spark.sql.legacy.keepCommandOutputSchema
设置为true
。 -
在 Spark 3.2 中,无论您是否指定表属性键,
SHOW TBLPROPERTIES
的输出模式都变为key: string, value: string
。在 Spark 3.1 及更早版本中,当您指定表属性键时,SHOW TBLPROPERTIES
的输出模式为value: string
。要恢复带有内置目录的旧模式,可以将spark.sql.legacy.keepCommandOutputSchema
设置为true
。 -
在 Spark 3.2 中,
DESCRIBE NAMESPACE
的输出模式变为info_name: string, info_value: string
。在 Spark 3.1 或更早版本中,对于内置目录,info_name
字段名为database_description_item
,info_value
字段名为database_description_value
。要恢复带有内置目录的旧模式,可以将spark.sql.legacy.keepCommandOutputSchema
设置为true
。 - 在 Spark 3.2 中,表刷新会清除表的缓存数据以及所有引用该表的依赖项(例如视图)的缓存数据,同时保持依赖项的缓存。以下命令执行表刷新:
ALTER TABLE .. ADD PARTITION
ALTER TABLE .. RENAME PARTITION
ALTER TABLE .. DROP PARTITION
ALTER TABLE .. RECOVER PARTITIONS
MSCK REPAIR TABLE
LOAD DATA
REFRESH TABLE
TRUNCATE TABLE
- 以及方法
spark.catalog.refreshTable
。在 Spark 3.1 及更早版本中,表刷新会使依赖项未缓存。
-
在 Spark 3.2 中,
count(tblName.*)
的使用被阻止,以避免产生歧义结果。因为如果存在任何 null 值,count(*)
和count(tblName.*)
将输出不同。要恢复 Spark 3.2 之前的行为,可以将spark.sql.legacy.allowStarWithSingleTableIdentifierInCount
设置为true
。 -
在 Spark 3.2 中,我们支持在 INSERT 和 ADD/DROP/RENAME PARTITION 的分区规范中使用类型化字面量。例如,
ADD PARTITION(dt = date'2020-01-01')
添加一个日期值为2020-01-01
的分区。在 Spark 3.1 及更早版本中,分区值将被解析为字符串值date '2020-01-01'
(这是一个非法的日期值),并且我们会在末尾添加一个值为 null 的分区。 -
在 Spark 3.2 中,
DataFrameNaFunctions.replace()
不再对输入列名使用精确字符串匹配,以匹配 SQL 语法并支持限定列名。名称中包含点的输入列名(非嵌套)需要用反引号 ` 转义。现在,如果数据框模式中找不到该列,它会抛出AnalysisException
。如果输入列名是嵌套列,它还会抛出IllegalArgumentException
。在 Spark 3.1 及更早版本中,它曾经忽略无效的输入列名和嵌套列名。 -
在 Spark 3.2 中,日期减法表达式(例如
date1 - date2
)返回DayTimeIntervalType
类型的值。在 Spark 3.1 及更早版本中,返回类型为CalendarIntervalType
。要恢复 Spark 3.2 之前的行为,可以将spark.sql.legacy.interval.enabled
设置为true
。 -
在 Spark 3.2 中,时间戳减法表达式(例如
timestamp '2021-03-31 23:48:00' - timestamp '2021-01-01 00:00:00'
)返回DayTimeIntervalType
类型的值。在 Spark 3.1 及更早版本中,同一表达式的类型是CalendarIntervalType
。要恢复 Spark 3.2 之前的行为,可以将spark.sql.legacy.interval.enabled
设置为true
。 -
在 Spark 3.2 中,
CREATE TABLE .. LIKE ..
命令不能使用保留属性。您需要它们特定的子句来指定它们,例如CREATE TABLE test1 LIKE test LOCATION 'some path'
。您可以将spark.sql.legacy.notReserveProperties
设置为true
以忽略ParseException
,在这种情况下,这些属性将被静默删除,例如:TBLPROPERTIES('owner'='yao')
将不起作用。在 Spark 3.1 及以下版本中,保留属性可以在CREATE TABLE .. LIKE ..
命令中使用,但没有副作用,例如TBLPROPERTIES('location'='/tmp')
不会改变表的 location,而只是创建一个无头属性,就像'a'='b'
一样。 -
在 Spark 3.2 中,
TRANSFORM
运算符不支持输入中的别名。在 Spark 3.1 及更早版本中,我们可以编写脚本转换,如SELECT TRANSFORM(a AS c1, b AS c2) USING 'cat' FROM TBL
。 -
在 Spark 3.2 中,
TRANSFORM
运算符可以在没有 Hive SerDe 的情况下支持ArrayType/MapType/StructType
,在此模式下,我们使用StructsToJson
将ArrayType/MapType/StructType
列转换为STRING
,并使用JsonToStructs
将STRING
解析为ArrayType/MapType/StructType
。在 Spark 3.1 中,Spark 只支持将ArrayType/MapType/StructType
列转换为STRING
,但不支持将STRING
解析为ArrayType/MapType/StructType
输出列。 -
在 Spark 3.2 中,单元到单元间隔字面量(例如
INTERVAL '1-1' YEAR TO MONTH
)和单元列表间隔字面量(例如INTERVAL '3' DAYS '1' HOUR
)被转换为 ANSI 间隔类型:YearMonthIntervalType
或DayTimeIntervalType
。在 Spark 3.1 及更早版本中,此类间隔字面量被转换为CalendarIntervalType
。要恢复 Spark 3.2 之前的行为,可以将spark.sql.legacy.interval.enabled
设置为true
。 -
在 Spark 3.2 中,单元列表间隔字面量不能混合年-月字段(YEAR 和 MONTH)和日-时字段(WEEK, DAY, ..., MICROSECOND)。例如,
INTERVAL 1 month 1 hour
在 Spark 3.2 中是无效的。在 Spark 3.1 及更早版本中,没有此限制,字面量返回CalendarIntervalType
类型的值。要恢复 Spark 3.2 之前的行为,可以将spark.sql.legacy.interval.enabled
设置为true
。 -
在 Spark 3.2 中,Spark 支持
DayTimeIntervalType
和YearMonthIntervalType
作为 HiveSERDE
模式下TRANSFORM
子句的输入和输出,当这两种类型用作输入时,HiveSERDE
模式和ROW FORMAT DELIMITED
模式之间的行为有所不同。在 HiveSERDE
模式下,DayTimeIntervalType
列被转换为HiveIntervalDayTime
,其字符串格式为[-]?d h:m:s.n
,但在ROW FORMAT DELIMITED
模式下,格式为INTERVAL '[-]?d h:m:s.n' DAY TO TIME
。在 HiveSERDE
模式下,YearMonthIntervalType
列被转换为HiveIntervalYearMonth
,其字符串格式为[-]?y-m
,但在ROW FORMAT DELIMITED
模式下,格式为INTERVAL '[-]?y-m' YEAR TO MONTH
。 -
在 Spark 3.2 中,浮点类型的
hash(0) == hash(-0)
。以前,会生成不同的值。 -
在 Spark 3.2 中,使用非空
LOCATION
的CREATE TABLE AS SELECT
将抛出AnalysisException
。要恢复 Spark 3.2 之前的行为,可以将spark.sql.legacy.allowNonEmptyLocationInCTAS
设置为true
。 -
在 Spark 3.2 中,特殊日期时间值(例如
epoch
、today
、yesterday
、tomorrow
和now
)仅在类型化字面量或可折叠字符串的强制转换中受支持,例如select timestamp'now'
或select cast('today' as date)
。在 Spark 3.1 和 3.0 中,此类特殊值在任何字符串到日期/时间戳的强制转换中都受支持。要在 Spark 3.1 和 3.0 中保留这些特殊值作为日期/时间戳,您应该手动替换它们,例如if (c in ('now', 'today'), current_date(), cast(c as date))
。 -
在 Spark 3.2 中,
FloatType
映射到 MySQL 中的FLOAT
。在此之前,它曾映射到REAL
,这在 MySQL 中默认是DOUBLE PRECISION
的同义词。 -
在 Spark 3.2 中,当发送到
QueryExecutionListener
时,由DataFrameWriter
触发的查询执行总是命名为command
。在 Spark 3.1 及更早版本中,名称是save
、insertInto
、saveAsTable
之一。 -
在 Spark 3.2 中,当
allowMissingColumns
设置为 true 时,Dataset.unionByName
会将缺失的嵌套字段添加到结构体的末尾。在 Spark 3.1 中,嵌套结构体字段按字母顺序排序。 -
在 Spark 3.2 中,如果输入查询输出列包含自动生成的别名,则创建/更改视图将失败。这是为了确保查询输出列名在不同 Spark 版本之间保持稳定。要恢复 Spark 3.2 之前的行为,请将
spark.sql.legacy.allowAutoGeneratedAliasForView
设置为true
。 - 在 Spark 3.2 中,日期 +/- 仅包含日时字段的间隔(例如
date '2011-11-11' + interval 12 hours
)返回时间戳。在 Spark 3.1 及更早版本中,相同的表达式返回日期。要恢复 Spark 3.2 之前的行为,可以使用cast
将时间戳转换为日期。
从 Spark SQL 3.0 升级到 3.1
-
在 Spark 3.1 中,统计聚合函数包括
std
、stddev
、stddev_samp
、variance
、var_samp
、skewness
、kurtosis
、covar_samp
、corr
,当表达式计算期间发生DivideByZero
时,例如当stddev_samp
应用于单个元素集时,将返回NULL
而不是Double.NaN
。在 Spark 3.0 及更早版本中,在这种情况下将返回Double.NaN
。要恢复 Spark 3.1 之前的行为,可以将spark.sql.legacy.statisticalAggregate
设置为true
。 -
在 Spark 3.1 中,grouping_id() 返回 long 值。在 Spark 3.0 及更早版本中,此函数返回 int 值。要恢复 Spark 3.1 之前的行为,可以将
spark.sql.legacy.integerGroupingId
设置为true
。 -
在 Spark 3.1 中,SQL UI 数据采用
formatted
模式来解释查询计划结果。要恢复 Spark 3.1 之前的行为,可以将spark.sql.ui.explainMode
设置为extended
。 -
在 Spark 3.1 中,如果指定的日期时间模式无效,
from_unixtime
、unix_timestamp
、to_unix_timestamp
、to_timestamp
和to_date
将失败。在 Spark 3.0 或更早版本中,它们会返回NULL
。 -
在 Spark 3.1 中,Parquet、ORC、Avro 和 JSON 数据源在读取时检测到顶级列以及嵌套结构中存在重复名称时,将抛出
org.apache.spark.sql.AnalysisException: Found duplicate column(s) in the data schema
异常。数据源在检测列名重复时会考虑 SQL 配置spark.sql.caseSensitive
。 -
在 Spark 3.1 中,结构体和映射在强制转换为字符串时被
{}
括号包裹。例如,show()
操作和CAST
表达式使用此类括号。在 Spark 3.0 及更早版本中,[]
括号用于相同目的。要恢复 Spark 3.1 之前的行为,可以将spark.sql.legacy.castComplexTypesToString.enabled
设置为true
。 -
在 Spark 3.1 中,结构体、数组和映射的 NULL 元素在强制转换为字符串时转换为“null”。在 Spark 3.0 或更早版本中,NULL 元素转换为空字符串。要恢复 Spark 3.1 之前的行为,可以将
spark.sql.legacy.castComplexTypesToString.enabled
设置为true
。 -
在 Spark 3.1 中,当
spark.sql.ansi.enabled
为 false 时,如果十进制类型列的总和溢出,Spark 总是返回 null。在 Spark 3.0 或更早版本中,在这种情况下,十进制类型列的总和可能返回 null 或不正确的结果,甚至在运行时失败(取决于实际的查询计划执行)。 -
在 Spark 3.1 中,当以下方法带有路径参数调用时,
path
选项不能共存:DataFrameReader.load()
、DataFrameWriter.save()
、DataStreamReader.load()
或DataStreamWriter.start()
。此外,paths
选项不能与DataFrameReader.load()
共存。例如,spark.read.format("csv").option("path", "/tmp").load("/tmp2")
或spark.read.option("path", "/tmp").csv("/tmp2")
将抛出org.apache.spark.sql.AnalysisException
。在 Spark 3.0 及以下版本中,如果一个路径参数传递给上述方法,path
选项会被覆盖;如果多个路径参数传递给DataFrameReader.load()
,path
选项会被添加到总路径中。要恢复 Spark 3.1 之前的行为,可以将spark.sql.legacy.pathOptionBehavior.enabled
设置为true
。 -
在 Spark 3.1 中,不完整的间隔字面量(例如
INTERVAL '1'
、INTERVAL '1 DAY 2'
)将返回IllegalArgumentException
,因为它们是无效的。在 Spark 3.0 中,这些字面量会返回NULL
。 -
在 Spark 3.1 中,我们移除了内置的 Hive 1.2。您需要将您的自定义 SerDe 迁移到 Hive 2.3。更多详情请参阅 HIVE-15167。
-
在 Spark 3.1 中,如果时间戳在 1900-01-01 00:00:00Z 之前,并且作为 INT96 类型加载(保存),则从/到 parquet 文件加载和保存时间戳会失败。在 Spark 3.0 中,这些操作不会失败,但由于从儒略历到/从原始公历的重设基准,可能会导致输入时间戳的偏移。要恢复 Spark 3.1 之前的行为,可以将
spark.sql.legacy.parquet.int96RebaseModeInRead
或/和spark.sql.legacy.parquet.int96RebaseModeInWrite
设置为LEGACY
。 -
在 Spark 3.1 中,
schema_of_json
和schema_of_csv
函数以 SQL 格式返回 schema,其中字段名带引号。在 Spark 3.0 中,该函数返回不带字段引号且小写的目录字符串。 -
在 Spark 3.1 中,刷新表将触发对所有引用该表的其他缓存的解缓存操作,即使表本身未缓存。在 Spark 3.0 中,只有当表本身被缓存时才会触发此操作。
-
在 Spark 3.1 中,创建或修改永久视图将捕获运行时 SQL 配置并将其存储为视图属性。这些配置将在视图解析的解析和分析阶段应用。要恢复 Spark 3.1 之前的行为,可以将
spark.sql.legacy.useCurrentConfigsForView
设置为true
。 -
在 Spark 3.1 中,临时视图将与永久视图具有相同的行为,即捕获并存储运行时 SQL 配置、SQL 文本、目录和命名空间。捕获的视图属性将在视图解析的解析和分析阶段应用。要恢复 Spark 3.1 之前的行为,可以将
spark.sql.legacy.storeAnalyzedPlanForView
设置为true
。 -
在 Spark 3.1 中,通过
CACHE TABLE ... AS SELECT
创建的临时视图也将具有与永久视图相同的行为。特别是,当临时视图被删除时,Spark 将使其所有缓存依赖项以及临时视图本身的缓存失效。这与 Spark 3.0 及更早版本不同,后者只做后者。要恢复之前的行为,可以将spark.sql.legacy.storeAnalyzedPlanForView
设置为true
。 -
自 Spark 3.1 起,表模式中支持 CHAR/CHARACTER 和 VARCHAR 类型。表扫描/插入将遵循 char/varchar 语义。如果在表模式之外使用 char/varchar,将抛出异常(CAST 是一个例外,它像以前一样简单地将 char/varchar 视为字符串)。要恢复 Spark 3.1 之前的行为(将它们视为 STRING 类型并忽略长度参数,例如
CHAR(4)
),可以将spark.sql.legacy.charVarcharAsString
设置为true
。 -
在 Spark 3.1 中,
AnalysisException
被其子类取代,这些子类在以下情况下针对 Hive 外部目录中的表抛出:ALTER TABLE .. ADD PARTITION
如果新分区已存在,则抛出PartitionsAlreadyExistException
。ALTER TABLE .. DROP PARTITION
对于不存在的分区,抛出NoSuchPartitionsException
。
从 Spark SQL 3.0.1 升级到 3.0.2
- 在 Spark 3.0.2 中,
AnalysisException
被其子类取代,这些子类在以下情况下针对 Hive 外部目录中的表抛出:ALTER TABLE .. ADD PARTITION
如果新分区已存在,则抛出PartitionsAlreadyExistException
。ALTER TABLE .. DROP PARTITION
对于不存在的分区,抛出NoSuchPartitionsException
。
-
在 Spark 3.0.2 中,
PARTITION(col=null)
在分区规范中始终被解析为 null 字面量。在 Spark 3.0.1 或更早版本中,如果分区列是字符串类型,它将被解析为其文本表示的字符串字面量,例如字符串“null”。要恢复旧版行为,可以将spark.sql.legacy.parseNullPartitionSpecAsStringLiteral
设置为 true。 - 在 Spark 3.0.2 中,
SHOW DATABASES
的输出模式变为namespace: string
。在 Spark 3.0.1 及更早版本中,模式是databaseName: string
。自 Spark 3.0.2 起,可以通过将spark.sql.legacy.keepCommandOutputSchema
设置为true
来恢复旧模式。
从 Spark SQL 3.0 升级到 3.0.1
-
在 Spark 3.0 中,如果 JSON 数据源和 JSON 函数
schema_of_json
中的字符串值与 JSON 选项timestampFormat
定义的模式匹配,则会推断出 TimestampType。自 3.0.1 版本起,时间戳类型推断默认禁用。将 JSON 选项inferTimestamp
设置为true
以启用此类型推断。 -
在 Spark 3.0 中,将字符串转换为整数类型(tinyint、smallint、int 和 bigint)、日期时间类型(date、timestamp 和 interval)和布尔类型时,前导和尾随字符(<= ASCII 32)将被修剪。例如,
cast('\b1\b' as int)
结果为1
。自 Spark 3.0.1 起,只修剪前导和尾随空格 ASCII 字符。例如,cast('\t1\t' as int)
结果为1
,但cast('\b1\b' as int)
结果为NULL
。
从 Spark SQL 2.4 升级到 3.0
Dataset/DataFrame API
-
在 Spark 3.0 中,Dataset 和 DataFrame API
unionAll
不再被弃用。它是union
的别名。 -
在 Spark 2.4 及更早版本中,如果键是非结构类型(例如 int、string、array 等),
Dataset.groupByKey
产生的分组数据集的键属性错误地命名为“value”。这违反直觉,并使聚合查询的模式出乎意料。例如,ds.groupByKey(...).count()
的模式是(value, count)
。自 Spark 3.0 起,我们将分组属性命名为“key”。通过新添加的配置spark.sql.legacy.dataset.nameNonStructGroupingKeyAsValue
(默认值为false
)保留了旧行为。 -
在 Spark 3.0 中,列元数据将始终在 API
Column.name
和Column.as
中传播。在 Spark 2.4 及更早版本中,NamedExpression
的元数据在 API 调用时被设置为新列的explicitMetadata
,即使底层NamedExpression
更改元数据,它也不会改变。要恢复 Spark 3.0 之前的行为,可以使用带显式元数据的 APIas(alias: String, metadata: Metadata)
。 -
将一个 Dataset 转换为另一个 Dataset 时,Spark 会将原始 Dataset 中的字段向上转换为目标 DataSet 中相应字段的类型。在 2.4 及更早版本中,这种向上转换不是很严格,例如
Seq("str").toDS.as[Int]
失败,但Seq("str").toDS.as[Boolean]
可以工作并在执行期间抛出 NPE。在 Spark 3.0 中,向上转换更严格,不允许将 String 转换为其他类型,即Seq("str").toDS.as[Boolean]
将在分析期间失败。要恢复 Spark 3.0 之前的行为,请将spark.sql.legacy.doLooseUpcast
设置为true
。
DDL 语句
-
在 Spark 3.0 中,当将值插入到具有不同数据类型的表列中时,类型强制转换按照 ANSI SQL 标准执行。某些不合理的类型转换,例如将
string
转换为int
以及将double
转换为boolean
,是不允许的。如果值超出列数据类型的范围,将抛出运行时异常。在 Spark 2.4 及以下版本中,只要它们是有效的Cast
,就允许在表插入期间进行类型转换。当将超出范围的值插入到整数字段时,将插入值的低位(与 Java/Scala 数字类型转换相同)。例如,如果将 257 插入到字节类型的字段中,结果是 1。该行为由选项spark.sql.storeAssignmentPolicy
控制,默认值为“ANSI”。将选项设置为“Legacy”可恢复之前的行为。 -
ADD JAR
命令以前返回一个结果集,其单个值为 0。它现在返回一个空结果集。 -
Spark 2.4 及更早版本:
SET
命令即使指定的键是SparkConf
条目并且没有效果(因为该命令不更新SparkConf
)也可以工作而没有任何警告,但这可能会让用户感到困惑。在 3.0 中,如果使用SparkConf
键,则该命令会失败。可以通过将spark.sql.legacy.setCommandRejectsSparkCoreConfs
设置为false
来禁用此检查。 -
刷新缓存表将触发表取消缓存操作,然后是表缓存(惰性)操作。在 Spark 2.4 及更早版本中,缓存名称和存储级别在取消缓存操作之前不会被保留。因此,缓存名称和存储级别可能会意外更改。在 Spark 3.0 中,缓存名称和存储级别首先保留以供缓存重新创建。这有助于在表刷新时保持一致的缓存行为。
-
在 Spark 3.0 中,以下列出的属性变为保留属性;如果您在
CREATE DATABASE ... WITH DBPROPERTIES
和ALTER TABLE ... SET TBLPROPERTIES
等位置指定保留属性,命令将失败。您需要使用它们的特定子句来指定它们,例如CREATE DATABASE test COMMENT 'any comment' LOCATION 'some path'
。您可以将spark.sql.legacy.notReserveProperties
设置为true
以忽略ParseException
,在这种情况下,这些属性将被静默删除,例如:SET DBPROPERTIES('location'='/tmp')
将无效。在 Spark 2.4 及更早版本中,这些属性既不保留也不具有副作用,例如SET DBPROPERTIES('location'='/tmp')
不会更改数据库的位置,而只会创建一个无头属性,就像'a'='b'
一样。属性(区分大小写) 数据库保留 表保留 备注 provider 否 是 对于表,使用 USING
子句指定它。一旦设置,不能更改。location 是 是 对于数据库和表,使用 LOCATION
子句指定它。owner 是 是 对于数据库和表,它由运行 Spark 并创建表的用户决定。 -
在 Spark 3.0 中,您可以使用
ADD FILE
添加文件目录。以前,您只能使用此命令添加单个文件。要恢复早期版本的行为,请将spark.sql.legacy.addSingleFileInAddFile
设置为true
。 -
在 Spark 3.0 中,如果表不存在,
SHOW TBLPROPERTIES
会抛出AnalysisException
。在 Spark 2.4 及更早版本中,这种情况会导致NoSuchTableException
。 -
在 Spark 3.0 中,
SHOW CREATE TABLE table_identifier
总是返回 Spark DDL,即使给定表是 Hive SerDe 表。要生成 Hive DDL,请改用SHOW CREATE TABLE table_identifier AS SERDE
命令。 -
在 Spark 3.0 中,非 Hive-Serde 表中不允许使用 CHAR 类型列,如果检测到 CHAR 类型,CREATE/ALTER TABLE 命令将失败。请改用 STRING 类型。在 Spark 2.4 及更早版本中,CHAR 类型被视为 STRING 类型,长度参数被简单忽略。
UDF 和内置函数
-
在 Spark 3.0 中,
date_add
和date_sub
函数只接受 int、smallint、tinyint 作为第二个参数;小数和非字面量字符串不再有效,例如:date_add(cast('1964-05-23' as date), '12.34')
会导致AnalysisException
。请注意,字符串字面量仍然允许,但如果字符串内容不是有效整数,Spark 将抛出AnalysisException
。在 Spark 2.4 及更早版本中,如果第二个参数是小数或字符串值,它将被强制转换为 int 值,结果是日期值1964-06-04
。 -
在 Spark 3.0 中,函数
percentile_approx
及其别名approx_percentile
只接受范围在[1, 2147483647]
的整数值作为其第三个参数accuracy
,小数和字符串类型不允许,例如,percentile_approx(10.0, 0.2, 1.8D)
会导致AnalysisException
。在 Spark 2.4 及更早版本中,如果accuracy
是小数或字符串值,它将被强制转换为 int 值,percentile_approx(10.0, 0.2, 1.8D)
的操作方式是percentile_approx(10.0, 0.2, 1)
,结果为10.0
。 -
在 Spark 3.0 中,当哈希表达式应用于
MapType
的元素时,会抛出分析异常。要恢复 Spark 3.0 之前的行为,请将spark.sql.legacy.allowHashOnMapType
设置为true
。 -
在 Spark 3.0 中,当调用
array
/map
函数时没有参数,它返回一个元素类型为NullType
的空集合。在 Spark 2.4 及更早版本中,它返回一个元素类型为StringType
的空集合。要恢复 Spark 3.0 之前的行为,可以将spark.sql.legacy.createEmptyCollectionUsingStringType
设置为true
。 -
在 Spark 3.0 中,
from_json
函数支持两种模式 -PERMISSIVE
和FAILFAST
。模式可以通过mode
选项设置。默认模式变为PERMISSIVE
。在以前的版本中,from_json
的行为既不符合PERMISSIVE
也不符合FAILFAST
,尤其是在处理格式错误的 JSON 记录时。例如,JSON 字符串{"a" 1}
和模式a INT
在以前的版本中被转换为null
,但 Spark 3.0 将其转换为Row(null)
。 -
在 Spark 2.4 及更早版本中,您可以使用内置函数(如
CreateMap
、MapFromArrays
等)创建键为 Map 类型的 Map 值。在 Spark 3.0 中,不允许使用这些内置函数创建键为 Map 类型的 Map 值。用户可以使用map_entries
函数将 Map 转换为 array<struct<key, value>> 作为解决方法。此外,用户仍然可以从数据源或 Java/Scala 集合中读取键为 Map 类型的 Map 值,尽管不建议这样做。 -
在 Spark 2.4 及更早版本中,您可以通过内置函数(如
CreateMap
、StringToMap
等)创建具有重复键的 Map。具有重复键的 Map 的行为是未定义的,例如,Map 查找尊重首先出现的重复键,Dataset.collect
只保留最后出现的重复键,MapKeys
返回重复键等。在 Spark 3.0 中,当发现重复键时,Spark 会抛出RuntimeException
。您可以将spark.sql.mapKeyDedupPolicy
设置为LAST_WIN
以使用“后赢”策略去重 Map 键。用户仍然可以从不强制执行此操作的数据源(例如 Parquet)读取具有重复键的 Map 值,但行为是未定义的。 -
在 Spark 3.0 中,默认情况下不允许使用
org.apache.spark.sql.functions.udf(AnyRef, DataType)
。建议删除返回类型参数以自动切换到类型化 Scala UDF,或者将spark.sql.legacy.allowUntypedScalaUDF
设置为 true 以继续使用它。在 Spark 2.4 及更早版本中,如果org.apache.spark.sql.functions.udf(AnyRef, DataType)
接收一个带有原始类型参数的 Scala 闭包,如果输入值为 null,则返回的 UDF 返回 null。但是,在 Spark 3.0 中,如果输入值为 null,UDF 返回 Java 类型的默认值。例如,val f = udf((x: Int) => x, IntegerType)
,如果列x
为 null,f($"x")
在 Spark 2.4 及更早版本中返回 null,在 Spark 3.0 中返回 0。此行为更改是由于 Spark 3.0 默认使用 Scala 2.12 构建。 -
在 Spark 3.0 中,高阶函数
exists
遵循三值布尔逻辑,即如果predicate
返回任何null
且未获得true
,则exists
返回null
而不是false
。例如,exists(array(1, null, 3), x -> x % 2 == 0)
为null
。通过将spark.sql.legacy.followThreeValuedLogicInArrayExists
设置为false
可以恢复之前的行为。 -
在 Spark 3.0 中,如果原始日期是一个月的最后一天,
add_months
函数不会将结果日期调整为当月的最后一天。例如,select add_months(DATE'2019-02-28', 1)
的结果是2019-03-28
。在 Spark 2.4 及更早版本中,如果原始日期是一个月的最后一天,则会调整结果日期。例如,向2019-02-28
添加一个月会得到2019-03-31
。 -
在 Spark 2.4 及更早版本中,
current_timestamp
函数只返回毫秒分辨率的时间戳。在 Spark 3.0 中,如果系统上可用的底层时钟提供微秒分辨率,该函数可以返回微秒分辨率的结果。 -
在 Spark 3.0 中,0 参数的 Java UDF 在执行器端与其他 UDF 一致地执行。在 Spark 2.4 及更早版本中,0 参数的 Java UDF 单独在驱动器端执行,结果传播到执行器,这在某些情况下可能性能更好,但在某些情况下会导致一致性问题和正确性问题。
-
java.lang.Math
的log
、log1p
、exp
、expm1
和pow
的结果可能因平台而异。在 Spark 3.0 中,等效 SQL 函数(包括相关的 SQL 函数如LOG10
)的结果与java.lang.StrictMath
保持一致。在几乎所有情况下,这都不会改变返回值,并且差异非常小,但可能与 x86 平台上的java.lang.Math
不完全匹配,例如log(3.0)
的值在Math.log()
和StrictMath.log()
之间有所不同。 -
在 Spark 3.0 中,
cast
函数在将字面量转换为Double
或Float
类型时,会不区分大小写地处理字符串字面量,例如 'Infinity'、'+Infinity'、'-Infinity'、'NaN'、'Inf'、'+Inf'、'-Inf',以确保与其它数据库系统更好地兼容。此行为更改如下表所示:操作 Spark 3.0 之前的结果 Spark 3.0 中的结果 CAST(‘infinity’ AS DOUBLE) NULL Double.PositiveInfinity CAST(‘+infinity’ AS DOUBLE) NULL Double.PositiveInfinity CAST(‘inf’ AS DOUBLE) NULL Double.PositiveInfinity CAST(‘inf’ AS DOUBLE) NULL Double.PositiveInfinity CAST(‘-infinity’ AS DOUBLE) NULL Double.NegativeInfinity CAST(‘-inf’ AS DOUBLE) NULL Double.NegativeInfinity CAST(‘infinity’ AS FLOAT) NULL Float.PositiveInfinity CAST(‘+infinity’ AS FLOAT) NULL Float.PositiveInfinity CAST(‘inf’ AS FLOAT) NULL Float.PositiveInfinity CAST(‘+inf’ AS FLOAT) NULL Float.PositiveInfinity CAST(‘-infinity’ AS FLOAT) NULL Float.NegativeInfinity CAST(‘-inf’ AS FLOAT) NULL Float.NegativeInfinity CAST(‘nan’ AS DOUBLE) NULL Double.NaN CAST(‘nan’ AS FLOAT) NULL Float.NaN -
在 Spark 3.0 中,将间隔值转换为字符串类型时,没有“interval”前缀,例如
1 days 2 hours
。在 Spark 2.4 及更早版本中,字符串包含“interval”前缀,例如interval 1 days 2 hours
。 -
在 Spark 3.0 中,将字符串值转换为整数类型(tinyint、smallint、int 和 bigint)、日期时间类型(date、timestamp 和 interval)和布尔类型时,前导和尾随空格(<= ASCII 32)将被修剪,然后转换为这些类型的值。例如,
cast(' 1\t' as int)
结果为1
,cast(' 1\t' as boolean)
结果为true
,cast('2019-10-10\t as date)
结果为日期值2019-10-10
。在 Spark 2.4 及更早版本中,将字符串转换为整数和布尔值时,它不会修剪两端的空格;上述结果为null
,而对于日期时间,只删除尾随空格(= ASCII 32)。
查询引擎
-
在 Spark 2.4 及更早版本中,意外地支持
FROM <table>
或FROM <table> UNION ALL FROM <table>
等 SQL 查询。在 Hive 风格的FROM <table> SELECT <expr>
中,SELECT
子句不可忽略。Hive 和 Presto 都不支持这种语法。这些查询在 Spark 3.0 中被视为无效。 -
在 Spark 3.0 中,间隔字面量语法不再允许使用多个 from-to 单元。例如,
SELECT INTERVAL '1-1' YEAR TO MONTH '2-2' YEAR TO MONTH'
会抛出解析器异常。 -
在 Spark 3.0 中,以科学计数法表示的数字(例如
1E2
)将被解析为 Double。在 Spark 2.4 及更早版本中,它们被解析为 Decimal。要恢复 Spark 3.0 之前的行为,可以将spark.sql.legacy.exponentLiteralAsDecimal.enabled
设置为true
。 -
在 Spark 3.0 中,日时间隔字符串根据
from
和to
界限转换为间隔。如果输入字符串与指定界限定义的模式不匹配,则会抛出ParseException
异常。例如,interval '2 10:20' hour to minute
会引发异常,因为预期的格式是[+|-]h[h]:[m]m
。在 Spark 2.4 中,from
界限未被考虑,并且to
界限用于截断结果间隔。例如,所示示例中的日时间隔字符串被转换为interval 10 hours 20 minutes
。要恢复 Spark 3.0 之前的行为,可以将spark.sql.legacy.fromDayTimeString.enabled
设置为true
。 -
在 Spark 3.0 中,默认情况下不允许小数的负刻度,例如,字面量
1E10BD
的数据类型是DecimalType(11, 0)
。在 Spark 2.4 及更早版本中,它是DecimalType(2, -9)
。要恢复 Spark 3.0 之前的行为,可以将spark.sql.legacy.allowNegativeScaleOfDecimal
设置为true
。 -
在 Spark 3.0 中,一元算术运算符加号(
+
)只接受字符串、数字和间隔类型值作为输入。此外,带有整数字符串表示形式的+
被强制转换为双精度值,例如+'1'
返回1.0
。在 Spark 2.4 及更早版本中,此运算符被忽略。它没有类型检查,因此,所有带有+
前缀的类型值都有效,例如+ array(1, 2)
有效并返回[1, 2]
。此外,它根本没有类型强制转换,例如,在 Spark 2.4 中,+'1'
的结果是字符串1
。 -
在 Spark 3.0 中,如果 Dataset 查询包含由自连接引起的歧义列引用,则查询会失败。一个典型的例子:
val df1 = ...; val df2 = df1.filter(...);
,然后df1.join(df2, df1("a") > df2("a"))
返回一个空结果,这非常令人困惑。这是因为 Spark 无法解析指向自连接表的 Dataset 列引用,并且在 Spark 中df1("a")
与df2("a")
完全相同。要恢复 Spark 3.0 之前的行为,可以将spark.sql.analyzer.failAmbiguousSelfJoin
设置为false
。 -
在 Spark 3.0 中,引入了配置
spark.sql.legacy.ctePrecedencePolicy
来控制嵌套 WITH 子句中名称冲突的行为。默认值EXCEPTION
会使 Spark 抛出 AnalysisException,强制用户选择他们想要的特定替换顺序。如果设置为CORRECTED
(推荐),内部 CTE 定义优先于外部定义。例如,将配置设置为false
,WITH t AS (SELECT 1), t2 AS (WITH t AS (SELECT 2) SELECT * FROM t) SELECT * FROM t2
返回2
,而将其设置为LEGACY
,结果是1
,这是 2.4 及更早版本中的行为。 -
在 Spark 3.0 中,配置
spark.sql.crossJoin.enabled
变为内部配置,并且默认值为 true,因此默认情况下 Spark 不会在带有隐式交叉连接的 SQL 上引发异常。 -
在 Spark 2.4 及更早版本中,浮点/双精度 -0.0 在语义上等于 0.0,但 -0.0 和 0.0 在聚合分组键、窗口分区键和连接键中使用时被认为是不同的值。在 Spark 3.0 中,此错误已修复。例如,
Seq(-0.0, 0.0).toDF("d").groupBy("d").count()
在 Spark 3.0 中返回[(0.0, 2)]
,而在 Spark 2.4 及更早版本中返回[(0.0, 1), (-0.0, 1)]
。 -
在 Spark 2.4 及更早版本中,无效的时区 ID 会被静默忽略并替换为 GMT 时区,例如在 `from_utc_timestamp` 函数中。在 Spark 3.0 中,此类时区 ID 将被拒绝,并且 Spark 会抛出
java.time.DateTimeException
。 -
在 Spark 3.0 中,在解析、格式化和转换日期和时间戳以及提取子组件(如年、日等)时,使用儒略历/格里高利历。Spark 3.0 使用基于 ISO 历法 的
java.time
包中的 Java 8 API 类。在 Spark 2.4 及更早版本中,这些操作是使用混合历法(儒略历 + 格里高利历)执行的。这些更改影响了 1582 年 10 月 15 日(格里高利历)之前的日期结果,并影响了以下 Spark 3.0 API:-
时间戳/日期字符串的解析/格式化。这影响了 CSV/JSON 数据源以及
unix_timestamp
、date_format
、to_unix_timestamp
、from_unixtime
、to_date
、to_timestamp
函数,当用户指定的模式用于解析和格式化时。在 Spark 3.0 中,我们在 日期时间格式和解析模式 中定义了我们自己的模式字符串,其底层通过 DateTimeFormatter 实现。新实现对其输入执行严格检查。例如,如果模式是yyyy-MM-dd
,则时间戳2015-07-22 10:00:00
无法解析,因为解析器不会消耗整个输入。另一个例子是输入31/01/2015 00:00
无法通过dd/MM/yyyy hh:mm
模式解析,因为hh
假定小时范围为1-12
。在 Spark 2.4 及更早版本中,java.text.SimpleDateFormat
用于时间戳/日期字符串转换,支持的模式在 SimpleDateFormat 中描述。可以通过将spark.sql.legacy.timeParserPolicy
设置为LEGACY
来恢复旧行为。 -
weekofyear
、weekday
、dayofweek
、date_trunc
、from_utc_timestamp
、to_utc_timestamp
和unix_timestamp
函数使用 java.time API 计算年份的周数、星期的天数以及从/到 UTC 时区 TimestampType 值的转换。 -
JDBC 选项
lowerBound
和upperBound
以与将字符串转换为 TimestampType/DateType 值相同的方式转换为 TimestampType/DateType 值。转换基于儒略历/格里高利历以及由 SQL 配置spark.sql.session.timeZone
定义的时区。在 Spark 2.4 及更早版本中,转换基于混合历法(儒略历 + 格里高利历)和默认系统时区。 -
格式化
TIMESTAMP
和DATE
字面量。 -
从字符串创建类型化的
TIMESTAMP
和DATE
字面量。在 Spark 3.0 中,字符串到类型化TIMESTAMP
/DATE
字面量的转换通过强制转换为TIMESTAMP
/DATE
值执行。例如,TIMESTAMP '2019-12-23 12:59:30'
在语义上等同于CAST('2019-12-23 12:59:30' AS TIMESTAMP)
。当输入字符串不包含时区信息时,在这种情况下使用 SQL 配置spark.sql.session.timeZone
中的时区。在 Spark 2.4 及更早版本中,转换基于 JVM 系统时区。默认时区的不同来源可能会改变类型化TIMESTAMP
和DATE
字面量的行为。
-
-
在 Spark 3.0 中,
TIMESTAMP
字面量使用 SQL 配置spark.sql.session.timeZone
转换为字符串。在 Spark 2.4 及更早版本中,转换使用 Java 虚拟机的默认时区。 -
在 Spark 3.0 中,Spark 在与日期/时间戳进行二进制比较时,将
String
转换为Date/Timestamp
。通过将spark.sql.legacy.typeCoercion.datetimeToString.enabled
设置为true
可以恢复将Date/Timestamp
转换为String
的先前行为。 - 在 Spark 3.0 中,从字符串到日期和时间戳的转换支持特殊值。这些值只是符号速记,在读取时转换为普通的日期或时间戳值。支持以下日期字符串值:
epoch [zoneId]
- 1970-01-01today [zoneId]
- 由spark.sql.session.timeZone
指定的时区中的当前日期yesterday [zoneId]
- 当前日期 - 1tomorrow [zoneId]
- 当前日期 + 1now
- 执行当前查询的日期。它与 today 具有相同的概念
例如
SELECT date 'tomorrow' - date 'yesterday';
应该输出2
。以下是特殊的时间戳值:epoch [zoneId]
- 1970-01-01 00:00:00+00 (Unix 系统时间零点)today [zoneId]
- 今天午夜yesterday [zoneId]
- 昨天午夜tomorrow [zoneId]
- 明天午夜now
- 当前查询开始时间
例如
SELECT timestamp 'tomorrow';
。 -
自 Spark 3.0 起,当使用
EXTRACT
表达式从日期/时间戳值中提取秒字段时,结果将是DecimalType(8, 6)
值,其中秒部分有 2 位数字,小数部分有 6 位数字,精度为微秒。例如,extract(second from to_timestamp('2019-09-20 10:10:10.1'))
的结果是10.100000
。在 Spark 2.4 及更早版本中,它返回一个IntegerType
值,前一个示例的结果是10
。 -
在 Spark 3.0 中,日期时间模式字母
F
表示**月份中对齐的周日**,它表示一周内天数的计数概念,其中周与月份的开始对齐。在 Spark 2.4 及更早版本中,它表示**月份的周数**,它表示月份内周的计数概念,其中周从固定的周日开始,例如,2020-07-30
是该月第一天之后的 30 天(4 周零 2 天),因此date_format(date '2020-07-30', 'F')
在 Spark 3.0 中返回 2,但在 Spark 2.x 中作为周数,它返回 5,因为它位于 2020 年 7 月的第 5 周,其中第一周是 2020-07-01 至 07-04。 -
在 Spark 3.0 中,Spark 将尝试在
CTAS
中使用内置数据源写入器而不是 Hive serde。此行为仅在spark.sql.hive.convertMetastoreParquet
或spark.sql.hive.convertMetastoreOrc
分别为 Parquet 和 ORC 格式启用时才有效。要恢复 Spark 3.0 之前的行为,可以将spark.sql.hive.convertMetastoreCtas
设置为false
。 - 在 Spark 3.0 中,Spark 将尝试使用内置数据源写入器而不是 Hive serde 来处理使用 HiveSQL 语法创建的插入到分区 ORC/Parquet 表中的操作。此行为仅在
spark.sql.hive.convertMetastoreParquet
或spark.sql.hive.convertMetastoreOrc
分别为 Parquet 和 ORC 格式启用时才有效。要恢复 Spark 3.0 之前的行为,可以将spark.sql.hive.convertInsertingPartitionedTable
设置为false
。
数据源
-
在 Spark 2.4 及更早版本中,当使用 Spark 本机数据源 (parquet/orc) 读取 Hive SerDe 表时,Spark 会推断实际文件模式并更新元存储中的表模式。在 Spark 3.0 中,Spark 不再推断模式。这不应该对最终用户造成任何问题,但如果确实造成了问题,请将
spark.sql.hive.caseSensitiveInferenceMode
设置为INFER_AND_SAVE
。 -
在 Spark 2.4 及更早版本中,如果分区列值无法转换为相应的用户提供模式,则会被转换为 null。在 3.0 中,分区列值将根据用户提供的模式进行验证。如果验证失败,则会抛出异常。可以通过将
spark.sql.sources.validatePartitionColumns
设置为false
来禁用此验证。 -
在 Spark 3.0 中,如果在递归目录列表期间文件或子目录消失(即,它们出现在中间列表中,但在递归目录列表的后续阶段无法读取或列出,原因可能是并发文件删除或对象存储一致性问题),那么除非
spark.sql.files.ignoreMissingFiles
为true
(默认false
),否则列表将因异常而失败。在以前的版本中,这些丢失的文件或子目录将被忽略。请注意,此行为更改仅适用于初始表文件列表(或REFRESH TABLE
期间),而不适用于查询执行:净变化是spark.sql.files.ignoreMissingFiles
现在在表文件列表/查询规划期间也遵循,而不仅仅在查询执行时。 -
在 Spark 2.4 及更早版本中,JSON 数据源的解析器将空字符串视为某些数据类型(如
IntegerType
)的 null。对于FloatType
、DoubleType
、DateType
和TimestampType
,它在空字符串上会失败并抛出异常。Spark 3.0 不允许空字符串,并且会为除StringType
和BinaryType
之外的数据类型抛出异常。通过将spark.sql.legacy.json.allowEmptyString.enabled
设置为true
可以恢复允许空字符串的先前行为。 -
在 Spark 2.4 及更早版本中,当指定模式为
StructType
时,JSON 数据源和from_json
等 JSON 函数在 PERMISSIVE 模式下将错误的 JSON 记录转换为所有字段为null
的行。在 Spark 3.0 中,如果某些 JSON 列值已成功解析并转换为所需类型,则返回的行可以包含非null
字段。 -
在 Spark 3.0 中,JSON 数据源和 JSON 函数
schema_of_json
如果字符串值与 JSON 选项timestampFormat
定义的模式匹配,则会推断出 TimestampType。将 JSON 选项inferTimestamp
设置为false
以禁用此类类型推断。 -
在 Spark 2.4 及更早版本中,CSV 数据源在 PERMISSIVE 模式下将格式错误的 CSV 字符串转换为所有字段为
null
的行。在 Spark 3.0 中,如果某些 CSV 列值已成功解析并转换为所需类型,则返回的行可以包含非null
字段。 -
在 Spark 3.0 中,当使用用户提供的模式写入 Avro 文件时,字段通过 Catalyst 模式和 Avro 模式之间的字段名称而不是位置进行匹配。
-
在 Spark 3.0 中,当使用用户提供的非空模式写入 Avro 文件时,即使 Catalyst 模式可为空,Spark 仍然能够写入文件。但是,如果任何记录包含 null,Spark 会抛出运行时 NullPointerException。
-
在 Spark 2.4 及更早版本中,当文件开头包含 BOM 时,CSV 数据源可以自动检测输入文件的编码。例如,CSV 数据源可以在多行模式下识别 UTF-8、UTF-16BE、UTF-16LE、UTF-32BE 和 UTF-32LE(CSV 选项
multiLine
设置为true
)。在 Spark 3.0 中,CSV 数据源根据 CSV 选项encoding
指定的编码读取输入文件,该选项的默认值为 UTF-8。这样,如果文件编码与 CSV 选项指定的编码不匹配,Spark 将错误地加载文件。要解决此问题,用户应通过 CSV 选项encoding
设置正确的编码,或者将该选项设置为null
,这将回退到 Spark 3.0 之前版本中的编码自动检测。
其他
-
在 Spark 2.4 中,当通过
cloneSession()
创建 Spark 会话时,新创建的 Spark 会话会从其父SparkContext
继承其配置,即使相同的配置可能以不同值存在于其父 Spark 会话中。在 Spark 3.0 中,父SparkSession
的配置优先级高于父SparkContext
。您可以通过将spark.sql.legacy.sessionInitWithConfigDefaults
设置为true
来恢复旧行为。 -
在 Spark 3.0 中,如果在
Spark SQL 配置
中找不到hive.default.fileformat
,则它会回退到SparkContext
的Hadoop 配置
中存在的hive-site.xml
文件。 -
在 Spark 3.0 中,对于
spark-sql
接口,我们用尾随零填充小数,使其达到列的刻度,例如:查询 Spark 2.4 Spark 3.0 SELECT CAST(1 AS decimal(38, 18));
1 1.000000000000000000 -
在 Spark 3.0 中,我们将内置 Hive 从 1.2 升级到 2.3,这带来了以下影响:
-
您可能需要根据要连接的 Hive 元存储版本设置
spark.sql.hive.metastore.version
和spark.sql.hive.metastore.jars
。例如:如果您的 Hive 元存储版本是 1.2.1,则将spark.sql.hive.metastore.version
设置为1.2.1
,将spark.sql.hive.metastore.jars
设置为maven
。 -
您需要将自定义 SerDes 迁移到 Hive 2.3 或使用
hive-1.2
配置文件构建自己的 Spark。有关更多详细信息,请参阅 HIVE-15167。 -
当在 SQL 中使用
TRANSFORM
运算符进行脚本转换时,Hive 1.2 和 Hive 2.3 之间的小数字符串表示可能不同,这取决于 Hive 的行为。在 Hive 1.2 中,字符串表示省略了尾随零。但在 Hive 2.3 中,如有必要,它总是用尾随零填充到 18 位。
-
从 Spark SQL 2.4.7 升级到 2.4.8
- 在 Spark 2.4.8 中,
AnalysisException
被其子类取代,这些子类在以下情况下针对 Hive 外部目录中的表抛出:ALTER TABLE .. ADD PARTITION
如果新分区已存在,则抛出PartitionsAlreadyExistException
。ALTER TABLE .. DROP PARTITION
对于不存在的分区,抛出NoSuchPartitionsException
。
从 Spark SQL 2.4.5 升级到 2.4.6
- 在 Spark 2.4.6 中,
RESET
命令不会将静态 SQL 配置值重置为默认值。它只清除运行时 SQL 配置值。
从 Spark SQL 2.4.4 升级到 2.4.5
-
自 Spark 2.4.5 起,
TRUNCATE TABLE
命令在重新创建表/分区路径时尝试恢复原始权限和 ACL。要恢复早期版本的行为,请将spark.sql.truncateTable.ignorePermissionAcl.enabled
设置为true
。 -
自 Spark 2.4.5 起,新增了
spark.sql.legacy.mssqlserver.numericMapping.enabled
配置,以支持使用 IntegerType 和 DoubleType 分别映射 SMALLINT 和 REAL JDBC 类型的旧版 MsSQLServer 数据库方言行为。要恢复 2.4.3 及更早版本的行为,请将spark.sql.legacy.mssqlserver.numericMapping.enabled
设置为true
。
从 Spark SQL 2.4.3 升级到 2.4.4
- 自 Spark 2.4.4 起,根据 MsSqlServer 指南,MsSQLServer JDBC 方言分别使用 ShortType 和 FloatType 用于 SMALLINT 和 REAL。以前使用 IntegerType 和 DoubleType。
从 Spark SQL 2.4 升级到 2.4.1
- 在 Spark 2.4.0 中,
spark.executor.heartbeatInterval
的值,当不带单位(如“30”而不是“30s”)指定时,在代码的不同部分被不一致地解释为秒和毫秒。现在无单位值始终被解释为毫秒。设置了“30”等值的应用程序现在需要指定带单位的值,如“30s”,以避免被解释为毫秒;否则,由此产生的极短间隔很可能会导致应用程序失败。
从 Spark SQL 2.3 升级到 2.4
- 在 Spark 2.3 及更早版本中,array_contains 函数的第二个参数隐式提升为第一个数组类型参数的元素类型。这种类型提升可能会导致数据丢失,并可能导致
array_contains
函数返回错误的结果。这个问题已在 2.4 中通过采用更安全的类型提升机制解决。这可能会导致一些行为变化,如下表所示。查询 Spark 2.3 或更早版本 Spark 2.4 备注 SELECT array_contains(array(1), 1.34D);
true
false
在 Spark 2.4 中,左参数和右参数分别提升为双精度类型的数组类型和双精度类型。 SELECT array_contains(array(1), '1');
true
抛出 AnalysisException
。参数中可以使用显式类型转换来避免异常。在 Spark 2.4 中,抛出 AnalysisException
,因为整数类型无法以无损方式提升为字符串类型。SELECT array_contains(array(1), 'anystring');
null
抛出 AnalysisException
。参数中可以使用显式类型转换来避免异常。在 Spark 2.4 中,抛出 AnalysisException
,因为整数类型无法以无损方式提升为字符串类型。 -
自 Spark 2.4 起,当 IN 运算符之前有结构体字段时,子查询也必须包含结构体字段。在以前的版本中,结构体的字段会与内部查询的输出进行比较。例如,如果
a
是struct(a string, b int)
,在 Spark 2.4 中a in (select (1 as a, 'a' as b) from range(1))
是有效查询,而a in (select 1, 'a' from range(1))
则不是。在以前的版本中则相反。 -
在 2.2.1+ 和 2.3 版本中,如果
spark.sql.caseSensitive
设置为 true,则CURRENT_DATE
和CURRENT_TIMESTAMP
函数会错误地变为区分大小写并解析为列(除非以小写形式输入)。在 Spark 2.4 中,此问题已修复,这些函数不再区分大小写。 -
自 Spark 2.4 起,Spark 将根据 SQL 标准的优先级规则评估查询中引用的集合操作。如果未通过括号指定顺序,则集合操作从左到右执行,但所有 INTERSECT 操作都在任何 UNION、EXCEPT 或 MINUS 操作之前执行。通过新添加的配置
spark.sql.legacy.setopsPrecedence.enabled
(默认值为false
)保留了所有集合操作具有相同优先级的旧行为。当此属性设置为true
时,如果查询中没有显式地强制执行顺序,Spark 将从左到右评估集合运算符。 -
自 Spark 2.4 起,当表的上次访问值为 1970 年 1 月 1 日时,Spark 会将表描述列中的“上次访问”值显示为 UNKNOWN。
-
自 Spark 2.4 起,Spark 默认最大化使用矢量化 ORC 读取器读取 ORC 文件。为此,
spark.sql.orc.impl
和spark.sql.orc.filterPushdown
的默认值分别更改为native
和true
。由本机 ORC 写入器创建的 ORC 文件无法被某些旧的 Apache Hive 版本读取。使用spark.sql.orc.impl=hive
可创建与 Hive 2.1.1 及更早版本共享的文件。 -
自 Spark 2.4 起,将空 DataFrame 写入目录会启动至少一个写入任务,即使物理上 DataFrame 没有分区。这引入了一个小的行为更改,即对于 Parquet 和 Orc 等自描述文件格式,当写入 0 分区 DataFrame 时,Spark 会在目标目录中创建一个仅包含元数据的文件,以便用户以后读取该目录时仍能进行模式推断。新行为更合理,并且在写入空 DataFrame 方面更一致。
-
自 Spark 2.4 起,UDF 参数中的表达式 ID 不再出现在列名称中。例如,Spark 2.4 中的列名称不是
UDF:f(col0 AS colA#28)
而是UDF:f(col0 AS `colA`)
。 -
自 Spark 2.4 起,不允许使用任何文件格式(parquet、orc、json、text、csv 等)写入具有空或嵌套空模式的 DataFrame。尝试写入具有空模式的 DataFrame 时会抛出异常。
-
自 Spark 2.4 起,Spark 在将 DATE 类型与 TIMESTAMP 类型进行比较时,会先将两者都提升为 TIMESTAMP。将
spark.sql.legacy.compareDateTimestampInTimestamp
设置为false
可以恢复以前的行为。此选项将在 Spark 3.0 中移除。 -
自 Spark 2.4 起,不允许创建具有非空位置的托管表。尝试创建具有非空位置的托管表时会抛出异常。将
spark.sql.legacy.allowCreatingManagedTableUsingNonemptyLocation
设置为true
可以恢复以前的行为。此选项将在 Spark 3.0 中移除。 -
自 Spark 2.4 起,不允许将托管表重命名到现有位置。尝试将托管表重命名到现有位置时会抛出异常。
-
自 Spark 2.4 起,类型强制转换规则可以自动将可变 SQL 函数(例如 IN/COALESCE)的参数类型提升为最宽的通用类型,无论输入参数的顺序如何。在之前的 Spark 版本中,这种提升在某些特定顺序(例如 TimestampType、IntegerType 和 StringType)下可能会失败并抛出异常。
-
自 Spark 2.4 起,Spark 在传统的缓存失效机制之外,还启用了非级联 SQL 缓存失效。非级联缓存失效机制允许用户删除缓存而不影响其依赖缓存。这种新的缓存失效机制用于需要删除的缓存数据仍然有效的情况,例如,对 Dataset 调用 unpersist(),或删除临时视图。这允许用户释放内存,同时保持所需的缓存有效。
-
在 2.3 及更早版本中,Spark 默认转换 Parquet Hive 表,但忽略表属性,例如
TBLPROPERTIES (parquet.compression 'NONE')
。在spark.sql.hive.convertMetastoreOrc=true
的情况下,ORC Hive 表属性(例如TBLPROPERTIES (orc.compress 'NONE')
)也存在这种情况。自 Spark 2.4 起,Spark 在转换 Parquet/ORC Hive 表时尊重 Parquet/ORC 特定的表属性。例如,在 Spark 2.3 中,CREATE TABLE t(id int) STORED AS PARQUET TBLPROPERTIES (parquet.compression 'NONE')
在插入期间会生成 Snappy parquet 文件,而在 Spark 2.4 中,结果将是未压缩的 parquet 文件。 -
自 Spark 2.0 起,Spark 默认转换 Parquet Hive 表以获得更好的性能。自 Spark 2.4 起,Spark 也默认转换 ORC Hive 表。这意味着 Spark 默认使用自己的 ORC 支持而不是 Hive SerDe。例如,
CREATE TABLE t(id int) STORED AS ORC
在 Spark 2.3 中将由 Hive SerDe 处理,而在 Spark 2.4 中,它将被转换为 Spark 的 ORC 数据源表并应用 ORC 矢量化。将spark.sql.hive.convertMetastoreOrc
设置为false
可恢复以前的行为。 -
在 2.3 及更早版本中,如果行中至少有一个列值格式不正确,则 CSV 行被视为格式不正确。CSV 解析器在 DROPMALFORMED 模式下会删除此类行,或在 FAILFAST 模式下输出错误。自 Spark 2.4 起,CSV 行仅在包含从 CSV 数据源请求的格式不正确的列值时才被视为格式不正确,其他值可以被忽略。例如,CSV 文件包含“id,name”标题和一行“1234”。在 Spark 2.4 中,id 列的选择由一个列值为 1234 的行组成,但在 Spark 2.3 及更早版本中,它在 DROPMALFORMED 模式下为空。要恢复以前的行为,请将
spark.sql.csv.parser.columnPruning.enabled
设置为false
。 -
自 Spark 2.4 起,计算统计信息的文件列表默认并行执行。可以通过将
spark.sql.statistics.parallelFileListingInStatsComputation.enabled
设置为False
来禁用此功能。 -
自 Spark 2.4 起,在计算统计信息期间,元数据文件(例如 Parquet 摘要文件)和临时文件不计为数据文件。
-
自 Spark 2.4 起,空字符串将保存为带引号的空字符串
""
。在 2.3 及更早版本中,空字符串等于null
值,并且不反映到已保存的 CSV 文件中的任何字符。例如,行"a", null, "", 1
被写入为a,,,1
。自 Spark 2.4 起,相同的行被保存为a,,"",1
。要恢复以前的行为,请将 CSV 选项emptyValue
设置为空(不带引号的)字符串。 -
自 Spark 2.4 起,LOAD DATA 命令支持通配符
?
和*
,它们分别匹配任意一个字符和零个或多个字符。示例:LOAD DATA INPATH '/tmp/folder*/'
或LOAD DATA INPATH '/tmp/part-?'
。特殊字符(如space
)现在也可以在路径中使用。示例:LOAD DATA INPATH '/tmp/folder name/'
。 -
在 2.3 及更早版本中,没有 GROUP BY 的 HAVING 被视为 WHERE。这意味着,
SELECT 1 FROM range(10) HAVING true
被执行为SELECT 1 FROM range(10) WHERE true
并返回 10 行。这违反了 SQL 标准,并在 Spark 2.4 中已修复。自 Spark 2.4 起,没有 GROUP BY 的 HAVING 被视为全局聚合,这意味着SELECT 1 FROM range(10) HAVING true
将只返回一行。要恢复以前的行为,请将spark.sql.legacy.parser.havingWithoutGroupByAsWhere
设置为true
。 - 在 2.3 及更早版本中,当从 Parquet 数据源表读取时,如果 Hive 元存储模式和 Parquet 模式中的列名大小写不同,Spark 总是为任何列返回 null,无论
spark.sql.caseSensitive
是设置为true
还是false
。自 2.4 起,当spark.sql.caseSensitive
设置为false
时,Spark 会在 Hive 元存储模式和 Parquet 模式之间进行不区分大小写的列名解析,因此即使列名大小写不同,Spark 也会返回相应的列值。如果存在歧义,即匹配多个 Parquet 列,则会抛出异常。此更改也适用于spark.sql.hive.convertMetastoreParquet
设置为true
时的 Parquet Hive 表。
从 Spark SQL 2.2 升级到 2.3
-
自 Spark 2.3 起,当引用的列只包含内部损坏记录列(默认为
_corrupt_record
)时,不允许从原始 JSON/CSV 文件进行查询。例如,spark.read.schema(schema).json(file).filter($"_corrupt_record".isNotNull).count()
和spark.read.schema(schema).json(file).select("_corrupt_record").show()
。相反,您可以缓存或保存解析结果,然后发送相同的查询。例如,val df = spark.read.schema(schema).json(file).cache()
,然后df.filter($"_corrupt_record".isNotNull).count()
。 -
percentile_approx
函数以前接受数字类型输入并输出双精度类型结果。现在它支持日期类型、时间戳类型和数字类型作为输入类型。结果类型也更改为与输入类型相同,这对于百分位数更合理。 -
从 Spark 2.3 开始,Join/Filter 的确定性谓词(位于第一个非确定性谓词之后)如果可能,也会下推/穿透到子操作符中。在之前的 Spark 版本中,这些过滤器不符合谓词下推的条件。
-
分区列推断之前会为不同的推断类型找到不正确的公共类型,例如,之前它会将 double 类型和 date 类型的公共类型推断为 double 类型。现在它会为这些冲突找到正确的公共类型。冲突解决遵循下表:
输入A \ 输入B NullType IntegerType LongType DecimalType(38,0)* DoubleType DateType TimestampType StringType NullType NullType IntegerType LongType DecimalType(38,0) DoubleType DateType TimestampType StringType IntegerType IntegerType IntegerType LongType DecimalType(38,0) DoubleType StringType StringType StringType LongType LongType LongType LongType DecimalType(38,0) StringType StringType StringType StringType DecimalType(38,0)* DecimalType(38,0) DecimalType(38,0) DecimalType(38,0) DecimalType(38,0) StringType StringType StringType StringType DoubleType DoubleType DoubleType StringType StringType DoubleType StringType StringType StringType DateType DateType StringType StringType StringType StringType DateType TimestampType StringType TimestampType TimestampType StringType StringType StringType StringType TimestampType TimestampType StringType StringType StringType StringType StringType StringType StringType StringType StringType StringType 请注意,对于 DecimalType(38,0)*,上表有意未涵盖所有其他比例和精度的组合,因为目前我们只推断像
BigInteger
/BigInt
这样的十进制类型。例如,1.1 被推断为 double 类型。 -
从 Spark 2.3 开始,当广播哈希连接或广播嵌套循环连接适用时,我们优先广播在广播提示中明确指定的表。有关详细信息,请参阅SQL 查询的连接策略提示部分和 SPARK-22489。
-
从 Spark 2.3 开始,当所有输入都是二进制时,
functions.concat()
返回二进制输出。否则,返回字符串。直到 Spark 2.3,无论输入类型如何,它总是返回字符串。为了保持旧的行为,请将spark.sql.function.concatBinaryAsString
设置为true
。 -
从 Spark 2.3 开始,当所有输入都是二进制时,SQL
elt()
返回二进制输出。否则,返回字符串。直到 Spark 2.3,无论输入类型如何,它总是返回字符串。为了保持旧的行为,请将spark.sql.function.eltOutputAsString
设置为true
。 -
从 Spark 2.3 开始,默认情况下,如果无法精确表示,十进制数之间的算术运算将返回一个四舍五入的值(而不是返回 NULL)。这符合 SQL ANSI 2011 规范以及 Hive 2.2 中引入的 Hive 新行为 (HIVE-15331)。这包括以下更改:
-
用于确定算术运算结果类型的规则已更新。特别地,如果所需的精度/小数位数超出可用值的范围,则小数位数会减少到 6,以防止十进制数的整数部分被截断。所有算术运算都受此更改影响,即加法 (
+
)、减法 (-
)、乘法 (*
)、除法 (/
)、取余 (%
) 和正模 (pmod
)。 -
SQL 操作中使用的字面值会转换为具有其所需精确精度和小数位数的 DECIMAL 类型。
-
引入了配置项
spark.sql.decimalOperations.allowPrecisionLoss
。它默认为true
,这意味着此处描述的新行为;如果设置为false
,Spark 将使用以前的规则,即它不会调整所需的小数位数来表示值,并且如果无法精确表示值,它将返回 NULL。
-
-
未加别名的子查询语义定义不明确,导致行为混乱。从 Spark 2.3 开始,我们废弃了这些令人困惑的用例,例如:
SELECT v.i from (SELECT i FROM v)
,在这种情况下,Spark 将抛出分析异常,因为用户不应在子查询内部使用限定符。有关更多详细信息,请参阅 SPARK-20690 和 SPARK-21335。 -
当使用
SparkSession.builder.getOrCreate()
创建SparkSession
时,如果存在一个SparkContext
,构建器会尝试用为构建器指定的配置来更新现有SparkContext
的SparkConf
,但SparkContext
由所有SparkSession
共享,因此我们不应更新它们。从 2.3 开始,构建器不再更新这些配置。如果要更新它们,需要在创建SparkSession
之前更新它们。
从 Spark SQL 2.1 升级到 2.2
-
Spark 2.1.1 引入了一个新的配置键:
spark.sql.hive.caseSensitiveInferenceMode
。它有一个默认设置NEVER_INFER
,这使得行为与 2.1.0 保持一致。然而,Spark 2.2.0 将此设置的默认值更改为INFER_AND_SAVE
,以恢复与读取底层文件模式具有混合大小写列名的 Hive metastore 表的兼容性。使用INFER_AND_SAVE
配置值,Spark 在首次访问时将对任何尚未保存推断模式的 Hive metastore 表执行模式推断。请注意,对于具有数千个分区的表,模式推断可能是一个非常耗时的操作。如果不关心与混合大小写列名的兼容性,您可以安全地将spark.sql.hive.caseSensitiveInferenceMode
设置为NEVER_INFER
,以避免模式推断的初始开销。请注意,使用新的默认INFER_AND_SAVE
设置,模式推断的结果将作为 metastore 键保存以供将来使用。因此,初始模式推断只在表首次访问时发生。 -
从 Spark 2.2.1 和 2.3.0 开始,当数据源表具有分区模式和数据模式中都存在的列时,模式总是在运行时推断。推断出的模式不包含分区列。读取表时,Spark 遵循这些重叠列的分区值,而不是数据源文件中存储的值。在 2.2.0 和 2.1.x 版本中,推断出的模式是分区的,但表的数据对用户不可见(即,结果集为空)。
-
从 Spark 2.2 开始,视图定义以与以前版本不同的方式存储。这可能导致 Spark 无法读取由以前版本创建的视图。在这种情况下,您需要使用较新的 Spark 版本,通过
ALTER VIEW AS
或CREATE OR REPLACE VIEW AS
重新创建视图。
从 Spark SQL 2.0 升级到 2.1
-
数据源表现在将分区元数据存储在 Hive metastore 中。这意味着像
ALTER TABLE PARTITION ... SET LOCATION
这样的 Hive DDL 现在可用于使用 Datasource API 创建的表。-
传统数据源表可以通过
MSCK REPAIR TABLE
命令迁移到此格式。建议迁移传统表,以利用 Hive DDL 支持和改进的规划性能。 -
要确定表是否已迁移,请在对表执行
DESCRIBE FORMATTED
时查找PartitionProvider: Catalog
属性。
-
-
数据源表的
INSERT OVERWRITE TABLE ... PARTITION ...
行为更改。-
在之前的 Spark 版本中,`INSERT OVERWRITE` 会覆盖整个数据源表,即使给定了分区规范。现在,只有与规范匹配的分区才会被覆盖。
-
请注意,这仍然与 Hive 表的行为不同,Hive 表只覆盖与新插入数据重叠的分区。
-
从 Spark SQL 1.6 升级到 2.0
-
SparkSession
现在是 Spark 的新入口点,它取代了旧的SQLContext
和HiveContext
。请注意,旧的 SQLContext 和 HiveContext 保留用于向后兼容。SparkSession
中现在可以访问新的catalog
接口——现有的数据库和表访问 API,例如listTables
、createExternalTable
、dropTempView
、cacheTable
已移至此处。 -
Dataset API 和 DataFrame API 已统一。在 Scala 中,
DataFrame
成为Dataset[Row]
的类型别名,而 Java API 用户必须将DataFrame
替换为Dataset<Row>
。Dataset 类上同时提供类型化转换(例如map
、filter
和groupByKey
)和非类型化转换(例如select
和groupBy
)。由于 Python 和 R 中编译时类型安全不是语言特性,因此 Dataset 的概念不适用于这些语言的 API。相反,DataFrame
仍然是主要的编程抽象,这类似于这些语言中的单节点数据帧概念。 -
Dataset 和 DataFrame API
unionAll
已被弃用并由union
取代 -
Dataset 和 DataFrame API
explode
已被弃用,作为替代,请结合select
或flatMap
使用functions.explode()
-
Dataset 和 DataFrame API
registerTempTable
已被弃用并由createOrReplaceTempView
取代 -
Hive 表的
CREATE TABLE ... LOCATION
行为更改。-
从 Spark 2.0 开始,
CREATE TABLE ... LOCATION
等同于CREATE EXTERNAL TABLE ... LOCATION
,以防止意外删除用户提供位置中的现有数据。这意味着,在 Spark SQL 中使用用户指定位置创建的 Hive 表始终是 Hive 外部表。删除外部表不会删除数据。用户不允许为 Hive 管理表指定位置。请注意,这与 Hive 的行为不同。 -
因此,对这些表执行
DROP TABLE
语句不会删除数据。
-
-
spark.sql.parquet.cacheMetadata
不再使用。有关详细信息,请参阅 SPARK-13664。
从 Spark SQL 1.5 升级到 1.6
- 从 Spark 1.6 开始,默认情况下,Thrift 服务器以多会话模式运行。这意味着每个 JDBC/ODBC 连接都拥有自己的 SQL 配置和临时函数注册表副本。不过,缓存表仍然是共享的。如果您更喜欢以旧的单会话模式运行 Thrift 服务器,请将选项
spark.sql.hive.thriftServer.singleSession
设置为true
。您可以将此选项添加到spark-defaults.conf
,也可以通过--conf
将其传递给start-thriftserver.sh
。
./sbin/start-thriftserver.sh \
--conf spark.sql.hive.thriftServer.singleSession=true \
...
- 从 Spark 1.6 开始,LongType 转换为 TimestampType 时预期的是秒而不是微秒。此更改是为了与 Hive 1.2 的行为匹配,从而使数字类型到 TimestampType 的类型转换更加一致。有关详细信息,请参阅 SPARK-11724。
从 Spark SQL 1.4 升级到 1.5
-
默认情况下,使用手动管理内存 (Tungsten) 的优化执行现已启用,以及表达式评估的代码生成。通过将
spark.sql.tungsten.enabled
设置为false
,可以禁用这两个功能。 -
Parquet 模式合并不再默认启用。通过将
spark.sql.parquet.mergeSchema
设置为true
,可以重新启用它。 -
内存中列式存储分区剪枝默认开启。通过将
spark.sql.inMemoryColumnarStorage.partitionPruning
设置为false
,可以禁用它。 -
不再支持无限精度十进制列,Spark SQL 现在强制执行最大精度为 38。从
BigDecimal
对象推断模式时,现在使用精度 (38, 18)。如果在 DDL 中未指定精度,则默认仍为Decimal(10, 0)
。 -
时间戳现在以 1 微秒的精度存储,而不是 1 纳秒
-
在
sql
方言中,浮点数现在被解析为十进制数。HiveQL 解析保持不变。 -
SQL/DataFrame 函数的规范名称现在是小写(例如,sum vs SUM)。
-
JSON 数据源不会自动加载由其他应用程序创建的新文件(即,未通过 Spark SQL 插入到数据集的文件)。对于 JSON 持久化表(即表的元数据存储在 Hive Metastore 中),用户可以使用
REFRESH TABLE
SQL 命令或HiveContext
的refreshTable
方法将这些新文件包含到表中。对于表示 JSON 数据集的 DataFrame,用户需要重新创建 DataFrame,并且新的 DataFrame 将包含新文件。
从 Spark SQL 1.3 升级到 1.4
DataFrame 数据读写器接口
根据用户反馈,我们创建了一个新的、更流畅的 API,用于读取数据 (SQLContext.read
) 和写入数据 (DataFrame.write
),并弃用了旧的 API(例如,SQLContext.parquetFile
、SQLContext.jsonFile
)。
有关更多信息,请参阅 SQLContext.read
( Scala、 Java、 Python )和 DataFrame.write
( Scala、 Java、 Python )的 API 文档。
DataFrame.groupBy 保留分组列
根据用户反馈,我们更改了 DataFrame.groupBy().agg()
的默认行为,以在结果 DataFrame
中保留分组列。为了保持 1.3 中的行为,请将 spark.sql.retainGroupColumns
设置为 false
。
import pyspark.sql.functions as func
# In 1.3.x, in order for the grouping column "department" to show up,
# it must be included explicitly as part of the agg function call.
df.groupBy("department").agg(df["department"], func.max("age"), func.sum("expense"))
# In 1.4+, grouping column "department" is included automatically.
df.groupBy("department").agg(func.max("age"), func.sum("expense"))
# Revert to 1.3.x behavior (not retaining grouping column) by:
sqlContext.setConf("spark.sql.retainGroupColumns", "false")
// In 1.3.x, in order for the grouping column "department" to show up,
// it must be included explicitly as part of the agg function call.
df.groupBy("department").agg($"department", max("age"), sum("expense"))
// In 1.4+, grouping column "department" is included automatically.
df.groupBy("department").agg(max("age"), sum("expense"))
// Revert to 1.3 behavior (not retaining grouping column) by:
sqlContext.setConf("spark.sql.retainGroupColumns", "false")
// In 1.3.x, in order for the grouping column "department" to show up,
// it must be included explicitly as part of the agg function call.
df.groupBy("department").agg(col("department"), max("age"), sum("expense"));
// In 1.4+, grouping column "department" is included automatically.
df.groupBy("department").agg(max("age"), sum("expense"));
// Revert to 1.3 behavior (not retaining grouping column) by:
sqlContext.setConf("spark.sql.retainGroupColumns", "false");
DataFrame.withColumn 的行为变化
在 1.4 之前,DataFrame.withColumn() 仅支持添加列。即使存在任何同名列,该列也将始终以其指定名称作为新列添加到结果 DataFrame 中。从 1.4 开始,DataFrame.withColumn() 支持添加名称与所有现有列名称不同的列,或替换同名现有列。
请注意,此更改仅适用于 Scala API,不适用于 PySpark 和 SparkR。
从 Spark SQL 1.0-1.2 升级到 1.3
在 Spark 1.3 中,我们从 Spark SQL 中移除了“Alpha”标签,并以此为契机清理了可用的 API。从 Spark 1.3 开始,Spark SQL 将与 1.X 系列中的其他版本提供二进制兼容性。此兼容性保证不包括明确标记为不稳定的 API(即 DeveloperAPI 或 Experimental)。
将 SchemaRDD 重命名为 DataFrame
用户升级到 Spark SQL 1.3 时会注意到的最大变化是 SchemaRDD
已被重命名为 DataFrame
。这主要是因为 DataFrame 不再直接继承自 RDD,而是通过自己的实现提供 RDD 提供的大部分功能。DataFrame 仍然可以通过调用 .rdd
方法转换为 RDD。
在 Scala 中,存在从 SchemaRDD
到 DataFrame
的类型别名,以便为某些用例提供源兼容性。仍然建议用户更新其代码以使用 DataFrame
。Java 和 Python 用户需要更新其代码。
Java 和 Scala API 的统一
在 Spark 1.3 之前,存在独立的 Java 兼容类(JavaSQLContext
和 JavaSchemaRDD
),它们镜像了 Scala API。在 Spark 1.3 中,Java API 和 Scala API 已统一。两种语言的用户都应使用 SQLContext
和 DataFrame
。通常,这些类尝试使用两种语言都可用的类型(即 Array
而不是特定于语言的集合)。在某些不存在通用类型的情况下(例如,用于传递闭包或 Map),则使用函数重载代替。
此外,Java 特定类型 API 已被移除。Scala 和 Java 用户都应使用 org.apache.spark.sql.types
中存在的类来以编程方式描述模式。
隐式转换的隔离和 dsl 包的移除(仅限 Scala)
在 Spark 1.3 之前,许多代码示例都以 import sqlContext._
开头,这会将 sqlContext 中的所有函数引入作用域。在 Spark 1.3 中,我们已将用于将 RDD
转换为 DataFrame
的隐式转换隔离到 SQLContext
内的一个对象中。用户现在应该编写 import sqlContext.implicits._
。
此外,隐式转换现在只为由 Product
(即 case 类或元组)组成的 RDD 增加一个 toDF
方法,而不是自动应用。
在 DSL(现在已被 DataFrame
API 取代)内部使用函数时,用户以前会导入 org.apache.spark.sql.catalyst.dsl
。现在应该使用公共 DataFrame 函数 API:import org.apache.spark.sql.functions._
。
移除 org.apache.spark.sql 中 DataType 的类型别名(仅限 Scala)
Spark 1.3 移除了基本 sql 包中 DataType
的类型别名。用户应改为导入 org.apache.spark.sql.types
中的类。
UDF 注册已移至 sqlContext.udf
(Java 和 Scala)
用于注册 UDF(无论是在 DataFrame DSL 还是 SQL 中使用)的函数已移至 SQLContext
中的 udf 对象。
sqlContext.udf.register("strLen", (s: String) => s.length())
sqlContext.udf().register("strLen", (String s) -> s.length(), DataTypes.IntegerType);
Python UDF 注册保持不变。
与 Apache Hive 的兼容性
Spark SQL 旨在与 Hive Metastore、SerDes 和 UDF 兼容。目前,Hive SerDes 和 UDF 基于内置 Hive,并且 Spark SQL 可以连接到不同版本的 Hive Metastore(从 2.0.0 到 2.3.10 以及 3.0.0 到 3.1.3。另请参阅 与不同版本的 Hive Metastore 交互)。
部署到现有 Hive 数据仓库
Spark SQL Thrift JDBC 服务器旨在“开箱即用”地与现有 Hive 安装兼容。您无需修改现有的 Hive Metastore,也无需更改表的数据放置或分区。
支持的 Hive 功能
Spark SQL 支持绝大多数 Hive 功能,例如:
- Hive 查询语句,包括
SELECT
GROUP BY
ORDER BY
DISTRIBUTE BY
CLUSTER BY
SORT BY
- 所有 Hive 操作符,包括
- 关系操作符 (
=
,<=>
,==
,<>
,<
,>
,>=
,<=
, 等) - 算术操作符 (
+
,-
,*
,/
,%
, 等) - 逻辑操作符 (
AND
,OR
, 等) - 复杂类型构造函数
- 数学函数 (
sign
,ln
,cos
, 等) - 字符串函数 (
instr
,length
,printf
, 等)
- 关系操作符 (
- 用户定义函数 (UDF)
- 用户定义聚合函数 (UDAF)
- 用户定义序列化格式 (SerDes)
- 窗口函数
- 连接 (Joins)
JOIN
{LEFT|RIGHT|FULL} OUTER JOIN
LEFT SEMI JOIN
LEFT ANTI JOIN
CROSS JOIN
- 联合 (Unions)
- 子查询
-
FROM 子句中的子查询
SELECT col FROM (SELECT a + b AS col FROM t1) t2
-
WHERE 子句中的子查询
-
WHERE 子句中相关或非相关的 IN 和 NOT IN 语句
SELECT col FROM t1 WHERE col IN (SELECT a FROM t2 WHERE t1.a = t2.a) SELECT col FROM t1 WHERE col IN (SELECT a FROM t2)
-
WHERE 子句中相关或非相关的 EXISTS 和 NOT EXISTS 语句
SELECT col FROM t1 WHERE EXISTS (SELECT t2.a FROM t2 WHERE t1.a = t2.a AND t2.a > 10) SELECT col FROM t1 WHERE EXISTS (SELECT t2.a FROM t2 WHERE t2.a > 10)
-
JOIN 条件中非相关的 IN 和 NOT IN 语句
SELECT t1.col FROM t1 JOIN t2 ON t1.a = t2.a AND t1.a IN (SELECT a FROM t3)
-
JOIN 条件中非相关的 EXISTS 和 NOT EXISTS 语句
SELECT t1.col FROM t1 JOIN t2 ON t1.a = t2.a AND EXISTS (SELECT * FROM t3 WHERE t3.a > 10)
-
-
- 抽样
- Explain
- 分区表,包括动态分区插入
- 视图
-
如果在视图定义查询中未指定列别名,Spark 和 Hive 都将生成别名,但方式不同。为了使 Spark 能够读取由 Hive 创建的视图,用户应在视图定义查询中显式指定列别名。例如,Spark 无法读取 Hive 如下创建的
v1
。CREATE VIEW v1 AS SELECT * FROM (SELECT c + 1 FROM (SELECT 1 c) t1) t2;
相反,您应该像下面这样创建
v1
,并显式指定列别名。CREATE VIEW v1 AS SELECT * FROM (SELECT c + 1 AS inc_c FROM (SELECT 1 c) t1) t2;
-
- 所有 Hive DDL 函数,包括
CREATE TABLE
CREATE TABLE AS SELECT
CREATE TABLE LIKE
ALTER TABLE
- 大多数 Hive 数据类型,包括
TINYINT
SMALLINT
INT
BIGINT
BOOLEAN
FLOAT
DOUBLE
STRING
BINARY
TIMESTAMP
DATE
ARRAY<>
MAP<>
STRUCT<>
不支持的 Hive 功能
以下是我们尚不支持的 Hive 功能列表。这些功能中的大多数在 Hive 部署中很少使用。
不常用的 Hive 功能
UNION
类型- 唯一连接
- 列统计信息收集:Spark SQL 目前不会附加扫描来收集列统计信息,并且只支持填充 hive metastore 的 sizeInBytes 字段。
Hive 输入/输出格式
- CLI 文件格式:对于显示回 CLI 的结果,Spark SQL 仅支持 TextOutputFormat。
- Hadoop 归档
Hive 优化
少数 Hive 优化尚未包含在 Spark 中。其中一些(例如索引)由于 Spark SQL 的内存计算模型而变得不那么重要。其他则计划在 Spark SQL 的未来版本中实现。
- 块级位图索引和虚拟列(用于构建索引)
- 自动确定连接和分组的 reducer 数量:目前,在 Spark SQL 中,您需要使用 “
SET spark.sql.shuffle.partitions=[num_tasks];
” 来控制 shuffle 后的并行度。 - 仅元数据查询:对于仅使用元数据即可回答的查询,Spark SQL 仍然会启动任务来计算结果。
- 数据倾斜标志:Spark SQL 不遵循 Hive 中的数据倾斜标志。
- 连接中的
STREAMTABLE
提示:Spark SQL 不遵循STREAMTABLE
提示。 - 合并查询结果的多个小文件:如果结果输出包含多个小文件,Hive 可以选择将小文件合并为少量大文件,以避免 HDFS 元数据溢出。Spark SQL 不支持此功能。
Hive UDF/UDTF/UDAF
并非所有 Hive UDF/UDTF/UDAF 的 API 都受 Spark SQL 支持。以下是不受支持的 API:
getRequiredJars
和getRequiredFiles
(UDF
和GenericUDF
)是自动包含此 UDF 所需额外资源的函数。GenericUDTF
中的initialize(StructObjectInspector)
尚不支持。Spark SQL 目前仅使用已弃用的接口initialize(ObjectInspector[])
。configure
(GenericUDF
、GenericUDTF
和GenericUDAFEvaluator
)是一个使用MapredContext
初始化函数的函数,这不适用于 Spark。close
(GenericUDF
和GenericUDAFEvaluator
)是一个释放相关资源的函数。Spark SQL 在任务完成时不会调用此函数。reset
(GenericUDAFEvaluator
)是一个重新初始化聚合以重用相同聚合的函数。Spark SQL 目前不支持聚合的重用。getWindowingEvaluator
(GenericUDAFEvaluator
)是一个通过在固定窗口上评估聚合来优化聚合的函数。
不兼容的 Hive UDF
以下是 Hive 和 Spark 生成不同结果的场景:
SQRT(n)
如果 n < 0,Hive 返回 null,Spark SQL 返回 NaN。ACOS(n)
如果 n < -1 或 n > 1,Hive 返回 null,Spark SQL 返回 NaN。ASIN(n)
如果 n < -1 或 n > 1,Hive 返回 null,Spark SQL 返回 NaN。CAST(n AS TIMESTAMP)
如果 n 是整数,Hive 将 n 视为毫秒,Spark SQL 将 n 视为秒。