You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@spark.apache.org by rx...@apache.org on 2016/01/04 05:48:59 UTC

spark git commit: [SPARK-12562][SQL] DataFrame.write.format(text) requires the column name to be called value

Repository: spark
Updated Branches:
  refs/heads/master 13dab9c38 -> 84f8492c1


[SPARK-12562][SQL] DataFrame.write.format(text) requires the column name to be called value

Author: Xiu Guo <xg...@gmail.com>

Closes #10515 from xguo27/SPARK-12562.


Project: http://git-wip-us.apache.org/repos/asf/spark/repo
Commit: http://git-wip-us.apache.org/repos/asf/spark/commit/84f8492c
Tree: http://git-wip-us.apache.org/repos/asf/spark/tree/84f8492c
Diff: http://git-wip-us.apache.org/repos/asf/spark/diff/84f8492c

Branch: refs/heads/master
Commit: 84f8492c1555bf8ab44c9818752278f61768eb16
Parents: 13dab9c
Author: Xiu Guo <xg...@gmail.com>
Authored: Sun Jan 3 20:48:56 2016 -0800
Committer: Reynold Xin <rx...@databricks.com>
Committed: Sun Jan 3 20:48:56 2016 -0800

----------------------------------------------------------------------
 .../sql/execution/datasources/text/DefaultSource.scala      | 9 +++++----
 .../spark/sql/execution/datasources/text/TextSuite.scala    | 4 ++--
 2 files changed, 7 insertions(+), 6 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/spark/blob/84f8492c/sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/text/DefaultSource.scala
----------------------------------------------------------------------
diff --git a/sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/text/DefaultSource.scala b/sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/text/DefaultSource.scala
index 248467a..fe69c72 100644
--- a/sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/text/DefaultSource.scala
+++ b/sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/text/DefaultSource.scala
@@ -48,7 +48,7 @@ class DefaultSource extends HadoopFsRelationProvider with DataSourceRegister {
       partitionColumns: Option[StructType],
       parameters: Map[String, String]): HadoopFsRelation = {
     dataSchema.foreach(verifySchema)
-    new TextRelation(None, partitionColumns, paths)(sqlContext)
+    new TextRelation(None, dataSchema, partitionColumns, paths)(sqlContext)
   }
 
   override def shortName(): String = "text"
@@ -68,15 +68,16 @@ class DefaultSource extends HadoopFsRelationProvider with DataSourceRegister {
 
 private[sql] class TextRelation(
     val maybePartitionSpec: Option[PartitionSpec],
+    val textSchema: Option[StructType],
     override val userDefinedPartitionColumns: Option[StructType],
     override val paths: Array[String] = Array.empty[String],
     parameters: Map[String, String] = Map.empty[String, String])
     (@transient val sqlContext: SQLContext)
   extends HadoopFsRelation(maybePartitionSpec, parameters) {
 
-  /** Data schema is always a single column, named "value". */
-  override def dataSchema: StructType = new StructType().add("value", StringType)
-
+  /** Data schema is always a single column, named "value" if original Data source has no schema. */
+  override def dataSchema: StructType =
+    textSchema.getOrElse(new StructType().add("value", StringType))
   /** This is an internal data source that outputs internal row format. */
   override val needConversion: Boolean = false
 

http://git-wip-us.apache.org/repos/asf/spark/blob/84f8492c/sql/core/src/test/scala/org/apache/spark/sql/execution/datasources/text/TextSuite.scala
----------------------------------------------------------------------
diff --git a/sql/core/src/test/scala/org/apache/spark/sql/execution/datasources/text/TextSuite.scala b/sql/core/src/test/scala/org/apache/spark/sql/execution/datasources/text/TextSuite.scala
index 914e516..02c416a 100644
--- a/sql/core/src/test/scala/org/apache/spark/sql/execution/datasources/text/TextSuite.scala
+++ b/sql/core/src/test/scala/org/apache/spark/sql/execution/datasources/text/TextSuite.scala
@@ -33,8 +33,8 @@ class TextSuite extends QueryTest with SharedSQLContext {
     verifyFrame(sqlContext.read.text(testFile))
   }
 
-  test("writing") {
-    val df = sqlContext.read.text(testFile)
+  test("SPARK-12562 verify write.text() can handle column name beyond `value`") {
+    val df = sqlContext.read.text(testFile).withColumnRenamed("value", "adwrasdf")
 
     val tempFile = Utils.createTempDir()
     tempFile.delete()


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