Skip to content
Closed
Show file tree
Hide file tree
Changes from 3 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 @@ -82,7 +82,7 @@ statement
(TBLPROPERTIES tablePropertyList)?
(AS? query)? #createHiveTable
| CREATE TABLE (IF NOT EXISTS)? target=tableIdentifier
LIKE source=tableIdentifier #createTableLike
LIKE source=tableIdentifier locationSpec? #createTableLike
| ANALYZE TABLE tableIdentifier partitionSpec? COMPUTE STATISTICS
(identifier | FOR COLUMNS identifierSeq)? #analyze
| ALTER (TABLE | VIEW) from=tableIdentifier
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1141,13 +1141,14 @@ class SparkSqlAstBuilder(conf: SQLConf) extends AstBuilder {
* For example:
* {{{
* CREATE TABLE [IF NOT EXISTS] [db_name.]table_name
* LIKE [other_db_name.]existing_table_name
* LIKE [other_db_name.]existing_table_name [locationSpec]
* }}}
*/
override def visitCreateTableLike(ctx: CreateTableLikeContext): LogicalPlan = withOrigin(ctx) {
val targetTable = visitTableIdentifier(ctx.target)
val sourceTable = visitTableIdentifier(ctx.source)
CreateTableLikeCommand(targetTable, sourceTable, ctx.EXISTS != null)
val location = Option(ctx.locationSpec).map(visitLocationSpec)
CreateTableLikeCommand(targetTable, sourceTable, location, ctx.EXISTS != null)
}

/**
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -42,7 +42,7 @@ import org.apache.spark.sql.types._
import org.apache.spark.util.Utils

/**
* A command to create a MANAGED table with the same definition of the given existing table.
* A command to create a table with the same definition of the given existing table.
* In the target table definition, the table comment is always empty but the column comments
* are identical to the ones defined in the source table.
*
Expand All @@ -52,12 +52,13 @@ import org.apache.spark.util.Utils
* The syntax of using this command in SQL is:
* {{{
* CREATE TABLE [IF NOT EXISTS] [db_name.]table_name
* LIKE [other_db_name.]existing_table_name
* LIKE [other_db_name.]existing_table_name [locationSpec]
* }}}
*/
case class CreateTableLikeCommand(
targetTable: TableIdentifier,
sourceTable: TableIdentifier,
location: Option[String],
ifNotExists: Boolean) extends RunnableCommand {

override def run(sparkSession: SparkSession): Seq[Row] = {
Expand All @@ -70,12 +71,19 @@ case class CreateTableLikeCommand(
sourceTableDesc.provider
}

// If location is specified, we create an external table internally.
Copy link
Member

Choose a reason for hiding this comment

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

-> If the

// Else create managed table.
Copy link
Member

Choose a reason for hiding this comment

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

Otherwise, create a

val tblType = if (location.isEmpty) {
CatalogTableType.MANAGED
} else {
CatalogTableType.EXTERNAL
}
Copy link
Member

Choose a reason for hiding this comment

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

Shorten it to one line?

  val tblType = if (location.isEmpty) CatalogTableType.MANAGED else CatalogTableType.EXTERNAL


val newTableDesc =
CatalogTable(
identifier = targetTable,
tableType = CatalogTableType.MANAGED,
// We are creating a new managed table, which should not have custom table location.
storage = sourceTableDesc.storage.copy(locationUri = None),
tableType = tblType,
storage = sourceTableDesc.storage.copy(locationUri = location),
schema = sourceTableDesc.schema,
provider = newProvider,
partitionColumnNames = sourceTableDesc.partitionColumnNames,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -524,24 +524,48 @@ class HiveDDLCommandSuite extends PlanTest with SQLTestUtils with TestHiveSingle

test("create table like") {
val v1 = "CREATE TABLE table1 LIKE table2"
val (target, source, exists) = parser.parsePlan(v1).collect {
case CreateTableLikeCommand(t, s, allowExisting) => (t, s, allowExisting)
val (target, source, location, exists) = parser.parsePlan(v1).collect {
case CreateTableLikeCommand(t, s, l, allowExisting) => (t, s, l, allowExisting)
}.head
assert(exists == false)
assert(target.database.isEmpty)
assert(target.table == "table1")
assert(source.database.isEmpty)
assert(source.table == "table2")
assert(location.isEmpty)

val v2 = "CREATE TABLE IF NOT EXISTS table1 LIKE table2"
val (target2, source2, exists2) = parser.parsePlan(v2).collect {
case CreateTableLikeCommand(t, s, allowExisting) => (t, s, allowExisting)
val (target2, source2, location2, exists2) = parser.parsePlan(v2).collect {
case CreateTableLikeCommand(t, s, l, allowExisting) => (t, s, l, allowExisting)
}.head
assert(exists2)
assert(target2.database.isEmpty)
assert(target2.table == "table1")
assert(source2.database.isEmpty)
assert(source2.table == "table2")
assert(location2.isEmpty)

val v3 = "CREATE TABLE table1 LIKE table2 LOCATION '/spark/warehouse'"
val (target3, source3, location3, exists3) = parser.parsePlan(v3).collect {
case CreateTableLikeCommand(t, s, l, allowExisting) => (t, s, l, allowExisting)
}.head
assert(!exists3)
assert(target3.database.isEmpty)
assert(target3.table == "table1")
assert(source3.database.isEmpty)
assert(source3.table == "table2")
assert(location3 == Some("/spark/warehouse"))

val v4 = "CREATE TABLE IF NOT EXISTS table1 LIKE table2 LOCATION '/spark/warehouse'"
val (target4, source4, location4, exists4) = parser.parsePlan(v4).collect {
case CreateTableLikeCommand(t, s, l, allowExisting) => (t, s, l, allowExisting)
}.head
assert(exists4)
assert(target4.database.isEmpty)
assert(target4.table == "table1")
assert(source4.database.isEmpty)
assert(source4.table == "table2")
assert(location4 == Some("/spark/warehouse"))
}

test("load data") {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -833,54 +833,95 @@ class HiveDDLSuite
}

test("CREATE TABLE LIKE a temporary view") {
// CREATE TABLE LIKE a temporary view.
withCreateTableLikeTempView(None)
Copy link
Member

Choose a reason for hiding this comment

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

withCreateTableLikeTempView(None) -> withCreateTableLikeTempView(location = None)


// CREATE TABLE LIKE a temporary view location ...
withTempDir {tmpDir =>
Copy link
Member

Choose a reason for hiding this comment

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

{tmpDir -> { tmpDir

withCreateTableLikeTempView(Some(tmpDir.toURI.toString))
}
}

private def withCreateTableLikeTempView(location : Option[String]): Unit = {
val sourceViewName = "tab1"
val targetTabName = "tab2"
val tableType = if (location.isDefined) CatalogTableType.EXTERNAL else CatalogTableType.MANAGED
withTempView(sourceViewName) {
withTable(targetTabName) {
spark.range(10).select('id as 'a, 'id as 'b, 'id as 'c, 'id as 'd)
.createTempView(sourceViewName)
sql(s"CREATE TABLE $targetTabName LIKE $sourceViewName")

val locationClause = if (location.nonEmpty) s"LOCATION '${location.getOrElse("")}'" else ""
sql(s"CREATE TABLE $targetTabName LIKE $sourceViewName $locationClause")

val sourceTable = spark.sessionState.catalog.getTempViewOrPermanentTableMetadata(
TableIdentifier(sourceViewName))
val targetTable = spark.sessionState.catalog.getTableMetadata(
TableIdentifier(targetTabName, Some("default")))

checkCreateTableLike(sourceTable, targetTable)
checkCreateTableLike(sourceTable, targetTable, tableType)
}
}
}

test("CREATE TABLE LIKE a data source table") {
// CREATE TABLE LIKE a data source table.
withCreateTableLikeDSTable(None)
Copy link
Member

Choose a reason for hiding this comment

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

The same here.


// CREATE TABLE LIKE a data source table location ...
withTempDir { tmpDir =>
withCreateTableLikeDSTable(Some(tmpDir.toURI.toString))
}
}

private def withCreateTableLikeDSTable(location : Option[String]): Unit = {
val sourceTabName = "tab1"
val targetTabName = "tab2"
val tableType = if (location.isDefined) CatalogTableType.EXTERNAL else CatalogTableType.MANAGED
withTable(sourceTabName, targetTabName) {
spark.range(10).select('id as 'a, 'id as 'b, 'id as 'c, 'id as 'd)
.write.format("json").saveAsTable(sourceTabName)
sql(s"CREATE TABLE $targetTabName LIKE $sourceTabName")

val locationClause = if (location.nonEmpty) s"LOCATION '${location.getOrElse("")}'" else ""
sql(s"CREATE TABLE $targetTabName LIKE $sourceTabName $locationClause")

val sourceTable =
spark.sessionState.catalog.getTableMetadata(TableIdentifier(sourceTabName, Some("default")))
spark.sessionState.catalog.getTableMetadata(
TableIdentifier(sourceTabName, Some("default")))
val targetTable =
spark.sessionState.catalog.getTableMetadata(TableIdentifier(targetTabName, Some("default")))
spark.sessionState.catalog.getTableMetadata(
TableIdentifier(targetTabName, Some("default")))
// The table type of the source table should be a Hive-managed data source table
assert(DDLUtils.isDatasourceTable(sourceTable))
assert(sourceTable.tableType == CatalogTableType.MANAGED)

checkCreateTableLike(sourceTable, targetTable)
checkCreateTableLike(sourceTable, targetTable, tableType)
}
}

test("CREATE TABLE LIKE an external data source table") {
// CREATE TABLE LIKE an external data source table.
withCreateTableLikeExtDSTable(None)
Copy link
Member

Choose a reason for hiding this comment

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

The same here


// CREATE TABLE LIKE an external data source table location ...
withTempDir { tmpDir =>
withCreateTableLikeExtDSTable(Some(tmpDir.toURI.toString))
}
}

private def withCreateTableLikeExtDSTable(location : Option[String]): Unit = {
val sourceTabName = "tab1"
val targetTabName = "tab2"
val tableType = if (location.isDefined) CatalogTableType.EXTERNAL else CatalogTableType.MANAGED
withTable(sourceTabName, targetTabName) {
withTempPath { dir =>
val path = dir.getCanonicalPath
spark.range(10).select('id as 'a, 'id as 'b, 'id as 'c, 'id as 'd)
.write.format("parquet").save(path)
sql(s"CREATE TABLE $sourceTabName USING parquet OPTIONS (PATH '${dir.toURI}')")
sql(s"CREATE TABLE $targetTabName LIKE $sourceTabName")

val locationClause = if (location.nonEmpty) s"LOCATION '${location.getOrElse("")}'" else ""
sql(s"CREATE TABLE $targetTabName LIKE $sourceTabName $locationClause")

// The source table should be an external data source table
val sourceTable = spark.sessionState.catalog.getTableMetadata(
Expand All @@ -891,32 +932,58 @@ class HiveDDLSuite
assert(DDLUtils.isDatasourceTable(sourceTable))
assert(sourceTable.tableType == CatalogTableType.EXTERNAL)

checkCreateTableLike(sourceTable, targetTable)
checkCreateTableLike(sourceTable, targetTable, tableType)
}
}
}

test("CREATE TABLE LIKE a managed Hive serde table") {
val catalog = spark.sessionState.catalog
// CREATE TABLE LIKE a managed Hive serde table.
withCreateTableLikeManagedHiveTable(None)

// CREATE TABLE LIKE a managed Hive serde table location ...
withTempDir { tmpDir =>
withCreateTableLikeManagedHiveTable(Some(tmpDir.toURI.toString))
}
}

private def withCreateTableLikeManagedHiveTable(location : Option[String]): Unit = {
val sourceTabName = "tab1"
val targetTabName = "tab2"
val tableType = if (location.isDefined) CatalogTableType.EXTERNAL else CatalogTableType.MANAGED
val catalog = spark.sessionState.catalog
withTable(sourceTabName, targetTabName) {
sql(s"CREATE TABLE $sourceTabName TBLPROPERTIES('prop1'='value1') AS SELECT 1 key, 'a'")
sql(s"CREATE TABLE $targetTabName LIKE $sourceTabName")

val sourceTable = catalog.getTableMetadata(TableIdentifier(sourceTabName, Some("default")))
val locationClause = if (location.nonEmpty) s"LOCATION '${location.getOrElse("")}'" else ""
sql(s"CREATE TABLE $targetTabName LIKE $sourceTabName $locationClause")

val sourceTable = catalog.getTableMetadata(
TableIdentifier(sourceTabName, Some("default")))
assert(sourceTable.tableType == CatalogTableType.MANAGED)
assert(sourceTable.properties.get("prop1").nonEmpty)
val targetTable = catalog.getTableMetadata(TableIdentifier(targetTabName, Some("default")))
val targetTable = catalog.getTableMetadata(
TableIdentifier(targetTabName, Some("default")))

checkCreateTableLike(sourceTable, targetTable)
checkCreateTableLike(sourceTable, targetTable, tableType)
}
}

test("CREATE TABLE LIKE an external Hive serde table") {
// CREATE TABLE LIKE an external Hive serde table.
withCreateTableLikeExtHiveTable(None)

// CREATE TABLE LIKE an external Hive serde table location ...
withTempDir { tmpDir =>
withCreateTableLikeExtHiveTable(Some(tmpDir.toURI.toString))
}
}

private def withCreateTableLikeExtHiveTable(location : Option[String]): Unit = {
val catalog = spark.sessionState.catalog
val tableType = if (location.isDefined) CatalogTableType.EXTERNAL else CatalogTableType.MANAGED
withTempDir { tmpDir =>
val basePath = tmpDir.toURI
val basePath1 = tmpDir.toURI
Copy link
Member

Choose a reason for hiding this comment

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

revert it back?

val sourceTabName = "tab1"
val targetTabName = "tab2"
withTable(sourceTabName, targetTabName) {
Expand All @@ -926,38 +993,55 @@ class HiveDDLSuite
|CREATE EXTERNAL TABLE $sourceTabName (key INT comment 'test', value STRING)
|COMMENT 'Apache Spark'
|PARTITIONED BY (ds STRING, hr STRING)
|LOCATION '$basePath'
""".stripMargin)
|LOCATION '$basePath1'
""".stripMargin)
Copy link
Member

Choose a reason for hiding this comment

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

revert it back

for (ds <- Seq("2008-04-08", "2008-04-09"); hr <- Seq("11", "12")) {
sql(
s"""
|INSERT OVERWRITE TABLE $sourceTabName
|partition (ds='$ds',hr='$hr')
|SELECT 1, 'a'
""".stripMargin)
""".stripMargin)
Copy link
Member

Choose a reason for hiding this comment

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

revert it back

}
sql(s"CREATE TABLE $targetTabName LIKE $sourceTabName")

val sourceTable = catalog.getTableMetadata(TableIdentifier(sourceTabName, Some("default")))
val locationClause = if (location.nonEmpty) s"LOCATION '${location.getOrElse("")}'" else ""
sql(s"CREATE TABLE $targetTabName LIKE $sourceTabName $locationClause")

val sourceTable = catalog.getTableMetadata(
TableIdentifier(sourceTabName, Some("default")))
assert(sourceTable.tableType == CatalogTableType.EXTERNAL)
assert(sourceTable.comment == Option("Apache Spark"))
val targetTable = catalog.getTableMetadata(TableIdentifier(targetTabName, Some("default")))
val targetTable = catalog.getTableMetadata(
TableIdentifier(targetTabName, Some("default")))

checkCreateTableLike(sourceTable, targetTable)
checkCreateTableLike(sourceTable, targetTable, tableType)
}
}
}

test("CREATE TABLE LIKE a view") {
// CREATE TABLE LIKE a view.
withCreateTableLikeView(None)
Copy link
Member

Choose a reason for hiding this comment

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

The same here.


// CREATE TABLE LIKE a view location ...
withTempDir { tmpDir =>
withCreateTableLikeView(Some(tmpDir.toURI.toString))
}
}

private def withCreateTableLikeView(location : Option[String]): Unit = {
val sourceTabName = "tab1"
val sourceViewName = "view"
val targetTabName = "tab2"
val tableType = if (location.isDefined) CatalogTableType.EXTERNAL else CatalogTableType.MANAGED
withTable(sourceTabName, targetTabName) {
withView(sourceViewName) {
spark.range(10).select('id as 'a, 'id as 'b, 'id as 'c, 'id as 'd)
.write.format("json").saveAsTable(sourceTabName)
sql(s"CREATE VIEW $sourceViewName AS SELECT * FROM $sourceTabName")
sql(s"CREATE TABLE $targetTabName LIKE $sourceViewName")

val locationClause = if (location.nonEmpty) s"LOCATION '${location.getOrElse("")}'" else ""
sql(s"CREATE TABLE $targetTabName LIKE $sourceViewName $locationClause")

val sourceView = spark.sessionState.catalog.getTableMetadata(
TableIdentifier(sourceViewName, Some("default")))
Expand All @@ -969,15 +1053,19 @@ class HiveDDLSuite
val targetTable = spark.sessionState.catalog.getTableMetadata(
TableIdentifier(targetTabName, Some("default")))

checkCreateTableLike(sourceView, targetTable)
checkCreateTableLike(sourceView, targetTable, tableType)
}
}
}

private def checkCreateTableLike(sourceTable: CatalogTable, targetTable: CatalogTable): Unit = {
// The created table should be a MANAGED table with empty view text and original text.
assert(targetTable.tableType == CatalogTableType.MANAGED,
"the created table must be a Hive managed table")
private def checkCreateTableLike(
sourceTable: CatalogTable,
targetTable: CatalogTable,
tableType: CatalogTableType): Unit = {
// The created table should be a MANAGED table or EXTERNAL table with empty view text
// and original text.
assert(targetTable.tableType == tableType,
s"the created table must be a Hive ${tableType.name} table")
Copy link
Member

Choose a reason for hiding this comment

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

For a Hive table? It could be a data source table too.

Copy link
Author

Choose a reason for hiding this comment

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

Sorry for the late reply.Ok, i will follow your suggestion.

assert(targetTable.viewText.isEmpty,
"the view text in the created table must be empty")
assert(targetTable.viewDefaultDatabase.isEmpty,
Expand Down