[ 
https://issues.apache.org/jira/browse/SPARK-7276?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14522153#comment-14522153
 ] 

Reynold Xin commented on SPARK-7276:
------------------------------------

The problem is having too many projects (each new column adds a project). You 
can speed this up significantly right now by using a single .select to add all 
the columns.

For a fix quick, I think we can have dataframe.select collapses Projects, i.e. 
if the last logical operator is a Project, simply merge the two Projects. 
However, this will break caching semantics, but we can make caching detection 
smarter in the future.

[~cloud_fan] would you like to give it a shot to collapse projects in 
dataframe.select?


> withColumn is very slow on dataframe with large number of columns
> -----------------------------------------------------------------
>
>                 Key: SPARK-7276
>                 URL: https://issues.apache.org/jira/browse/SPARK-7276
>             Project: Spark
>          Issue Type: Bug
>          Components: SQL
>    Affects Versions: 1.3.1
>            Reporter: Alexandre CLEMENT
>
> The code snippet demonstrates the problem.
> {code}
>     import org.apache.spark.sql._
>     import org.apache.spark.sql.types._
>     val sparkConf = new SparkConf().setAppName("Spark 
> Test").setMaster(System.getProperty("spark.master", "local[4]"))
>     val sc = new SparkContext(sparkConf)
>     val sqlContext = new SQLContext(sc)
>     import sqlContext.implicits._
>     val custs = Seq(
>       Row(1, "Bob", 21, 80.5),
>       Row(2, "Bobby", 21, 80.5),
>       Row(3, "Jean", 21, 80.5),
>       Row(4, "Fatime", 21, 80.5)
>     )
>     var fields = List(
>       StructField("id", IntegerType, true),
>       StructField("a", IntegerType, true),
>       StructField("b", StringType, true),
>       StructField("target", DoubleType, false))
>     val schema = StructType(fields)
>     var rdd = sc.parallelize(custs)
>     var df = sqlContext.createDataFrame(rdd, schema)
>     for (i <- 1 to 200) {
>       val now = System.currentTimeMillis
>       df = df.withColumn("a_new_col_" + i, df("a") + i)
>       println(s"$i -> " + (System.currentTimeMillis - now))
>     }
>     df.show()
> {code}



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)

---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to