Skip to content

Commit 37e387a

Browse files
planga82cloud-fan
authored andcommitted
[SPARK-29519][SQL] SHOW TBLPROPERTIES should do multi-catalog resolution
### What changes were proposed in this pull request? Add ShowTablePropertiesStatement and make SHOW TBLPROPERTIES go through the same catalog/table resolution framework of v2 commands. ### Why are the changes needed? It's important to make all the commands have the same table resolution behavior, to avoid confusing end-users. e.g. USE my_catalog DESC t // success and describe the table t from my_catalog SHOW TBLPROPERTIES t // report table not found as there is no table t in the session catalog ### Does this PR introduce any user-facing change? yes. When running SHOW TBLPROPERTIES Spark fails the command if the current catalog is set to a v2 catalog, or the table name specified a v2 catalog. ### How was this patch tested? Unit tests. Closes #26176 from planga82/feature/SPARK-29519_SHOW_TBLPROPERTIES_datasourceV2. Authored-by: Pablo Langa <soypab@gmail.com> Signed-off-by: Wenchen Fan <wenchen@databricks.com>
1 parent df08e90 commit 37e387a

File tree

13 files changed

+169
-27
lines changed

13 files changed

+169
-27
lines changed

sql/catalyst/src/main/antlr4/org/apache/spark/sql/catalyst/parser/SqlBase.g4

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -189,7 +189,7 @@ statement
189189
(LIKE? pattern=STRING)? #showTables
190190
| SHOW TABLE EXTENDED ((FROM | IN) db=errorCapturingIdentifier)?
191191
LIKE pattern=STRING partitionSpec? #showTable
192-
| SHOW TBLPROPERTIES table=tableIdentifier
192+
| SHOW TBLPROPERTIES table=multipartIdentifier
193193
('(' key=tablePropertyKey ')')? #showTblProperties
194194
| SHOW COLUMNS (FROM | IN) table=multipartIdentifier
195195
((FROM | IN) namespace=multipartIdentifier)? #showColumns

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/Analyzer.scala

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -691,6 +691,11 @@ class Analyzer(
691691
.map(rel => alter.copy(table = rel))
692692
.getOrElse(alter)
693693

694+
case show @ ShowTableProperties(u: UnresolvedV2Relation, _) =>
695+
CatalogV2Util.loadRelation(u.catalog, u.tableName)
696+
.map(rel => show.copy(table = rel))
697+
.getOrElse(show)
698+
694699
case u: UnresolvedV2Relation =>
695700
CatalogV2Util.loadRelation(u.catalog, u.tableName).getOrElse(u)
696701
}

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/ResolveCatalogs.scala

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -207,6 +207,11 @@ class ResolveCatalogs(val catalogManager: CatalogManager)
207207

208208
case ShowCurrentNamespaceStatement() =>
209209
ShowCurrentNamespace(catalogManager)
210+
211+
case ShowTablePropertiesStatement(
212+
nameParts @ NonSessionCatalog(catalog, tableName), propertyKey) =>
213+
val r = UnresolvedV2Relation(nameParts, catalog.asTableCatalog, tableName.asIdentifier)
214+
ShowTableProperties(r, propertyKey)
210215
}
211216

212217
object NonSessionCatalog {

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/parser/AstBuilder.scala

Lines changed: 16 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -3193,4 +3193,20 @@ class AstBuilder(conf: SQLConf) extends SqlBaseBaseVisitor[AnyRef] with Logging
31933193
originalText = source(ctx.query),
31943194
query = plan(ctx.query))
31953195
}
3196+
3197+
/**
3198+
* A command for users to list the properties for a table. If propertyKey is specified, the value
3199+
* for the propertyKey is returned. If propertyKey is not specified, all the keys and their
3200+
* corresponding values are returned.
3201+
* The syntax of using this command in SQL is:
3202+
* {{{
3203+
* SHOW TBLPROPERTIES multi_part_name[('propertyKey')];
3204+
* }}}
3205+
*/
3206+
override def visitShowTblProperties(
3207+
ctx: ShowTblPropertiesContext): LogicalPlan = withOrigin(ctx) {
3208+
ShowTablePropertiesStatement(
3209+
visitMultipartIdentifier(ctx.table),
3210+
Option(ctx.key).map(visitTablePropertyKey))
3211+
}
31963212
}

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/plans/logical/statements.scala

Lines changed: 7 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -448,3 +448,10 @@ case class ShowColumnsStatement(
448448
* A SHOW CURRENT NAMESPACE statement, as parsed from SQL
449449
*/
450450
case class ShowCurrentNamespaceStatement() extends ParsedStatement
451+
452+
/**
453+
* A SHOW TBLPROPERTIES statement, as parsed from SQL
454+
*/
455+
case class ShowTablePropertiesStatement(
456+
tableName: Seq[String],
457+
propertyKey: Option[String]) extends ParsedStatement

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/plans/logical/v2Commands.scala

Lines changed: 11 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -412,3 +412,14 @@ case class ShowCurrentNamespace(catalogManager: CatalogManager) extends Command
412412
AttributeReference("catalog", StringType, nullable = false)(),
413413
AttributeReference("namespace", StringType, nullable = false)())
414414
}
415+
416+
/**
417+
* The logical plan of the SHOW TBLPROPERTIES command that works for v2 catalogs.
418+
*/
419+
case class ShowTableProperties(
420+
table: NamedRelation,
421+
propertyKey: Option[String]) extends Command{
422+
override val output: Seq[Attribute] = Seq(
423+
AttributeReference("key", StringType, nullable = false)(),
424+
AttributeReference("value", StringType, nullable = false)())
425+
}

sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/parser/DDLParserSuite.scala

Lines changed: 10 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -1586,6 +1586,16 @@ class DDLParserSuite extends AnalysisTest {
15861586
comparePlans(parsed, expected)
15871587
}
15881588

1589+
test("SHOW TBLPROPERTIES table") {
1590+
comparePlans(
1591+
parsePlan("SHOW TBLPROPERTIES a.b.c"),
1592+
ShowTablePropertiesStatement(Seq("a", "b", "c"), None))
1593+
1594+
comparePlans(
1595+
parsePlan("SHOW TBLPROPERTIES a.b.c('propKey1')"),
1596+
ShowTablePropertiesStatement(Seq("a", "b", "c"), Some("propKey1")))
1597+
}
1598+
15891599
private case class TableSpec(
15901600
name: Seq[String],
15911601
schema: Option[StructType],

sql/core/src/main/scala/org/apache/spark/sql/catalyst/analysis/ResolveSessionCatalog.scala

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -428,6 +428,11 @@ class ResolveSessionCatalog(
428428
v1TableName.asTableIdentifier,
429429
originalText,
430430
query)
431+
432+
case ShowTablePropertiesStatement(SessionCatalog(_, tableName), propertyKey) =>
433+
ShowTablePropertiesCommand(
434+
tableName.asTableIdentifier,
435+
propertyKey)
431436
}
432437

433438
private def parseV1Table(tableName: Seq[String], sql: String): Seq[String] = {

sql/core/src/main/scala/org/apache/spark/sql/execution/SparkSqlParser.scala

Lines changed: 0 additions & 16 deletions
Original file line numberDiff line numberDiff line change
@@ -106,22 +106,6 @@ class SparkSqlAstBuilder(conf: SQLConf) extends AstBuilder(conf) {
106106
partitionSpec = partitionSpec)
107107
}
108108

109-
/**
110-
* A command for users to list the properties for a table. If propertyKey is specified, the value
111-
* for the propertyKey is returned. If propertyKey is not specified, all the keys and their
112-
* corresponding values are returned.
113-
* The syntax of using this command in SQL is:
114-
* {{{
115-
* SHOW TBLPROPERTIES table_name[('propertyKey')];
116-
* }}}
117-
*/
118-
override def visitShowTblProperties(
119-
ctx: ShowTblPropertiesContext): LogicalPlan = withOrigin(ctx) {
120-
ShowTablePropertiesCommand(
121-
visitTableIdentifier(ctx.tableIdentifier),
122-
Option(ctx.key).map(visitTablePropertyKey))
123-
}
124-
125109
/**
126110
* Create a [[RefreshResource]] logical plan.
127111
*/

sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/v2/DataSourceV2Strategy.scala

Lines changed: 4 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -22,7 +22,7 @@ import scala.collection.JavaConverters._
2222
import org.apache.spark.sql.{AnalysisException, Strategy}
2323
import org.apache.spark.sql.catalyst.expressions.{And, PredicateHelper, SubqueryExpression}
2424
import org.apache.spark.sql.catalyst.planning.PhysicalOperation
25-
import org.apache.spark.sql.catalyst.plans.logical.{AlterTable, AppendData, CreateNamespace, CreateTableAsSelect, CreateV2Table, DeleteFromTable, DescribeTable, DropNamespace, DropTable, LogicalPlan, OverwriteByExpression, OverwritePartitionsDynamic, RefreshTable, Repartition, ReplaceTable, ReplaceTableAsSelect, SetCatalogAndNamespace, ShowCurrentNamespace, ShowNamespaces, ShowTables}
25+
import org.apache.spark.sql.catalyst.plans.logical.{AlterTable, AppendData, CreateNamespace, CreateTableAsSelect, CreateV2Table, DeleteFromTable, DescribeTable, DropNamespace, DropTable, LogicalPlan, OverwriteByExpression, OverwritePartitionsDynamic, RefreshTable, Repartition, ReplaceTable, ReplaceTableAsSelect, SetCatalogAndNamespace, ShowCurrentNamespace, ShowNamespaces, ShowTableProperties, ShowTables}
2626
import org.apache.spark.sql.connector.catalog.{StagingTableCatalog, TableCapability}
2727
import org.apache.spark.sql.connector.read.streaming.{ContinuousStream, MicroBatchStream}
2828
import org.apache.spark.sql.execution.{FilterExec, ProjectExec, SparkPlan}
@@ -213,6 +213,9 @@ object DataSourceV2Strategy extends Strategy with PredicateHelper {
213213
case r: ShowCurrentNamespace =>
214214
ShowCurrentNamespaceExec(r.output, r.catalogManager) :: Nil
215215

216+
case r @ ShowTableProperties(DataSourceV2Relation(table, _, _), propertyKey) =>
217+
ShowTablePropertiesExec(r.output, table, propertyKey) :: Nil
218+
216219
case _ => Nil
217220
}
218221
}

0 commit comments

Comments
 (0)