Skip to content

Commit

Permalink
[SPARK-29279][SQL] Merge SHOW NAMESPACES and SHOW DATABASES code path
Browse files Browse the repository at this point in the history
### What changes were proposed in this pull request?
Currently,  `SHOW NAMESPACES` and `SHOW DATABASES` are separate code paths. This PR merges two implementations.

### Why are the changes needed?
To remove code/behavior duplication

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

### How was this patch tested?
Added new unit tests.

Closes #26006 from imback82/combine_show.

Authored-by: Terry Kim <yuminkim@gmail.com>
Signed-off-by: Wenchen Fan <wenchen@databricks.com>
  • Loading branch information
imback82 authored and cloud-fan committed Oct 14, 2019
1 parent 67e1360 commit ef6dce2
Show file tree
Hide file tree
Showing 11 changed files with 55 additions and 137 deletions.
Expand Up @@ -93,8 +93,7 @@ statement
SET locationSpec #setDatabaseLocation
| DROP database (IF EXISTS)? db=errorCapturingIdentifier
(RESTRICT | CASCADE)? #dropDatabase
| SHOW DATABASES (LIKE? pattern=STRING)? #showDatabases
| SHOW NAMESPACES ((FROM | IN) multipartIdentifier)?
| SHOW (DATABASES | NAMESPACES) ((FROM | IN) multipartIdentifier)?
(LIKE? pattern=STRING)? #showNamespaces
| createTableHeader ('(' colTypeList ')')? tableProvider
((OPTIONS options=tablePropertyList) |
Expand Down
Expand Up @@ -169,13 +169,11 @@ class ResolveCatalogs(val catalogManager: CatalogManager)
s"Can not specify catalog `${catalog.name}` for view ${viewName.quoted} " +
s"because view support in catalog has not been implemented yet")

case ShowNamespacesStatement(Some(NonSessionCatalog(catalog, nameParts)), pattern) =>
val namespace = if (nameParts.isEmpty) None else Some(nameParts)
case ShowNamespacesStatement(Some(CatalogAndNamespace(catalog, namespace)), pattern) =>
ShowNamespaces(catalog.asNamespaceCatalog, namespace, pattern)

// TODO (SPARK-29014): we should check if the current catalog is not session catalog here.
case ShowNamespacesStatement(None, pattern) if defaultCatalog.isDefined =>
ShowNamespaces(defaultCatalog.get.asNamespaceCatalog, None, pattern)
case ShowNamespacesStatement(None, pattern) =>
ShowNamespaces(currentCatalog.asNamespaceCatalog, None, pattern)

case ShowTablesStatement(Some(NonSessionCatalog(catalog, nameParts)), pattern) =>
ShowTables(catalog.asTableCatalog, nameParts, pattern)
Expand All @@ -188,9 +186,8 @@ class ResolveCatalogs(val catalogManager: CatalogManager)
if (isNamespaceSet) {
SetCatalogAndNamespace(catalogManager, None, Some(nameParts))
} else {
val CurrentCatalogAndNamespace(catalog, namespace) = nameParts
val ns = if (namespace.isEmpty) { None } else { Some(namespace) }
SetCatalogAndNamespace(catalogManager, Some(catalog.name()), ns)
val CatalogAndNamespace(catalog, namespace) = nameParts
SetCatalogAndNamespace(catalogManager, Some(catalog.name()), namespace)
}
}

Expand Down
Expand Up @@ -2304,6 +2304,10 @@ class AstBuilder(conf: SQLConf) extends SqlBaseBaseVisitor[AnyRef] with Logging
* Create a [[ShowNamespacesStatement]] command.
*/
override def visitShowNamespaces(ctx: ShowNamespacesContext): LogicalPlan = withOrigin(ctx) {
if (ctx.DATABASES != null && ctx.multipartIdentifier != null) {
throw new ParseException(s"FROM/IN operator is not allowed in SHOW DATABASES", ctx)
}

ShowNamespacesStatement(
Option(ctx.multipartIdentifier).map(visitMultipartIdentifier),
Option(ctx.pattern).map(string))
Expand Down
Expand Up @@ -84,38 +84,19 @@ private[sql] trait LookupCatalog extends Logging {
}
}

type DefaultCatalogAndNamespace = (Option[CatalogPlugin], Seq[String])

/**
* Extract catalog and namespace from a multi-part identifier with the default catalog if needed.
* Catalog name takes precedence over namespaces.
*/
object DefaultCatalogAndNamespace {
def unapply(parts: Seq[String]): Some[DefaultCatalogAndNamespace] = parts match {
case Seq(catalogName, tail @ _*) =>
try {
Some((Some(catalogManager.catalog(catalogName)), tail))
} catch {
case _: CatalogNotFoundException =>
Some((defaultCatalog, parts))
}
}
}

type CurrentCatalogAndNamespace = (CatalogPlugin, Seq[String])

/**
* Extract catalog and namespace from a multi-part identifier with the current catalog if needed.
* Catalog name takes precedence over namespaces.
*/
object CurrentCatalogAndNamespace {
def unapply(parts: Seq[String]): Some[CurrentCatalogAndNamespace] = parts match {
object CatalogAndNamespace {
def unapply(parts: Seq[String]): Some[(CatalogPlugin, Option[Seq[String]])] = parts match {
case Seq(catalogName, tail @ _*) =>
try {
Some((catalogManager.catalog(catalogName), tail))
Some(
(catalogManager.catalog(catalogName), if (tail.isEmpty) { None } else { Some(tail) }))
} catch {
case _: CatalogNotFoundException =>
Some((currentCatalog, parts))
Some((currentCatalog, Some(parts)))
}
}
}
Expand Down
Expand Up @@ -846,6 +846,25 @@ class DDLParserSuite extends AnalysisTest {
ShowTablesStatement(Some(Seq("tbl")), Some("*dog*")))
}

test("show databases: basic") {
comparePlans(
parsePlan("SHOW DATABASES"),
ShowNamespacesStatement(None, None))
comparePlans(
parsePlan("SHOW DATABASES LIKE 'defau*'"),
ShowNamespacesStatement(None, Some("defau*")))
}

test("show databases: FROM/IN operator is not allowed") {
def verify(sql: String): Unit = {
val exc = intercept[ParseException] { parsePlan(sql) }
assert(exc.getMessage.contains("FROM/IN operator is not allowed in SHOW DATABASES"))
}

verify("SHOW DATABASES FROM testcat.ns1.ns2")
verify("SHOW DATABASES IN testcat.ns1.ns2")
}

test("show namespaces") {
comparePlans(
parsePlan("SHOW NAMESPACES"),
Expand Down
Expand Up @@ -40,7 +40,7 @@ class ParserUtilsSuite extends SparkFunSuite {
}

val showDbsContext = buildContext("show databases like 'identifier_with_wildcards'") { parser =>
parser.statement().asInstanceOf[ShowDatabasesContext]
parser.statement().asInstanceOf[ShowNamespacesContext]
}

val createDbContext = buildContext(
Expand Down
Expand Up @@ -256,15 +256,6 @@ class ResolveSessionCatalog(
case DropViewStatement(SessionCatalog(catalog, viewName), ifExists) =>
DropTableCommand(viewName.asTableIdentifier, ifExists, isView = true, purge = false)

case ShowNamespacesStatement(Some(SessionCatalog(catalog, nameParts)), pattern) =>
throw new AnalysisException(
"SHOW NAMESPACES is not supported with the session catalog.")

// TODO (SPARK-29014): we should check if the current catalog is session catalog here.
case ShowNamespacesStatement(None, pattern) if defaultCatalog.isEmpty =>
throw new AnalysisException(
"SHOW NAMESPACES is not supported with the session catalog.")

case ShowTablesStatement(Some(SessionCatalog(catalog, nameParts)), pattern) =>
if (nameParts.length != 1) {
throw new AnalysisException(
Expand Down
Expand Up @@ -155,17 +155,6 @@ class SparkSqlAstBuilder(conf: SQLConf) extends AstBuilder(conf) {
partitionSpec = partitionSpec)
}

/**
* Create a [[ShowDatabasesCommand]] logical plan.
* Example SQL:
* {{{
* SHOW (DATABASES|SCHEMAS) [LIKE 'identifier_with_wildcards'];
* }}}
*/
override def visitShowDatabases(ctx: ShowDatabasesContext): LogicalPlan = withOrigin(ctx) {
ShowDatabasesCommand(Option(ctx.pattern).map(string))
}

/**
* A command for users to list the properties for a table. If propertyKey is specified, the value
* for the propertyKey is returned. If propertyKey is not specified, all the keys and their
Expand Down

This file was deleted.

Expand Up @@ -781,12 +781,8 @@ class DataSourceV2SQLSuite
test("ShowNamespaces: default v2 catalog is not set") {
spark.sql("CREATE TABLE testcat.ns.table (id bigint) USING foo")

val exception = intercept[AnalysisException] {
sql("SHOW NAMESPACES")
}

assert(exception.getMessage.contains(
"SHOW NAMESPACES is not supported with the session catalog"))
// The current catalog is resolved to a v2 session catalog.
testShowNamespaces("SHOW NAMESPACES", Seq("default"))
}

test("ShowNamespaces: default v2 catalog doesn't support namespace") {
Expand Down Expand Up @@ -814,13 +810,28 @@ class DataSourceV2SQLSuite
assert(exception.getMessage.contains("does not support namespaces"))
}

test("ShowNamespaces: session catalog") {
test("ShowNamespaces: session catalog is used and namespace doesn't exist") {
val exception = intercept[AnalysisException] {
sql("SHOW NAMESPACES in dummy")
}

assert(exception.getMessage.contains(
"SHOW NAMESPACES is not supported with the session catalog"))
assert(exception.getMessage.contains("Namespace 'dummy' not found"))
}

test("ShowNamespaces: change catalog and namespace with USE statements") {
sql("CREATE TABLE testcat.ns1.ns2.table (id bigint) USING foo")

// Initially, the current catalog is a v2 session catalog.
testShowNamespaces("SHOW NAMESPACES", Seq("default"))

// Update the current catalog to 'testcat'.
sql("USE testcat")
testShowNamespaces("SHOW NAMESPACES", Seq("ns1"))

// Update the current namespace to 'ns1'.
sql("USE ns1")
// 'SHOW NAMESPACES' is not affected by the current namespace and lists root namespaces.
testShowNamespaces("SHOW NAMESPACES", Seq("ns1"))
}

private def testShowNamespaces(
Expand Down
Expand Up @@ -811,17 +811,6 @@ class DDLParserSuite extends AnalysisTest with SharedSparkSession {
""".stripMargin)
}

test("show databases") {
val sql1 = "SHOW DATABASES"
val sql2 = "SHOW DATABASES LIKE 'defau*'"
val parsed1 = parser.parsePlan(sql1)
val expected1 = ShowDatabasesCommand(None)
val parsed2 = parser.parsePlan(sql2)
val expected2 = ShowDatabasesCommand(Some("defau*"))
comparePlans(parsed1, expected1)
comparePlans(parsed2, expected2)
}

test("show tblproperties") {
val parsed1 = parser.parsePlan("SHOW TBLPROPERTIES tab1")
val expected1 = ShowTablePropertiesCommand(TableIdentifier("tab1", None), None)
Expand Down

0 comments on commit ef6dce2

Please sign in to comment.