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 2019/12/06 17:11:25 UTC

[GitHub] [spark] attilapiros commented on a change in pull request #26016: [SPARK-24914][SQL] New statistic to improve data size estimate for columnar storage formats

attilapiros commented on a change in pull request #26016: [SPARK-24914][SQL] New statistic to improve data size estimate for columnar storage formats
URL: https://github.com/apache/spark/pull/26016#discussion_r354938632
 
 

 ##########
 File path: sql/core/src/main/scala/org/apache/spark/sql/execution/command/CommandUtils.scala
 ##########
 @@ -45,42 +48,101 @@ object CommandUtils extends Logging {
     val catalog = sparkSession.sessionState.catalog
     if (sparkSession.sessionState.conf.autoSizeUpdateEnabled) {
       val newTable = catalog.getTableMetadata(table.identifier)
-      val newSize = CommandUtils.calculateTotalSize(sparkSession, newTable)
-      val newStats = CatalogStatistics(sizeInBytes = newSize)
+      val oldDeserFactor = newTable.stats.flatMap(_.deserFactor)
+      val newSizeWithDeserFactor = CommandUtils.calculateTotalSize(sparkSession, newTable)
+      val newStats = CatalogStatistics(
+        sizeInBytes = newSizeWithDeserFactor.sizeInBytes,
+        deserFactor = newSizeWithDeserFactor.deserFactor.orElse(oldDeserFactor))
       catalog.alterTableStats(table.identifier, Some(newStats))
     } else if (table.stats.nonEmpty) {
       catalog.alterTableStats(table.identifier, None)
     }
   }
 
-  def calculateTotalSize(spark: SparkSession, catalogTable: CatalogTable): BigInt = {
+  def calculateTotalSize(
+      spark: SparkSession,
+      catalogTable: CatalogTable): SizeInBytesWithDeserFactor = {
     val sessionState = spark.sessionState
     if (catalogTable.partitionColumnNames.isEmpty) {
-      calculateLocationSize(sessionState, catalogTable.identifier, catalogTable.storage.locationUri)
+      calculateLocationSize(
+        sessionState,
+        catalogTable.identifier,
+        catalogTable.storage.locationUri,
+        catalogTable.storage.serde)
     } else {
       // Calculate table size as a sum of the visible partitions. See SPARK-21079
       val partitions = sessionState.catalog.listPartitions(catalogTable.identifier)
-      if (spark.sessionState.conf.parallelFileListingInStatsComputation) {
-        val paths = partitions.map(x => new Path(x.storage.locationUri.get))
-        val stagingDir = sessionState.conf.getConfString("hive.exec.stagingdir", ".hive-staging")
-        val pathFilter = new PathFilter with Serializable {
-          override def accept(path: Path): Boolean = isDataPath(path, stagingDir)
+      val sizeWithDeserFactorsForPartitions =
+        if (sessionState.conf.parallelFileListingInStatsComputation) {
+          val paths = partitions.map(x => new Path(x.storage.locationUri.get))
+          val stagingDir = sessionState.conf.getConfString("hive.exec.stagingdir", ".hive-staging")
+          val pathFilter = new PathFilter with Serializable {
+            override def accept(path: Path): Boolean = isDataPath(path, stagingDir)
+          }
+          val deserFactCalcEnabled = sessionState.conf.deserFactorStatCalcEnabled
+          val hadoopConf = sessionState.newHadoopConf()
+          val fileStatusSeq = InMemoryFileIndex.bulkListLeafFiles(
+            paths, hadoopConf, pathFilter, spark, areRootPaths = true)
+          fileStatusSeq.flatMap { case (_, fileStatuses) =>
+            fileStatuses.map { fileStatus =>
+              sizeInBytesWithDeserFactor(
+                deserFactCalcEnabled,
+                hadoopConf,
+                fileStatus,
+                catalogTable.storage.serde)
+            }
+          }
+        } else {
+          partitions.map { p =>
+            calculateLocationSize(
+              sessionState,
+              catalogTable.identifier,
+              p.storage.locationUri,
+              p.storage.serde)
+          }
         }
-        val fileStatusSeq = InMemoryFileIndex.bulkListLeafFiles(
-          paths, sessionState.newHadoopConf(), pathFilter, spark, areRootPaths = true)
-        fileStatusSeq.flatMap(_._2.map(_.getLen)).sum
-      } else {
-        partitions.map { p =>
-          calculateLocationSize(sessionState, catalogTable.identifier, p.storage.locationUri)
-        }.sum
+      sumSizeWithMaxDeserializationFactor(sizeWithDeserFactorsForPartitions)
+    }
+  }
+
+  def sumSizeWithMaxDeserializationFactor(
+      sizesWithFactors: Seq[SizeInBytesWithDeserFactor]): SizeInBytesWithDeserFactor = {
+    val definedFactors = sizesWithFactors.filter(_.deserFactor.isDefined).map(_.deserFactor.get)
+    SizeInBytesWithDeserFactor(
+      sizesWithFactors.map(_.sizeInBytes).sum,
+      if (definedFactors.isEmpty) None else Some(definedFactors.max))
+  }
+
+  def sizeInBytesWithDeserFactor(
+      calcDeserFact: Boolean,
+      hadoopConf: Configuration,
+      fStatus: FileStatus,
+      serde: Option[String]): SizeInBytesWithDeserFactor = {
+    assert(fStatus.isFile)
+    val factor = if (calcDeserFact) {
+      val isOrc = serde.contains("org.apache.hadoop.hive.ql.io.orc.OrcSerde") ||
 
 Review comment:
   Resolved in a9a673372eb224207cb950780fa8930a48154ab0.

----------------------------------------------------------------
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


With regards,
Apache Git Services

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