layout: global title: Error Conditions displayTitle: Error Conditions license: | Licensed to the Apache Software Foundation (ASF) under one or more contributor license agreements. See the NOTICE file distributed with this work for additional information regarding copyright ownership. The ASF licenses this file to You under the Apache License, Version 2.0 (the “License”); you may not use this file except in compliance with the License. You may obtain a copy of the License at
http://www.apache.org/licenses/LICENSE-2.0
This is a list of common, named error conditions returned by Spark SQL.
Also see SQLSTATE Codes.
Non-deterministic expression <sqlExpr>
should not appear in the arguments of an aggregate function.
SQLSTATE: KD005
Cannot use all columns for partition columns.
ALTER TABLE <type>
column <columnName>
specifies descriptor “<optionName>
” more than once, which is invalid.
Name <name>
is ambiguous in nested CTE. Please set <config>
to “CORRECTED” so that name defined in inner CTE takes precedence. If set it to “LEGACY”, outer CTE definitions will take precedence. See ‘<docroot>
/sql-migration-guide.html#query-engine’.
Column or field <name>
is ambiguous and has <n>
matches.
Lateral column alias <name>
is ambiguous and has <n>
matches.
Reference <name>
is ambiguous, could be: <referenceNames>
.
Ambiguous reference to the field <field>
. It appears <count>
times in the schema.
<message>
.<alternative>
If necessary set <config>
to “false” to bypass this error.
The number of columns or variables assigned or aliased: <numTarget>
does not match the number of source expressions: <numExpr>
.
Invalid as-of join.
For more details see AS_OF_JOIN
Cannot convert Avro <avroPath>
to SQL <sqlPath>
because the original encoded data type is <avroType>
, however you're trying to read the field as <sqlType>
, which would lead to an incorrect answer. To allow reading this field, enable the SQL configuration: “spark.sql.legacy.avro.allowIncompatibleSchema”.
Unable to find batch <batchMetadataFile>
.
<value1>
<symbol>
<value2>
caused overflow.
The method <methodName>
can not be called on streaming Dataset/DataFrame.
ALTER TABLE (ALTER|CHANGE) COLUMN is not supported for partition columns, but found the partition column <columnName>
in the table <tableName>
.
Cannot cast <sourceType>
to <targetType>
.
Cannot convert Protobuf <protobufColumn>
to SQL <sqlColumn>
because schema is incompatible (protobufType = <protobufType>
, sqlType = <sqlType>
).
Unable to convert <protobufType>
of Protobuf to SQL type <toType>
.
Cannot convert SQL <sqlColumn>
to Protobuf <protobufColumn>
because schema is incompatible (protobufType = <protobufType>
, sqlType = <sqlType>
).
Cannot convert SQL <sqlColumn>
to Protobuf <protobufColumn>
because <data>
is not in defined values for enum: <enumString>
.
The provided URL cannot be decoded: <url>
. Please ensure that the URL is properly formatted and try again.
Dataset transformations and actions can only be invoked by the driver, not inside of other Dataset transformations; for example, dataset1.map(x => dataset2.values.count() * x) is invalid because the values transformation and count action cannot be performed inside of the dataset1.map transformation. For more information, see SPARK-28702.
Cannot load class <className>
when registering the function <functionName>
, please make sure it is on the classpath.
Could not load Protobuf class with name <protobufClassName>
. <explanation>
.
SQLSTATE: 58030
An error occurred during loading state.
For more details see CANNOT_LOAD_STATE_STORE
Failed to merge incompatible data types <left>
and <right>
. Please check the data types of the columns being merged and ensure that they are compatible. If necessary, consider casting the columns to compatible data types before attempting the merge.
Failed merging schemas: Initial schema: <left>
Schema that cannot be merged with the initial schema: <right>
.
Cannot modify the value of the Spark config: <key>
. See also ‘<docroot>
/sql-migration-guide.html#ddl-statements’.
Cannot parse decimal. Please ensure that the input is a valid number with optional decimal point or comma separators.
Unable to parse <intervalString>
. Please ensure that the value provided is in a valid format for defining an interval. You can reference the documentation for the correct format. If the issue persists, please double check that the input value is not null or empty and try again.
Cannot parse the field name <fieldName>
and the value <fieldValue>
of the JSON token type <jsonType>
to target Spark data type <dataType>
.
Error parsing descriptor bytes into Protobuf FileDescriptorSet.
<message>
. If necessary set <ansiConfig>
to “false” to bypass this error.
SQLSTATE: KD001
Could not read footer for file: <file>
. Please ensure that the file is in either ORC or Parquet format. If not, please convert it to a valid format. If the file is in the valid format, please check if it is corrupt. If it is, you can choose to either ignore it or fix the corruption.
Cannot recognize hive type string: <fieldType>
, column: <fieldName>
. The specified data type for the field cannot be recognized by Spark SQL. Please check the data type of the specified field and ensure that it is a valid Spark SQL data type. Refer to the Spark SQL documentation for a list of valid data types and their format. If the data type is correct, please ensure that you are using a supported version of Spark SQL.
Renaming a <type>
across schemas is not allowed.
Cannot resolve <targetString>
.* given input columns <columns>
. Please check that the specified table or struct exists and is accessible in the input columns.
SQLSTATE: 58030
Failed to set permissions on created path <path>
back to <permission>
.
Cannot update <table>
field <fieldName>
type:
For more details see CANNOT_UPDATE_FIELD
Cannot up cast <expression>
from <sourceType>
to <targetType>
. <details>
SQLSTATE: 58030
Error writing state store files for provider <providerClass>
.
For more details see CANNOT_WRITE_STATE_STORE
The value <expression>
of the type <sourceType>
cannot be cast to <targetType>
because it is malformed. Correct the value as per the syntax, or change its target type. Use try_cast
to tolerate malformed input and return NULL instead. If necessary set <ansiConfig>
to “false” to bypass this error.
The value <value>
of the type <sourceType>
cannot be cast to <targetType>
due to an overflow. Use try_cast
to tolerate overflow and return NULL instead. If necessary set <ansiConfig>
to “false” to bypass this error.
Fail to assign a value of <sourceType>
type to the <targetType>
type column or variable <columnName>
due to an overflow. Use try_cast
on the input value to tolerate overflow and return NULL instead.
SQLSTATE: 56000
Checkpoint block <rddBlockId>
not found! Either the executor that originally checkpointed this partition is no longer alive, or the original RDD is unpersisted. If this problem persists, you may consider using rdd.checkpoint()
instead, which is slower than local checkpointing but more fault-tolerant.
MapObjects
does not support the class <cls>
as resulting collection.
SQLSTATE: 56038
The codec <codecName>
is not available. Consider to set the config <configKey>
to <configVal>
.
Cannot find a short name for the codec <codecName>
.
Columns aliases are not allowed in <op>
.
The column <columnName>
already exists. Consider to choose another name or rename the existing column.
<colType>
column <colName>
is not defined in table <tableName>
, defined table columns are: <tableCols>
.
The column <colName>
cannot be found. Verify the spelling and correctness of the column name according to the SQL config <caseSensitiveConfig>
.
The comparator has returned a NULL for a comparison between <firstValue>
and <secondValue>
. It should return a positive integer for “greater than”, 0 for “equal” and a negative integer for “less than”. To revert to deprecated behavior where NULL is treated as 0 (equal), you must set “spark.sql.legacy.allowNullComparisonResultInArraySort” to “true”.
Another instance of this query was just started by a concurrent session.
SQLSTATE: 40000
Concurrent update to the log. Multiple streaming jobs detected for <batchId>
. Please make sure only one streaming job runs on a specific checkpoint location at a time.
SQLSTATE: 56K00
Generic Spark Connect error.
For more details see CONNECT
The value <str>
(<fmt>
) cannot be converted to <targetType>
because it is malformed. Correct the value as per the syntax, or change its format. Use <suggestion>
to tolerate malformed input and return NULL instead.
Not allowed to create the permanent view <name>
without explicitly assigning an alias for the expression <attr>
.
CREATE TABLE column <columnName>
specifies descriptor “<optionName>
” more than once, which is invalid.
Cannot create view <viewName>
, the reason is
For more details see CREATE_VIEW_COLUMN_ARITY_MISMATCH
Cannot resolve <sqlExpr>
due to data type mismatch:
For more details see DATATYPE_MISMATCH
DataType <type>
requires a length parameter, for example <type>
(10). Please specify the length.
Failed to find the data source: <provider>
. Please find packages at https://spark.apache.org/third-party-projects.html
.
Datetime operation overflow: <operation>
.
Decimal precision <precision>
exceeds max precision <maxPrecision>
.
Default database <defaultDatabase>
does not exist, please create it first or change default database to <defaultDatabase>
.
A DEFAULT keyword in a MERGE, INSERT, UPDATE, or SET VARIABLE command could not be directly assigned to a target column because it was part of an expression. For example: UPDATE SET c1 = DEFAULT
is allowed, but UPDATE T SET c1 = DEFAULT + 1
is not allowed.
Distinct window functions are not supported: <windowExpr>
.
Division by zero. Use try_divide
to tolerate divisor being 0 and return NULL instead. If necessary set <config>
to “false” to bypass this error.
Duplicated field names in Arrow Struct are not allowed, got <fieldNames>
.
Duplicate map key <key>
was found, please check the input data. If you want to remove the duplicated keys, you can set <mapKeyDedupPolicy>
to “LAST_WIN” so that the key inserted at last takes precedence.
The metric name is not unique: <metricName>
. The same name cannot be used for metrics with different results. However multiple instances of metrics with with same result and name are allowed (e.g. self-joins).
The columns or variables <nameList>
appear more than once as assignment targets.
Found duplicate clauses: <clauseName>
. Please, remove one of them.
Found duplicate keys <keyColumn>
.
Call to function <functionName>
is invalid because it includes multiple argument assignments to the same parameter name <parameterName>
.
For more details see DUPLICATE_ROUTINE_PARAMETER_ASSIGNMENT
Failed to parse an empty string for data type <dataType>
.
Not found an encoder of the type <typeName>
to Spark SQL internal representation. Consider to change the input type to one of supported at ‘<docroot>
/sql-ref-datatypes.html’.
SQLSTATE: KD00B
Error reading avro data -- encountered an unknown fingerprint: <fingerprint>
, not sure what schema to use. This could happen if you registered additional schemas after starting your spark context.
The event time <eventName>
has the invalid type <eventType>
, but expected “TIMESTAMP”.
Exceeds char/varchar type length limitation: <limit>
.
‘<operation>
’ expects a permanent view but <viewName>
is a temp view.
‘<operation>
’ expects a table but <viewName>
is a view.
For more details see EXPECT_TABLE_NOT_VIEW
The table <tableName>
does not support <operation>
.
For more details see EXPECT_VIEW_NOT_TABLE
Failed to decode a row to a value of the expressions: <expressions>
.
Failed to encode a value of the expressions: <expressions>
to a row.
Column expression <expr>
cannot be sorted because its type <exprType>
is not orderable.
Failed to execute user defined function (<functionName>
: (<signature>
) => <result>
).
Failed preparing of the function <funcName>
for call. Please, double check function's arguments.
Failed parsing struct: <raw>
.
SQLSTATE: KD000
Failed to register classes with Kryo.
Failed to rename <sourcePath>
to <targetPath>
as destination already exists.
SQLSTATE: 58030
Failed to rename temp file <srcPath>
to <dstPath>
as FileSystem.rename returned false.
Cannot <op>
column, because <fieldNames>
already exists in <struct>
.
No such struct field <fieldName>
in <fields>
.
The operation <statement>
is not allowed on the <objectType>
: <objectName>
.
A column cannot have both a default value and a generation expression but column <colName>
has default value: (<defaultValue>
) and generation expression: (<genExpr>
).
SQLSTATE: 56038
Hive 2.2 and lower versions don't support getTablesByType. Please use Hive 2.3 or higher version.
SQLSTATE: KD000
Invalid Graphite protocol: <protocol>
.
SQLSTATE: KD000
Graphite sink requires ‘<property>
’ property.
Column of grouping (<grouping>
) can't be found in grouping columns <groupingColumns>
.
Columns of grouping_id (<groupingIdColumn>
) does not match grouping columns (<groupByColumns>
).
Grouping sets size cannot be greater than <maxSize>
.
Aggregate functions are not allowed in GROUP BY, but found <sqlExpr>
.
GROUP BY <index>
refers to an expression <aggExpr>
that contains an aggregate function. Aggregate functions are not allowed in GROUP BY.
GROUP BY position <index>
is not in select list (valid range is [1, <size>
]).
The expression <sqlExpr>
cannot be used as a grouping expression because its data type <dataType>
is not an orderable data type.
Invalid call to <function>
; only valid HLL sketch buffers are supported as inputs (such as those produced by the hll_sketch_agg
function).
Invalid call to <function>
; the lgConfigK
value must be between <min>
and <max>
, inclusive: <value>
.
Sketches have different lgConfigK
values: <left>
and <right>
. Set the allowDifferentLgConfigK
parameter to true to call <function>
with different lgConfigK
values.
<identifier>
is not a valid identifier as it has more than 2 name parts.
Invalid pivot column <columnName>
. Pivot columns must be comparable.
<operator>
can only be performed on tables with compatible column types. The <columnOrdinalNumber>
column of the <tableOrdinalNumber>
table is <dataType1>
type which is not compatible with <dataType2>
at the same column of the first table.<hint>
.
SQLSTATE: 56038
Detected an incompatible DataSourceRegister. Please remove the incompatible library from classpath or upgrade it. Error: <message>
SQLSTATE: KD000
Cannot write incompatible data for the table <tableName>
:
For more details see INCOMPATIBLE_DATA_FOR_TABLE
The join types <joinType1>
and <joinType2>
are incompatible.
SQLSTATE: 51024
The SQL query of view <viewName>
has an incompatible schema change and column <colName>
cannot be resolved. Expected <expectedNum>
columns named <colName>
but got <actualCols>
. Please try to re-create the view by running: <suggestion>
.
Incomplete complex type:
For more details see INCOMPLETE_TYPE_DEFINITION
You may get a different result due to the upgrading to
For more details see INCONSISTENT_BEHAVIOR_CROSS_VERSION
Max offset with <rowsPerSecond>
rowsPerSecond is <maxSeconds>
, but it's <endSeconds>
now.
Max offset with <rowsPerSecond>
rowsPerSecond is <maxSeconds>
, but ‘rampUpTimeSeconds’ is <rampUpTimeSeconds>
.
Cannot create the index <indexName>
on table <tableName>
because it already exists.
Cannot find the index <indexName>
on table <tableName>
.
Cannot write to <tableName>
, the reason is
For more details see INSERT_COLUMN_ARITY_MISMATCH
Cannot write to ‘<tableName>
’, <reason>
: Table columns: <tableColumns>
. Partition columns with static values: <staticPartCols>
. Data columns: <dataColumns>
.
Can't find table property:
For more details see INSUFFICIENT_TABLE_PROPERTY
<message>
<message>
<message>
<message>
An object in the metadata catalog has been corrupted:
For more details see INTERNAL_ERROR_METADATA_CATALOG
<message>
<message>
<message>
<message>
.<alternative>
Division by zero. Use try_divide
to tolerate divisor being 0 and return NULL instead.
The index <indexValue>
is out of bounds. The array has <arraySize>
elements. Use the SQL function get()
to tolerate accessing element at invalid index and return NULL instead. If necessary set <ansiConfig>
to “false” to bypass this error.
The index <indexValue>
is out of bounds. The array has <arraySize>
elements. Use try_element_at
to tolerate accessing element at invalid index and return NULL instead. If necessary set <ansiConfig>
to “false” to bypass this error.
The 0-indexed bitmap position <bitPosition>
is out of bounds. The bitmap has <bitmapNumBits>
bits (<bitmapNumBytes>
bytes).
The boundary <boundary>
is invalid: <invalidValue>
.
For more details see INVALID_BOUNDARY
SQLSTATE: 58030
Invalid bucket file: <path>
.
The expected format is ByteString, but was <unsupported>
(<class>
).
The datasource <datasource>
cannot save the column <columnName>
because its name contains some characters that are not allowed in file paths. Please, use an alias to rename it.
Column or field <name>
is of type <type>
while it's required to be <expectedType>
.
The cursor is invalid.
For more details see INVALID_CURSOR
Failed to execute <statement>
command because the destination column or variable <colName>
has a DEFAULT value <defaultValue>
,
For more details see INVALID_DEFAULT_VALUE
SQLSTATE: F0000
System memory <systemMemory>
must be at least <minSystemMemory>
. Please increase heap size using the --driver-memory option or “<config>
” in Spark configuration.
The location name cannot be empty string, but <location>
was given.
Found an invalid escape string: <invalidEscape>
. The escape string must contain only one character.
EscapeChar
should be a string literal of length one, but got <sqlExpr>
.
SQLSTATE: F0000
Executor memory <executorMemory>
must be at least <minSystemMemory>
. Please increase executor memory using the --executor-memory option or “<config>
” in Spark configuration.
Can't extract a value from <base>
. Need a complex type [STRUCT, ARRAY, MAP] but got <other>
.
Cannot extract <field>
from <expr>
.
Field name should be a non-null string literal, but it's <extraction>
.
Field name <fieldName>
is invalid: <path>
is not a struct.
The format is invalid: <format>
.
For more details see INVALID_FORMAT
The fraction of sec must be zero. Valid range is [0, 60]. If necessary set <ansiConfig>
to “false” to bypass this error.
The handle <handle>
is invalid.
For more details see INVALID_HANDLE
Cannot create the table <tableName>
having the column <columnName>
whose name contains invalid characters <invalidChars>
in Hive metastore.
The identifier <ident>
is invalid. Please, consider quoting it with back-quotes as <ident>
.
The index 0 is invalid. An index shall be either < 0 or >
0 (the first element has index 1).
Invalid inline table.
For more details see INVALID_INLINE_TABLE
Cannot convert JSON root field to target Spark type.
Input schema <jsonSchema>
can only contain STRING as a key type for a MAP.
SQLSTATE: F0000
The value of the config “<bufferSizeConfKey>
” must be less than 2048 MiB, but got <bufferSizeConfValue>
MiB.
Invalid lambda function call.
For more details see INVALID_LAMBDA_FUNCTION_CALL
The <joinType>
JOIN with LATERAL correlation is not allowed because an OUTER subquery cannot correlate to its join partner. Remove the LATERAL correlation or use an INNER JOIN, or LEFT OUTER JOIN instead.
The limit like expression <expr>
is invalid.
For more details see INVALID_LIMIT_LIKE_EXPRESSION
The operator expects a deterministic expression, but the actual expression is <sqlExprs>
.
Numeric literal <rawStrippedQualifier>
is outside the valid range for <typeName>
with minimum value of <minValue>
and maximum value of <maxValue>
. Please adjust the value accordingly.
Invalid observed metrics.
For more details see INVALID_OBSERVED_METRICS
Invalid options:
For more details see INVALID_OPTIONS
The group aggregate pandas UDF <functionList>
cannot be invoked together with as other, non-pandas aggregate functions.
The value of parameter(s) <parameter>
in <functionName>
is invalid:
For more details see INVALID_PARAMETER_VALUE
The partition command is invalid.
For more details see INVALID_PARTITION_OPERATION
<key>
is an invalid property key, please use quotes, e.g. SET <key>
=<value>
.
<value>
is an invalid property value, please use quotes, e.g. SET <key>
=<value>
The input schema <inputSchema>
is not a valid schema string.
For more details see INVALID_SCHEMA
<name>
is not a valid name for tables/schemas. Valid names only contain alphabet characters, numbers and _.
Expected format is ‘SET’, ‘SET key’, or ‘SET key=value’. If you want to include special characters in key, or include semicolon in value, please use backquotes, e.g., SET key
=value
.
The argument <name>
of sql()
is invalid. Consider to replace it either by a SQL literal or by collection constructor functions such as map()
, array()
, struct()
.
Invalid SQL syntax:
For more details see INVALID_SQL_SYNTAX
Invalid subquery:
For more details see INVALID_SUBQUERY_EXPRESSION
Cannot create the persistent object <objName>
of the type <obj>
because it references to the temporary object <tempObjName>
of the type <tempObj>
. Please make the temporary object <tempObjName>
persistent, or make the persistent object <objName>
temporary.
The time travel timestamp expression <expr>
is invalid.
For more details see INVALID_TIME_TRAVEL_TIMESTAMP_EXPR
The value of the typed literal <valueType>
is invalid: <value>
.
Function <funcName>
does not implement a ScalarFunction or AggregateFunction.
The url is invalid: <url>
. If necessary set <ansiConfig>
to “false” to bypass this error.
Invalid usage of <elem>
in <prettyName>
.
The view <viewName>
cannot be displayed due to invalid view text: <viewText>
. This may be caused by an unauthorized modification of the view or an incorrect query syntax. Please check your query syntax and verify that the view has not been tampered with.
The WHERE condition <condition>
contains invalid expressions: <expressionList>
. Rewrite the query to avoid window functions, aggregate functions, and generator functions in the WHERE clause.
Cannot specify ORDER BY or a window frame for <aggFunc>
.
The requested write distribution is invalid.
For more details see INVALID_WRITE_DISTRIBUTION
The join condition <joinCondition>
has the invalid type <conditionType>
, expected “BOOLEAN”.
Kryo serialization failed: <exceptionMsg>
. To avoid this, increase “<bufferSizeConfKey>
” value.
LOAD DATA input path does not exist: <path>
.
LOCAL must be used together with the schema of file
, but got: <actualSchema>
.
Cannot name the managed table as <identifier>
, as its associated location <location>
already exists. Please pick a different table name, or remove the existing location first.
SQLSTATE: KD000
Malformed CSV record: <badRecord>
Malformed Protobuf messages are detected in message deserialization. Parse Mode: <failFastMode>
. To process malformed protobuf message as null result, try setting the option ‘mode’ as ‘PERMISSIVE’.
Malformed records are detected in record parsing: <badRecord>
. Parse Mode: <failFastMode>
. To process malformed records as null result, try setting the option ‘mode’ as ‘PERMISSIVE’.
For more details see MALFORMED_RECORD_IN_PARSING
The ON search condition of the MERGE statement matched a single row from the target table with multiple rows of the source table. This could result in the target row being operated on more than once with an update or delete operation and is not allowed.
The non-aggregating expression <expression>
is based on columns which are not participating in the GROUP BY clause. Add the columns or the expression to the GROUP BY, aggregate the expression, or use <expressionAnyValue>
if you do not care which of the values within a group is returned.
Resolved attribute(s) <missingAttributes>
missing from <input>
in operator <operator>
.
For more details see MISSING_ATTRIBUTES
The query does not include a GROUP BY clause. Add GROUP BY or turn it into the window functions using OVER clauses.
The expression <expr>
does not support more than one source.
Not allowed to implement multiple UDF interfaces, UDF class <className>
.
Named parameters are not supported for function <functionName>
; please retry the query with positional arguments to the function call instead.
Cannot call function <functionName>
because named argument references are not enabled here. In this case, the named argument reference was <argument>
. Set “spark.sql.allowNamedFunctionArguments” to “true” to turn on feature.
It is not allowed to use an aggregate function in the argument of another aggregate function. Please use the inner aggregate function in a sub-query.
The function <funcName>
requires the parameter <paramName>
to be a foldable expression of the type <paramType>
, but the actual argument is a non-foldable.
When there are more than one MATCHED clauses in a MERGE statement, only the last MATCHED clause can omit the condition.
When there are more than one NOT MATCHED BY SOURCE clauses in a MERGE statement, only the last NOT MATCHED BY SOURCE clause can omit the condition.
When there are more than one NOT MATCHED [BY TARGET] clauses in a MERGE statement, only the last NOT MATCHED [BY TARGET] clause can omit the condition.
Literal expressions required for pivot values, found <expression>
.
PARTITION clause cannot contain the non-partition column: <columnName>
.
Window function is not supported in <windowFunc>
(as column <columnName>
) on streaming DataFrames/Datasets. Structured Streaming only supports time-window aggregation using the WINDOW function. (window specification: <windowSpec>
)
Not allowed in the FROM clause:
For more details see NOT_ALLOWED_IN_FROM
The expression <expr>
used for the routine or clause <name>
must be a constant STRING which is NOT NULL.
For more details see NOT_A_CONSTANT_STRING
Operation <operation>
is not allowed for <tableIdentWithDB>
because it is not a partitioned table.
Assigning a NULL is not allowed here.
For more details see NOT_NULL_CONSTRAINT_VIOLATION
ALTER TABLE ALTER/CHANGE COLUMN is not supported for changing <table>
's column <originName>
with type <originType>
to <newName>
with type <newType>
.
<cmd>
is not supported for v2 tables.
<cmd>
is not supported, if you want to enable it, please set “spark.sql.catalogImplementation” to “hive”.
Not supported command in JDBC catalog:
For more details see NOT_SUPPORTED_IN_JDBC_CATALOG
Can't determine the default value for <colName>
since it is not nullable and it has no default value.
No handler for UDAF ‘<functionName>
’. Use sparkSession.udf.register(...) instead.
Cannot find <catalystFieldPath>
in Protobuf schema.
UDF class <className>
doesn't implement any UDF interface.
Column or field <name>
is nullable while it's required to be non-nullable.
Row ID attributes cannot be nullable: <nullableRowIdAttrs>
.
Cannot use null as map key.
The value <value>
cannot be interpreted as a numeric since it has more than 38 digits.
<value>
cannot be represented as Decimal(<precision>
, <scale>
). If necessary set <config>
to “false” to bypass this error, and return NULL instead.
<operator>
can only be performed on inputs with the same number of columns, but the first input has <firstNumColumns>
columns and the <invalidOrdinalNum>
input has <invalidNumColumns>
columns.
Number of given aliases does not match number of output columns. Function name: <funcName>
; number of aliases: <aliasesNum>
; number of output columns: <outColsNum>
.
Operation has been canceled.
ORDER BY position <index>
is not in select list (valid range is [1, <size>
]).
Syntax error, unexpected empty statement.
Syntax error at or near <error>``<hint>
.
Cannot ADD or RENAME TO partition(s) <partitionList>
in table <tableName>
because they already exist. Choose a different name, drop the existing partition, or add the IF NOT EXISTS clause to tolerate a pre-existing partition.
The partition(s) <partitionList>
cannot be found in table <tableName>
. Verify the partition specification and table name. To tolerate the error on drop use ALTER TABLE … DROP IF EXISTS PARTITION.
Path <outputPath>
already exists. Set mode as “overwrite” to overwrite the existing path.
Path does not exist: <path>
.
Invalid pivot value ‘<value>
’: value data type <valueType>
does not match pivot column data type <pivotType>
.
The input plan of <ruleExecutor>
is invalid: <reason>
Rule <rule>
in batch <batch>
generated an invalid plan: <reason>
Could not find dependency: <dependencyName>
.
Error reading Protobuf descriptor file at path: <filePath>
.
Searching for <field>
in Protobuf schema at <protobufSchema>
gave <matchSize>
matches. Candidates: <matches>
.
Found <field>
in Protobuf schema but there is no match in the SQL schema.
Type mismatch encountered for field: <field>
.
Unable to locate Message <messageName>
in Descriptor.
Protobuf type not yet supported: <protobufType>
.
Found recursive reference in Protobuf schema, which can not be processed by Spark by default: <fieldDescriptor>
. try setting the option recursive.fields.max.depth
0 to 10. Going beyond 10 levels of recursion is not allowed.
Recursive view <viewIdent>
detected (cycle: <newPath>
).
References to DEFAULT column values are not allowed within the PARTITION clause.
Failed to rename as <sourcePath>
was not found.
The <clause>
clause may be used at most once per <operation>
operation.
Cannot invoke function <functionName>
because the parameter named <parameterName>
is required, but the function call did not supply a value. Please update the function call to supply an argument value (either positionally at index <index>
or by name) and retry the query again.
<sessionCatalog>
requires a single-part namespace, but got <namespace>
.
Cannot create the function <routineName>
because it already exists. Choose a different name, drop or replace the existing function, or add the IF NOT EXISTS clause to tolerate a pre-existing function.
The function <routineName>
cannot be found. Verify the spelling and correctness of the schema and catalog. If you did not qualify the name with a schema and catalog, verify the current_schema() output, or qualify the name with the correct schema and catalog. To tolerate the error on drop use DROP FUNCTION IF EXISTS.
More than one row returned by a subquery used as a row.
Not found an id for the rule name “<ruleName>
”. Please modify RuleIdCollection.scala if you are adding a new rule.
The correlated scalar subquery ‘<sqlExpr>
’ is neither present in GROUP BY, nor in an aggregate function. Add it to GROUP BY using ordinal position or wrap it in first()
(or first_value
) if you don't care which value you get.
More than one row returned by a subquery used as an expression.
Cannot create schema <schemaName>
because it already exists. Choose a different name, drop the existing schema, or add the IF NOT EXISTS clause to tolerate pre-existing schema.
Cannot drop a schema <schemaName>
because it contains objects. Use DROP SCHEMA ... CASCADE to drop the schema and all its objects.
The schema <schemaName>
cannot be found. Verify the spelling and correctness of the schema and catalog. If you did not qualify the name with a catalog, verify the current_schema() output, or qualify the name with the correct catalog. To tolerate the error on drop use DROP SCHEMA IF EXISTS.
The second argument of <functionName>
function needs to be an integer.
The seed expression <seedExpr>
of the expression <exprWithSeed>
must be foldable.
sortBy must be used together with bucketBy.
A CREATE TABLE without explicit column list cannot specify bucketing information. Please use the form with explicit column list and specify bucketing information. Alternatively, allow bucketing information to be inferred by omitting the clause.
A CREATE TABLE without explicit column list cannot specify PARTITIONED BY. Please use the form with explicit column list and specify PARTITIONED BY. Alternatively, allow partitioning to be inferred by omitting the PARTITION BY clause.
The SQL config <sqlConf>
cannot be found. Please verify that the config exists.
Star (*) is not allowed in a select list when GROUP BY an ordinal position is used.
Static partition column <staticName>
is also specified in the column list.
Query [id = <id>
, runId = <runId>
] terminated with exception: <message>
The sum of the LIMIT clause and the OFFSET clause must not be greater than the maximum 32-bit integer value (2,147,483,647) but found limit = <limit>
, offset = <offset>
.
Cannot create table or view <relationName>
because it already exists. Choose a different name, drop or replace the existing object, or add the IF NOT EXISTS clause to tolerate pre-existing objects.
The table or view <relationName>
cannot be found. Verify the spelling and correctness of the schema and catalog. If you did not qualify the name with a schema, verify the current_schema() output, or qualify the name with the correct schema and catalog. To tolerate the error on drop use DROP VIEW IF EXISTS or DROP TABLE IF EXISTS.
Failed to analyze the Python user defined table function: <msg>
Failed to evaluate the table function <functionName>
because its table metadata <requestedMetadata>
, but the function call <invalidFunctionCallProperty>
.
Failed to evaluate the table function <functionName>
because its table metadata was invalid; <reason>
.
There are too many table arguments for table-valued function. It allows one table argument, but got: <num>
. If you want to allow it, please set “spark.sql.allowMultipleTableArguments.enabled” to “true”
SQLSTATE: 58030
Task failed while writing rows to <path>
.
Cannot create the temporary view <relationName>
because it already exists. Choose a different name, drop or replace the existing view, or add the IF NOT EXISTS clause to tolerate pre-existing views.
CREATE TEMPORARY VIEW or the corresponding Dataset APIs only accept single-part view names, but got: <actualName>
.
Cannot initialize array with <numElements>
elements of size <size>
.
The number of aliases supplied in the AS clause does not match the number of columns output by the UDTF. Expected <aliasesSize>
aliases, but got <aliasesNames>
. Please ensure that the number of aliases provided matches the number of columns output by the UDTF.
Unable to acquire <requestedBytes>
bytes of memory, got <receivedBytes>
.
Unable to convert SQL type <toType>
to Protobuf type <protobufType>
.
SQLSTATE: 58030
Unable to fetch tables of Hive database: <dbName>
.
Unable to infer schema for <format>
. It must be specified manually.
Found the unbound parameter: <name>
. Please, fix args
and provide a mapping of the parameter to either a SQL literal or collection constructor functions such as map()
, array()
, struct()
.
Found an unclosed bracketed comment. Please, append */ at the end of the comment.
Parameter <paramIndex>
of function <functionName>
requires the <requiredType>
type, however <inputSql>
has the type <inputType>
.
Cannot invoke function <functionName>
because it contains positional argument(s) following the named argument assigned to <parameterName>
; please rearrange them so the positional arguments come first and then retry the query again.
The class <className>
has an unexpected expression serializer. Expects “STRUCT” or “IF” which returns “STRUCT” but found <expr>
.
Attempting to treat <descriptorName>
as a Message, but it was <containingType>
.
UNPIVOT requires all given <given>
expressions to be columns when no <empty>
expressions are given. These are not columns: [<expressions>
].
At least one value column needs to be specified for UNPIVOT, all columns specified as ids.
Unpivot value columns must share a least common type, some types do not: [<types>
].
All unpivot value columns must have the same size as there are value column names (<names>
).
Cannot invoke function <functionName>
because the function call included a named argument reference for the argument named <argumentName>
, but this function does not include any signature containing an argument with this name. Did you mean one of the following? [<proposal>
].
Unrecognized SQL type - name: <typeName>
, id: <jdbcType>
.
Could not resolve <name>
to a table-valued function. Please make sure that <name>
is defined as a table-valued function and that all required parameters are provided correctly. If <name>
is not defined, please create the table-valued function before using it. For more information about defining table-valued functions, please refer to the Apache Spark documentation.
Cannot infer grouping columns for GROUP BY ALL based on the select clause. Please explicitly specify the grouping columns.
A column, variable, or function parameter with name <objectName>
cannot be resolved.
For more details see UNRESOLVED_COLUMN
A field with name <fieldName>
cannot be resolved with the struct-type column <columnPath>
.
For more details see UNRESOLVED_FIELD
Cannot resolve column <objectName>
as a map key. If the key is a string literal, add the single quotes '' around it.
For more details see UNRESOLVED_MAP_KEY
Cannot resolve function <routineName>
on search path <searchPath>
.
USING column <colName>
cannot be resolved on the <side>
side of the join. The <side>
-side columns: [<suggestion>
].
Cannot resolve variable <variableName>
on search path <searchPath>
.
Attempted to unset non-existent properties [<properties>
] in table <table>
.
Don't support add file.
For more details see UNSUPPORTED_ADD_FILE
Unsupported arrow type <typeName>
.
The char/varchar type can't be used in the table schema. If you want Spark treat them as string type as same as Spark 3.0 and earlier, please set “spark.sql.legacy.charVarcharAsString” to “true”.
Unsupported data source type for direct query on files: <dataSourceType>
Unsupported data type <typeName>
.
The <format>
datasource doesn't support the column <columnName>
of the type <columnType>
.
DEFAULT column values is not supported.
For more details see UNSUPPORTED_DEFAULT_VALUE
The deserializer is not supported:
For more details see UNSUPPORTED_DESERIALIZER
Cannot create generated column <fieldName>
with generation expression <expressionStr>
because <reason>
.
A query operator contains one or more unsupported expressions. Consider to rewrite it to avoid window functions, aggregate functions, and generator functions in the WHERE clause. Invalid expressions: [<invalidExprSqls>
]
Expression <sqlExpr>
not supported within a window function.
The feature is not supported:
For more details see UNSUPPORTED_FEATURE
The generator is not supported:
For more details see UNSUPPORTED_GENERATOR
grouping()/grouping_id() can only be used with GroupingSets/Cube/Rollup.
Can't insert into the target.
For more details see UNSUPPORTED_INSERT
MERGE operation contains unsupported <condName>
condition.
For more details see UNSUPPORTED_MERGE_CONDITION
Can't overwrite the target that is also being read from.
For more details see UNSUPPORTED_OVERWRITE
The save mode <saveMode>
is not supported for:
For more details see UNSUPPORTED_SAVE_MODE
Unsupported subquery expression:
For more details see UNSUPPORTED_SUBQUERY_EXPRESSION_CATEGORY
Literals of the type <unsupportedType>
are not supported. Supported types are <supportedTypes>
.
You're using untyped Scala UDF, which does not have the input type information. Spark may blindly pass null to the Scala closure with primitive-type argument, and the closure will see the default value of the Java type for the null argument, e.g. udf((x: Int) => x, IntegerType)
, the result is 0 for null input. To get rid of this error, you could:
udf((x: Int) => x)
.udf(new UDF1[String, Integer] { override def call(s: String): Integer = s.length() }, IntegerType)
, if input types are all non primitive.SQLSTATE: P0001
<errorMessage>
SQLSTATE: P0001
The raise_error()
function was used to raise error class: <errorClass>
which expects parameters: <expectedParms>
. The provided parameters <providedParms>
do not match the expected parameters. Please make sure to provide all expected parameters.
SQLSTATE: P0001
The raise_error()
function was used to raise an unknown error class: <errorClass>
Cannot create the variable <variableName>
because it already exists. Choose a different name, or drop or replace the existing variable.
The variable <variableName>
cannot be found. Verify the spelling and correctness of the schema and catalog. If you did not qualify the name with a schema and catalog, verify the current_schema() output, or qualify the name with the correct schema and catalog. To tolerate the error on drop use DROP VARIABLE IF EXISTS.
Cannot create view <relationName>
because it already exists. Choose a different name, drop or replace the existing object, or add the IF NOT EXISTS clause to tolerate pre-existing objects.
The depth of view <viewName>
exceeds the maximum view resolution depth (<maxNestedDepth>
). Analysis is aborted to avoid errors. If you want to work around this, please try to increase the value of “spark.sql.view.maxNestedViewDepth”.
The view <relationName>
cannot be found. Verify the spelling and correctness of the schema and catalog. If you did not qualify the name with a schema, verify the current_schema() output, or qualify the name with the correct schema and catalog. To tolerate the error on drop use DROP VIEW IF EXISTS.
<funcName>
function can only be evaluated in an ordered row-based window frame with a single offset: <windowExpr>
.
Window function <funcName>
requires an OVER clause.
writeStream
can be called only on streaming Dataset/DataFrame.
The operation <operation>
requires a <requiredType>
. But <objectName>
is a <foundType>
. Use <alternative>
instead.
The <functionName>
requires <expectedNum>
parameters but the actual number is <actualNum>
.
For more details see WRONG_NUM_ARGS