Skip to content
Original file line number Diff line number Diff line change
Expand Up @@ -17,50 +17,18 @@

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

import org.scalactic.source.Position
import org.scalatest.Tag

import org.apache.spark.sql.{AnalysisException, QueryTest, Row}
import org.apache.spark.sql.{AnalysisException, QueryTest}
import org.apache.spark.sql.catalyst.analysis.PartitionsAlreadyExistException
import org.apache.spark.sql.catalyst.catalog.CatalogTypes.TablePartitionSpec
import org.apache.spark.sql.execution.datasources.PartitioningUtils
import org.apache.spark.sql.internal.SQLConf
import org.apache.spark.sql.test.SQLTestUtils

trait AlterTableAddPartitionSuiteBase extends QueryTest with SQLTestUtils {
protected def version: String
protected def catalog: String
protected def defaultUsing: String

override def test(testName: String, testTags: Tag*)(testFun: => Any)
(implicit pos: Position): Unit = {
super.test(s"ALTER TABLE .. ADD PARTITION $version: " + testName, testTags: _*)(testFun)
}
trait AlterTableAddPartitionSuiteBase extends QueryTest with DDLCommandTestUtils {
override val command = "ALTER TABLE .. ADD PARTITION"

protected def checkPartitions(t: String, expected: Map[String, String]*): Unit = {
val partitions = sql(s"SHOW PARTITIONS $t")
.collect()
.toSet
.map((row: Row) => row.getString(0))
.map(PartitioningUtils.parsePathFragment)
assert(partitions === expected.toSet)
}
protected def checkLocation(t: String, spec: TablePartitionSpec, expected: String): Unit

protected def withNsTable(ns: String, tableName: String, cat: String = catalog)
(f: String => Unit): Unit = {
val nsCat = s"$cat.$ns"
withNamespace(nsCat) {
sql(s"CREATE NAMESPACE $nsCat")
val t = s"$nsCat.$tableName"
withTable(t) {
f(t)
}
}
}

test("one partition") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"CREATE TABLE $t (id bigint, data string) $defaultUsing PARTITIONED BY (id)")
Seq("", "IF NOT EXISTS").foreach { exists =>
sql(s"ALTER TABLE $t ADD $exists PARTITION (id=1) LOCATION 'loc'")
Expand All @@ -72,7 +40,7 @@ trait AlterTableAddPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("multiple partitions") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"CREATE TABLE $t (id bigint, data string) $defaultUsing PARTITIONED BY (id)")
Seq("", "IF NOT EXISTS").foreach { exists =>
sql(s"""
Expand All @@ -88,7 +56,7 @@ trait AlterTableAddPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("multi-part partition") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"CREATE TABLE $t (id bigint, a int, b string) $defaultUsing PARTITIONED BY (a, b)")
Seq("", "IF NOT EXISTS").foreach { exists =>
sql(s"ALTER TABLE $t ADD $exists PARTITION (a=2, b='abc')")
Expand All @@ -99,7 +67,7 @@ trait AlterTableAddPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("table to alter does not exist") {
withNsTable("ns", "does_not_exist") { t =>
withNamespaceAndTable("ns", "does_not_exist") { t =>
val errMsg = intercept[AnalysisException] {
sql(s"ALTER TABLE $t ADD IF NOT EXISTS PARTITION (a='4', b='9')")
}.getMessage
Expand All @@ -108,7 +76,7 @@ trait AlterTableAddPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("case sensitivity in resolving partition specs") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
spark.sql(s"CREATE TABLE $t (id bigint, data string) $defaultUsing PARTITIONED BY (id)")
withSQLConf(SQLConf.CASE_SENSITIVE.key -> "true") {
val errMsg = intercept[AnalysisException] {
Expand All @@ -125,7 +93,7 @@ trait AlterTableAddPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("SPARK-33521: universal type conversions of partition values") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"""
|CREATE TABLE $t (
| id int,
Expand Down Expand Up @@ -173,7 +141,7 @@ trait AlterTableAddPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("SPARK-33676: not fully specified partition spec") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"""
|CREATE TABLE $t (id bigint, part0 int, part1 string)
|$defaultUsing
Expand All @@ -187,7 +155,7 @@ trait AlterTableAddPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("partition already exists") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"CREATE TABLE $t (id bigint, data string) $defaultUsing PARTITIONED BY (id)")
sql(s"ALTER TABLE $t ADD PARTITION (id=2) LOCATION 'loc1'")

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,48 +17,15 @@

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

import org.scalactic.source.Position
import org.scalatest.Tag

import org.apache.spark.sql.{AnalysisException, QueryTest, Row}
import org.apache.spark.sql.{AnalysisException, QueryTest}
import org.apache.spark.sql.catalyst.analysis.NoSuchPartitionsException
import org.apache.spark.sql.execution.datasources.PartitioningUtils
import org.apache.spark.sql.internal.SQLConf
import org.apache.spark.sql.test.SQLTestUtils

trait AlterTableDropPartitionSuiteBase extends QueryTest with SQLTestUtils {
protected def version: String
protected def catalog: String
protected def defaultUsing: String
trait AlterTableDropPartitionSuiteBase extends QueryTest with DDLCommandTestUtils {
override val command = "ALTER TABLE .. DROP PARTITION"

protected def notFullPartitionSpecErr: String

override def test(testName: String, testTags: Tag*)(testFun: => Any)
(implicit pos: Position): Unit = {
super.test(s"ALTER TABLE .. DROP PARTITION $version: " + testName, testTags: _*)(testFun)
}

protected def withNsTable(ns: String, tableName: String, cat: String = catalog)
(f: String => Unit): Unit = {
val nsCat = s"$cat.$ns"
withNamespace(nsCat) {
sql(s"CREATE NAMESPACE $nsCat")
val t = s"$nsCat.$tableName"
withTable(t) {
f(t)
}
}
}

protected def checkPartitions(t: String, expected: Map[String, String]*): Unit = {
val partitions = sql(s"SHOW PARTITIONS $t")
.collect()
.toSet
.map((row: Row) => row.getString(0))
.map(PartitioningUtils.parsePathFragment)
assert(partitions === expected.toSet)
}

protected def checkDropPartition(
t: String,
ifExists: String,
Expand All @@ -75,7 +42,7 @@ trait AlterTableDropPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("single partition") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"CREATE TABLE $t (id bigint, data string) $defaultUsing PARTITIONED BY (id)")
Seq("", "IF EXISTS").foreach { ifExists =>
sql(s"ALTER TABLE $t ADD PARTITION (id=1) LOCATION 'loc'")
Expand All @@ -85,7 +52,7 @@ trait AlterTableDropPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("multiple partitions") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"CREATE TABLE $t (id bigint, data string) $defaultUsing PARTITIONED BY (id)")
Seq("", "IF EXISTS").foreach { ifExists =>
sql(s"""
Expand All @@ -98,7 +65,7 @@ trait AlterTableDropPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("multi-part partition") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"CREATE TABLE $t (id bigint, a int, b string) $defaultUsing PARTITIONED BY (a, b)")
Seq("", "IF EXISTS").foreach { ifExists =>
sql(s"ALTER TABLE $t ADD PARTITION (a = 2, b = 'abc')")
Expand All @@ -108,7 +75,7 @@ trait AlterTableDropPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("table to alter does not exist") {
withNsTable("ns", "does_not_exist") { t =>
withNamespaceAndTable("ns", "does_not_exist") { t =>
val errMsg = intercept[AnalysisException] {
sql(s"ALTER TABLE $t DROP PARTITION (a='4', b='9')")
}.getMessage
Expand All @@ -117,7 +84,7 @@ trait AlterTableDropPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("case sensitivity in resolving partition specs") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"CREATE TABLE $t (id bigint, data string) $defaultUsing PARTITIONED BY (id)")
withSQLConf(SQLConf.CASE_SENSITIVE.key -> "true") {
val errMsg = intercept[AnalysisException] {
Expand All @@ -136,7 +103,7 @@ trait AlterTableDropPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("SPARK-33676: not fully specified partition spec") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"""
|CREATE TABLE $t (id bigint, part0 int, part1 string)
|$defaultUsing
Expand All @@ -149,7 +116,7 @@ trait AlterTableDropPartitionSuiteBase extends QueryTest with SQLTestUtils {
}

test("partition not exists") {
withNsTable("ns", "tbl") { t =>
withNamespaceAndTable("ns", "tbl") { t =>
sql(s"CREATE TABLE $t (id bigint, data string) $defaultUsing PARTITIONED BY (id)")
sql(s"ALTER TABLE $t ADD PARTITION (id=1) LOCATION 'loc'")

Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,60 @@
/*
* 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.scalactic.source.Position
import org.scalatest.Tag

import org.apache.spark.sql.Row
import org.apache.spark.sql.execution.datasources.PartitioningUtils
import org.apache.spark.sql.test.SQLTestUtils

trait DDLCommandTestUtils extends SQLTestUtils {
// The version of the catalog under testing such as "V1", "V2", "Hive V1".
protected def version: String
// Name of the command as SQL statement, for instance "SHOW PARTITIONS"
protected def command: String
protected def catalog: String
protected def defaultUsing: String

override def test(testName: String, testTags: Tag*)(testFun: => Any)
(implicit pos: Position): Unit = {
super.test(s"$command $version: " + testName, testTags: _*)(testFun)
}

protected def withNamespaceAndTable(ns: String, tableName: String, cat: String = catalog)
Copy link
Contributor

Choose a reason for hiding this comment

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

very nit: the parameter names for namespace and catalog are both very short(ns and cat), we should probably use t to be consistent.

Copy link
Member Author

Choose a reason for hiding this comment

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

If the build fails, I will change this to re-trigger GA or Jenkins.

(f: String => Unit): Unit = {
val nsCat = s"$cat.$ns"
withNamespace(nsCat) {
sql(s"CREATE NAMESPACE $nsCat")
val t = s"$nsCat.$tableName"
withTable(t) {
f(t)
}
}
}

protected def checkPartitions(t: String, expected: Map[String, String]*): Unit = {
val partitions = sql(s"SHOW PARTITIONS $t")
.collect()
.toSet
.map((row: Row) => row.getString(0))
.map(PartitioningUtils.parsePathFragment)
assert(partitions === expected.toSet)
}
}
Loading