You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by breakdawn <gi...@git.apache.org> on 2016/07/23 04:46:19 UTC

[GitHub] spark pull request #14324: [SPARK-16664][SQL] Fix persist call on Data frame...

GitHub user breakdawn opened a pull request:

    https://github.com/apache/spark/pull/14324

    [SPARK-16664][SQL] Fix persist call on Data frames with more than 200\u2026

    ## What changes were proposed in this pull request?
    
    f12f11e578169b47e3f8b18b299948c0670ba585 introduced this bug, missed foreach as map
    
    ## How was this patch tested?
    
    manual tests done with following:
    test("test data frame with 201 columns") {
        val sparkConfig = new SparkConf()
        val parallelism = 5
        sparkConfig.set("spark.default.parallelism", s"$parallelism")
        sparkConfig.set("spark.sql.shuffle.partitions", s"$parallelism")
    
        val sc = new SparkContext(s"local[3]", "TestNestedJson", sparkConfig)
        val sqlContext = new SQLContext(sc)
    
        // create dataframe with 201 columns and fake 201 values
        val size = 201
        val rdd: RDD[Seq[Long]] = sc.parallelize(Seq(Seq.range(0, size)))
        val rowRdd: RDD[Row] = rdd.map(d => Row.fromSeq(d))
    
        val schemas = List.range(0, size).map(a => StructField("name"+ a, LongType, true))
        val testSchema = StructType(schemas)
        val testDf = sqlContext.createDataFrame(rowRdd, testSchema)
    
    
        // test value
        assert(testDf.persist.take(1).apply(0).toSeq(100).asInstanceOf[Long] == 100)
        sc.stop()
      }

You can merge this pull request into a Git repository by running:

    $ git pull https://github.com/breakdawn/spark master

Alternatively you can review and apply these changes as the patch at:

    https://github.com/apache/spark/pull/14324.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

    This closes #14324
    
----
commit 7040dc9f45eae56cb706cb44cd48bea16217db1e
Author: Wesley Tang <ta...@mininglamp.com>
Date:   2016-07-23T04:35:48Z

    [SPARK-16664][SQL] Fix persist call on Data frames with more than 200 columns is wiping out the data.

----


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    **[Test build #62876 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/62876/consoleFull)** for PR 14324 at commit [`0d6c29b`](https://github.com/apache/spark/commit/0d6c29b96d3d270880b3d0913e2df7d1f3913441).
     * This patch **fails Scala style tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by srowen <gi...@git.apache.org>.
Github user srowen commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Merged to master/2.0/1.6


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by breakdawn <gi...@git.apache.org>.
Github user breakdawn commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    @srowen sure, please refer to https://github.com/apache/spark/pull/14404


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by srowen <gi...@git.apache.org>.
Github user srowen commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    There are actually _55_ occurrences of this type of problem in the code base. I think I will open a PR separately to fix them. It might or might not cause a problem in practice in other cases, but many are in examples or tests, where we might not observe the consequence.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by breakdawn <gi...@git.apache.org>.
Github user breakdawn commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    @lw-lin Personally, multiple classes way is smoother base on current implementation. But no matter in what way, it's a big change, maybe it's better to open another jira issue to involve more discussions.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Test PASSed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/62920/
    Test PASSed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Merged build finished. Test FAILed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by lw-lin <gi...@git.apache.org>.
Github user lw-lin commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    @breakdawn yes that's a different issue and I'm looking into it.
    
    Regarding what this PR tries to fix, could you run this PR's change against [this test case](https://github.com/apache/spark/blob/master/sql/core/src/test/scala/org/apache/spark/sql/execution/columnar/InMemoryColumnarQuerySuite.scala#L225) to see if there's more needs to be done?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by srowen <gi...@git.apache.org>.
Github user srowen commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Darn, this breaks 1.6, because the test doesn't compile. I'll revert it in 1.6. @breakdawn if you're willing, could you open a PR vs 1.6 that updates the test to work in that branch?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by breakdawn <gi...@git.apache.org>.
Github user breakdawn commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    8118 cols limit due to janino,  the exception like following, might be another story
    	at org.apache.spark.sql.catalyst.expressions.codegen.CodeGenerator$.org$apache$spark$sql$catalyst$expressions$codegen$CodeGenerator$$doCompile(CodeGenerator.scala:889)
    	at org.apache.spark.sql.catalyst.expressions.codegen.CodeGenerator$$anon$1.load(CodeGenerator.scala:941)
    	at org.apache.spark.sql.catalyst.expressions.codegen.CodeGenerator$$anon$1.load(CodeGenerator.scala:938)
    	at com.google.common.cache.LocalCache$LoadingValueReference.loadFuture(LocalCache.java:3599)
    	at com.google.common.cache.LocalCache$Segment.loadSync(LocalCache.java:2379)
    	... 25 more
    Caused by: java.io.EOFException
    	at java.io.DataInputStream.readFully(DataInputStream.java:197)
    	at java.io.DataInputStream.readFully(DataInputStream.java:169)
    	at org.codehaus.janino.util.ClassFile.loadAttribute(ClassFile.java:1509)
    	at org.codehaus.janino.util.ClassFile.loadAttributes(ClassFile.java:644)
    	at org.codehaus.janino.util.ClassFile.loadFields(ClassFile.java:623)
    	at org.codehaus.janino.util.ClassFile.<init>(ClassFile.java:280)
    	at org.apache.spark.sql.catalyst.expressions.codegen.CodeGenerator$$anonfun$recordCompilationStats$1.apply(CodeGenerator.scala:914)
    	at org.apache.spark.sql.catalyst.expressions.codegen.CodeGenerator$$anonfun$recordCompilationStats$1.apply(CodeGenerator.scala:912)
    	at scala.collection.Iterator$class.foreach(Iterator.scala:893)
    	at scala.collection.AbstractIterator.foreach(Iterator.scala:1336)
    	at scala.collection.IterableLike$class.foreach(IterableLike.scala:72)
    	at scala.collection.AbstractIterable.foreach(Iterable.scala:54)
    	at org.apache.spark.sql.catalyst.expressions.codegen.CodeGenerator$.recordCompilationStats(CodeGenerator.scala:912)
    	at org.apache.spark.sql.catalyst.expressions.codegen.CodeGenerator$.org$apache$spark$sql$catalyst$expressions$codegen$CodeGenerator$$doCompile(CodeGenerator.scala:884)
    	... 29 more


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Test FAILed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/62876/
    Test FAILed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Merged build finished. Test PASSed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by srowen <gi...@git.apache.org>.
Github user srowen commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Jenkins add to whitelist


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request #14324: [SPARK-16664][SQL] Fix persist call on Data frame...

Posted by srowen <gi...@git.apache.org>.
Github user srowen commented on a diff in the pull request:

    https://github.com/apache/spark/pull/14324#discussion_r71971259
  
    --- Diff: sql/core/src/test/scala/org/apache/spark/sql/DataFrameSuite.scala ---
    @@ -1571,4 +1571,12 @@ class DataFrameSuite extends QueryTest with SharedSQLContext {
         checkAnswer(joined, Row("x", null, null))
         checkAnswer(joined.filter($"new".isNull), Row("x", null, null))
       }
    +
    +  test("SPARK-16664: persist with more than 200 columns") {
    +    val size = 201l
    --- End diff --
    
    Nit: write 201L for a long literal; it's too easy to read this as 2011.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request #14324: [SPARK-16664][SQL] Fix persist call on Data frame...

Posted by breakdawn <gi...@git.apache.org>.
Github user breakdawn commented on a diff in the pull request:

    https://github.com/apache/spark/pull/14324#discussion_r72221726
  
    --- Diff: sql/core/src/test/scala/org/apache/spark/sql/execution/columnar/InMemoryColumnarQuerySuite.scala ---
    @@ -227,7 +227,8 @@ class InMemoryColumnarQuerySuite extends QueryTest with SharedSQLContext {
         val columnTypes1 = List.fill(length1)(IntegerType)
         val columnarIterator1 = GenerateColumnAccessor.generate(columnTypes1)
     
    -    val length2 = 10000
    +    //SPARK-16664: the limit of janino is 8117
    --- End diff --
    
    @srowen Sorry for that.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by breakdawn <gi...@git.apache.org>.
Github user breakdawn commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    @lw-lin umm, thanks for pointing it out. Since the limit is 8117, 10000 will fail, that case needs a update.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by rxin <gi...@git.apache.org>.
Github user rxin commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Can you add a test case?



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request #14324: [SPARK-16664][SQL] Fix persist call on Data frame...

Posted by asfgit <gi...@git.apache.org>.
Github user asfgit closed the pull request at:

    https://github.com/apache/spark/pull/14324


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    **[Test build #62920 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/62920/consoleFull)** for PR 14324 at commit [`b3f60fa`](https://github.com/apache/spark/commit/b3f60fad6e80f5cb9f73de2ccb9ccafe14f3c4c1).
     * This patch passes all tests.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by srowen <gi...@git.apache.org>.
Github user srowen commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Jenkins test this please


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by lw-lin <gi...@git.apache.org>.
Github user lw-lin commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    @breakdawn it'd be great to do more tests when you open a request. As I'm investigate into this too, I found that my same fix works for 201 cols but fails for 8118 cols. The exact limit is 8117.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by srowen <gi...@git.apache.org>.
Github user srowen commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Jenkins retest this please


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by breakdawn <gi...@git.apache.org>.
Github user breakdawn commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Yes, working on that


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    Can one of the admins verify this patch?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    **[Test build #62876 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/62876/consoleFull)** for PR 14324 at commit [`0d6c29b`](https://github.com/apache/spark/commit/0d6c29b96d3d270880b3d0913e2df7d1f3913441).


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by breakdawn <gi...@git.apache.org>.
Github user breakdawn commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    @rxin @srowen Anything i can follow?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    **[Test build #62920 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/62920/consoleFull)** for PR 14324 at commit [`b3f60fa`](https://github.com/apache/spark/commit/b3f60fad6e80f5cb9f73de2ccb9ccafe14f3c4c1).


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by lw-lin <gi...@git.apache.org>.
Github user lw-lin commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    @breakdawn what else can we do to actually fix the \u2265 8118 cols issue? We're actually running out of the constant pool when we compile the generated code. So maybe compile it into multiple classes? Or just fall back to the non-code-gen path?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark issue #14324: [SPARK-16664][SQL] Fix persist call on Data frames with ...

Posted by breakdawn <gi...@git.apache.org>.
Github user breakdawn commented on the issue:

    https://github.com/apache/spark/pull/14324
  
    @lw-lin You're right, thanks for your suggestion.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request #14324: [SPARK-16664][SQL] Fix persist call on Data frame...

Posted by srowen <gi...@git.apache.org>.
Github user srowen commented on a diff in the pull request:

    https://github.com/apache/spark/pull/14324#discussion_r72212135
  
    --- Diff: sql/core/src/test/scala/org/apache/spark/sql/execution/columnar/InMemoryColumnarQuerySuite.scala ---
    @@ -227,7 +227,8 @@ class InMemoryColumnarQuerySuite extends QueryTest with SharedSQLContext {
         val columnTypes1 = List.fill(length1)(IntegerType)
         val columnarIterator1 = GenerateColumnAccessor.generate(columnTypes1)
     
    -    val length2 = 10000
    +    //SPARK-16664: the limit of janino is 8117
    --- End diff --
    
    Oh, this needs a space after `//`


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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