You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@spark.apache.org by GitBox <gi...@apache.org> on 2019/01/14 17:14:11 UTC

[spark] Diff for: [GitHub] gatorsmile closed pull request #23535: [SPARK-26613][SQL] Add another rename table grammar for spark sql

diff --git a/sql/catalyst/src/main/antlr4/org/apache/spark/sql/catalyst/parser/SqlBase.g4 b/sql/catalyst/src/main/antlr4/org/apache/spark/sql/catalyst/parser/SqlBase.g4
index b39681d886c5c..de18fc93cba2d 100644
--- a/sql/catalyst/src/main/antlr4/org/apache/spark/sql/catalyst/parser/SqlBase.g4
+++ b/sql/catalyst/src/main/antlr4/org/apache/spark/sql/catalyst/parser/SqlBase.g4
@@ -105,6 +105,7 @@ statement
         ADD COLUMNS '(' columns=colTypeList ')'                        #addTableColumns
     | ALTER (TABLE | VIEW) from=tableIdentifier
         RENAME TO to=tableIdentifier                                   #renameTable
+    | RENAME (TABLE | VIEW) from=tableIdentifier TO to=tableIdentifier #renameTable
     | ALTER (TABLE | VIEW) tableIdentifier
         SET TBLPROPERTIES tablePropertyList                            #setTableProperties
     | ALTER (TABLE | VIEW) tableIdentifier
diff --git a/sql/core/src/test/scala/org/apache/spark/sql/execution/command/DDLParserSuite.scala b/sql/core/src/test/scala/org/apache/spark/sql/execution/command/DDLParserSuite.scala
index e0ccae15f1d05..0d8e75947c1a8 100644
--- a/sql/core/src/test/scala/org/apache/spark/sql/execution/command/DDLParserSuite.scala
+++ b/sql/core/src/test/scala/org/apache/spark/sql/execution/command/DDLParserSuite.scala
@@ -636,8 +636,12 @@ class DDLParserSuite extends PlanTest with SharedSQLContext {
   test("alter table/view: rename table/view") {
     val sql_table = "ALTER TABLE table_name RENAME TO new_table_name"
     val sql_view = sql_table.replace("TABLE", "VIEW")
+    val sql_table2 = "RENAME TABLE table_name TO new_table_name"
+    val sql_view2 = sql_table2.replace("TABLE", "VIEW")
     val parsed_table = parser.parsePlan(sql_table)
     val parsed_view = parser.parsePlan(sql_view)
+    val parsed_table2 = parser.parsePlan(sql_table2)
+    val parsed_view2 = parser.parsePlan(sql_view2)
     val expected_table = AlterTableRenameCommand(
       TableIdentifier("table_name"),
       TableIdentifier("new_table_name"),
@@ -646,15 +650,29 @@ class DDLParserSuite extends PlanTest with SharedSQLContext {
       TableIdentifier("table_name"),
       TableIdentifier("new_table_name"),
       isView = true)
+    val expected_table2 = AlterTableRenameCommand(
+      TableIdentifier("table_name"),
+      TableIdentifier("new_table_name"),
+      isView = false)
+    val expected_view2 = AlterTableRenameCommand(
+      TableIdentifier("table_name"),
+      TableIdentifier("new_table_name"),
+      isView = true)
     comparePlans(parsed_table, expected_table)
     comparePlans(parsed_view, expected_view)
+    comparePlans(parsed_table2, expected_table2)
+    comparePlans(parsed_view2, expected_view2)
   }
 
   test("alter table: rename table with database") {
     val query = "ALTER TABLE db1.tbl RENAME TO db1.tbl2"
+    val query2 = "RENAME TABLE db1.tbl TO db1.tbl2"
     val plan = parseAs[AlterTableRenameCommand](query)
+    val plan2 = parseAs[AlterTableRenameCommand](query2)
     assert(plan.oldName == TableIdentifier("tbl", Some("db1")))
     assert(plan.newName == TableIdentifier("tbl2", Some("db1")))
+    assert(plan2.oldName == TableIdentifier("tbl", Some("db1")))
+    assert(plan2.newName == TableIdentifier("tbl2", Some("db1")))
   }
 
   // ALTER TABLE table_name SET TBLPROPERTIES ('comment' = new_comment);
diff --git a/sql/core/src/test/scala/org/apache/spark/sql/execution/command/DDLSuite.scala b/sql/core/src/test/scala/org/apache/spark/sql/execution/command/DDLSuite.scala
index 052a5e757c445..d5012cdc6e3cc 100644
--- a/sql/core/src/test/scala/org/apache/spark/sql/execution/command/DDLSuite.scala
+++ b/sql/core/src/test/scala/org/apache/spark/sql/execution/command/DDLSuite.scala
@@ -945,6 +945,37 @@ abstract class DDLSuite extends QueryTest with SQLTestUtils {
     }
   }
 
+  test("rename table to") {
+    val catalog = spark.sessionState.catalog
+    val tableIdent1 = TableIdentifier("tab1", Some("dbx"))
+    val tableIdent2 = TableIdentifier("tab2", Some("dbx"))
+    createDatabase(catalog, "dbx")
+    createDatabase(catalog, "dby")
+    createTable(catalog, tableIdent1)
+
+    assert(catalog.listTables("dbx") == Seq(tableIdent1))
+    sql("RENAME TABLE dbx.tab1 TO dbx.tab2")
+    assert(catalog.listTables("dbx") == Seq(tableIdent2))
+
+    // The database in destination table name can be omitted, and we will use the database of source
+    // table for it.
+    sql("RENAME TABLE dbx.tab2 TO tab1")
+    assert(catalog.listTables("dbx") == Seq(tableIdent1))
+
+    catalog.setCurrentDatabase("dbx")
+    // rename without explicitly specifying database
+    sql("RENAME TABLE tab1 TO tab2")
+    assert(catalog.listTables("dbx") == Seq(tableIdent2))
+    // table to rename does not exist
+    intercept[AnalysisException] {
+      sql("RENAME TABLE dbx.does_not_exist TO dbx.tab2")
+    }
+    // destination database is different
+    intercept[AnalysisException] {
+      sql("RENAME TABLE dbx.tab1 TO dby.tab2")
+    }
+  }
+
   test("alter table: rename cached table") {
     import testImplicits._
     sql("CREATE TABLE students (age INT, name STRING) USING parquet")
@@ -963,7 +994,25 @@ abstract class DDLSuite extends QueryTest with SQLTestUtils {
     checkAnswer(spark.table("teachers"), df)
   }
 
-  test("rename temporary view - destination table with database name") {
+  test("rename cached table") {
+    import testImplicits._
+    sql("CREATE TABLE students (age INT, name STRING) USING parquet")
+    val df = (1 to 2).map { i => (i, i.toString) }.toDF("age", "name")
+    df.write.insertInto("students")
+    spark.catalog.cacheTable("students")
+    checkAnswer(spark.table("students"), df)
+    assume(spark.catalog.isCached("students"), "bad test: table was not cached in the first place")
+    sql("RENAME TABLE students TO teachers")
+    sql("CREATE TABLE students (age INT, name STRING) USING parquet")
+    // Now we have both students and teachers.
+    // The cached data for the old students table should not be read by the new students table.
+    assert(!spark.catalog.isCached("students"))
+    assert(spark.catalog.isCached("teachers"))
+    assert(spark.table("students").collect().isEmpty)
+    checkAnswer(spark.table("teachers"), df)
+  }
+
+  test("alter view: rename temporary view - destination table with database name") {
     withTempView("tab1") {
       sql(
         """
@@ -988,7 +1037,32 @@ abstract class DDLSuite extends QueryTest with SQLTestUtils {
     }
   }
 
-  test("rename temporary view - destination table with database name,with:CREATE TEMPORARY view") {
+  test("rename temporary view - destination table with database name") {
+    withTempView("tab1") {
+      sql(
+        """
+          |CREATE TEMPORARY TABLE tab1
+          |USING org.apache.spark.sql.sources.DDLScanSource
+          |OPTIONS (
+          |  From '1',
+          |  To '10',
+          |  Table 'test1'
+          |)
+        """.stripMargin)
+
+      val e = intercept[AnalysisException] {
+        sql("RENAME TABLE tab1 TO default.tab2")
+      }
+      assert(e.getMessage.contains(
+        "RENAME TEMPORARY VIEW from '`tab1`' to '`default`.`tab2`': " +
+          "cannot specify database name 'default' in the destination table"))
+
+      val catalog = spark.sessionState.catalog
+      assert(catalog.listTables("default") == Seq(TableIdentifier("tab1", None)))
+    }
+  }
+
+  test("alter view: rename temporary view - destination table with database name,with:CREATE TEMPORARY view") {
     withTempView("view1") {
       sql(
         """
@@ -1013,7 +1087,32 @@ abstract class DDLSuite extends QueryTest with SQLTestUtils {
     }
   }
 
-  test("rename temporary view") {
+  test("rename temporary view - destination table with database name,with:CREATE TEMPORARY view") {
+    withTempView("view1") {
+      sql(
+        """
+          |CREATE TEMPORARY VIEW view1
+          |USING org.apache.spark.sql.sources.DDLScanSource
+          |OPTIONS (
+          |  From '1',
+          |  To '10',
+          |  Table 'test1'
+          |)
+        """.stripMargin)
+
+      val e = intercept[AnalysisException] {
+        sql("RENAME TABLE view1 TO default.tab2")
+      }
+      assert(e.getMessage.contains(
+        "RENAME TEMPORARY VIEW from '`view1`' to '`default`.`tab2`': " +
+          "cannot specify database name 'default' in the destination table"))
+
+      val catalog = spark.sessionState.catalog
+      assert(catalog.listTables("default") == Seq(TableIdentifier("view1", None)))
+    }
+  }
+
+  test("ralter view: ename temporary view") {
     withTempView("tab1", "tab2") {
       spark.range(10).createOrReplaceTempView("tab1")
       sql("ALTER TABLE tab1 RENAME TO tab2")
@@ -1026,7 +1125,20 @@ abstract class DDLSuite extends QueryTest with SQLTestUtils {
     }
   }
 
-  test("rename temporary view - destination table already exists") {
+  test("rename temporary view") {
+    withTempView("tab1", "tab2") {
+      spark.range(10).createOrReplaceTempView("tab1")
+      sql("RENAME TABLE tab1 TO tab2")
+      checkAnswer(spark.table("tab2"), spark.range(10).toDF())
+      val e = intercept[AnalysisException](spark.table("tab1")).getMessage
+      assert(e.contains("Table or view not found"))
+      sql("RENAME VIEW tab2 TO tab1")
+      checkAnswer(spark.table("tab1"), spark.range(10).toDF())
+      intercept[AnalysisException] { spark.table("tab2") }
+    }
+  }
+
+  test("alter view: rename temporary view - destination table already exists") {
     withTempView("tab1", "tab2") {
       sql(
         """
@@ -1061,7 +1173,42 @@ abstract class DDLSuite extends QueryTest with SQLTestUtils {
     }
   }
 
-  test("rename temporary view - destination table already exists, with: CREATE TEMPORARY view") {
+  test("rename temporary view - destination table already exists") {
+    withTempView("tab1", "tab2") {
+      sql(
+        """
+          |CREATE TEMPORARY TABLE tab1
+          |USING org.apache.spark.sql.sources.DDLScanSource
+          |OPTIONS (
+          |  From '1',
+          |  To '10',
+          |  Table 'test1'
+          |)
+        """.stripMargin)
+
+      sql(
+        """
+          |CREATE TEMPORARY TABLE tab2
+          |USING org.apache.spark.sql.sources.DDLScanSource
+          |OPTIONS (
+          |  From '1',
+          |  To '10',
+          |  Table 'test1'
+          |)
+        """.stripMargin)
+
+      val e = intercept[AnalysisException] {
+        sql("RENAME TABLE tab1 TO tab2")
+      }
+      assert(e.getMessage.contains(
+        "RENAME TEMPORARY VIEW from '`tab1`' to '`tab2`': destination table already exists"))
+
+      val catalog = spark.sessionState.catalog
+      assert(catalog.listTables("default") == Seq(TableIdentifier("tab1", None), TableIdentifier("tab2", None)))
+    }
+  }
+
+  test("alter view: rename temporary view - destination table already exists, with: CREATE TEMPORARY view") {
     withTempView("view1", "view2") {
       sql(
         """
@@ -1097,6 +1244,42 @@ abstract class DDLSuite extends QueryTest with SQLTestUtils {
     }
   }
 
+  test("rename temporary view - destination table already exists, with: CREATE TEMPORARY view") {
+    withTempView("view1", "view2") {
+      sql(
+        """
+          |CREATE TEMPORARY VIEW view1
+          |USING org.apache.spark.sql.sources.DDLScanSource
+          |OPTIONS (
+          |  From '1',
+          |  To '10',
+          |  Table 'test1'
+          |)
+        """.stripMargin)
+
+      sql(
+        """
+          |CREATE TEMPORARY VIEW view2
+          |USING org.apache.spark.sql.sources.DDLScanSource
+          |OPTIONS (
+          |  From '1',
+          |  To '10',
+          |  Table 'test1'
+          |)
+        """.stripMargin)
+
+      val e = intercept[AnalysisException] {
+        sql("RENAME TABLE view1 TO view2")
+      }
+      assert(e.getMessage.contains(
+        "RENAME TEMPORARY VIEW from '`view1`' to '`view2`': destination table already exists"))
+
+      val catalog = spark.sessionState.catalog
+      assert(catalog.listTables("default") ==
+        Seq(TableIdentifier("view1", None), TableIdentifier("view2", None)))
+    }
+  }
+
   test("alter table: bucketing is not supported") {
     val catalog = spark.sessionState.catalog
     val tableIdent = TableIdentifier("tab1", Some("dbx"))
diff --git a/sql/core/src/test/scala/org/apache/spark/sql/sources/PathOptionSuite.scala b/sql/core/src/test/scala/org/apache/spark/sql/sources/PathOptionSuite.scala
index 85da3f0e38468..a56854f21c1e8 100644
--- a/sql/core/src/test/scala/org/apache/spark/sql/sources/PathOptionSuite.scala
+++ b/sql/core/src/test/scala/org/apache/spark/sql/sources/PathOptionSuite.scala
@@ -131,6 +131,12 @@ class PathOptionSuite extends DataSourceTest with SharedSQLContext {
       sql("ALTER TABLE src RENAME TO src2")
       assert(getPathOption("src2").map(makeQualifiedPath) == Some(defaultTablePath("src2")))
     }
+
+    withTable("src", "src2") {
+      sql(s"CREATE TABLE src(i int) USING ${classOf[TestOptionsSource].getCanonicalName}")
+      sql("RENAME TABLE src TO src2")
+      assert(getPathOption("src2").map(makeQualifiedPath) == Some(defaultTablePath("src2")))
+    }
   }
 
   private def getPathOption(tableName: String): Option[String] = {
diff --git a/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/HiveDDLSuite.scala b/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/HiveDDLSuite.scala
index 6abdc4054cb0c..52453647687ed 100644
--- a/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/HiveDDLSuite.scala
+++ b/sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/HiveDDLSuite.scala
@@ -718,11 +718,14 @@ class HiveDDLSuite
         assert(!catalog.tableExists(TableIdentifier(newViewName)))
 
         assertErrorForAlterViewOnTable(s"ALTER VIEW $tabName RENAME TO $newViewName")
+        assertErrorForAlterViewOnTable(s"RENAME VIEW $tabName TO $newViewName")
 
         assertErrorForAlterTableOnView(s"ALTER TABLE $oldViewName RENAME TO $newViewName")
+        assertErrorForAlterTableOnView(s"RENAME TABLE $oldViewName TO $newViewName")
 
         assertErrorForAlterViewOnTable(s"ALTER VIEW $tabName SET TBLPROPERTIES ('p' = 'an')")
 
+
         assertErrorForAlterTableOnView(s"ALTER TABLE $oldViewName SET TBLPROPERTIES ('p' = 'an')")
 
         assertErrorForAlterViewOnTable(s"ALTER VIEW $tabName UNSET TBLPROPERTIES ('p')")


With regards,
Apache Git Services

---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@spark.apache.org
For additional commands, e-mail: commits-help@spark.apache.org