1717
1818package org .apache .spark .sql .execution
1919
20+ import org .apache .commons .lang .StringUtils
21+
2022import org .apache .spark .rdd .RDD
2123import org .apache .spark .sql .{AnalysisException , Row , SparkSession , SQLContext }
22- import org .apache .spark .sql .catalyst .{CatalystTypeConverters , InternalRow }
24+ import org .apache .spark .sql .catalyst .{CatalystTypeConverters , InternalRow , TableIdentifier }
2325import org .apache .spark .sql .catalyst .analysis .MultiInstanceRelation
2426import org .apache .spark .sql .catalyst .expressions ._
2527import org .apache .spark .sql .catalyst .expressions .codegen .{CodegenContext , ExprCode }
@@ -127,8 +129,13 @@ private[sql] case class RDDScanExec(
127129private [sql] trait DataSourceScanExec extends LeafExecNode {
128130 val rdd : RDD [InternalRow ]
129131 val relation : BaseRelation
132+ val metastoreTableIdentifier : Option [TableIdentifier ]
130133
131- override val nodeName : String = relation.toString
134+ override val nodeName : String = if (metastoreTableIdentifier.isEmpty) {
135+ relation.toString
136+ } else {
137+ relation.toString + " " + metastoreTableIdentifier.get.unquotedString
138+ }
132139
133140 // Ignore rdd when checking results
134141 override def sameResult (plan : SparkPlan ): Boolean = plan match {
@@ -143,7 +150,8 @@ private[sql] case class RowDataSourceScanExec(
143150 rdd : RDD [InternalRow ],
144151 @ transient relation : BaseRelation ,
145152 override val outputPartitioning : Partitioning ,
146- override val metadata : Map [String , String ] = Map .empty)
153+ override val metadata : Map [String , String ],
154+ override val metastoreTableIdentifier : Option [TableIdentifier ])
147155 extends DataSourceScanExec with CodegenSupport {
148156
149157 private [sql] override lazy val metrics =
@@ -174,7 +182,10 @@ private[sql] case class RowDataSourceScanExec(
174182 }
175183
176184 override def simpleString : String = {
177- val metadataEntries = for ((key, value) <- metadata.toSeq.sorted) yield s " $key: $value"
185+ val metadataEntries = for ((key, value) <- metadata.toSeq.sorted) yield {
186+ key + " : " + StringUtils .abbreviate(value, 100 )
187+ }
188+
178189 s " Scan $nodeName${output.mkString(" [" , " ," , " ]" )}${metadataEntries.mkString(" " , " , " , " " )}"
179190 }
180191
@@ -212,7 +223,8 @@ private[sql] case class BatchedDataSourceScanExec(
212223 rdd : RDD [InternalRow ],
213224 @ transient relation : BaseRelation ,
214225 override val outputPartitioning : Partitioning ,
215- override val metadata : Map [String , String ] = Map .empty)
226+ override val metadata : Map [String , String ],
227+ override val metastoreTableIdentifier : Option [TableIdentifier ])
216228 extends DataSourceScanExec with CodegenSupport {
217229
218230 private [sql] override lazy val metrics =
@@ -224,7 +236,9 @@ private[sql] case class BatchedDataSourceScanExec(
224236 }
225237
226238 override def simpleString : String = {
227- val metadataEntries = for ((key, value) <- metadata.toSeq.sorted) yield s " $key: $value"
239+ val metadataEntries = for ((key, value) <- metadata.toSeq.sorted) yield {
240+ key + " : " + StringUtils .abbreviate(value, 100 )
241+ }
228242 val metadataStr = metadataEntries.mkString(" " , " , " , " " )
229243 s " BatchedScan $nodeName${output.mkString(" [" , " ," , " ]" )}$metadataStr"
230244 }
@@ -325,7 +339,8 @@ private[sql] object DataSourceScanExec {
325339 output : Seq [Attribute ],
326340 rdd : RDD [InternalRow ],
327341 relation : BaseRelation ,
328- metadata : Map [String , String ] = Map .empty): DataSourceScanExec = {
342+ metadata : Map [String , String ] = Map .empty,
343+ metastoreTableIdentifier : Option [TableIdentifier ] = None ): DataSourceScanExec = {
329344 val outputPartitioning = {
330345 val bucketSpec = relation match {
331346 // TODO: this should be closer to bucket planning.
@@ -351,9 +366,11 @@ private[sql] object DataSourceScanExec {
351366 relation match {
352367 case r : HadoopFsRelation
353368 if r.fileFormat.supportBatch(r.sparkSession, StructType .fromAttributes(output)) =>
354- BatchedDataSourceScanExec (output, rdd, relation, outputPartitioning, metadata)
369+ BatchedDataSourceScanExec (
370+ output, rdd, relation, outputPartitioning, metadata, metastoreTableIdentifier)
355371 case _ =>
356- RowDataSourceScanExec (output, rdd, relation, outputPartitioning, metadata)
372+ RowDataSourceScanExec (
373+ output, rdd, relation, outputPartitioning, metadata, metastoreTableIdentifier)
357374 }
358375 }
359376}
0 commit comments