Skip to content

Commit 54f3f70

Browse files
author
RoryQi
committed
[SPARK-36011][SQL][3.0] Disallow altering permanent views based on temporary views or UDFs
1 parent 845e750 commit 54f3f70

File tree

3 files changed

+82
-35
lines changed

3 files changed

+82
-35
lines changed

sql/core/src/main/scala/org/apache/spark/sql/execution/command/views.scala

Lines changed: 41 additions & 35 deletions
Original file line numberDiff line numberDiff line change
@@ -107,7 +107,7 @@ case class CreateViewCommand(
107107

108108
// When creating a permanent view, not allowed to reference temporary objects.
109109
// This should be called after `qe.assertAnalyzed()` (i.e., `child` can be resolved)
110-
verifyTemporaryObjectsNotExists(catalog)
110+
verifyTemporaryObjectsNotExists(catalog, isTemporary, name, child)
111111

112112
if (viewType == LocalTempView) {
113113
val aliasedPlan = aliasPlan(sparkSession, analyzedPlan)
@@ -161,39 +161,7 @@ case class CreateViewCommand(
161161
Seq.empty[Row]
162162
}
163163

164-
/**
165-
* Permanent views are not allowed to reference temp objects, including temp function and views
166-
*/
167-
private def verifyTemporaryObjectsNotExists(catalog: SessionCatalog): Unit = {
168-
import org.apache.spark.sql.connector.catalog.CatalogV2Implicits._
169-
if (!isTemporary) {
170-
// This func traverses the unresolved plan `child`. Below are the reasons:
171-
// 1) Analyzer replaces unresolved temporary views by a SubqueryAlias with the corresponding
172-
// logical plan. After replacement, it is impossible to detect whether the SubqueryAlias is
173-
// added/generated from a temporary view.
174-
// 2) The temp functions are represented by multiple classes. Most are inaccessible from this
175-
// package (e.g., HiveGenericUDF).
176-
def verify(child: LogicalPlan) {
177-
child.collect {
178-
// Disallow creating permanent views based on temporary views.
179-
case UnresolvedRelation(nameParts) if catalog.isTempView(nameParts) =>
180-
throw new AnalysisException(s"Not allowed to create a permanent view $name by " +
181-
s"referencing a temporary view ${nameParts.quoted}. " +
182-
"Please create a temp view instead by CREATE TEMP VIEW")
183-
case w: With if !w.resolved => w.innerChildren.foreach(verify)
184-
case other if !other.resolved => other.expressions.flatMap(_.collect {
185-
// Traverse subquery plan for any unresolved relations.
186-
case e: SubqueryExpression => verify(e.plan)
187-
// Disallow creating permanent views based on temporary UDFs.
188-
case e: UnresolvedFunction if catalog.isTemporaryFunction(e.name) =>
189-
throw new AnalysisException(s"Not allowed to create a permanent view $name by " +
190-
s"referencing a temporary function `${e.name}`")
191-
})
192-
}
193-
}
194-
verify(child)
195-
}
196-
}
164+
197165

198166
/**
199167
* If `userSpecifiedColumns` is defined, alias the analyzed plan to the user specified columns,
@@ -266,7 +234,8 @@ case class AlterViewAsCommand(
266234
val qe = session.sessionState.executePlan(query)
267235
qe.assertAnalyzed()
268236
val analyzedPlan = qe.analyzed
269-
237+
val isTemporary = session.sessionState.catalog.isTemporaryTable(name)
238+
verifyTemporaryObjectsNotExists(session.sessionState.catalog, isTemporary, name, query)
270239
if (session.sessionState.catalog.alterTempViewDefinition(name, analyzedPlan)) {
271240
// a local/global temp view has been altered, we are done.
272241
} else {
@@ -441,4 +410,41 @@ object ViewHelper {
441410
}
442411
}
443412
}
413+
414+
/**
415+
* Permanent views are not allowed to reference temp objects, including temp function and views
416+
*/
417+
def verifyTemporaryObjectsNotExists(catalog: SessionCatalog,
418+
isTemporary: Boolean,
419+
name: TableIdentifier,
420+
child: LogicalPlan): Unit = {
421+
import org.apache.spark.sql.connector.catalog.CatalogV2Implicits._
422+
if (!isTemporary) {
423+
// This func traverses the unresolved plan `child`. Below are the reasons:
424+
// 1) Analyzer replaces unresolved temporary views by a SubqueryAlias with the corresponding
425+
// logical plan. After replacement, it is impossible to detect whether the SubqueryAlias is
426+
// added/generated from a temporary view.
427+
// 2) The temp functions are represented by multiple classes. Most are inaccessible from this
428+
// package (e.g., HiveGenericUDF).
429+
def verify(child: LogicalPlan) {
430+
child.collect {
431+
// Disallow creating permanent views based on temporary views.
432+
case UnresolvedRelation(nameParts) if catalog.isTempView(nameParts) =>
433+
throw new AnalysisException(s"Not allowed to create a permanent view $name by " +
434+
s"referencing a temporary view ${nameParts.quoted}. " +
435+
"Please create a temp view instead by CREATE TEMP VIEW")
436+
case w: With if !w.resolved => w.innerChildren.foreach(verify)
437+
case other if !other.resolved => other.expressions.flatMap(_.collect {
438+
// Traverse subquery plan for any unresolved relations.
439+
case e: SubqueryExpression => verify(e.plan)
440+
// Disallow creating permanent views based on temporary UDFs.
441+
case e: UnresolvedFunction if catalog.isTemporaryFunction(e.name) =>
442+
throw new AnalysisException(s"Not allowed to create a permanent view $name by " +
443+
s"referencing a temporary function `${e.name}`")
444+
})
445+
}
446+
}
447+
verify(child)
448+
}
449+
}
444450
}

sql/core/src/test/scala/org/apache/spark/sql/execution/SQLViewSuite.scala

Lines changed: 16 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -17,6 +17,8 @@
1717

1818
package org.apache.spark.sql.execution
1919

20+
import org.scalatest.Assertions.intercept
21+
2022
import org.apache.spark.sql._
2123
import org.apache.spark.sql.catalyst.TableIdentifier
2224
import org.apache.spark.sql.catalyst.analysis.NoSuchTableException
@@ -795,4 +797,18 @@ abstract class SQLViewSuite extends QueryTest with SQLTestUtils {
795797
}
796798
}
797799
}
800+
801+
test("SPARK-36011: Disallow altering permanent views based on temporary views") {
802+
withView("jtv1") {
803+
withTempView("jtv2") {
804+
sql(s"CREATE VIEW jtv1 AS SELECT * FROM jt WHERE id > 3")
805+
sql(s"CREATE TEMPORARY VIEW jtv2 AS SELECT * FROM jt where id < 3")
806+
val e = intercept[AnalysisException] {
807+
sql(s"ALTER VIEW jtv1 AS SELECT * FROM jtv2")
808+
}.getMessage
809+
assert(e.contains("Not allowed to create a permanent view `default`.`jtv1` by " +
810+
"referencing a temporary view jtv2"))
811+
}
812+
}
813+
}
798814
}

sql/hive/src/test/scala/org/apache/spark/sql/hive/execution/HiveSQLViewSuite.scala

Lines changed: 25 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -17,6 +17,8 @@
1717

1818
package org.apache.spark.sql.hive.execution
1919

20+
import org.scalatest.Assertions.intercept
21+
2022
import org.apache.spark.sql.{AnalysisException, Row, SaveMode, SparkSession}
2123
import org.apache.spark.sql.catalyst.TableIdentifier
2224
import org.apache.spark.sql.catalyst.catalog.{CatalogStorageFormat, CatalogTable, CatalogTableType}
@@ -137,4 +139,27 @@ class HiveSQLViewSuite extends SQLViewSuite with TestHiveSingleton {
137139
}
138140
}
139141
}
142+
143+
test("SPARK-36011: Disallow altering permanent views based on temporary UDFs") {
144+
val tempFunctionName = "temp"
145+
val functionClass =
146+
classOf[org.apache.hadoop.hive.ql.udf.generic.GenericUDFUpper].getCanonicalName
147+
withUserDefinedFunction(tempFunctionName -> true) {
148+
sql(s"CREATE TEMPORARY FUNCTION $tempFunctionName AS '$functionClass'")
149+
withView("view1") {
150+
withTempView("tempView1") {
151+
withTable("tab1") {
152+
(1 to 10).map(i => s"$i").toDF("id").write.saveAsTable("tab1")
153+
sql("CREATE VIEW view1 AS SELECT id from tab1")
154+
155+
val e = intercept[AnalysisException] {
156+
sql(s"ALTER VIEW view1 AS SELECT $tempFunctionName(id) from tab1")
157+
}.getMessage
158+
assert(e.contains("Not allowed to create a permanent view `default`.`view1` by " +
159+
s"referencing a temporary function `$tempFunctionName`"))
160+
}
161+
}
162+
}
163+
}
164+
}
140165
}

0 commit comments

Comments
 (0)