Skip to content

Commit

Permalink
[SPARK-33849][SQL][TESTS] Unify v1 and v2 DROP TABLE tests
Browse files Browse the repository at this point in the history
### What changes were proposed in this pull request?
1. Move the `DROP TABLE` parsing tests to `DropTableParserSuite`
2. Place the v1 tests for `DROP TABLE` from `DDLSuite` and v2 tests from `DataSourceV2SQLSuite` to the common trait `DropTableSuiteBase`, so, the tests will run for V1, Hive V1 and V2 DS.

### Why are the changes needed?
- The unification will allow to run common `DROP TABLE` tests for both DSv1 and Hive DSv1, DSv2
- We can detect missing features and differences between DSv1 and DSv2 implementations.

### Does this PR introduce _any_ user-facing change?
No

### How was this patch tested?
By running new test suites:
```
$ build/sbt -Phive-2.3 -Phive-thriftserver "test:testOnly *DropTableParserSuite"
$ build/sbt -Phive-2.3 -Phive-thriftserver "test:testOnly *DropTableSuite"
```

Closes #30854 from MaxGekk/unify-drop-table-tests.

Authored-by: Max Gekk <max.gekk@gmail.com>
Signed-off-by: Wenchen Fan <wenchen@databricks.com>
  • Loading branch information
MaxGekk authored and cloud-fan committed Dec 21, 2020
1 parent 1c7b79c commit b313a1e
Show file tree
Hide file tree
Showing 9 changed files with 312 additions and 126 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -694,32 +694,6 @@ class DDLParserSuite extends AnalysisTest {
}
}

test("drop table") {
parseCompare("DROP TABLE testcat.ns1.ns2.tbl",
DropTable(
UnresolvedTableOrView(Seq("testcat", "ns1", "ns2", "tbl"), "DROP TABLE"),
ifExists = false,
purge = false))
parseCompare(s"DROP TABLE db.tab",
DropTable(
UnresolvedTableOrView(Seq("db", "tab"), "DROP TABLE"), ifExists = false, purge = false))
parseCompare(s"DROP TABLE IF EXISTS db.tab",
DropTable(
UnresolvedTableOrView(Seq("db", "tab"), "DROP TABLE"), ifExists = true, purge = false))
parseCompare(s"DROP TABLE tab",
DropTable(
UnresolvedTableOrView(Seq("tab"), "DROP TABLE"), ifExists = false, purge = false))
parseCompare(s"DROP TABLE IF EXISTS tab",
DropTable(
UnresolvedTableOrView(Seq("tab"), "DROP TABLE"), ifExists = true, purge = false))
parseCompare(s"DROP TABLE tab PURGE",
DropTable(
UnresolvedTableOrView(Seq("tab"), "DROP TABLE"), ifExists = false, purge = true))
parseCompare(s"DROP TABLE IF EXISTS tab PURGE",
DropTable(
UnresolvedTableOrView(Seq("tab"), "DROP TABLE"), ifExists = true, purge = true))
}

test("drop view") {
val cmd = "DROP VIEW"
val hint = Some("Please use DROP TABLE instead.")
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -711,81 +711,6 @@ class DataSourceV2SQLSuite
assert(t2.v1Table.provider == Some(conf.defaultDataSourceName))
}

test("DropTable: basic") {
val tableName = "testcat.ns1.ns2.tbl"
val ident = Identifier.of(Array("ns1", "ns2"), "tbl")
sql(s"CREATE TABLE $tableName USING foo AS SELECT id, data FROM source")
assert(catalog("testcat").asTableCatalog.tableExists(ident) === true)
sql(s"DROP TABLE $tableName")
assert(catalog("testcat").asTableCatalog.tableExists(ident) === false)
}

test("DropTable: table qualified with the session catalog name") {
val ident = Identifier.of(Array("default"), "tbl")
sql("CREATE TABLE tbl USING json AS SELECT 1 AS i")
assert(catalog("spark_catalog").asTableCatalog.tableExists(ident) === true)
sql("DROP TABLE spark_catalog.default.tbl")
assert(catalog("spark_catalog").asTableCatalog.tableExists(ident) === false)
}

test("DropTable: if exists") {
val ex = intercept[AnalysisException] {
sql("DROP TABLE testcat.db.notbl")
}
assert(ex.getMessage.contains("Table or view not found: testcat.db.notbl"))
sql("DROP TABLE IF EXISTS testcat.db.notbl")
}

test("DropTable: purge option") {
withTable("testcat.ns.t") {
sql("CREATE TABLE testcat.ns.t (id bigint) USING foo")
val ex = intercept[UnsupportedOperationException] {
sql ("DROP TABLE testcat.ns.t PURGE")
}
// The default TableCatalog.dropTable implementation doesn't support the purge option.
assert(ex.getMessage.contains("Purge option is not supported"))
}
}

test("SPARK-33174: DROP TABLE should resolve to a temporary view first") {
withTable("testcat.ns.t") {
withTempView("t") {
sql("CREATE TABLE testcat.ns.t (id bigint) USING foo")
sql("CREATE TEMPORARY VIEW t AS SELECT 2")
sql("USE testcat.ns")

// Check the temporary view 't' exists.
runShowTablesSql(
"SHOW TABLES FROM spark_catalog.default LIKE 't'",
Seq(Row("", "t", true)),
expectV2Catalog = false)
sql("DROP TABLE t")
// Verify that the temporary view 't' is resolved first and dropped.
runShowTablesSql(
"SHOW TABLES FROM spark_catalog.default LIKE 't'",
Nil,
expectV2Catalog = false)
}
}
}

test("SPARK-33305: DROP TABLE should also invalidate cache") {
val t = "testcat.ns.t"
val view = "view"
withTable(t) {
withTempView(view) {
sql(s"CREATE TABLE $t USING foo AS SELECT id, data FROM source")
sql(s"CACHE TABLE $view AS SELECT id FROM $t")
checkAnswer(sql(s"SELECT * FROM $t"), spark.table("source"))
checkAnswer(sql(s"SELECT * FROM $view"), spark.table("source").select("id"))

assert(!spark.sharedState.cacheManager.lookupCachedData(spark.table(view)).isEmpty)
sql(s"DROP TABLE $t")
assert(spark.sharedState.cacheManager.lookupCachedData(spark.table(view)).isEmpty)
}
}
}

test("SPARK-33492: ReplaceTableAsSelect (atomic or non-atomic) should invalidate cache") {
Seq("testcat.ns.t", "testcat_atomic.ns.t").foreach { t =>
val view = "view"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -338,10 +338,6 @@ abstract class DDLSuite extends QueryTest with SQLTestUtils {
testRenamePartitions(isDatasourceTable = true)
}

test("drop table - data source table") {
testDropTable(isDatasourceTable = true)
}

test("the qualified path of a database is stored in the catalog") {
val catalog = spark.sessionState.catalog

Expand Down Expand Up @@ -1332,23 +1328,6 @@ abstract class DDLSuite extends QueryTest with SQLTestUtils {
assert(catalog.listTables("default") == Nil)
}

protected def testDropTable(isDatasourceTable: Boolean): Unit = {
if (!isUsingHiveMetastore) {
assert(isDatasourceTable, "InMemoryCatalog only supports data source tables")
}
val catalog = spark.sessionState.catalog
val tableIdent = TableIdentifier("tab1", Some("dbx"))
createDatabase(catalog, "dbx")
createTable(catalog, tableIdent, isDatasourceTable)
assert(catalog.listTables("dbx") == Seq(tableIdent))
sql("DROP TABLE dbx.tab1")
assert(catalog.listTables("dbx") == Nil)
sql("DROP TABLE IF EXISTS dbx.tab1")
intercept[AnalysisException] {
sql("DROP TABLE dbx.tab1")
}
}

test("drop view") {
val catalog = spark.sessionState.catalog
val tableIdent = TableIdentifier("tab1", Some("dbx"))
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,55 @@
/*
* 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
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package org.apache.spark.sql.execution.command

import org.apache.spark.sql.catalyst.analysis.{AnalysisTest, UnresolvedTableOrView}
import org.apache.spark.sql.catalyst.parser.CatalystSqlParser.parsePlan
import org.apache.spark.sql.catalyst.plans.logical.{DropTable, LogicalPlan}
import org.apache.spark.sql.test.SharedSparkSession

class DropTableParserSuite extends AnalysisTest with SharedSparkSession {
private def parseCompare(sql: String, expected: LogicalPlan): Unit = {
comparePlans(parsePlan(sql), expected, checkAnalysis = false)
}

test("drop table") {
parseCompare("DROP TABLE testcat.ns1.ns2.tbl",
DropTable(
UnresolvedTableOrView(Seq("testcat", "ns1", "ns2", "tbl"), "DROP TABLE"),
ifExists = false,
purge = false))
parseCompare(s"DROP TABLE db.tab",
DropTable(
UnresolvedTableOrView(Seq("db", "tab"), "DROP TABLE"), ifExists = false, purge = false))
parseCompare(s"DROP TABLE IF EXISTS db.tab",
DropTable(
UnresolvedTableOrView(Seq("db", "tab"), "DROP TABLE"), ifExists = true, purge = false))
parseCompare(s"DROP TABLE tab",
DropTable(
UnresolvedTableOrView(Seq("tab"), "DROP TABLE"), ifExists = false, purge = false))
parseCompare(s"DROP TABLE IF EXISTS tab",
DropTable(
UnresolvedTableOrView(Seq("tab"), "DROP TABLE"), ifExists = true, purge = false))
parseCompare(s"DROP TABLE tab PURGE",
DropTable(
UnresolvedTableOrView(Seq("tab"), "DROP TABLE"), ifExists = false, purge = true))
parseCompare(s"DROP TABLE IF EXISTS tab PURGE",
DropTable(
UnresolvedTableOrView(Seq("tab"), "DROP TABLE"), ifExists = true, purge = true))
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,99 @@
/*
* 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
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package org.apache.spark.sql.execution.command

import org.apache.spark.sql.{AnalysisException, QueryTest, Row}

trait DropTableSuiteBase extends QueryTest with DDLCommandTestUtils {
override val command = "DROP TABLE"

protected def createTable(tableName: String): Unit = {
sql(s"CREATE TABLE $tableName (c int) $defaultUsing")
sql(s"INSERT INTO $tableName SELECT 0")
}

protected def checkTables(namespace: String, expectedTables: String*): Unit = {
val tables = sql(s"SHOW TABLES IN $catalog.$namespace").select("tableName")
val rows = expectedTables.map(Row(_))
checkAnswer(tables, rows)
}

test("basic") {
withNamespace(s"$catalog.ns") {
sql(s"CREATE NAMESPACE $catalog.ns")

createTable(s"$catalog.ns.tbl")
checkTables("ns", "tbl")

sql(s"DROP TABLE $catalog.ns.tbl")
checkTables("ns") // no tables
}
}

test("try to drop a nonexistent table") {
withNamespace(s"$catalog.ns") {
sql(s"CREATE NAMESPACE $catalog.ns")
checkTables("ns") // no tables

val errMsg = intercept[AnalysisException] {
sql(s"DROP TABLE $catalog.ns.tbl")
}.getMessage
assert(errMsg.contains("Table or view not found"))
}
}

test("with IF EXISTS") {
withNamespace(s"$catalog.ns") {
sql(s"CREATE NAMESPACE $catalog.ns")

createTable(s"$catalog.ns.tbl")
checkTables("ns", "tbl")
sql(s"DROP TABLE IF EXISTS $catalog.ns.tbl")
checkTables("ns")

// It must not throw any exceptions
sql(s"DROP TABLE IF EXISTS $catalog.ns.tbl")
checkTables("ns")
}
}

test("SPARK-33174: DROP TABLE should resolve to a temporary view first") {
withNamespaceAndTable("ns", "t") { t =>
withTempView("t") {
sql(s"CREATE TABLE $t (id bigint) $defaultUsing")
sql("CREATE TEMPORARY VIEW t AS SELECT 2")
sql(s"USE $catalog.ns")
try {
// Check the temporary view 't' exists.
checkAnswer(
sql("SHOW TABLES FROM spark_catalog.default LIKE 't'")
.select("tableName", "isTemporary"),
Row("t", true))
sql("DROP TABLE t")
// Verify that the temporary view 't' is resolved first and dropped.
checkAnswer(
sql("SHOW TABLES FROM spark_catalog.default LIKE 't'")
.select("tableName", "isTemporary"),
Seq.empty)
} finally {
sql(s"USE spark_catalog")
}
}
}
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,62 @@
/*
* 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
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package org.apache.spark.sql.execution.command.v1

import org.apache.spark.sql.execution.command

trait DropTableSuiteBase extends command.DropTableSuiteBase {
test("purge option") {
withNamespace(s"$catalog.ns") {
sql(s"CREATE NAMESPACE $catalog.ns")

createTable(s"$catalog.ns.tbl")
checkTables("ns", "tbl")

sql(s"DROP TABLE $catalog.ns.tbl PURGE")
checkTables("ns") // no tables
}
}
}

class DropTableSuite extends DropTableSuiteBase with CommandSuiteBase {
// The test fails in Hive External catalog with:
// org.apache.spark.sql.AnalysisException:
// spark_catalog.ns.tbl is not a valid TableIdentifier as it has more than 2 name parts.
test("SPARK-33305: DROP TABLE should also invalidate cache") {
val t = s"$catalog.ns.tbl"
val view = "view"
withNamespace(s"$catalog.ns") {
sql(s"CREATE NAMESPACE $catalog.ns")
withTempView(view, "source") {
val df = spark.createDataFrame(Seq((1L, "a"), (2L, "b"), (3L, "c"))).toDF("id", "data")
df.createOrReplaceTempView("source")
sql(s"CREATE TABLE $t $defaultUsing AS SELECT id, data FROM source")
sql(s"CACHE TABLE $view AS SELECT id FROM $t")
checkAnswer(sql(s"SELECT * FROM $t"), spark.table("source").collect())
checkAnswer(
sql(s"SELECT * FROM $view"),
spark.table("source").select("id").collect())

assert(!spark.sharedState.cacheManager.lookupCachedData(spark.table(view)).isEmpty)
sql(s"DROP TABLE $t")
assert(spark.sharedState.cacheManager.lookupCachedData(spark.table(view)).isEmpty)
}
}
}
}

Loading

0 comments on commit b313a1e

Please sign in to comment.