Skip to content

Commit 9b53480

Browse files
viiryadongjoon-hyun
authored andcommitted
[SPARK-30312][SQL][2.4] Preserve path permission and acl when truncate table
### What changes were proposed in this pull request? This patch proposes to preserve existing permission/acls of paths when truncate table/partition. Note that this is backport of #26956 to branch-2.4. ### Why are the changes needed? When Spark SQL truncates table, it deletes the paths of table/partitions, then re-create new ones. If permission/acls were set on the paths, the existing permission/acls will be deleted. We should preserve the permission/acls if possible. ### Does this PR introduce any user-facing change? Yes. When truncate table/partition, Spark will keep permission/acls of paths. ### How was this patch tested? Unit test and manual test as shown in #26956. Closes #27173 from viirya/truncate-table-permission-2.4. Authored-by: Liang-Chi Hsieh <liangchi@uber.com> Signed-off-by: Dongjoon Hyun <dhyun@apple.com>
1 parent 3b029d9 commit 9b53480

File tree

3 files changed

+135
-1
lines changed
  • sql
    • catalyst/src/main/scala/org/apache/spark/sql/internal
    • core/src

3 files changed

+135
-1
lines changed

sql/catalyst/src/main/scala/org/apache/spark/sql/internal/SQLConf.scala

Lines changed: 11 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -1576,6 +1576,14 @@ object SQLConf {
15761576
"turning the flag on provides a way for these sources to see these partitionBy columns.")
15771577
.booleanConf
15781578
.createWithDefault(false)
1579+
1580+
val TRUNCATE_TABLE_IGNORE_PERMISSION_ACL =
1581+
buildConf("spark.sql.truncateTable.ignorePermissionAcl")
1582+
.internal()
1583+
.doc("When set to true, TRUNCATE TABLE command will not try to set back original " +
1584+
"permission and ACLs when re-creating the table/partition paths.")
1585+
.booleanConf
1586+
.createWithDefault(false)
15791587
}
15801588

15811589
/**
@@ -1983,6 +1991,9 @@ class SQLConf extends Serializable with Logging {
19831991

19841992
def setOpsPrecedenceEnforced: Boolean = getConf(SQLConf.LEGACY_SETOPS_PRECEDENCE_ENABLED)
19851993

1994+
def truncateTableIgnorePermissionAcl: Boolean =
1995+
getConf(SQLConf.TRUNCATE_TABLE_IGNORE_PERMISSION_ACL)
1996+
19861997
/** ********************** SQLConf functionality methods ************ */
19871998

19881999
/** Set Spark SQL configuration properties. */

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

Lines changed: 46 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -26,6 +26,7 @@ import scala.util.Try
2626
import scala.util.control.NonFatal
2727

2828
import org.apache.hadoop.fs.{FileContext, FsConstants, Path}
29+
import org.apache.hadoop.fs.permission.{AclEntry, FsPermission}
2930

3031
import org.apache.spark.sql.{AnalysisException, Row, SparkSession}
3132
import org.apache.spark.sql.catalyst.TableIdentifier
@@ -457,13 +458,58 @@ case class TruncateTableCommand(
457458
partLocations
458459
}
459460
val hadoopConf = spark.sessionState.newHadoopConf()
461+
val ignorePermissionAcl = SQLConf.get.truncateTableIgnorePermissionAcl
460462
locations.foreach { location =>
461463
if (location.isDefined) {
462464
val path = new Path(location.get)
463465
try {
464466
val fs = path.getFileSystem(hadoopConf)
467+
468+
// Not all fs impl. support these APIs.
469+
var optPermission: Option[FsPermission] = None
470+
var optAcls: Option[java.util.List[AclEntry]] = None
471+
if (!ignorePermissionAcl) {
472+
val fileStatus = fs.getFileStatus(path)
473+
try {
474+
optPermission = Some(fileStatus.getPermission())
475+
} catch {
476+
case NonFatal(_) => // do nothing
477+
}
478+
479+
try {
480+
optAcls = Some(fs.getAclStatus(path).getEntries)
481+
} catch {
482+
case NonFatal(_) => // do nothing
483+
}
484+
}
485+
465486
fs.delete(path, true)
487+
// We should keep original permission/acl of the path.
488+
// For owner/group, only super-user can set it, for example on HDFS. Because
489+
// current user can delete the path, we assume the user/group is correct or not an issue.
466490
fs.mkdirs(path)
491+
if (!ignorePermissionAcl) {
492+
optPermission.foreach { permission =>
493+
try {
494+
fs.setPermission(path, permission)
495+
} catch {
496+
case NonFatal(e) =>
497+
throw new SecurityException(
498+
s"Failed to set original permission $permission back to " +
499+
s"the created path: $path. Exception: ${e.getMessage}")
500+
}
501+
}
502+
optAcls.foreach { acls =>
503+
try {
504+
fs.setAcl(path, acls)
505+
} catch {
506+
case NonFatal(e) =>
507+
throw new SecurityException(
508+
s"Failed to set original ACL $acls back to " +
509+
s"the created path: $path. Exception: ${e.getMessage}")
510+
}
511+
}
512+
}
467513
} catch {
468514
case NonFatal(e) =>
469515
throw new AnalysisException(

sql/core/src/test/scala/org/apache/spark/sql/execution/command/DDLSuite.scala

Lines changed: 78 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -21,7 +21,8 @@ import java.io.{File, PrintWriter}
2121
import java.net.URI
2222
import java.util.Locale
2323

24-
import org.apache.hadoop.fs.Path
24+
import org.apache.hadoop.fs.{Path, RawLocalFileSystem}
25+
import org.apache.hadoop.fs.permission.{AclEntry, AclEntryScope, AclEntryType, AclStatus, FsAction, FsPermission}
2526
import org.scalatest.BeforeAndAfterEach
2627

2728
import org.apache.spark.sql.{AnalysisException, QueryTest, Row, SaveMode}
@@ -1935,6 +1936,60 @@ abstract class DDLSuite extends QueryTest with SQLTestUtils {
19351936
}
19361937
}
19371938

1939+
test("SPARK-30312: truncate table - keep acl/permission") {
1940+
import testImplicits._
1941+
val ignorePermissionAcl = Seq(true, false)
1942+
1943+
ignorePermissionAcl.foreach { ignore =>
1944+
withSQLConf(
1945+
"fs.file.impl" -> classOf[FakeLocalFsFileSystem].getName,
1946+
"fs.file.impl.disable.cache" -> "true",
1947+
SQLConf.TRUNCATE_TABLE_IGNORE_PERMISSION_ACL.key -> ignore.toString) {
1948+
withTable("tab1") {
1949+
sql("CREATE TABLE tab1 (col INT) USING parquet")
1950+
sql("INSERT INTO tab1 SELECT 1")
1951+
checkAnswer(spark.table("tab1"), Row(1))
1952+
1953+
val tablePath = new Path(spark.sessionState.catalog
1954+
.getTableMetadata(TableIdentifier("tab1")).storage.locationUri.get)
1955+
1956+
val hadoopConf = spark.sessionState.newHadoopConf()
1957+
val fs = tablePath.getFileSystem(hadoopConf)
1958+
val fileStatus = fs.getFileStatus(tablePath);
1959+
1960+
fs.setPermission(tablePath, new FsPermission("777"))
1961+
assert(fileStatus.getPermission().toString() == "rwxrwxrwx")
1962+
1963+
// Set ACL to table path.
1964+
val customAcl = new java.util.ArrayList[AclEntry]()
1965+
customAcl.add(new AclEntry.Builder()
1966+
.setType(AclEntryType.USER)
1967+
.setScope(AclEntryScope.ACCESS)
1968+
.setPermission(FsAction.READ).build())
1969+
fs.setAcl(tablePath, customAcl)
1970+
assert(fs.getAclStatus(tablePath).getEntries().get(0) == customAcl.get(0))
1971+
1972+
sql("TRUNCATE TABLE tab1")
1973+
assert(spark.table("tab1").collect().isEmpty)
1974+
1975+
val fileStatus2 = fs.getFileStatus(tablePath)
1976+
if (ignore) {
1977+
assert(fileStatus2.getPermission().toString() == "rwxr-xr-x")
1978+
} else {
1979+
assert(fileStatus2.getPermission().toString() == "rwxrwxrwx")
1980+
}
1981+
val aclEntries = fs.getAclStatus(tablePath).getEntries()
1982+
if (ignore) {
1983+
assert(aclEntries.size() == 0)
1984+
} else {
1985+
assert(aclEntries.size() == 1)
1986+
assert(aclEntries.get(0) == customAcl.get(0))
1987+
}
1988+
}
1989+
}
1990+
}
1991+
}
1992+
19381993
test("create temporary view with mismatched schema") {
19391994
withTable("tab1") {
19401995
spark.range(10).write.saveAsTable("tab1")
@@ -2752,3 +2807,25 @@ abstract class DDLSuite extends QueryTest with SQLTestUtils {
27522807
}
27532808
}
27542809
}
2810+
2811+
object FakeLocalFsFileSystem {
2812+
var aclStatus = new AclStatus.Builder().build()
2813+
}
2814+
2815+
// A fake test local filesystem used to test ACL. It keeps a ACL status. If deletes
2816+
// a path of this filesystem, it will clean up the ACL status. Note that for test purpose,
2817+
// it has only one ACL status for all paths.
2818+
class FakeLocalFsFileSystem extends RawLocalFileSystem {
2819+
import FakeLocalFsFileSystem._
2820+
2821+
override def delete(f: Path, recursive: Boolean): Boolean = {
2822+
aclStatus = new AclStatus.Builder().build()
2823+
super.delete(f, recursive)
2824+
}
2825+
2826+
override def getAclStatus(path: Path): AclStatus = aclStatus
2827+
2828+
override def setAcl(path: Path, aclSpec: java.util.List[AclEntry]): Unit = {
2829+
aclStatus = new AclStatus.Builder().addEntries(aclSpec).build()
2830+
}
2831+
}

0 commit comments

Comments
 (0)