You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by GitBox <gi...@apache.org> on 2021/02/25 18:21:31 UTC

[GitHub] [spark] imback82 opened a new pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

imback82 opened a new pull request #31652:
URL: https://github.com/apache/spark/pull/31652


   <!--
   Thanks for sending a pull request!  Here are some tips for you:
     1. If this is your first time, please read our contributor guidelines: https://spark.apache.org/contributing.html
     2. Ensure you have added or run the appropriate tests for your PR: https://spark.apache.org/developer-tools.html
     3. If the PR is unfinished, add '[WIP]' in your PR title, e.g., '[WIP][SPARK-XXXX] Your PR title ...'.
     4. Be sure to keep the PR description updated to reflect all changes.
     5. Please write your PR title to summarize what this PR proposes.
     6. If possible, provide a concise example to reproduce the issue for a faster review.
     7. If you want to add a new configuration, please read the guideline first for naming configurations in
        'core/src/main/scala/org/apache/spark/internal/config/ConfigEntry.scala'.
   -->
   
   ### What changes were proposed in this pull request?
   <!--
   Please clarify what changes you are proposing. The purpose of this section is to outline the changes and how this PR fixes the issue. 
   If possible, please consider writing useful notes for better and faster reviews in your PR. See the examples below.
     1. If you refactor some codes with changing classes, showing the class hierarchy will help reviewers.
     2. If you fix some SQL features, you can provide some references of other DBMSes.
     3. If there is design documentation, please add the link.
     4. If there is a discussion in the mailing list, please add the link.
   -->
   `AlterViewAs.query` is currently analyzed in the physical operator `AlterViewAsCommand`, but it should be analyzed during the analysis phase.
   
   ### Why are the changes needed?
   <!--
   Please clarify why the changes are needed. For instance,
     1. If you propose a new API, clarify the use case for a new API.
     2. If you fix a bug, you can clarify why it is a bug.
   -->
   Analyzing a plan should be done in the analysis phase if possible.
   
   ### Does this PR introduce _any_ user-facing change?
   <!--
   Note that it means *any* user-facing change including all aspects such as the documentation fix.
   If yes, please clarify the previous behavior and the change this PR proposes - provide the console output, description and/or an example to show the behavior difference if possible.
   If possible, please also clarify if this is a user-facing change compared to the released Spark versions or within the unreleased branches such as master.
   If no, write 'No'.
   -->
   No. The PR doesn't affect the behavior, just moving up the analysis to an earlier stage.
   
   ### How was this patch tested?
   <!--
   If tests were added, say they were added here. Please make sure to add some test cases that check the changes thoroughly including negative and positive cases if possible.
   If it was tested in a way different from regular unit tests, please clarify how you tested step by step, ideally copy and paste-able, so that other reviewers can test and check, and descendants can verify in the future.
   If tests were not added, please describe why they were not added and/or why it was difficult to add.
   -->
   The existing tests such as `SQLViewSuite` should cover the new changes.


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-796380796


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/135946/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-796380796


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/135946/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-794896899






----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-786281232


   **[Test build #135475 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135475/testReport)** for PR 31652 at commit [`ba766f7`](https://github.com/apache/spark/commit/ba766f74c218d97ca95b030f98049bb86b6fabd6).
    * This patch passes all tests.
    * This patch merges cleanly.
    * This patch adds no public classes.


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-796278133


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder-K8s/40530/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-786182938


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder-K8s/40055/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] cloud-fan commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
cloud-fan commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r591072759



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -596,19 +546,102 @@ object ViewHelper {
     (collectTempViews(child), collectTempFunctions(child))
   }
 
+  /**
+   * Returns a [[TemporaryViewRelation]] that contains information about a temporary view
+   * to create, given an analyzed plan of the view. If a temp view is to be replaced and it is
+   * cached, it will be uncached before being replaced.
+   *
+   * @param name the name of the temporary view to create/replace.
+   * @param session the spark session.
+   * @param replace if true and the existing view is cached, it will be uncached.
+   * @param getRawTempView the function that returns an optional raw plan of the local or
+   *                       global temporary view.
+   * @param originalText the original SQL text of this view, can be None if this view is created via
+   *                     Dataset API or spark.sql.legacy.storeAnalyzedPlanForView is set to true.
+   * @param userSpecifiedColumns the output column names and optional comments specified by users,
+   *                             can be Nil if not specified.
+   * @param analyzedPlan the logical plan that represents the view; this is used to generate the
+   *                     logical plan for temporary view and the view schema.
+   */
+  def createTemporaryViewRelation(
+      name: TableIdentifier,
+      session: SparkSession,
+      replace: Boolean,
+      getRawTempView: String => Option[LogicalPlan],
+      originalText: Option[String],
+      userSpecifiedColumns: Seq[(String, Option[String])],
+      analyzedPlan: LogicalPlan): TemporaryViewRelation = {
+    val aliasedPlan = aliasPlan(session, analyzedPlan, userSpecifiedColumns)
+    val uncache = getRawTempView(name.table).map { r =>
+      needsToUncache(r, aliasedPlan)
+    }.getOrElse(false)
+    if (replace && uncache) {
+      logInfo(s"Try to uncache ${name.quotedString} before replacing.")
+      checkCyclicViewReference(analyzedPlan, Seq(name), name)
+      CommandUtils.uncacheTableOrView(session, name.quotedString)
+    }
+    if (!conf.storeAnalyzedPlanForView && originalText.nonEmpty) {
+      TemporaryViewRelation(
+        prepareTemporaryView(
+          name,
+          session,
+          analyzedPlan,
+          aliasedPlan.schema,
+          originalText.get))
+    } else {
+      TemporaryViewRelation(
+        prepareTemporaryViewStoringAnalyzedPlan(name, aliasedPlan),
+        Some(aliasedPlan))
+    }
+  }
+
+  /**
+   * If `userSpecifiedColumns` is defined, alias the analyzed plan to the user specified columns,
+   * else return the analyzed plan directly.
+   */
+  def aliasPlan(

Review comment:
       shall we still keep this in `CreateViewCommand` and let it pass the `aliasedPlan` to `createTemporaryViewRelation`? This logic is not needed for `AlterViewAsCommand`.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-786110215


   **[Test build #135475 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135475/testReport)** for PR 31652 at commit [`ba766f7`](https://github.com/apache/spark/commit/ba766f74c218d97ca95b030f98049bb86b6fabd6).


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] cloud-fan commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
cloud-fan commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r589382099



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -268,30 +268,32 @@ case class AlterViewAsCommand(
     name: TableIdentifier,
     originalText: String,
     query: LogicalPlan) extends RunnableCommand {
+  require(query.resolved)
 
   import ViewHelper._
 
   override def innerChildren: Seq[QueryPlan[_]] = Seq(query)
 
   override def run(session: SparkSession): Seq[Row] = {
-    // If the plan cannot be analyzed, throw an exception and don't proceed.
-    val qe = session.sessionState.executePlan(query)
-    qe.assertAnalyzed()
-    val analyzedPlan = qe.analyzed
-
     if (session.sessionState.catalog.isTempView(name)) {
-      alterTemporaryView(session, analyzedPlan)
+      alterTemporaryView(session, query)
     } else {
-      alterPermanentView(session, analyzedPlan)
+      alterPermanentView(session, query)
     }
     Seq.empty[Row]
   }
 
   private def alterTemporaryView(session: SparkSession, analyzedPlan: LogicalPlan): Unit = {
+    checkCyclicViewReference(analyzedPlan, Seq(name), name)
+
+    logDebug(s"Try to uncache ${name.quotedString} before altering.")
+    CommandUtils.uncacheTableOrView(session, name.quotedString)

Review comment:
       yea I think so, since the cached data is stale. It's similar to CREATE OR REPLACE TEMP VIEW, seems we have an extra check `needsToUncache`. Shall we do it here as well?




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterView should invalidate the cache

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792428162


   **[Test build #135849 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135849/testReport)** for PR 31652 at commit [`51c66e8`](https://github.com/apache/spark/commit/51c66e881eec8f64b85654d5156da82f9098f7d0).
    * This patch **fails PySpark unit tests**.
    * This patch merges cleanly.
    * This patch adds no public classes.


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-795944068






----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-794919044


   **[Test build #135921 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135921/testReport)** for PR 31652 at commit [`9302be9`](https://github.com/apache/spark/commit/9302be9b872e816f9470bd0ef88cb1f7c82ee4c2).


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-795201546


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/135921/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] imback82 commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
imback82 commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r589544449



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -268,30 +268,32 @@ case class AlterViewAsCommand(
     name: TableIdentifier,
     originalText: String,
     query: LogicalPlan) extends RunnableCommand {
+  require(query.resolved)
 
   import ViewHelper._
 
   override def innerChildren: Seq[QueryPlan[_]] = Seq(query)
 
   override def run(session: SparkSession): Seq[Row] = {
-    // If the plan cannot be analyzed, throw an exception and don't proceed.
-    val qe = session.sessionState.executePlan(query)
-    qe.assertAnalyzed()
-    val analyzedPlan = qe.analyzed

Review comment:
       We cannot yet because of the following: https://github.com/apache/spark/blob/f340857757034ac955862b34e60322ca5ee81758/sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/v2/CacheTableExec.scala#L96-L105
   
   Here, `query` is not resolved yet. I will get to this though.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-786182938


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder-K8s/40055/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-796176563


   **[Test build #135946 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135946/testReport)** for PR 31652 at commit [`7e4815a`](https://github.com/apache/spark/commit/7e4815a03d0d423ac377b6c3f7ce05f102ab0087).


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterView should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792428808


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/135849/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA removed a comment on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterView should invalidate the cache

Posted by GitBox <gi...@apache.org>.
SparkQA removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792397775


   **[Test build #135849 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135849/testReport)** for PR 31652 at commit [`51c66e8`](https://github.com/apache/spark/commit/51c66e881eec8f64b85654d5156da82f9098f7d0).


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-786301980


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/135475/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-796245549


   Kubernetes integration test starting
   URL: https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder-K8s/40530/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] cloud-fan commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
cloud-fan commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r589382947



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -268,30 +268,32 @@ case class AlterViewAsCommand(
     name: TableIdentifier,
     originalText: String,
     query: LogicalPlan) extends RunnableCommand {
+  require(query.resolved)
 
   import ViewHelper._
 
   override def innerChildren: Seq[QueryPlan[_]] = Seq(query)
 
   override def run(session: SparkSession): Seq[Row] = {
-    // If the plan cannot be analyzed, throw an exception and don't proceed.
-    val qe = session.sessionState.executePlan(query)
-    qe.assertAnalyzed()
-    val analyzedPlan = qe.analyzed

Review comment:
       shall we remove the similar code in `CreateViewCommand`?




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] imback82 commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
imback82 commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r591735546



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -596,19 +546,102 @@ object ViewHelper {
     (collectTempViews(child), collectTempFunctions(child))
   }
 
+  /**
+   * Returns a [[TemporaryViewRelation]] that contains information about a temporary view
+   * to create, given an analyzed plan of the view. If a temp view is to be replaced and it is
+   * cached, it will be uncached before being replaced.
+   *
+   * @param name the name of the temporary view to create/replace.
+   * @param session the spark session.
+   * @param replace if true and the existing view is cached, it will be uncached.
+   * @param getRawTempView the function that returns an optional raw plan of the local or
+   *                       global temporary view.
+   * @param originalText the original SQL text of this view, can be None if this view is created via
+   *                     Dataset API or spark.sql.legacy.storeAnalyzedPlanForView is set to true.
+   * @param userSpecifiedColumns the output column names and optional comments specified by users,
+   *                             can be Nil if not specified.
+   * @param analyzedPlan the logical plan that represents the view; this is used to generate the
+   *                     logical plan for temporary view and the view schema.
+   */
+  def createTemporaryViewRelation(
+      name: TableIdentifier,
+      session: SparkSession,
+      replace: Boolean,
+      getRawTempView: String => Option[LogicalPlan],
+      originalText: Option[String],
+      userSpecifiedColumns: Seq[(String, Option[String])],
+      analyzedPlan: LogicalPlan): TemporaryViewRelation = {
+    val aliasedPlan = aliasPlan(session, analyzedPlan, userSpecifiedColumns)
+    val uncache = getRawTempView(name.table).map { r =>
+      needsToUncache(r, aliasedPlan)
+    }.getOrElse(false)
+    if (replace && uncache) {
+      logInfo(s"Try to uncache ${name.quotedString} before replacing.")
+      checkCyclicViewReference(analyzedPlan, Seq(name), name)
+      CommandUtils.uncacheTableOrView(session, name.quotedString)
+    }
+    if (!conf.storeAnalyzedPlanForView && originalText.nonEmpty) {
+      TemporaryViewRelation(
+        prepareTemporaryView(
+          name,
+          session,
+          analyzedPlan,
+          aliasedPlan.schema,
+          originalText.get))
+    } else {
+      TemporaryViewRelation(
+        prepareTemporaryViewStoringAnalyzedPlan(name, aliasedPlan),
+        Some(aliasedPlan))
+    }
+  }
+
+  /**
+   * If `userSpecifiedColumns` is defined, alias the analyzed plan to the user specified columns,
+   * else return the analyzed plan directly.
+   */
+  def aliasPlan(
+      session: SparkSession,
+      analyzedPlan: LogicalPlan,
+      userSpecifiedColumns: Seq[(String, Option[String])]): LogicalPlan = {
+    if (userSpecifiedColumns.isEmpty) {
+      analyzedPlan
+    } else {
+      val projectList = analyzedPlan.output.zip(userSpecifiedColumns).map {
+        case (attr, (colName, None)) => Alias(attr, colName)()
+        case (attr, (colName, Some(colComment))) =>
+          val meta = new MetadataBuilder().putString("comment", colComment).build()
+          Alias(attr, colName)(explicitMetadata = Some(meta))
+      }
+      session.sessionState.executePlan(Project(projectList, analyzedPlan)).analyzed
+    }
+  }
+
+  /**
+   * Checks if need to uncache the temp view being replaced.
+   */
+  private def needsToUncache(
+      rawTempView: LogicalPlan,
+      aliasedPlan: LogicalPlan): Boolean = rawTempView match {
+    // If TemporaryViewRelation doesn't store the analyzed view, always uncache.
+    case TemporaryViewRelation(_, None) => true
+    // Do not need to uncache if the to-be-replaced temp view plan and the new plan are the
+    // same-result plans.
+    case TemporaryViewRelation(_, Some(p)) => !p.sameResult(aliasedPlan)
+    case p => !p.sameResult(aliasedPlan)

Review comment:
       We need to first update `CREATE TEMP VIEW USING`:
   https://github.com/apache/spark/blob/3a299aa6480ac22501512cd0310d31a441d7dfdc/sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/ddl.scala#L93-L100
   , which I will get to right after this PR.
   
   Once the PR is done, we can update `createTempView` to take in `TemporaryViewRelation` (https://github.com/apache/spark/pull/31273/files#r580757641), then I can safely remove this line.
   
   Let me actually create a JIRA to capture these as subtasks.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-795181598


   **[Test build #135921 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135921/testReport)** for PR 31652 at commit [`9302be9`](https://github.com/apache/spark/commit/9302be9b872e816f9470bd0ef88cb1f7c82ee4c2).
    * This patch passes all tests.
    * This patch merges cleanly.
    * This patch adds no public classes.


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] imback82 commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
imback82 commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r590988931



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -192,38 +170,6 @@ case class CreateViewCommand(
     Seq.empty[Row]
   }
 
-  /**
-   * Checks if need to uncache the temp view being replaced.
-   */
-  private def needsToUncache(

Review comment:
       This is moved to `ViewHelper` and renamed to `produceSameResult` with a slight change.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792224042


   Kubernetes integration test starting
   URL: https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder-K8s/40421/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-796278133


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder-K8s/40530/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-796373992


   **[Test build #135946 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135946/testReport)** for PR 31652 at commit [`7e4815a`](https://github.com/apache/spark/commit/7e4815a03d0d423ac377b6c3f7ce05f102ab0087).
    * This patch passes all tests.
    * This patch merges cleanly.
    * This patch adds no public classes.


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] imback82 commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
imback82 commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r591029660



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -596,19 +546,102 @@ object ViewHelper {
     (collectTempViews(child), collectTempFunctions(child))
   }
 
+  /**
+   * Returns a [[TemporaryViewRelation]] that contains information about a temporary view
+   * to create, given an analyzed plan of the view. If a temp view is to be replaced and it is
+   * cached, it will be uncached before being replaced.
+   *
+   * @param name the name of the temporary view to create/replace.
+   * @param session the spark session.
+   * @param replace if true and the existing view is cached, it will be uncached.
+   * @param getRawTempView the function that returns an optional raw plan of the local or
+   *                       global temporary view.
+   * @param originalText the original SQL text of this view, can be None if this view is created via
+   *                     Dataset API or spark.sql.legacy.storeAnalyzedPlanForView is set to true.
+   * @param userSpecifiedColumns the output column names and optional comments specified by users,
+   *                             can be Nil if not specified.
+   * @param analyzedPlan the logical plan that represents the view; this is used to generate the
+   *                     logical plan for temporary view and the view schema.
+   */
+  def createTemporaryViewRelation(
+      name: TableIdentifier,
+      session: SparkSession,
+      replace: Boolean,
+      getRawTempView: String => Option[LogicalPlan],
+      originalText: Option[String],
+      userSpecifiedColumns: Seq[(String, Option[String])],
+      analyzedPlan: LogicalPlan): TemporaryViewRelation = {
+    val aliasedPlan = aliasPlan(session, analyzedPlan, userSpecifiedColumns)
+    val uncache = getRawTempView(name.table).map { r =>
+      needsToUncache(r, aliasedPlan)
+    }.getOrElse(false)
+    if (replace && uncache) {
+      logInfo(s"Try to uncache ${name.quotedString} before replacing.")
+      checkCyclicViewReference(analyzedPlan, Seq(name), name)
+      CommandUtils.uncacheTableOrView(session, name.quotedString)
+    }
+    if (!conf.storeAnalyzedPlanForView && originalText.nonEmpty) {
+      TemporaryViewRelation(
+        prepareTemporaryView(
+          name,
+          session,
+          analyzedPlan,
+          aliasedPlan.schema,
+          originalText.get))
+    } else {
+      TemporaryViewRelation(
+        prepareTemporaryViewStoringAnalyzedPlan(name, aliasedPlan),
+        Some(aliasedPlan))
+    }
+  }
+
+  /**
+   * If `userSpecifiedColumns` is defined, alias the analyzed plan to the user specified columns,
+   * else return the analyzed plan directly.
+   */
+  def aliasPlan(
+      session: SparkSession,
+      analyzedPlan: LogicalPlan,
+      userSpecifiedColumns: Seq[(String, Option[String])]): LogicalPlan = {
+    if (userSpecifiedColumns.isEmpty) {
+      analyzedPlan
+    } else {
+      val projectList = analyzedPlan.output.zip(userSpecifiedColumns).map {
+        case (attr, (colName, None)) => Alias(attr, colName)()
+        case (attr, (colName, Some(colComment))) =>
+          val meta = new MetadataBuilder().putString("comment", colComment).build()
+          Alias(attr, colName)(explicitMetadata = Some(meta))
+      }
+      session.sessionState.executePlan(Project(projectList, analyzedPlan)).analyzed
+    }
+  }
+
+  /**
+   * Checks if need to uncache the temp view being replaced.
+   */
+  private def needsToUncache(
+      rawTempView: LogicalPlan,
+      aliasedPlan: LogicalPlan): Boolean = rawTempView match {
+    // If TemporaryViewRelation doesn't store the analyzed view, always uncache.
+    case TemporaryViewRelation(_, None) => true

Review comment:
       This is unnecessary because `case p => !p.sameResult(aliasedPlan)` should cover, but I added this case explicitly to make the intention clear.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] cloud-fan commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
cloud-fan commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r591074003



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -596,19 +546,102 @@ object ViewHelper {
     (collectTempViews(child), collectTempFunctions(child))
   }
 
+  /**
+   * Returns a [[TemporaryViewRelation]] that contains information about a temporary view
+   * to create, given an analyzed plan of the view. If a temp view is to be replaced and it is
+   * cached, it will be uncached before being replaced.
+   *
+   * @param name the name of the temporary view to create/replace.
+   * @param session the spark session.
+   * @param replace if true and the existing view is cached, it will be uncached.
+   * @param getRawTempView the function that returns an optional raw plan of the local or
+   *                       global temporary view.
+   * @param originalText the original SQL text of this view, can be None if this view is created via
+   *                     Dataset API or spark.sql.legacy.storeAnalyzedPlanForView is set to true.
+   * @param userSpecifiedColumns the output column names and optional comments specified by users,
+   *                             can be Nil if not specified.
+   * @param analyzedPlan the logical plan that represents the view; this is used to generate the
+   *                     logical plan for temporary view and the view schema.
+   */
+  def createTemporaryViewRelation(
+      name: TableIdentifier,
+      session: SparkSession,
+      replace: Boolean,
+      getRawTempView: String => Option[LogicalPlan],
+      originalText: Option[String],
+      userSpecifiedColumns: Seq[(String, Option[String])],
+      analyzedPlan: LogicalPlan): TemporaryViewRelation = {
+    val aliasedPlan = aliasPlan(session, analyzedPlan, userSpecifiedColumns)
+    val uncache = getRawTempView(name.table).map { r =>
+      needsToUncache(r, aliasedPlan)
+    }.getOrElse(false)
+    if (replace && uncache) {
+      logInfo(s"Try to uncache ${name.quotedString} before replacing.")
+      checkCyclicViewReference(analyzedPlan, Seq(name), name)
+      CommandUtils.uncacheTableOrView(session, name.quotedString)
+    }
+    if (!conf.storeAnalyzedPlanForView && originalText.nonEmpty) {
+      TemporaryViewRelation(
+        prepareTemporaryView(
+          name,
+          session,
+          analyzedPlan,
+          aliasedPlan.schema,
+          originalText.get))
+    } else {
+      TemporaryViewRelation(
+        prepareTemporaryViewStoringAnalyzedPlan(name, aliasedPlan),
+        Some(aliasedPlan))
+    }
+  }
+
+  /**
+   * If `userSpecifiedColumns` is defined, alias the analyzed plan to the user specified columns,
+   * else return the analyzed plan directly.
+   */
+  def aliasPlan(
+      session: SparkSession,
+      analyzedPlan: LogicalPlan,
+      userSpecifiedColumns: Seq[(String, Option[String])]): LogicalPlan = {
+    if (userSpecifiedColumns.isEmpty) {
+      analyzedPlan
+    } else {
+      val projectList = analyzedPlan.output.zip(userSpecifiedColumns).map {
+        case (attr, (colName, None)) => Alias(attr, colName)()
+        case (attr, (colName, Some(colComment))) =>
+          val meta = new MetadataBuilder().putString("comment", colComment).build()
+          Alias(attr, colName)(explicitMetadata = Some(meta))
+      }
+      session.sessionState.executePlan(Project(projectList, analyzedPlan)).analyzed
+    }
+  }
+
+  /**
+   * Checks if need to uncache the temp view being replaced.
+   */
+  private def needsToUncache(
+      rawTempView: LogicalPlan,
+      aliasedPlan: LogicalPlan): Boolean = rawTempView match {
+    // If TemporaryViewRelation doesn't store the analyzed view, always uncache.
+    case TemporaryViewRelation(_, None) => true
+    // Do not need to uncache if the to-be-replaced temp view plan and the new plan are the
+    // same-result plans.
+    case TemporaryViewRelation(_, Some(p)) => !p.sameResult(aliasedPlan)
+    case p => !p.sameResult(aliasedPlan)

Review comment:
       Do we still need it? `rawTempView` should always be `TemporaryViewRelation` right?




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] imback82 commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
imback82 commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r590988252



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -192,38 +170,6 @@ case class CreateViewCommand(
     Seq.empty[Row]
   }
 
-  /**
-   * Checks if need to uncache the temp view being replaced.
-   */
-  private def needsToUncache(
-      rawTempView: Option[LogicalPlan],
-      aliasedPlan: LogicalPlan): Boolean = rawTempView match {
-    // The temp view doesn't exist, no need to uncache.
-    case None => false
-    // Do not need to uncache if the to-be-replaced temp view plan and the new plan are the
-    // same-result plans.
-    case Some(TemporaryViewRelation(_, Some(p))) => !p.sameResult(aliasedPlan)
-    case Some(p) => !p.sameResult(aliasedPlan)
-  }
-
-  /**
-   * If `userSpecifiedColumns` is defined, alias the analyzed plan to the user specified columns,
-   * else return the analyzed plan directly.
-   */
-  private def aliasPlan(session: SparkSession, analyzedPlan: LogicalPlan): LogicalPlan = {

Review comment:
       This is moved to `ViewHelper`.

##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -268,30 +268,32 @@ case class AlterViewAsCommand(
     name: TableIdentifier,
     originalText: String,
     query: LogicalPlan) extends RunnableCommand {
+  require(query.resolved)
 
   import ViewHelper._
 
   override def innerChildren: Seq[QueryPlan[_]] = Seq(query)
 
   override def run(session: SparkSession): Seq[Row] = {
-    // If the plan cannot be analyzed, throw an exception and don't proceed.
-    val qe = session.sessionState.executePlan(query)
-    qe.assertAnalyzed()
-    val analyzedPlan = qe.analyzed
-
     if (session.sessionState.catalog.isTempView(name)) {
-      alterTemporaryView(session, analyzedPlan)
+      alterTemporaryView(session, query)
     } else {
-      alterPermanentView(session, analyzedPlan)
+      alterPermanentView(session, query)
     }
     Seq.empty[Row]
   }
 
   private def alterTemporaryView(session: SparkSession, analyzedPlan: LogicalPlan): Unit = {
+    checkCyclicViewReference(analyzedPlan, Seq(name), name)
+
+    logDebug(s"Try to uncache ${name.quotedString} before altering.")
+    CommandUtils.uncacheTableOrView(session, name.quotedString)

Review comment:
       Since the logic is very similar, I refactored this part to share code with `CreateViewCommand`.

##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -192,38 +170,6 @@ case class CreateViewCommand(
     Seq.empty[Row]
   }
 
-  /**
-   * Checks if need to uncache the temp view being replaced.
-   */
-  private def needsToUncache(

Review comment:
       This is moved to `ViewHelper` and renamed to `produceSameResult` with a slight change.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] imback82 commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
imback82 commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r591800082



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -596,19 +546,102 @@ object ViewHelper {
     (collectTempViews(child), collectTempFunctions(child))
   }
 
+  /**
+   * Returns a [[TemporaryViewRelation]] that contains information about a temporary view
+   * to create, given an analyzed plan of the view. If a temp view is to be replaced and it is
+   * cached, it will be uncached before being replaced.
+   *
+   * @param name the name of the temporary view to create/replace.
+   * @param session the spark session.
+   * @param replace if true and the existing view is cached, it will be uncached.
+   * @param getRawTempView the function that returns an optional raw plan of the local or
+   *                       global temporary view.
+   * @param originalText the original SQL text of this view, can be None if this view is created via
+   *                     Dataset API or spark.sql.legacy.storeAnalyzedPlanForView is set to true.
+   * @param userSpecifiedColumns the output column names and optional comments specified by users,
+   *                             can be Nil if not specified.
+   * @param analyzedPlan the logical plan that represents the view; this is used to generate the
+   *                     logical plan for temporary view and the view schema.
+   */
+  def createTemporaryViewRelation(
+      name: TableIdentifier,
+      session: SparkSession,
+      replace: Boolean,
+      getRawTempView: String => Option[LogicalPlan],
+      originalText: Option[String],
+      userSpecifiedColumns: Seq[(String, Option[String])],
+      analyzedPlan: LogicalPlan): TemporaryViewRelation = {
+    val aliasedPlan = aliasPlan(session, analyzedPlan, userSpecifiedColumns)
+    val uncache = getRawTempView(name.table).map { r =>
+      needsToUncache(r, aliasedPlan)
+    }.getOrElse(false)
+    if (replace && uncache) {
+      logInfo(s"Try to uncache ${name.quotedString} before replacing.")
+      checkCyclicViewReference(analyzedPlan, Seq(name), name)
+      CommandUtils.uncacheTableOrView(session, name.quotedString)
+    }
+    if (!conf.storeAnalyzedPlanForView && originalText.nonEmpty) {
+      TemporaryViewRelation(
+        prepareTemporaryView(
+          name,
+          session,
+          analyzedPlan,
+          aliasedPlan.schema,
+          originalText.get))
+    } else {
+      TemporaryViewRelation(
+        prepareTemporaryViewStoringAnalyzedPlan(name, aliasedPlan),
+        Some(aliasedPlan))
+    }
+  }
+
+  /**
+   * If `userSpecifiedColumns` is defined, alias the analyzed plan to the user specified columns,
+   * else return the analyzed plan directly.
+   */
+  def aliasPlan(
+      session: SparkSession,
+      analyzedPlan: LogicalPlan,
+      userSpecifiedColumns: Seq[(String, Option[String])]): LogicalPlan = {
+    if (userSpecifiedColumns.isEmpty) {
+      analyzedPlan
+    } else {
+      val projectList = analyzedPlan.output.zip(userSpecifiedColumns).map {
+        case (attr, (colName, None)) => Alias(attr, colName)()
+        case (attr, (colName, Some(colComment))) =>
+          val meta = new MetadataBuilder().putString("comment", colComment).build()
+          Alias(attr, colName)(explicitMetadata = Some(meta))
+      }
+      session.sessionState.executePlan(Project(projectList, analyzedPlan)).analyzed
+    }
+  }
+
+  /**
+   * Checks if need to uncache the temp view being replaced.
+   */
+  private def needsToUncache(
+      rawTempView: LogicalPlan,
+      aliasedPlan: LogicalPlan): Boolean = rawTempView match {
+    // If TemporaryViewRelation doesn't store the analyzed view, always uncache.
+    case TemporaryViewRelation(_, None) => true
+    // Do not need to uncache if the to-be-replaced temp view plan and the new plan are the
+    // same-result plans.
+    case TemporaryViewRelation(_, Some(p)) => !p.sameResult(aliasedPlan)
+    case p => !p.sameResult(aliasedPlan)

Review comment:
       I created https://issues.apache.org/jira/browse/SPARK-34698 to track these.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] cloud-fan commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
cloud-fan commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-796467635


   thanks, merging to master!


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-786301980


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/135475/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] cloud-fan commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
cloud-fan commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r591067365



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -596,19 +546,102 @@ object ViewHelper {
     (collectTempViews(child), collectTempFunctions(child))
   }
 
+  /**
+   * Returns a [[TemporaryViewRelation]] that contains information about a temporary view
+   * to create, given an analyzed plan of the view. If a temp view is to be replaced and it is
+   * cached, it will be uncached before being replaced.
+   *
+   * @param name the name of the temporary view to create/replace.
+   * @param session the spark session.
+   * @param replace if true and the existing view is cached, it will be uncached.
+   * @param getRawTempView the function that returns an optional raw plan of the local or
+   *                       global temporary view.
+   * @param originalText the original SQL text of this view, can be None if this view is created via
+   *                     Dataset API or spark.sql.legacy.storeAnalyzedPlanForView is set to true.
+   * @param userSpecifiedColumns the output column names and optional comments specified by users,
+   *                             can be Nil if not specified.
+   * @param analyzedPlan the logical plan that represents the view; this is used to generate the
+   *                     logical plan for temporary view and the view schema.
+   */
+  def createTemporaryViewRelation(
+      name: TableIdentifier,
+      session: SparkSession,
+      replace: Boolean,
+      getRawTempView: String => Option[LogicalPlan],
+      originalText: Option[String],
+      userSpecifiedColumns: Seq[(String, Option[String])],
+      analyzedPlan: LogicalPlan): TemporaryViewRelation = {
+    val aliasedPlan = aliasPlan(session, analyzedPlan, userSpecifiedColumns)
+    val uncache = getRawTempView(name.table).map { r =>
+      needsToUncache(r, aliasedPlan)
+    }.getOrElse(false)
+    if (replace && uncache) {
+      logInfo(s"Try to uncache ${name.quotedString} before replacing.")

Review comment:
       logDebug?




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792229808


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder-K8s/40421/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA removed a comment on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
SparkQA removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-796176563


   **[Test build #135946 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135946/testReport)** for PR 31652 at commit [`7e4815a`](https://github.com/apache/spark/commit/7e4815a03d0d423ac377b6c3f7ce05f102ab0087).


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] cloud-fan closed pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
cloud-fan closed pull request #31652:
URL: https://github.com/apache/spark/pull/31652


   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterView should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792428717


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder-K8s/40431/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA removed a comment on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
SparkQA removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-794919044


   **[Test build #135921 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135921/testReport)** for PR 31652 at commit [`9302be9`](https://github.com/apache/spark/commit/9302be9b872e816f9470bd0ef88cb1f7c82ee4c2).


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-795201546


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/135921/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterView should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792428717


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder-K8s/40431/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792223756


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/135839/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792223756


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/135839/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterView should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792428808


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/135849/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA removed a comment on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
SparkQA removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-786110215


   **[Test build #135475 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135475/testReport)** for PR 31652 at commit [`ba766f7`](https://github.com/apache/spark/commit/ba766f74c218d97ca95b030f98049bb86b6fabd6).


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-786144698


   Kubernetes integration test starting
   URL: https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder-K8s/40055/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-795944068






----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterView should invalidate the cache

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792397775


   **[Test build #135849 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/135849/testReport)** for PR 31652 at commit [`51c66e8`](https://github.com/apache/spark/commit/51c66e881eec8f64b85654d5156da82f9098f7d0).


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] imback82 commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterView should invalidate the cache

Posted by GitBox <gi...@apache.org>.
imback82 commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r589113144



##########
File path: sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/Analyzer.scala
##########
@@ -879,7 +879,7 @@ class Analyzer(override val catalogManager: CatalogManager)
 
   private def unwrapRelationPlan(plan: LogicalPlan): LogicalPlan = {
     EliminateSubqueryAliases(plan) match {
-      case v: View if v.isDataFrameTempView => v.child
+      case v: View if v.isTempViewStoringAnalyzedPlan => v.child

Review comment:
       I renamed because this type of view can also be created not from a dataframe - e.g., `ALTER VIEW ... AS` with `spark.sql.legacy.storeAnalyzedPlanForView` set to true.

##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -268,30 +268,32 @@ case class AlterViewAsCommand(
     name: TableIdentifier,
     originalText: String,
     query: LogicalPlan) extends RunnableCommand {
+  require(query.resolved)
 
   import ViewHelper._
 
   override def innerChildren: Seq[QueryPlan[_]] = Seq(query)
 
   override def run(session: SparkSession): Seq[Row] = {
-    // If the plan cannot be analyzed, throw an exception and don't proceed.
-    val qe = session.sessionState.executePlan(query)
-    qe.assertAnalyzed()
-    val analyzedPlan = qe.analyzed

Review comment:
       `query` will be an analyzed plan since `AlterViewAs` is converted to `AlterViewAsCommand` in Analyzer phase (`ResolveSessionCatalog`).

##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -268,30 +268,32 @@ case class AlterViewAsCommand(
     name: TableIdentifier,
     originalText: String,
     query: LogicalPlan) extends RunnableCommand {
+  require(query.resolved)
 
   import ViewHelper._
 
   override def innerChildren: Seq[QueryPlan[_]] = Seq(query)
 
   override def run(session: SparkSession): Seq[Row] = {
-    // If the plan cannot be analyzed, throw an exception and don't proceed.
-    val qe = session.sessionState.executePlan(query)
-    qe.assertAnalyzed()
-    val analyzedPlan = qe.analyzed
-
     if (session.sessionState.catalog.isTempView(name)) {
-      alterTemporaryView(session, analyzedPlan)
+      alterTemporaryView(session, query)
     } else {
-      alterPermanentView(session, analyzedPlan)
+      alterPermanentView(session, query)
     }
     Seq.empty[Row]
   }
 
   private def alterTemporaryView(session: SparkSession, analyzedPlan: LogicalPlan): Unit = {
+    checkCyclicViewReference(analyzedPlan, Seq(name), name)
+
+    logDebug(s"Try to uncache ${name.quotedString} before altering.")
+    CommandUtils.uncacheTableOrView(session, name.quotedString)

Review comment:
       @cloud-fan Uncaching seems like a right thing to do when you alter a view?

##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -116,7 +116,7 @@ case class CreateViewCommand(
     if (viewType == LocalTempView) {
       val aliasedPlan = aliasPlan(sparkSession, analyzedPlan)
       if (replace && needsToUncache(catalog.getRawTempView(name.table), aliasedPlan)) {
-        logInfo(s"Try to uncache ${name.quotedString} before replacing.")
+        logDebug(s"Try to uncache ${name.quotedString} before replacing.")

Review comment:
       I changed this to `logDebug` because the permanent view was using it: https://github.com/apache/spark/blob/9ec8696f117a5c43d24aae3188f07ddd91b69d2d/sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala#L174
   
   If `logInfo` is better, I can change these to `logInfo` instead.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792226837


   Kubernetes integration test status success
   URL: https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder-K8s/40421/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-796269982


   Kubernetes integration test status success
   URL: https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder-K8s/40530/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] cloud-fan commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
cloud-fan commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r591075732



##########
File path: sql/core/src/test/scala/org/apache/spark/sql/CachedTableSuite.scala
##########
@@ -1445,4 +1445,68 @@ class CachedTableSuite extends QueryTest with SQLTestUtils
       }
     }
   }
+
+  test("SPARK-34546: ALTER VIEW AS should uncache if a temp view is cached") {
+    Seq(true, false).foreach { storeAnalyzed =>
+      withSQLConf(SQLConf.STORE_ANALYZED_PLAN_FOR_VIEW.key -> storeAnalyzed.toString) {
+        withTempView("tv") {
+          testAlterTemporaryViewAsWithCache(TableIdentifier("tv"), storeAnalyzed)
+        }
+      }
+    }
+  }
+
+  test("SPARK-34546: ALTER VIEW AS should uncache if a global temp view is cached") {
+    Seq(true, false).foreach { storeAnalyzed =>
+      withSQLConf(SQLConf.STORE_ANALYZED_PLAN_FOR_VIEW.key -> storeAnalyzed.toString) {
+        withGlobalTempView("global_tv") {
+          val db = spark.sharedState.globalTempViewManager.database
+          testAlterTemporaryViewAsWithCache(TableIdentifier("global_tv", Some(db)), storeAnalyzed)
+        }
+      }
+    }
+  }
+
+  private def testAlterTemporaryViewAsWithCache(
+      ident: TableIdentifier,
+      storeAnalyzed: Boolean): Unit = {
+    val (tempViewStr, viewName) = if (ident.database.nonEmpty) {
+      ("GLOBAL TEMPORARY", s"${ident.database.get}.${ident.table}")
+    } else {
+      ("TEMPORARY", ident.table)
+    }
+
+    sql(s"CREATE $tempViewStr VIEW ${ident.table} AS SELECT 1")
+
+    sql(s"CACHE TABLE $viewName")
+    assert(spark.catalog.isCached(viewName))
+    assert(spark.sharedState.cacheManager.lookupCachedData(sql("SELECT 1")).nonEmpty)
+
+    if (storeAnalyzed) {
+      // Altered temporary view will have the same plan, thus it will not be uncached.
+      // Note that this check is done only if a temporary view stores an analyzed view.
+      sql(s"ALTER VIEW $viewName as SELECT 1")
+      assert(spark.catalog.isCached(viewName))
+      assert(spark.sharedState.cacheManager.lookupCachedData(sql("SELECT 1")).nonEmpty)
+    }
+
+    sql(s"ALTER VIEW $viewName as SELECT 2")
+    assert(!spark.catalog.isCached(viewName))
+    assert(spark.sharedState.cacheManager.lookupCachedData(sql("SELECT 1")).isEmpty)
+  }
+
+  test("SPARK-34546: ALTER VIEW AS should uncache if a permanent temp view is cached") {

Review comment:
       `permanent temp view` -> `permanent view`




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] SparkQA commented on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
SparkQA commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-786161142


   Kubernetes integration test status success
   URL: https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder-K8s/40055/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] cloud-fan commented on a change in pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
cloud-fan commented on a change in pull request #31652:
URL: https://github.com/apache/spark/pull/31652#discussion_r589381265



##########
File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala
##########
@@ -268,30 +268,32 @@ case class AlterViewAsCommand(
     name: TableIdentifier,
     originalText: String,
     query: LogicalPlan) extends RunnableCommand {
+  require(query.resolved)

Review comment:
       This is not needed. `query` is a child now, and the analyzer will make sure the entire query plan tree is analyzed.




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-794910486


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder-K8s/40504/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins commented on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins commented on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-794910486


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder-K8s/40504/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [WIP][SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-792229808


   
   Refer to this link for build results (access rights to CI server needed): 
   https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder-K8s/40421/
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] AmplabJenkins removed a comment on pull request #31652: [SPARK-34546][SQL] AlterViewAs.query should be analyzed during the analysis phase, and AlterViewAs should invalidate the cache

Posted by GitBox <gi...@apache.org>.
AmplabJenkins removed a comment on pull request #31652:
URL: https://github.com/apache/spark/pull/31652#issuecomment-794896895






----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org