Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
Show all changes
44 commits
Select commit Hold shift + click to select a range
39f3c42
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in DSv2
panbingkun Aug 20, 2022
cbe7433
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Aug 20, 2022
023414b
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v
panbingkun Aug 20, 2022
0697773
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Aug 28, 2022
676455e
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Feb 5, 2023
9876120
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Feb 5, 2023
7d71eb0
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Feb 20, 2023
5c40c7c
Merge branch 'master' into v2_SHOW_TABLE_EXTENDED
panbingkun Feb 20, 2023
3c3651a
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Feb 20, 2023
b1cc56c
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Feb 21, 2023
ab50fb0
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Feb 21, 2023
9c4a9ac
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Feb 21, 2023
56ceeac
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Jun 29, 2023
be4199c
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Oct 18, 2023
3a296c0
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Oct 18, 2023
f54a832
Apply suggestions from code review
panbingkun Oct 19, 2023
892de53
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Oct 19, 2023
f0b2db4
Merge branch 'master' into v2_SHOW_TABLE_EXTENDED
panbingkun Oct 20, 2023
d898932
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Oct 22, 2023
ffd3148
Merge branch 'master' into v2_SHOW_TABLE_EXTENDED
panbingkun Oct 22, 2023
54531d8
Merge branch 'master' into v2_SHOW_TABLE_EXTENDED
panbingkun Oct 23, 2023
59369a5
Merge branch 'master' into v2_SHOW_TABLE_EXTENDED
panbingkun Oct 23, 2023
482dea1
Merge branch 'master' into v2_SHOW_TABLE_EXTENDED
panbingkun Oct 23, 2023
3b12d76
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Oct 27, 2023
7684a94
Merge branch 'master' into v2_SHOW_TABLE_EXTENDED
panbingkun Oct 27, 2023
474b963
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Oct 30, 2023
ac7586f
Merge branch 'master' into v2_SHOW_TABLE_EXTENDED
panbingkun Oct 30, 2023
bcb5db7
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Oct 30, 2023
74e52ba
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Oct 30, 2023
723e664
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Nov 1, 2023
0f5c3c7
Merge branch 'master' into v2_SHOW_TABLE_EXTENDED
panbingkun Nov 1, 2023
e06f867
Trigger build
panbingkun Nov 1, 2023
4af1c5e
Trigger build
panbingkun Nov 1, 2023
ee254f4
Trigger build
panbingkun Nov 1, 2023
02065eb
debug appveyor
panbingkun Nov 2, 2023
9c3940c
Temporarily set R-win version to 4.3.2
panbingkun Nov 2, 2023
b47e619
Apply suggestions from code review
panbingkun Nov 3, 2023
be5b337
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Nov 3, 2023
73ff15b
Merge branch 'master' into v2_SHOW_TABLE_EXTENDED
panbingkun Nov 3, 2023
e6978e3
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Nov 6, 2023
1bab140
Merge branch 'master' into v2_SHOW_TABLE_EXTENDED
panbingkun Nov 6, 2023
26ae679
fix minor
panbingkun Nov 7, 2023
ef4213a
[SPARK-33393][SQL] Support SHOW TABLE EXTENDED in v2
panbingkun Nov 12, 2023
d318f6b
Trigger build
panbingkun Nov 12, 2023
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
5 changes: 0 additions & 5 deletions common/utils/src/main/resources/error/error-classes.json
Original file line number Diff line number Diff line change
Expand Up @@ -4770,11 +4770,6 @@
"Invalid bound function '<bound>: there are <argsLen> arguments but <inputTypesLen> parameters returned from 'inputTypes()'."
]
},
"_LEGACY_ERROR_TEMP_1200" : {
"message" : [
"<name> is not supported for v2 tables."
]
},
"_LEGACY_ERROR_TEMP_1201" : {
"message" : [
"Cannot resolve column name \"<colName>\" among (<fieldNames>)."
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -271,10 +271,6 @@ trait CheckAnalysis extends PredicateHelper with LookupCatalog with QueryErrorsB
case _ =>
}

// `ShowTableExtended` should have been converted to the v1 command if the table is v1.
case _: ShowTableExtended =>
throw QueryCompilationErrors.commandUnsupportedInV2TableError("SHOW TABLE EXTENDED")

case operator: LogicalPlan =>
operator transformExpressionsDown {
// Check argument data types of higher-order functions downwards first.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -50,7 +50,7 @@ class ResolveCatalogs(val catalogManager: CatalogManager)
case s @ ShowTables(UnresolvedNamespace(Seq()), _, _) =>
s.copy(namespace = ResolvedNamespace(currentCatalog,
catalogManager.currentNamespace.toImmutableArraySeq))
case s @ ShowTableExtended(UnresolvedNamespace(Seq()), _, _, _) =>
case s @ ShowTablesExtended(UnresolvedNamespace(Seq()), _, _) =>
s.copy(namespace = ResolvedNamespace(currentCatalog,
catalogManager.currentNamespace.toImmutableArraySeq))
case s @ ShowViews(UnresolvedNamespace(Seq()), _, _) =>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1090,6 +1090,25 @@ class SessionCatalog(
dbViews ++ listLocalTempViews(pattern)
}

/**
* List all matching temp views in the specified database, including global/local temporary views.
*/
def listTempViews(db: String, pattern: String): Seq[CatalogTable] = {
val globalTempViews = if (format(db) == globalTempViewManager.database) {
globalTempViewManager.listViewNames(pattern).flatMap { viewName =>
globalTempViewManager.get(viewName).map(_.tableMeta)
}
} else {
Seq.empty
}

val localTempViews = listLocalTempViews(pattern).flatMap { viewIndent =>
tempViews.get(viewIndent.table).map(_.tableMeta)
}

globalTempViews ++ localTempViews
}

/**
* List all matching local temporary views.
*/
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -4086,19 +4086,31 @@ class AstBuilder extends DataTypeAstBuilder with SQLConfHelper with Logging {
}

/**
* Create a [[ShowTableExtended]] command.
* Create a [[ShowTablesExtended]] or [[ShowTablePartition]] command.
*/
override def visitShowTableExtended(
Copy link
Contributor

Choose a reason for hiding this comment

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

Because we change ShowTableExtended to ShowTablesExtended, shall we update the name in g4 ?

Copy link
Contributor Author

Choose a reason for hiding this comment

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

@cloud-fan
Do we need this related modification?
Because renaming the name to ShowTablesExtended or ShowTablePartition does not seem to convey the meaning of the following two commands at the same time.

| SHOW TABLE EXTENDED ((FROM | IN) ns=identifierReference)?
LIKE pattern=stringLit partitionSpec? #showTableExtended

Copy link
Contributor

Choose a reason for hiding this comment

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

Let's not rename the parser rule now. It matches both commands.

ctx: ShowTableExtendedContext): LogicalPlan = withOrigin(ctx) {
val partitionKeys = Option(ctx.partitionSpec).map { specCtx =>
UnresolvedPartitionSpec(visitNonOptionalPartitionSpec(specCtx), None)
}
val ns = if (ctx.identifierReference() != null) {
withIdentClause(ctx.identifierReference, UnresolvedNamespace(_))
} else {
UnresolvedNamespace(Seq.empty[String])
Option(ctx.partitionSpec).map { spec =>
val table = withOrigin(ctx.pattern) {
if (ctx.identifierReference() != null) {
withIdentClause(ctx.identifierReference(), ns => {
val names = ns :+ string(visitStringLit(ctx.pattern))
UnresolvedTable(names, "SHOW TABLE EXTENDED ... PARTITION ...")
})
} else {
val names = Seq.empty[String] :+ string(visitStringLit(ctx.pattern))
UnresolvedTable(names, "SHOW TABLE EXTENDED ... PARTITION ...")
}
}
ShowTablePartition(table, UnresolvedPartitionSpec(visitNonOptionalPartitionSpec(spec)))
}.getOrElse {
val ns = if (ctx.identifierReference() != null) {
withIdentClause(ctx.identifierReference, UnresolvedNamespace)
} else {
UnresolvedNamespace(Seq.empty[String])
}
ShowTablesExtended(ns, string(visitStringLit(ctx.pattern)))
}
ShowTableExtended(ns, string(visitStringLit(ctx.pattern)), partitionKeys)
}

/**
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -886,26 +886,37 @@ object ShowTables {
}

/**
* The logical plan of the SHOW TABLE EXTENDED command.
* The logical plan of the SHOW TABLE EXTENDED (without PARTITION) command.
*/
case class ShowTableExtended(
case class ShowTablesExtended(
namespace: LogicalPlan,
pattern: String,
partitionSpec: Option[PartitionSpec],
override val output: Seq[Attribute] = ShowTableExtended.getOutputAttrs) extends UnaryCommand {
override val output: Seq[Attribute] = ShowTablesUtils.getOutputAttrs) extends UnaryCommand {
override def child: LogicalPlan = namespace
override protected def withNewChildInternal(newChild: LogicalPlan): ShowTableExtended =
override protected def withNewChildInternal(newChild: LogicalPlan): ShowTablesExtended =
copy(namespace = newChild)
}

object ShowTableExtended {
object ShowTablesUtils {
def getOutputAttrs: Seq[Attribute] = Seq(
AttributeReference("namespace", StringType, nullable = false)(),
AttributeReference("tableName", StringType, nullable = false)(),
AttributeReference("isTemporary", BooleanType, nullable = false)(),
AttributeReference("information", StringType, nullable = false)())
}

/**
* The logical plan of the SHOW TABLE EXTENDED ... PARTITION ... command.
*/
case class ShowTablePartition(
table: LogicalPlan,
partitionSpec: PartitionSpec,
override val output: Seq[Attribute] = ShowTablesUtils.getOutputAttrs)
extends V2PartitionCommand {
override protected def withNewChildInternal(newChild: LogicalPlan): ShowTablePartition =
copy(table = newChild)
}

/**
* The logical plan of the SHOW VIEWS command.
*
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -23,7 +23,7 @@ import org.apache.hadoop.fs.Path

import org.apache.spark.{SPARK_DOC_ROOT, SparkException, SparkThrowable, SparkThrowableHelper, SparkUnsupportedOperationException}
import org.apache.spark.sql.AnalysisException
import org.apache.spark.sql.catalyst.{ExtendedAnalysisException, FunctionIdentifier, QualifiedTableName, TableIdentifier}
import org.apache.spark.sql.catalyst.{ExtendedAnalysisException, FunctionIdentifier, InternalRow, QualifiedTableName, TableIdentifier}
import org.apache.spark.sql.catalyst.analysis.{CannotReplaceMissingTableException, FunctionAlreadyExistsException, NamespaceAlreadyExistsException, NoSuchFunctionException, NoSuchNamespaceException, NoSuchPartitionException, NoSuchTableException, ResolvedTable, Star, TableAlreadyExistsException, UnresolvedRegex}
import org.apache.spark.sql.catalyst.catalog.{CatalogTable, InvalidUDFClassException}
import org.apache.spark.sql.catalyst.catalog.CatalogTypes.TablePartitionSpec
Expand Down Expand Up @@ -2139,12 +2139,6 @@ private[sql] object QueryCompilationErrors extends QueryErrorsBase with Compilat
"inputTypesLen" -> bound.inputTypes().length.toString))
}

def commandUnsupportedInV2TableError(name: String): Throwable = {
new AnalysisException(
errorClass = "_LEGACY_ERROR_TEMP_1200",
messageParameters = Map("name" -> name))
}
Copy link
Contributor

Choose a reason for hiding this comment

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

We can remove the error class too.

Copy link
Contributor Author

Choose a reason for hiding this comment

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

To reduce the complexity of this PR, it will be placed in a separate PR to complete the renaming or removing of the error class-related operations.

Copy link
Contributor

Choose a reason for hiding this comment

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

In fact, just remove the error class, it will not increase the complexity.

Copy link
Contributor Author

Choose a reason for hiding this comment

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

Yes, I misunderstood your meaning. That's right, we can remove it from error-classes. json.


def cannotResolveColumnNameAmongAttributesError(
colName: String, fieldNames: String): Throwable = {
new AnalysisException(
Expand Down Expand Up @@ -2471,7 +2465,7 @@ private[sql] object QueryCompilationErrors extends QueryErrorsBase with Compilat
errorClass = "_LEGACY_ERROR_TEMP_1231",
messageParameters = Map(
"key" -> key,
"tblName" -> tblName))
"tblName" -> toSQLId(tblName)))
Copy link
Contributor

Choose a reason for hiding this comment

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

It seems this change is not related to PR.

Copy link
Contributor

Choose a reason for hiding this comment

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

We can change it if we rename the error class.

}

def invalidPartitionSpecError(
Expand All @@ -2483,7 +2477,7 @@ private[sql] object QueryCompilationErrors extends QueryErrorsBase with Compilat
messageParameters = Map(
"specKeys" -> specKeys,
"partitionColumnNames" -> partitionColumnNames.mkString(", "),
"tableName" -> tableName))
"tableName" -> toSQLId(tableName)))
Copy link
Contributor

Choose a reason for hiding this comment

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

ditto

Copy link
Contributor Author

Choose a reason for hiding this comment

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

If this is not modified, the following issues will occur

  • When executes UT org.apache.spark.sql.execution.command.v1.ShowTablesSuite.show table extended in multi partition key - the command's partition parameters are incomplete, its error message is:
    org.apache.spark.sql.AnalysisException: Partition spec is invalid. The spec (id1) must match the partition spec (id1, id2) defined in table '`spark_catalog`.`ns1`.`tbl`'.
    

throw QueryCompilationErrors.invalidPartitionSpecError(
s"The spec (${s.keys.mkString(", ")}) must be contained " +
s"within the partition spec (${table.partitionColumnNames.mkString(", ")}) defined " +
s"in table '${table.identifier}'")

  • When executes org.apache.spark.sql.execution.command.v2.ShowTablesSuite.show table extended in multi partition key - the command's partition parameters are incomplete, its error message is:
    org.apache.spark.sql.AnalysisException: Partition spec is invalid. The spec (id1) must match the partition spec (id1, id2) defined in table 'test_catalog.ns1.tbl'.
    

requireExactMatchedPartitionSpec(tableName, normalizedSpec, partSchema.fieldNames)

The reason is that there are two entrances to calling the method PartitioningUtils.requireExactMatchedPartitionSpec,
image

and the format of the parameter tableName: String inside is not consistent, so it has been unified here. In this way, we can put this UT into show table extended in multi partition key - the command's partition parameters are incomplete to maintain consistency.

Copy link
Contributor

Choose a reason for hiding this comment

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

It seems there are two invalidPartitionSpecError with different signature.
Before this PR, why org.apache.spark.sql.execution.command.v1.ShowTablesSuite.show table extended in multi partition key - the command's partition parameters are incomplete not failed?

Copy link
Contributor Author

Choose a reason for hiding this comment

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

Hmm... this is a newly added UT. 😄

Copy link
Contributor

@beliefer beliefer Oct 31, 2023

Choose a reason for hiding this comment

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

I got it now. In fact, you added the test cases into the base class ShowTablesSuiteBase, so the output is different from v1 and v2.
So we can add this test case into v1.ShowTablesSuite and v2.ShowTablesSuite and avoid add toSQLId.

After this PR, you can create another PR to fix the error class.

Copy link
Contributor

Choose a reason for hiding this comment

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

Let's avoid unrelated change I said.

Copy link
Contributor Author

Choose a reason for hiding this comment

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

@cloud-fan
Do we really need to split a UT logic into subclasses in order not to modify the above logic? Separate into two subclasses, so it seems that the similarities cannot be seen anymore.

Copy link
Contributor

Choose a reason for hiding this comment

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

This change is necessary as it helps to unify the v1/v2 command behavior, which is an important goal of adding new v2 commands

}

def columnAlreadyExistsError(columnName: String): Throwable = {
Expand Down Expand Up @@ -2541,6 +2535,13 @@ private[sql] object QueryCompilationErrors extends QueryErrorsBase with Compilat
new NoSuchPartitionException(db, table, partition)
}

def notExistPartitionError(
table: Identifier,
partitionIdent: InternalRow,
partitionSchema: StructType): Throwable = {
new NoSuchPartitionException(table.toString, partitionIdent, partitionSchema)
}

def analyzingColumnStatisticsNotSupportedForColumnTypeError(
name: String,
dataType: DataType): Throwable = {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -65,6 +65,11 @@ object DataSourceV2Implicits {
}
}

def supportsPartitions: Boolean = table match {
case _: SupportsPartitionManagement => true
case _ => false
}

def asPartitionable: SupportsPartitionManagement = {
table match {
case support: SupportsPartitionManagement =>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -241,19 +241,31 @@ class ResolveSessionCatalog(val catalogManager: CatalogManager)
case ShowTables(DatabaseInSessionCatalog(db), pattern, output) if conf.useV1Command =>
ShowTablesCommand(Some(db), pattern, output)

case ShowTableExtended(
case ShowTablesExtended(
DatabaseInSessionCatalog(db),
pattern,
partitionSpec @ (None | Some(UnresolvedPartitionSpec(_, _))),
output) =>
val newOutput = if (conf.getConf(SQLConf.LEGACY_KEEP_COMMAND_OUTPUT_SCHEMA)) {
assert(output.length == 4)
output.head.withName("database") +: output.tail
} else {
output
}
val tablePartitionSpec = partitionSpec.map(_.asInstanceOf[UnresolvedPartitionSpec].spec)
ShowTablesCommand(Some(db), Some(pattern), newOutput, true, tablePartitionSpec)
ShowTablesCommand(Some(db), Some(pattern), newOutput, isExtended = true)

case ShowTablePartition(
ResolvedTable(catalog, _, table: V1Table, _),
partitionSpec,
output) if isSessionCatalog(catalog) =>
val newOutput = if (conf.getConf(SQLConf.LEGACY_KEEP_COMMAND_OUTPUT_SCHEMA)) {
output.head.withName("database") +: output.tail
} else {
output
}
val tablePartitionSpec = Option(partitionSpec).map(
_.asInstanceOf[UnresolvedPartitionSpec].spec)
ShowTablesCommand(table.catalogTable.identifier.database,
Some(table.catalogTable.identifier.table), newOutput,
isExtended = true, tablePartitionSpec)

// ANALYZE TABLE works on permanent views if the views are cached.
case AnalyzeTable(ResolvedV1TableOrViewIdentifier(ident), partitionSpec, noScan) =>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -406,6 +406,16 @@ class DataSourceV2Strategy(session: SparkSession) extends Strategy with Predicat
case ShowTables(ResolvedNamespace(catalog, ns), pattern, output) =>
ShowTablesExec(output, catalog.asTableCatalog, ns, pattern) :: Nil

case ShowTablesExtended(
ResolvedNamespace(catalog, ns),
pattern,
output) =>
ShowTablesExtendedExec(output, catalog.asTableCatalog, ns, pattern) :: Nil

case ShowTablePartition(r: ResolvedTable, part, output) =>
ShowTablePartitionExec(output, r.catalog, r.identifier,
r.table.asPartitionable, Seq(part).asResolvedPartitionSpecs.head) :: Nil

case SetCatalogAndNamespace(ResolvedNamespace(catalog, ns)) =>
val catalogManager = session.sessionState.catalogManager
val namespace = if (ns.nonEmpty) Some(ns) else None
Expand Down
Loading