Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -89,6 +89,8 @@ statement
SET TBLPROPERTIES tablePropertyList #setTableProperties
| ALTER (TABLE | VIEW) tableIdentifier
UNSET TBLPROPERTIES (IF EXISTS)? tablePropertyList #unsetTableProperties
| ALTER TABLE tableIdentifier partitionSpec?
CHANGE COLUMN? identifier colType colPosition? #changeColumn
| ALTER TABLE tableIdentifier (partitionSpec)?
SET SERDE STRING (WITH SERDEPROPERTIES tablePropertyList)? #setTableSerDe
| ALTER TABLE tableIdentifier (partitionSpec)?
Expand Down Expand Up @@ -192,7 +194,6 @@ unsupportedHiveNativeCommands
| kw1=ALTER kw2=TABLE tableIdentifier partitionSpec? kw3=CONCATENATE
| kw1=ALTER kw2=TABLE tableIdentifier partitionSpec? kw3=SET kw4=FILEFORMAT
| kw1=ALTER kw2=TABLE tableIdentifier partitionSpec? kw3=ADD kw4=COLUMNS
| kw1=ALTER kw2=TABLE tableIdentifier partitionSpec? kw3=CHANGE kw4=COLUMN?
| kw1=ALTER kw2=TABLE tableIdentifier partitionSpec? kw3=REPLACE kw4=COLUMNS
| kw1=START kw2=TRANSACTION
| kw1=COMMIT
Expand Down Expand Up @@ -578,6 +579,10 @@ intervalValue
| STRING
;

colPosition
: FIRST | AFTER identifier
;

dataType
: complex=ARRAY '<' dataType '>' #complexDataType
| complex=MAP '<' dataType ',' dataType '>' #complexDataType
Expand Down Expand Up @@ -669,7 +674,7 @@ number
nonReserved
: SHOW | TABLES | COLUMNS | COLUMN | PARTITIONS | FUNCTIONS | DATABASES
| ADD
| OVER | PARTITION | RANGE | ROWS | PRECEDING | FOLLOWING | CURRENT | ROW | LAST | FIRST
| OVER | PARTITION | RANGE | ROWS | PRECEDING | FOLLOWING | CURRENT | ROW | LAST | FIRST | AFTER
| MAP | ARRAY | STRUCT
| LATERAL | WINDOW | REDUCE | TRANSFORM | USING | SERDE | SERDEPROPERTIES | RECORDREADER
| DELIMITED | FIELDS | TERMINATED | COLLECTION | ITEMS | KEYS | ESCAPED | LINES | SEPARATED
Expand Down Expand Up @@ -759,6 +764,7 @@ PRECEDING: 'PRECEDING';
FOLLOWING: 'FOLLOWING';
CURRENT: 'CURRENT';
FIRST: 'FIRST';
AFTER: 'AFTER';
LAST: 'LAST';
ROW: 'ROW';
WITH: 'WITH';
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -31,7 +31,7 @@ import org.apache.spark.sql.catalyst.plans.logical.{LogicalPlan, OneRowRelation,
import org.apache.spark.sql.execution.command._
import org.apache.spark.sql.execution.datasources.{CreateTable, _}
import org.apache.spark.sql.internal.{HiveSerDe, SQLConf, VariableSubstitution}
import org.apache.spark.sql.types.StructType
import org.apache.spark.sql.types.{StructField, StructType}

/**
* Concrete parser for Spark SQL statements.
Expand Down Expand Up @@ -876,6 +876,33 @@ class SparkSqlAstBuilder(conf: SQLConf) extends AstBuilder {
visitLocationSpec(ctx.locationSpec))
}

/**
* Create a [[AlterTableChangeColumnCommand]] command.
*
* For example:
* {{{
* ALTER TABLE table [PARTITION partition_spec]
* CHANGE [COLUMN] column_old_name column_new_name column_dataType [COMMENT column_comment]
* [FIRST | AFTER column_name];
* }}}
*/
override def visitChangeColumn(ctx: ChangeColumnContext): LogicalPlan = withOrigin(ctx) {
if (ctx.partitionSpec != null) {
operationNotAllowed("ALTER TABLE table PARTITION partition_spec CHANGE COLUMN", ctx)
}

if (ctx.colPosition != null) {
operationNotAllowed(
"ALTER TABLE table [PARTITION partition_spec] CHANGE COLUMN ... FIRST | AFTER otherCol",
ctx)
}

AlterTableChangeColumnCommand(
tableName = visitTableIdentifier(ctx.tableIdentifier),
columnName = ctx.identifier.getText,
newColumn = visitColType(ctx.colType))
}

/**
* Create location string.
*/
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -274,6 +274,77 @@ case class AlterTableUnsetPropertiesCommand(

}


/**
* A command to change the column for a table, only support changing the comment of a non-partition
* column for now.
*
* The syntax of using this command in SQL is:
* {{{
* ALTER TABLE table_identifier
* CHANGE [COLUMN] column_old_name column_new_name column_dataType [COMMENT column_comment]
* [FIRST | AFTER column_name];
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This is the right Hive syntax, but SparkSqlParser.scala is using the different one.

* }}}
*/
case class AlterTableChangeColumnCommand(
tableName: TableIdentifier,
columnName: String,
newColumn: StructField) extends RunnableCommand {

// TODO: support change column name/dataType/metadata/position.
override def run(sparkSession: SparkSession): Seq[Row] = {
val catalog = sparkSession.sessionState.catalog
val table = catalog.getTableMetadata(tableName)
val resolver = sparkSession.sessionState.conf.resolver
DDLUtils.verifyAlterTableType(catalog, table, isView = false)

// Find the origin column from schema by column name.
val originColumn = findColumnByName(table.schema, columnName, resolver)
// Throw an AnalysisException if the column name/dataType is changed.
if (!columnEqual(originColumn, newColumn, resolver)) {
throw new AnalysisException(
"ALTER TABLE CHANGE COLUMN is not supported for changing column " +
s"'${originColumn.name}' with type '${originColumn.dataType}' to " +
s"'${newColumn.name}' with type '${newColumn.dataType}'")
}

val newSchema = table.schema.fields.map { field =>
if (field.name == originColumn.name) {
// Create a new column from the origin column with the new comment.
addComment(field, newColumn.getComment)
} else {
field
}
}
val newTable = table.copy(schema = StructType(newSchema))
catalog.alterTable(newTable)

Seq.empty[Row]
}

// Find the origin column from schema by column name, throw an AnalysisException if the column
// reference is invalid.
private def findColumnByName(
schema: StructType, name: String, resolver: Resolver): StructField = {
schema.fields.collectFirst {
case field if resolver(field.name, name) => field
}.getOrElse(throw new AnalysisException(
s"Invalid column reference '$name', table schema is '${schema}'"))
}

// Add the comment to a column, if comment is empty, return the original column.
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

if the comment is empty, shall we remove the existing comment? what's the behavior of hive?

Copy link
Contributor Author

@jiangxb1987 jiangxb1987 Dec 12, 2016

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Hive won't change the original comment if the COMMENT statement is empty. We keep the same with this behavior.

hive> desc t;
OK
a                   	string              	this is a           
Time taken: 7.811 seconds, Fetched: 1 row(s)
hive> alter table t change column a b string;
OK
Time taken: 10.781 seconds
hive> desc t;
OK
b                   	string              	this is a           
Time taken: 7.97 seconds, Fetched: 1 row(s)

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

as we only check name and data type, how about ALTER TABLE CHANGE COLUMN is not supported for changing column abc with type int to xyz with type long?

Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

To drop a comment, I think users can do something like

ALTER TABLE test_change CHANGE a A INT COMMENT ''

private def addComment(column: StructField, comment: Option[String]): StructField = {
comment.map(column.withComment(_)).getOrElse(column)
}

// Compare a [[StructField]] to another, return true if they have the same column
// name(by resolver) and dataType.
private def columnEqual(
field: StructField, other: StructField, resolver: Resolver): Boolean = {
resolver(field.name, other.name) && field.dataType == other.dataType
}
}

/**
* A command that sets the serde class and/or serde properties of a table/view.
*
Expand Down
55 changes: 55 additions & 0 deletions sql/core/src/test/resources/sql-tests/inputs/change-column.sql
Original file line number Diff line number Diff line change
@@ -0,0 +1,55 @@
-- Create the origin table
CREATE TABLE test_change(a INT, b STRING, c INT);
DESC test_change;

-- Change column name (not supported yet)
ALTER TABLE test_change CHANGE a a1 INT;
DESC test_change;

-- Change column dataType (not supported yet)
ALTER TABLE test_change CHANGE a a STRING;
DESC test_change;

-- Change column position (not supported yet)
ALTER TABLE test_change CHANGE a a INT AFTER b;
ALTER TABLE test_change CHANGE b b STRING FIRST;
DESC test_change;

-- Change column comment
ALTER TABLE test_change CHANGE a a INT COMMENT 'this is column a';
ALTER TABLE test_change CHANGE b b STRING COMMENT '#*02?`';
ALTER TABLE test_change CHANGE c c INT COMMENT '';
DESC test_change;

-- Don't change anything.
ALTER TABLE test_change CHANGE a a INT COMMENT 'this is column a';
DESC test_change;

-- Change a invalid column
ALTER TABLE test_change CHANGE invalid_col invalid_col INT;
DESC test_change;

-- Change column name/dataType/position/comment together (not supported yet)
ALTER TABLE test_change CHANGE a a1 STRING COMMENT 'this is column a1' AFTER b;
DESC test_change;

-- Check the behavior with different values of CASE_SENSITIVE
SET spark.sql.caseSensitive=false;
ALTER TABLE test_change CHANGE a A INT COMMENT 'this is column A';
SET spark.sql.caseSensitive=true;
ALTER TABLE test_change CHANGE a A INT COMMENT 'this is column A1';
DESC test_change;

-- Change column can't apply to a temporary/global_temporary view
CREATE TEMPORARY VIEW temp_view(a, b) AS SELECT 1, "one";
ALTER TABLE temp_view CHANGE a a INT COMMENT 'this is column a';
CREATE GLOBAL TEMPORARY VIEW global_temp_view(a, b) AS SELECT 1, "one";
ALTER TABLE global_temp.global_temp_view CHANGE a a INT COMMENT 'this is column a';

-- Change column in partition spec (not supported yet)
CREATE TABLE partition_table(a INT, b STRING) PARTITIONED BY (c INT, d STRING);
ALTER TABLE partition_table PARTITION (c = 1) CHANGE COLUMN a new_a INT;

-- DROP TEST TABLE
DROP TABLE test_change;
DROP TABLE partition_table;
Loading