移行ガイド: SQL、Datasets、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構文では、テーブルプロバイダーとしてHiveの代わりにspark.sql.sources.defaultの値が使用されます。以前の動作に戻すには、spark.sql.legacy.createHiveTableByDefaultをtrueに設定するか、SPARK_SQL_LEGACY_CREATE_HIVE_TABLEをtrueに設定してください。 - Spark 4.0 以降、マップに要素を挿入する際のデフォルトの動作が、キーの正規化 (-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 テーブルの読み込みでは、コア設定
spark.files.ignoreCorruptFiles/spark.files.ignoreMissingFilesではなく、SQL 設定spark.sql.files.ignoreCorruptFiles/spark.sql.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-commonjar をクラスパスに追加してください。 - 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に設定してください。たとえば、Latin1 でエンコードされた文字列値tést/ [116, -23, 115, 116] を ‘UTF-8’ でdecodeしようとすると、t�stが得られます。 - Spark 4.0 以降、
spark.sql.legacyプレフィックスを持つレガシーな日付/時刻リベース SQL 設定が削除されました。以前の動作に戻すには、次の設定を使用してください。spark.sql.legacy.parquet.int96RebaseModeInWriteの代わりにspark.sql.parquet.int96RebaseModeInWritespark.sql.legacy.parquet.datetimeRebaseModeInWriteの代わりにspark.sql.parquet.datetimeRebaseModeInWritespark.sql.legacy.parquet.int96RebaseModeInReadの代わりにspark.sql.parquet.int96RebaseModeInReadspark.sql.legacy.avro.datetimeRebaseModeInWriteの代わりにspark.sql.avro.datetimeRebaseModeInWritespark.sql.legacy.avro.datetimeRebaseModeInReadの代わりにspark.sql.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 番目の引数は2$のように参照する必要があります。 - Spark 4.0 以降、Postgres JDBC データソースは、JDBC 読み取りオプション
preferTimestampNTZの有無にかかわらず、JDBC 読み取り TIMESTAMP WITH TIME ZONE を TimestampType として読み取ります。一方、3.5 以前では、preferTimestampNTZ=trueの場合に TimestampNTZType として読み取っていました。以前の動作に戻すには、spark.sql.legacy.postgres.datetimeMapping.enabledをtrueに設定してください。 - Spark 4.0 以降、Postgres JDBC データソースは、3.5 以前では TIMESTAMP (つまり TIMESTAMP WITHOUT TIME ZONE) として書き込んでいたのに対し、TimestampType を TIMESTAMP WITH TIME ZONE として書き込みます。以前の動作に戻すには、
spark.sql.legacy.postgres.datetimeMapping.enabledをtrueに設定してください。 - Spark 4.0 以降、MySQL JDBC データソースは、JDBC 読み取りオプション
preferTimestampNTZの有無にかかわらず、TIMESTAMP を TimestampType として読み取ります。一方、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 モードが有効な場合、INCONSISTENT_BEHAVIOR_CROSS_VERSIONエラーを発生させる代わりにCANNOT_PARSE_TIMESTAMPが発生します。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 以降、
sentences関数は、languageパラメーターがNULLでなく、countryパラメーターがNULLの場合、Locale.USの代わりにLocale(language)を使用します。 - 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 へのアップグレード
- Spark 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 スパークスルーサーバーは、実行中のステートメントをキャンセルする際にタスクを中断します。以前の動作に戻すには、
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 以前では、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 は存在しないキーでマップ値を取得すると、常に 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入力に対してエラーをスローします。不正な入力を許容して NULL を返すには、try_to_binaryを使用してください。- 有効な Base64 文字列には、base64 アルファベット (A-Za-z0-9+/)、オプションのパディング (
=)、およびオプションの空白文字のシンボルが含まれている必要があります。空白文字は、パディングシンボルに先行する場合を除き、変換時にスキップされます。パディングが存在する場合、それは文字列の末尾にある必要があり、RFC 4648 § 4 に記載されている規則に従う必要があります。 - 有効な 16 進数文字列には、許可されたシンボル (0-9A-Fa-f) のみが含まれている必要があります。
fmtの有効な値は、大文字小文字を区別しないhex、base64、utf-8、utf8です。
- 有効な Base64 文字列には、base64 アルファベット (A-Za-z0-9+/)、オプションのパディング (
- Spark 3.4 以降、Spark は、パーティションを作成する際に一部のパーティションが既に存在する場合に、
PartitionsAlreadyExistExceptionのみをスローします。Spark 3.3 以前では、PartitionsAlreadyExistExceptionまたはPartitionAlreadyExistsExceptionのいずれかがスローされる可能性がありました。 - Spark 3.4 以降、Spark は ALTER PARTITION のパーティション仕様の検証を行い、
spark.sql.storeAssignmentPolicyの動作に従います。これにより、型変換が失敗した場合に例外が発生する可能性があります。例:ALTER TABLE .. ADD PARTITION(p='a')、ただし列pは int 型です。レガシーな動作に戻すには、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 以降、ブルームフィルター結合がデフォルトで有効になります。レガシーな動作に戻すには、
spark.sql.optimizer.runtime.bloomFilter.enabledをfalseに設定してください。 - Spark 3.4 以降、外部 Parquet ファイルのスキーマ推論において、注釈
isAdjustedToUTC=falseを持つ INT64 タイムスタンプは、Timestamp 型ではなく TimestampNTZ 型として推論されます。レガシーな動作に戻すには、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) の配列の出力型を返します。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 マッピングの型と一致しませんでした。たとえば、Java ではDayTimeIntervalTypeはjava.time.Durationにマッピングされます。 -
Spark 3.3 以降、
lpadおよびrpad関数は、バイトシーケンスをサポートするようにオーバーロードされました。最初の引数がバイトシーケンスの場合、オプションのパディングパターンもバイトシーケンスである必要があり、結果は BINARY 値になります。この場合のデフォルトのパディングパターンはゼロバイトです。常に文字列型を返すレガシーな動作に戻すには、spark.sql.legacy.lpadRpadAlwaysReturnStringをtrueに設定してください。 -
Spark 3.3 以降、ユーザーがスキーマを指定し、NULL 非許容フィールドが含まれている場合、API
DataFrameReader.schema(schema: StructType).json(jsonDataset: Dataset[String])およびDataFrameReader.schema(schema: StructType).csv(csvDataset: Dataset[String])のために、NULL 非許容スキーマが NULL 許容スキーマに変換されます。NULL 非許容性を尊重するレガシーな動作に戻すには、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 以降、JSON 属性から
FloatTypeまたはDoubleTypeとして定義された値を読み取る際に、文字列"+Infinity"、"+INF"、およびは、既にサポートされていた"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 以降、丸め関数などの返り値の精度が修正されました。これにより、以前のバージョンで作成されたビューを使用する際に、Spark が
CANNOT_UP_CAST_DATATYPEエラー クラスのAnalysisExceptionをスローする可能性があります。その場合、新しい Spark バージョンで ALTER VIEW AS または CREATE OR REPLACE VIEW AS を使用してビューを再作成する必要があります。 -
Spark 3.3 以降、
unbase64関数は、不正なstr入力に対してエラーをスローします。不正な入力を許容して NULL を返すには、try_to_binary(<str>, 'base64')を使用してください。Spark 3.2 以前では、unbase64関数は、不正なstr入力に対して最善を尽くした結果を返していました。 -
Spark 3.3 以降、Spark によって生成されなかった Parquet ファイルを読み取る場合、注釈
isAdjustedToUTC = falseを持つ Parquet タイムスタンプ列は、スキーマ推論中に 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 とは異なる値を返します。これは、ユーザー指定のグループバイ式にグルーピングセット列が追加されて計算されます。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 を使用し、他の Dialect は LongType を使用していました。
-
Spark 3.2 以降、タイムスタンプ型にナノ秒精度を持つ Parquet ファイル (
INT64 (TIMESTAMP(NANOS, true))) は読み取れません。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 では、ターゲットパーティションが既に存在する場合、
ALTER TABLE .. RENAME TO PARTITIONはAnalysisExceptionの代わりにPartitionAlreadyExistsExceptionをスローします。 -
Spark 3.2 では、スクリプト変換のデフォルト FIELD DELIMIT は、serde モードがない場合は
\u0001、ユーザーが serde を指定した場合の Hive serde モードではfield.delimが\tです。Spark 3.1 以前では、デフォルトの FIELD DELIMIT は\t、ユーザーが serde を指定した場合の Hive serde モードではfield.delimが\u0001でした。 -
Spark 3.2 では、(型変換ルールによって追加されるような) 自動生成された
Castは、列エイリアス名を生成する際に削除されます。例:sql("SELECT floor(1)").columnsは、FLOOR(CAST(1 AS DOUBLE))の代わりにFLOOR(1)になります。 -
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 PARTITIONALTER TABLE .. RENAME PARTITIONALTER TABLE .. DROP PARTITIONALTER TABLE .. RECOVER PARTITIONSMSCK REPAIR TABLELOAD DATAREFRESH TABLETRUNCATE TABLE- およびメソッド
spark.catalog.refreshTable。Spark 3.1 以前では、テーブルの更新は依存関係をキャッシュしませんでした。
-
Spark 3.2 では、曖昧な結果を生成するのを避けるために
count(tblName.*)の使用がブロックされています。これは、count(*)とcount(tblName.*)が、NULL 値が存在する場合に異なる結果を生成するためです。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'。ParseExceptionを無視するには、spark.sql.legacy.notReservePropertiesをtrueに設定できます。この場合、これらのプロパティはサイレントに削除されます。例:TBLPROPERTIES('owner'='yao')は効果がありません。Spark 3.1 以前のバージョンでは、予約プロパティはCREATE TABLE .. LIKE ..コマンドで使用できましたが、副作用はありませんでした。例:TBLPROPERTIES('location'='/tmp')はテーブルの場所を変更せず、'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) を混在させる単位リスト間隔リテラルはサポートされていません。例: Spark 3.2 では
INTERVAL 1 month 1 hourは無効です。Spark 3.1 以前では、このような制限はなく、リテラルはCalendarIntervalTypeの値を返していました。Spark 3.2 より前の動作に戻すには、spark.sql.legacy.interval.enabledをtrueに設定してください。 -
Spark 3.2 では、
TRANSFORM句の HiveSERDEモードでのDayTimeIntervalTypeおよびYearMonthIntervalTypeの入力と出力がサポートされています。これらの 2 つの型が入力として使用される場合、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 では、
DataFrameWriterによってトリガーされるクエリ実行は、QueryExecutionListenerに送信される際に常にcommandという名前が付けられます。Spark 3.1 以前では、名前はsave、insertInto、saveAsTableのいずれかでした。 -
Spark 3.2 では、
Dataset.unionByNameでallowMissingColumnsが true に設定されている場合、不足しているネストされたフィールドが構造体の末尾に追加されます。Spark 3.1 では、ネストされた構造体フィールドはアルファベット順にソートされていました。 -
Spark 3.2 では、作成/alter ビューは、入力クエリの出力列に自動生成されたエイリアスが含まれている場合、失敗します。これは、ビューの出力列名が 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を含む) は、式評価中にゼロ除算が発生した場合 (例: 単一要素セットにstddev_sampを適用した場合)、Double.NaNの代わりにNULLを返します。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 の場合、10 進数型列の合計がオーバーフローした場合、Spark は常に null を返します。Spark 3.0 以前では、この場合、10 進数型列の合計は null または不正な結果を返す可能性があり、実行時に失敗することさえありました (実際のクエリプランの実行による)。 -
Spark 3.1 では、
pathオプションは、DataFrameReader.load()、DataFrameWriter.save()、DataStreamReader.load()、またはDataStreamWriter.start()にパスパラメータ付きで呼び出されたときに、pathパラメータと共存できません。さらに、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オプションは、上記のメソッドに 1 つのパスパラメータが渡された場合に上書きされていました。複数のパスパラメータが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 では、これらのリテラルはNULLs を結果として返していました。 -
Spark 3.1 では、組み込みの Hive 1.2 を削除しました。カスタム SerDes を Hive 2.3 に移行する必要があります。詳細については、HIVE-15167 を参照してください。
-
Spark 3.1 では、タイムスタンプが 1900-01-01 00:00:00Z より前の場合、INT96 型として読み込まれた (保存された) 場合、parquet ファイルからのタイムスタンプの読み込みと保存が失敗します。Spark 3.0 では、これらの操作は失敗しませんでしたが、Julian から Proleptic Gregorian カレンダーへの/からのリベースによる入力タイムスタンプのシフトにつながる可能性がありました。Spark 3.1 より前の動作に戻すには、
spark.sql.legacy.parquet.int96RebaseModeInReadまたは/およびspark.sql.legacy.parquet.int96RebaseModeInWriteをLEGACYに設定してください。 -
Spark 3.1 では、
schema_of_jsonおよびschema_of_csv関数は、フィールド名が引用符で囲まれた SQL 形式でスキーマを返します。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.sql.legacy.charVarcharAsStringをtrueに設定してください。 -
Spark 3.1 では、Hive 外部カタログからテーブルの場合に、
AnalysisExceptionは、次の状況でサブクラスに置き換えられます。ALTER TABLE .. ADD PARTITIONは、新しいパーティションが既に存在する場合にPartitionsAlreadyExistExceptionをスローします。ALTER TABLE .. DROP PARTITIONは、存在しないパーティションに対してNoSuchPartitionsExceptionをスローします。
Spark SQL 3.0.1 から 3.0.2 へのアップグレード
- Spark 3.0.2 では、Hive 外部カタログからテーブルの場合に、
AnalysisExceptionは、次の状況でサブクラスに置き換えられます。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」と名付けられます。古い動作は、デフォルト値がfalseである新しく追加された設定spark.sql.legacy.dataset.nameNonStructGroupingKeyAsValueで維持されます。 -
Spark 3.0 では、列メタデータは常に
Column.nameおよびColumn.asAPI で伝播されます。Spark 2.4 以前のバージョンでは、NamedExpressionのメタデータは API が呼び出されたときに新しい列のexplicitMetadataとして設定され、基になるNamedExpressionのメタデータが変更されても変更されませんでした。Spark 3.0 より前の動作を復元するには、明示的なメタデータを使用してas(alias: String, metadata: Metadata)API を使用できます。 -
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 になります。この動作は、デフォルト値が「ANSI」であるオプションspark.sql.storeAssignmentPolicyによって制御されます。オプションを「Legacy」に設定すると、以前の動作が復元されます。 -
ADD JARコマンドは以前は単一の値 0 を持つ結果セットを返していましたが、現在は空の結果セットを返します。 -
Spark 2.4 以前では、
SETコマンドは、指定されたキーがSparkConfエントリのものであっても警告なしに機能し、コマンドはSparkConfを更新しないため効果がありませんでしたが、ユーザーを混乱させる可能性がありました。3.0 では、SparkConfキーが使用された場合、コマンドは失敗します。spark.sql.legacy.setCommandRejectsSparkCoreConfsをfalseに設定することで、そのようなチェックを無効にできます。 -
キャッシュされたテーブルをリフレッシュすると、テーブルのアンキャッシュ操作、その後テーブルのキャッシュ(遅延)操作がトリガーされます。Spark 2.2.0 以前のバージョンでは、キャッシュ名とストレージレベルはアンキャッシュ操作の前に保持されませんでした。そのため、キャッシュ名とストレージレベルが予期せず変更される可能性がありました。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'のようなヘッドレスプロパティを作成するだけでした。プロパティ(大文字小文字を区別) データベース予約済み テーブル予約済み 備考 プロバイダー いいえ はい テーブルの場合、 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は、指定されたテーブルが Hive SerDe テーブルであっても、常に Spark DDL を返します。Hive DDL を生成するには、代わりにSHOW CREATE TABLE table_identifier AS SERDEコマンドを使用してください。 -
Spark 3.0 では、CHAR 型の列は非 Hive SerDe テーブルでは許可されず、CHAR 型が検出されると CREATE/ALTER TABLE コマンドは失敗します。代わりに STRING 型を使用してください。Spark 2.4 以前のバージョンでは、CHAR 型は STRING 型として扱われ、長さパラメータは単に無視されました。
UDF および組み込み関数
-
Spark 3.0 では、
date_addおよびdate_sub関数は、2 番目の引数として int, smallint, tinyint のみを受け入れます。小数やリテラルではない文字列はもはや有効ではありません。例えば、date_add(cast('1964-05-23' as date), '12.34')はAnalysisExceptionを引き起こします。文字列リテラルは引き続き許可されますが、Spark は文字列の内容が無効な整数である場合にAnalysisExceptionをスローします。Spark 2.4 以前のバージョンでは、2 番目の引数が小数または文字列値の場合、int 値にキャストされ、結果は1964-06-04の日付値となります。 -
Spark 3.0 では、関数
percentile_approxおよびそのエイリアスapprox_percentileは、3 番目の引数accuracyとして[1, 2147483647]の範囲の整数値のみを受け入れます。小数型および文字列型は許可されません。例えば、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の 2 つのモードをサポートしています。モードはmodeオプションを介して設定できます。デフォルトモードはPERMISSIVEになりました。以前のバージョンでは、from_jsonの動作はPERMISSIVEまたはFAILFASTのいずれにも準拠しておらず、特に不正な JSON レコードの処理においてそうでした。例えば、スキーマa INTを持つ JSON 文字列{"a" 1}は、以前のバージョンではnullに変換されていましたが、Spark 3.0 ではRow(null)に変換されます。 -
Spark 2.4 以前のバージョンでは、
CreateMap、MapFromArraysなどの組み込み関数を介して、マップ型のキーを持つマップ値を作成できました。Spark 3.0 では、これらの組み込み関数を使用してマップ型のキーを持つマップ値を作成することは許可されていません。回避策として、ユーザーはmap_entries関数を使用してマップを array<struct<key, value» に変換できます。さらに、データソースまたは Java/Scala コレクションからマップ型のキーを持つマップ値は引き続き読み取ることができますが、推奨されません。 -
Spark 2.4 以前のバージョンでは、
CreateMap、StringToMapなどの組み込み関数を使用して、重複したキーを持つマップを作成できました。重複したキーを持つマップの動作は未定義です。例えば、マップのルックアップは最初に出現した重複キーを尊重しますが、Dataset.collectは最後に出現した重複キーのみを保持し、MapKeysは重複キーを返します。Spark 3.0 では、重複したキーが見つかると Spark はRuntimeExceptionをスローします。spark.sql.mapKeyDedupPolicyをLAST_WINに設定すると、最後に勝利したポリシーでマップキーを重複排除できます。データソースから重複したキーを持つマップ値(例: Parquet)を読み取ると、動作は未定義のままです。 -
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 クロージャを受け取った場合、返される UDF は入力値が null の場合に null を返します。しかし、Spark 3.0 では、UDF は入力値が null の場合、Java 型のデフォルト値を返します。例えば、val f = udf((x: Int) => x, IntegerType)、f($"x")は、列xが null の場合、Spark 2.4 以前では null を返しますが、Spark 3.0 では 0 を返します。この動作の変更は、Spark 3.0 がデフォルトで Scala 2.12 でビルドされているために導入されました。 -
Spark 3.0 では、高階関数
existsは 3 値ブール論理に従います。つまり、predicateがnullを返し、trueが取得されない場合、existsはfalseの代わりにnullを返します。例えば、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に 1 か月追加すると、2019-03-31になります。 -
Spark 2.4 以前のバージョンでは、
current_timestamp関数はミリ秒解像度のタイムスタンプのみを返していました。Spark 3.0 では、基になるシステムクロックがそのような解像度を提供する場合、関数はマイクロ秒解像度の結果を返すことができます。 -
Spark 3.0 では、0 引数の Java UDF は、他の UDF と同様に executor 側で実行されます。Spark 2.4 以前のバージョンでは、0 引数の Java UDF のみが driver 側で実行され、結果が executor に伝播されていました。これは場合によってはよりパフォーマンスが高かったかもしれませんが、場合によっては一貫性の問題や正確性の問題を引き起こしました。
-
java.lang.Mathのlog,log1p,exp,expm1,powの結果はプラットフォームによって異なる場合があります。Spark 3.0 では、同等の SQL 関数(LOG10のような関連 SQL 関数を含む)の結果は、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 値を文字列型にキャストする際、「interval」プレフィックスはありません。例えば、
1 days 2 hoursとなります。Spark 2.4 以前のバージョンでは、文字列にはinterval 1 days 2 hoursのような「interval」プレフィックスが含まれていました。 -
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 では、interval リテラルの構文は、複数の 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 では、日時の interval 文字列は、
fromおよびtoの境界に基づいて interval に変換されます。入力文字列が指定された境界によって定義されたパターンに一致しない場合、ParseException例外がスローされます。例えば、interval '2 10:20' hour to minuteは、期待される形式が[+|-]h[h]:[m]mであるため、例外が発生します。Spark 2.4 では、from境界は考慮されず、to境界が結果の interval を切り捨てるために使用されました。たとえば、上記例の日時の interval 文字列はinterval 10 hours 20 minutesに変換されました。Spark 3.0 より前の動作を復元するには、spark.sql.legacy.fromDayTimeString.enabledをtrueに設定してください。 -
Spark 3.0 では、デフォルトで負のスケールの decimal は許可されません。例えば、
1E10BDのようなリテラルのデータ型はDecimalType(11, 0)になります。Spark 2.4 以前のバージョンでは、DecimalType(2, -9)でした。Spark 3.0 より前の動作を復元するには、spark.sql.legacy.allowNegativeScaleOfDecimalをtrueに設定してください。 -
Spark 3.0 では、単項算術演算子プラス (
+) は、文字列、数値、および interval 型の値のみを入力として受け入れます。さらに、整数文字列表現の+は double 値にキャストされます。例えば、+'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 では、ネストされた WITH 句での名前の競合を制御するために
spark.sql.legacy.ctePrecedencePolicyが導入されました。デフォルト値の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 以前のバージョンでは、float/double の -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 では、Proleptic Gregorian カレンダーが、日付とタイムスタンプの解析、フォーマット、変換、および年、日などのサブコンポーネントの抽出に使用されます。Spark 3.0 は
java.timeパッケージの Java 8 API クラスを使用しており、これは ISO クロノロジーに基づいています。Spark 2.4 以前のバージョンでは、これらの操作はハイブリッドカレンダー(Julian + Gregorian)を使用して実行されていました。この変更は、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 では、Datetime Patterns for Formatting and Parsing で独自のパターン文字列を定義しており、これは内部的にDateTimeFormatterを介して実装されます。新しい実装は、入力の厳密なチェックを実行します。例えば、パターンがyyyy-MM-ddである場合、2015-07-22 10:00:00タイムスタンプは解析できません。これは、パーサーが入力全体を消費しないためです。別の例は、dd/MM/yyyy hh:mmパターンでは31/01/2015 00:00入力が解析できないことです。これは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関数は、年の週番号、週の日の番号の計算、および UTC タイムゾーンでの TimestampType 値との変換に java.time API を使用します。 -
JDBC オプション
lowerBoundおよびupperBoundは、文字列を TimestampType/DateType 値にキャストするのと同じ方法で TimestampType/DateType 値に変換されます。変換は、Proleptic Gregorian カレンダーと、SQL 設定spark.sql.session.timeZoneによって定義されるタイムゾーンに基づいています。Spark 2.4 以前のバージョンでは、変換はハイブリッドカレンダー(Julian + Gregorian)とデフォルトのシステムタイムゾーンに基づいています。 -
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に設定することで復元できます。 - 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 では、datetime パターン文字
Fは月内の整列された曜日であり、週内の日の数を表します。週は月の初めに整列されます。Spark 2.4 以前のバージョンでは、月内の週であり、週は固定の曜日から始まる月内の週数を表します。例えば、2020-07-30は月の初日から 30 日(4 週と 2 日)後です。そのため、date_format(date '2020-07-30', 'F')は Spark 3.0 では 2 を返しますが、Spark 2.x では週の数として、2020 年 7 月の 5 週目に位置するため 5 を返します(週 1 は 2020-07-01 から 07-04)。 -
Spark 3.0 では、
CTASで Hive serde の代わりに組み込みデータソースライターを使用しようとします。この動作は、Parquet および ORC フォーマットに対してそれぞれspark.sql.hive.convertMetastoreParquetまたはspark.sql.hive.convertMetastoreOrcが有効になっている場合にのみ有効です。Spark 3.0 より前の動作を復元するには、spark.sql.hive.convertMetastoreCtasをfalseに設定してください。 - Spark 3.0 では、HiveSQL 構文を使用して作成されたパーティション化された ORC/Parquet テーブルに挿入する際、Hive serde の代わりに組み込みデータソースライターを使用しようとします。この動作は、Parquet および ORC フォーマットに対してそれぞれ
spark.sql.hive.convertMetastoreParquetまたはspark.sql.hive.convertMetastoreOrcが有効になっている場合にのみ有効です。Spark 3.0 より前の動作を復元するには、spark.sql.hive.insertingPartitionedTableを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 以前のバージョンでは、JSON データソースおよび
from_jsonのような JSON 関数は、PERMISSIVE モードで、指定されたスキーマがStructTypeである場合、不正な 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 ファイルがユーザー提供スキーマで書き込まれる際、フィールドは、位置ではなく、触媒スキーマと Avro スキーマ間のフィールド名によって一致します。
-
Spark 3.0 では、Avro ファイルがユーザー提供の非 null スキーマで書き込まれる際、触媒スキーマが null 可能であっても、Spark はファイルを書き込むことができます。しかし、レコードのいずれかに null が含まれている場合、実行時 NullPointerException がスローされます。
-
Spark 2.4 以前のバージョンでは、CSV データソースは、ファイルに BOM が先頭にある場合、入力ファイルのエンコーディングを自動的に検出できます。例えば、CSV データソースはマルチラインモード(CSV オプション
multiLineがtrueに設定されている)で UTF-8、UTF-16BE、UTF-16LE、UTF-32BE、UTF-32LE を認識できます。Spark 3.0 では、CSV データソースは CSV オプションencodingを介して指定されたエンコーディングで入力ファイルを読み取ります。このオプションのデフォルト値は UTF-8 です。これにより、ファイルエンコーディングが CSV オプションで指定されたエンコーディングと一致しない場合、Spark はファイルを誤ってロードします。問題を解決するには、ユーザーは CSV オプションencodingを介して正しいエンコーディングを設定するか、オプションをnullに設定して Spark 3.0 より前のエンコーディング自動検出にフォールバックする必要があります。
その他
-
Spark 2.4 以前のバージョンでは、
cloneSession()を介して Spark セッションが作成される際、新しく作成された Spark セッションは、親 Spark セッションに同じ設定が存在する場合でも、親SparkContextから設定を継承します。Spark 3.0 では、親SparkSessionの設定が親SparkContextよりも優先されます。古い動作を復元するには、spark.sql.legacy.sessionInitWithConfigDefaultsをtrueに設定してください。 -
Spark 3.0 では、`hive.default.fileformat` が `Spark SQL configuration` に見つからない場合、`SparkContext` の `Hadoop configuration` に存在する `hive-site.xml` ファイルにフォールバックします。
-
Spark 3.0 では、`spark-sql` インターフェイスに対して、10 進数を列のスケールに合わせて末尾ゼロでパディングします。例えば
クエリ 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 を参照してください。 -
TRANSFORM演算子を SQL でスクリプト変換に使用する場合、Hive 1.2 と Hive 2.3 の間で 10 進数の文字列表現が異なる場合があります。これは Hive の動作に依存します。Hive 1.2 では、文字列表現は末尾のゼロを省略します。しかし、Hive 2.3 では、必要に応じて末尾ゼロで 18 桁にパディングされます。
-
Spark SQL 2.4.7 から 2.4.8 へのアップグレード
- Spark 2.4.8 では、Hive 外部カタログからのテーブルに対して、以下の状況で
AnalysisExceptionがそのサブクラスに置き換えられます。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設定が追加され、SMALLINT および REAL の JDBC 型に対して IntegerType および DoubleType を使用するレガシー 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 ダイアレクトは SMALLINT および REAL に対して ShortType および FloatType を使用します。以前は、IntegerType および DoubleType が使用されていました。
Spark SQL 2.4 から 2.4.1 へのアップグレード
spark.executor.heartbeatIntervalの値は、単位なし(「30」など、「30s」ではなく)で指定された場合、Spark 2.4 ではコードの異なる部分で秒とミリ秒の両方として解釈されていました。単位なしの値は、一貫してミリ秒として解釈されるようになりました。値「30」のような値を設定したアプリケーションは、ミリ秒として解釈されないように、現在では「30s」のような単位を持つ値を指定する必要があります。そうしないと、結果として得られる非常に短い間隔がアプリケーションの失敗を引き起こす可能性が高くなります。
Spark SQL 2.3 から 2.4 へのアップグレード
- Spark 2.3 以前のバージョンでは、array_contains 関数の 2 番目のパラメーターは、最初の配列型パラメーターの要素型に暗黙的に昇格されていました。この型昇格は損失を伴う可能性があり、
array_contains関数が誤った結果を返す原因となる可能性がありました。この問題は、より安全な型昇格メカニズムを採用することで 2.4 で解決されました。これにより、動作にいくつかの変更が生じる可能性があり、それは以下の表に示されています。クエリ Spark 2.3 以前 Spark 2.4 備考 SELECT array_contains(array(1), 1.34D);truefalseSpark 2.4 では、左側と右側のパラメーターは、それぞれ double 型および double 型の配列型に昇格されます。 SELECT array_contains(array(1), '1');trueAnalysisExceptionがスローされます。明示的なキャストを引数で使用することで例外を回避できます。Spark 2.4 では、整数型を文字列型に損失なく昇格できないため AnalysisExceptionがスローされます。SELECT array_contains(array(1), 'anystring');nullAnalysisExceptionがスローされます。明示的なキャストを引数で使用することで例外を回避できます。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 演算の前に実行されるという例外があります。すべてのセット演算に等しい優先順位を与える古い動作は、デフォルト値が
falseである新しく追加された設定spark.sql.legacy.setopsPrecedence.enabledで維持されます。このプロパティがtrueに設定されている場合、Spark は、括弧の使用によって明示的な順序付けが強制されていない場合、クエリに表示されるセット演算子を左から右に評価します。 -
Spark 2.4 以降、Spark はテーブル説明列の Last Access 値を、値が 1970 年 1 月 1 日の場合に UNKNOWN と表示します。
-
Spark 2.4 以降、Spark はデフォルトでベクトル化された ORC リーダーを ORC ファイルに最大限に活用します。これを行うために、
spark.sql.orc.implおよびspark.sql.orc.filterPushdownはデフォルト値をnativeおよびtrueに変更します。ネイティブ ORC ライターによって作成された ORC ファイルは、一部の古い Apache Hive リリースでは読み取ることができません。Hive 2.1.1 以前と共有されるファイルを作成するには、spark.sql.orc.impl=hiveを使用します。 -
Spark 2.4 以降、空の DataFrame をディレクトリに書き込むと、DataFrame が物理的にパーティションを持たない場合でも、少なくとも 1 つの書き込みタスクが起動します。これにより、自己記述型ファイル形式(Parquet や Orc)の場合、Spark は 0 パーティションの DataFrame を書き込む際にメタデータのみのファイルを作成し、後でユーザーがそのディレクトリを読み取る際にスキーマ推論が機能するように、わずかな動作変更が生じます。新しい動作はより合理的であり、空の DataFrame の書き込みに関して一貫性があります。
-
Spark 2.4 以降、UDF 引数内の式 ID は列名に表示されなくなりました。例えば、Spark 2.4 の列名は
UDF:f(col0 AS colA#28)ではなくUDF:f(col0 AS `colA`)となります。 -
Spark 2.4 以降、空またはネストされた空のスキーマを持つ DataFrame をファイル形式(parquet, orc, json, text, csv など)で書き込むことは許可されていません。空のスキーマを持つ 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 固有のテーブルプロパティを尊重します。例として、CREATE TABLE t(id int) STORED AS PARQUET TBLPROPERTIES (parquet.compression 'NONE')は Spark 2.3 で挿入中に Snappy Parquet ファイルを生成しますが、Spark 2.4 では、結果は圧縮されていない Parquet ファイルになります。 -
バージョン 2.3 以前では、Spark はデフォルトで Parquet Hive テーブルを変換してパフォーマンスを向上させます。Spark 2.4 以降、Spark は ORC Hive テーブルもデフォルトで変換します。つまり、Spark はデフォルトで Hive SerDe の代わりに独自の ORC サポートを使用します。例として、
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 行は、行内の少なくとも 1 つの列値が不正な場合に不正と見なされます。CSV パーサーは、
DROPMALFORMEDモードでそのような行をドロップするか、FAILFASTモードでエラーを出力します。Spark 2.4 以降、CSV 行は、CSV データソースから要求された不正な列値が含まれている場合にのみ不正と見なされます。他の値は無視される可能性があります。例として、CSV ファイルにヘッダー「id,name」と 1 行「1234」が含まれているとします。Spark 2.4 では、id 列の選択は 1234 という 1 つの列値を持つ行で構成されますが、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コマンドは、それぞれ任意の 1 文字、および 0 文字以上の文字に一致するワイルドカード?および*をサポートします。例:LOAD DATA INPATH '/tmp/folder*/'またはLOAD DATA INPATH '/tmp/part-?'。spaceのような特殊文字もパスで機能するようになりました。例:LOAD DATA INPATH '/tmp/folder name/'。 -
Spark 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は 1 行のみを返します。以前の動作を復元するには、spark.sql.legacy.parser.havingWithoutGroupByAsWhereをtrueに設定してください。 - バージョン 2.3 以前では、Parquet データソーステーブルから読み取る際、
spark.sql.caseSensitiveがtrueまたはfalseに設定されているかどうかにかかわらず、Hive メタストアスキーマと Parquet スキーマの列名の大文字小文字が異なる場合、Spark は常に null を返します。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関数は以前は数値型入力を受け入れ、double 型の結果を返していました。現在は、日付型、タイムスタンプ型、および数値型を入力型としてサポートしています。結果型も入力型と同じに変更され、パーセンタイルのために、より合理的になっています。 -
Spark 2.3 以降、Join/Filter の決定論的述語は、最初の非決定論的述語の後であっても、可能であれば子演算子にプッシュダウン/スルーされます。以前の Spark バージョンでは、これらのフィルターは述語プッシュダウンの対象ではありませんでした。
-
パーティション列の推論は、以前は異なる推論型に対して誤った共通型を見つけていました。例えば、以前は double 型と date 型の共通型として double 型になっていました。現在は、そのような競合に対して正しい共通型を見つけます。競合解決は以下の表に従います。
InputA \ InputB 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)* については、現在、Decimal 型は
BigInteger/BigIntのように推論されるため、上記の表はスケールと精度の他のすべての組み合わせを意図的にカバーしていません。例えば、1.1 は double 型として推論されます。 -
Spark 2.3 以降、ブロードキャストハッシュ結合またはブロードキャストネストドループ結合が適用可能な場合、ブロードキャストヒントで明示的に指定されたテーブルをブロードキャストすることを優先します。詳細については、Join Strategy Hints for SQL Queries および 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 以降、デフォルトで、10 進数間の算術演算は、正確な表現が不可能な場合に NULL を返すのではなく、丸められた値を返します。これは SQL ANSI 2011 仕様および Hive 2.2 で導入された Hive の新しい動作(HIVE-15331)に準拠しています。これには以下の変更が含まれます。
-
算術演算の結果の型を決定するルールが更新されました。特に、必要な精度/スケールが利用可能な値の範囲外の場合、整数部分の 10 進数の切り捨てを防ぐために、スケールが 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 は、大文字小文字が混在する列名を持つ基盤ファイルスキーマの Hive メタストアテーブルの読み取りとの互換性を回復するために、この設定のデフォルト値をINFER_AND_SAVEに変更しました。INFER_AND_SAVE構成値を使用すると、最初にアクセスしたときに Spark は、すでに推論されたスキーマを保存していない Hive メタストアテーブルのスキーマ推論を実行します。スキーマ推論は、数千のパーティションを持つテーブルでは非常に時間がかかる操作になる可能性があることに注意してください。大文字小文字の混在列名との互換性が懸念されない場合は、スキーマ推論の初期オーバーヘッドを回避するために、spark.sql.hive.caseSensitiveInferenceModeをNEVER_INFERに安全に設定できます。新しいデフォルトのINFER_AND_SAVE設定では、スキーマ推論の結果はメタストアキーとして保存されるため、初期スキーマ推論はテーブルの最初のアクセス時にのみ発生することに注意してください。 -
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 メタストアに保存するようになりました。これは、
ALTER TABLE PARTITION ... SET LOCATIONのような Hive DDL が、データソース 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 は後方互換性のために保持されていることに注意してください。新しいcatalogインターフェイスはSparkSessionからアクセスできます。データベースおよびテーブルアクセスに関する既存の API(listTables、createExternalTable、dropTempView、cacheTable)はここに移動されました。 -
Dataset API と DataFrame API が統合されました。Scala では、
DataFrameはDataset[Row]の型エイリアスになりました。Java API ユーザーはDataFrameをDataset<Row>に置き換える必要があります。型指定された変換(例:map、filter、groupByKey)と型指定されていない変換(例:select、groupBy)の両方が Dataset クラスで利用可能です。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 へのキャストは、マイクロ秒ではなく秒を期待します。この変更は、数値型から TimestampType へのより一貫した型キャストのために、Hive 1.2 の動作に合わせるために行われました。詳細については、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に設定することで無効にできます。 -
無制限の精度を持つ decimal 型の列はサポートされなくなり、代わりに Spark SQL は最大精度 38 を適用します。
BigDecimalオブジェクトからスキーマを推論する場合、精度 (38, 18) が使用されるようになりました。DDL で精度が指定されていない場合、デフォルトはDecimal(10, 0)のままです。 -
タイムスタンプは、1ns ではなく 1us の精度で格納されるようになりました。
-
sql言語では、浮動小数点数はデフォルトで decimal として解析されます。HiveQL の解析は変更されません。 -
SQL/DataFrame 関数の正準名は、(例: sum vs SUM) 小文字になりました。
-
JSON データソースは、他のアプリケーションによって作成された新しいファイル(つまり、Spark SQL を介してデータセットに挿入されていないファイル)を自動的にロードしなくなります。JSON 永続テーブル(つまり、テーブルのメタデータが Hive Metastore に格納されている)の場合、ユーザーは
REFRESH TABLESQL コマンドまたはHiveContextのrefreshTableメソッドを使用して、これらの新しいファイルをテーブルに含めることができます。JSON データセットを表す DataFrame の場合、ユーザーは DataFrame を再作成する必要があり、新しい DataFrame には新しいファイルが含まれます。
Spark SQL 1.3 から 1.4 へのアップグレード
DataFrame データリーダー/ライターインターフェース
ユーザーからのフィードバックに基づき、データの読み込み (SQLContext.read) とデータの書き出し (DataFrame.write) のための、より流動的な新しい API を作成し、古い 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 が提供する機能のほとんどを独自の G実装を通じて提供するようになったためです。DataFrame は、.rdd メソッドを呼び出すことで、依然として RDD に変換できます。
Scala では、一部のユースケースでソース互換性を提供するために、SchemaRDD から DataFrame への型エイリアスがあります。それでも、ユーザーはコードを DataFrame を使用するように更新することが推奨されます。Java および Python ユーザーはコードを更新する必要があります。
Java および Scala API の統合
Spark 1.3 より前は、Scala API をミラーリングする個別の Java 互換クラス (JavaSQLContext および JavaSchemaRDD) がありました。Spark 1.3 では、Java API と Scala API が統合されました。どちらの言語のユーザーも SQLContext および DataFrame を使用する必要があります。一般的に、これらのクラスは両方の言語で使用可能な型 (つまり、言語固有のコレクションではなく Array) を使用しようとします。共通の型が存在しない場合 (例: クロージャまたは Map の渡し方) は、代わりにオーバーロードされた関数が使用されます。
さらに、Java 固有の Types 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 を持つ RDD のみに適用されるようになり、自動的に適用されるのではなくようになりました。
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)
DataFrame DSL または SQL で使用するために UDF を登録するために使用される関数が、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 Warehouse へのデプロイ
Spark SQL Thrift JDBC サーバーは、既存の Hive インストールと「すぐに使える」互換性があるように設計されています。既存の Hive Metastore を変更したり、テーブルのデータ配置やパーティショニングを変更したりする必要はありません。
サポートされている Hive 機能
Spark SQL は、次のような大多数の Hive 機能もサポートしています。
- Hive クエリステートメント、以下を含む
SELECTGROUP BYORDER BYDISTRIBUTE BYCLUSTER BYSORT BY
- すべての Hive 演算子、以下を含む
- 関係演算子 (
=,<=>,==,<>,<,>,>=,<=など) - 算術演算子 (
+,-,*,/,%など) - 論理演算子 (
AND,ORなど) - 複合型コンストラクタ
- 数学関数 (
sign,ln,cosなど) - 文字列関数 (
instr,length,printfなど)
- 関係演算子 (
- ユーザー定義関数 (UDF)
- ユーザー定義集計関数 (UDAF)
- ユーザー定義シリアライゼーションフォーマット (SerDes)
- ウィンドウ関数
- 結合
JOIN{LEFT|RIGHT|FULL} OUTER JOINLEFT SEMI JOINLEFT ANTI JOINCROSS JOIN
- Union
- サブクエリ
-
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
- パーティション分割されたテーブル(動的パーティション挿入を含む)
- View
-
View 定義クエリで列エイリアスが指定されていない場合、Spark と Hive の両方がエイリアス名を生成しますが、その方法は異なります。Spark が Hive によって作成された View を読み取れるようにするには、ユーザーは View 定義クエリで列エイリアスを明示的に指定する必要があります。たとえば、Hive によって以下のように作成された
v1を Spark は読み取ることができません。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 TABLECREATE TABLE AS SELECTCREATE TABLE LIKEALTER TABLE
- ほとんどの Hive データ型、以下を含む
TINYINTSMALLINTINTBIGINTBOOLEANFLOATDOUBLESTRINGBINARYTIMESTAMPDATEARRAY<>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];」を使用して制御する必要があります。 - メタデータのみのクエリ: メタデータのみを使用して回答できるクエリの場合、Spark SQL は依然としてタスクを起動して結果を計算します。
- Skew data フラグ: Spark SQL は Hive の skew data フラグに従いません。
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 に必要な追加リソースを自動的に含めるための関数です。initialize(StructObjectInspector)(GenericUDTF) はまだサポートされていません。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 を秒として扱います。