You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by HyukjinKwon <gi...@git.apache.org> on 2016/04/29 13:50:15 UTC

[GitHub] spark pull request: [SPARK-14962][SQL] Do not push down isnotnull/...

GitHub user HyukjinKwon opened a pull request:

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

    [SPARK-14962][SQL] Do not push down isnotnull/isnull on unsuportted types.

    ## What changes were proposed in this pull request?
    
    https://issues.apache.org/jira/browse/SPARK-14962
    
    ORC filters were being pushed down for all types for both `IsNull` and `IsNotNull`.
    
    This is apparently OK because both `IsNull` and `IsNotNull` do not take a type as an argument (Hive 1.2.x) during building filters (`SearchArgument`)in Spark-side but they do not filter correctly because stored statistics always produces `null` for not supported types, eg `ArrayType` in ORC-side. So, it is always `true` for `IsNull` which ends up always `false` for `IsNotNull`.
    
    This looks prevented in Hive 1.3.x >= by forcing to give a type when building a filter (`SearchArgument`) but Hive 1.2.x seems not doing this. 
    
    This PR prevents ORC filter creation for `IsNull` and `IsNotNull` on unsupported types. `OrcFilters` resembles `ParquetFilters`.
    
    ## How was this patch tested?
    
    Unittests in `OrcQuerySuite` and `OrcFilterSuite` and `sbt scalastyle`.


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

    $ git pull https://github.com/HyukjinKwon/spark SPARK-14962

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

    https://github.com/apache/spark/pull/12777.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 #12777
    
----
commit c959f7e3c88d150bb61e314c8a01559a685c1208
Author: hyukjinkwon <gu...@gmail.com>
Date:   2016-04-29T11:42:16Z

    Do not push down isnotnull/isnull on unsuportted types.

----


---
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: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215705461
  
    **[Test build #57330 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/57330/consoleFull)** for PR 12777 at commit [`0cbfbe4`](https://github.com/apache/spark/commit/0cbfbe4b495549e0456337b61c68e5596b97807d).
     * This patch **fails Spark unit 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 pull request: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215701233
  
    **[Test build #57329 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/57329/consoleFull)** for PR 12777 at commit [`c959f7e`](https://github.com/apache/spark/commit/c959f7e3c88d150bb61e314c8a01559a685c1208).
     * This patch **fails Spark unit 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 pull request: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#discussion_r62393964
  
    --- Diff: sql/hive/src/main/scala/org/apache/spark/sql/hive/orc/OrcFilters.scala ---
    @@ -56,29 +55,35 @@ import org.apache.spark.sql.sources._
      * known to be convertible.
      */
     private[orc] object OrcFilters extends Logging {
    -  def createFilter(filters: Array[Filter]): Option[SearchArgument] = {
    +  def createFilter(schema: StructType, filters: Array[Filter]): Option[SearchArgument] = {
    +    val dataTypeMap = schema.map(f => f.name -> f.dataType).toMap
    +
         // First, tries to convert each filter individually to see whether it's convertible, and then
         // collect all convertible ones to build the final `SearchArgument`.
         val convertibleFilters = for {
           filter <- filters
    -      _ <- buildSearchArgument(filter, SearchArgumentFactory.newBuilder())
    +      _ <- buildSearchArgument(dataTypeMap, filter, SearchArgumentFactory.newBuilder())
         } yield filter
     
         for {
           // Combines all convertible filters using `And` to produce a single conjunction
           conjunction <- convertibleFilters.reduceOption(And)
           // Then tries to build a single ORC `SearchArgument` for the conjunction predicate
    -      builder <- buildSearchArgument(conjunction, SearchArgumentFactory.newBuilder())
    +      builder <- buildSearchArgument(dataTypeMap, conjunction, SearchArgumentFactory.newBuilder())
         } yield builder.build()
       }
     
    -  private def buildSearchArgument(expression: Filter, builder: Builder): Option[Builder] = {
    +  private def buildSearchArgument(
    +      dataTypeMap: Map[String, DataType],
    +      expression: Filter,
    +      builder: Builder): Option[Builder] = {
         def newBuilder = SearchArgumentFactory.newBuilder()
     
    -    def isSearchableLiteral(value: Any): Boolean = value match {
    -      // These are types recognized by the `SearchArgumentImpl.BuilderImpl.boxLiteral()` method.
    -      case _: String | _: Long | _: Double | _: Byte | _: Short | _: Integer | _: Float => true
    -      case _: DateWritable | _: HiveDecimal | _: HiveChar | _: HiveVarchar => true
    +    def isSearchableType(dataType: DataType): Boolean = dataType match {
    +      // Only the values in the Spark types below can be recognized by
    +      // the `SearchArgumentImpl.BuilderImpl.boxLiteral()` method.
    +      case ByteType | ShortType | FloatType | DoubleType => true
    --- End diff --
    
    What about BooleanType ?


---
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: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215701364
  
    Test FAILed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/57329/
    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 pull request: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#discussion_r62408994
  
    --- Diff: sql/hive/src/main/scala/org/apache/spark/sql/hive/orc/OrcFilters.scala ---
    @@ -56,29 +55,35 @@ import org.apache.spark.sql.sources._
      * known to be convertible.
      */
     private[orc] object OrcFilters extends Logging {
    -  def createFilter(filters: Array[Filter]): Option[SearchArgument] = {
    +  def createFilter(schema: StructType, filters: Array[Filter]): Option[SearchArgument] = {
    +    val dataTypeMap = schema.map(f => f.name -> f.dataType).toMap
    +
         // First, tries to convert each filter individually to see whether it's convertible, and then
         // collect all convertible ones to build the final `SearchArgument`.
         val convertibleFilters = for {
           filter <- filters
    -      _ <- buildSearchArgument(filter, SearchArgumentFactory.newBuilder())
    +      _ <- buildSearchArgument(dataTypeMap, filter, SearchArgumentFactory.newBuilder())
         } yield filter
     
         for {
           // Combines all convertible filters using `And` to produce a single conjunction
           conjunction <- convertibleFilters.reduceOption(And)
           // Then tries to build a single ORC `SearchArgument` for the conjunction predicate
    -      builder <- buildSearchArgument(conjunction, SearchArgumentFactory.newBuilder())
    +      builder <- buildSearchArgument(dataTypeMap, conjunction, SearchArgumentFactory.newBuilder())
         } yield builder.build()
       }
     
    -  private def buildSearchArgument(expression: Filter, builder: Builder): Option[Builder] = {
    +  private def buildSearchArgument(
    +      dataTypeMap: Map[String, DataType],
    +      expression: Filter,
    +      builder: Builder): Option[Builder] = {
         def newBuilder = SearchArgumentFactory.newBuilder()
     
    -    def isSearchableLiteral(value: Any): Boolean = value match {
    -      // These are types recognized by the `SearchArgumentImpl.BuilderImpl.boxLiteral()` method.
    -      case _: String | _: Long | _: Double | _: Byte | _: Short | _: Integer | _: Float => true
    -      case _: DateWritable | _: HiveDecimal | _: HiveChar | _: HiveVarchar => true
    +    def isSearchableType(dataType: DataType): Boolean = dataType match {
    +      // Only the values in the Spark types below can be recognized by
    +      // the `SearchArgumentImpl.BuilderImpl.boxLiteral()` method.
    +      case ByteType | ShortType | FloatType | DoubleType => true
    --- End diff --
    
    @tedyu Let me test and will make a followup-up or another PR.


---
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: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215692092
  
    **[Test build #57329 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/57329/consoleFull)** for PR 12777 at commit [`c959f7e`](https://github.com/apache/spark/commit/c959f7e3c88d150bb61e314c8a01559a685c1208).


---
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: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215705583
  
    Test FAILed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/57330/
    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 pull request: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215705582
  
    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 pull request: [SPARK-14962][SQL] Do not push down isnotnull/...

Posted by liancheng <gi...@git.apache.org>.
Github user liancheng commented on the pull request:

    https://github.com/apache/spark/pull/12777#issuecomment-217511291
  
    LGTM, merging to master and branch-2.0. Thanks for fixing this!
    
    And yes, `OrcTableScan` should be safe to remove now.


---
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: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215741756
  
    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 pull request: [SPARK-14962][SQL] Do not push down isnotnull/...

Posted by HyukjinKwon <gi...@git.apache.org>.
Github user HyukjinKwon commented on the pull request:

    https://github.com/apache/spark/pull/12777#issuecomment-215692521
  
    @liancheng @yhuai Could you please take a look 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 pull request: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215701361
  
    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 pull request: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#discussion_r61565675
  
    --- Diff: sql/hive/src/main/scala/org/apache/spark/sql/hive/orc/OrcFilters.scala ---
    @@ -56,29 +55,35 @@ import org.apache.spark.sql.sources._
      * known to be convertible.
      */
     private[orc] object OrcFilters extends Logging {
    -  def createFilter(filters: Array[Filter]): Option[SearchArgument] = {
    +  def createFilter(schema: StructType, filters: Array[Filter]): Option[SearchArgument] = {
    +    val dataTypeMap = schema.map(f => f.name -> f.dataType).toMap
    +
         // First, tries to convert each filter individually to see whether it's convertible, and then
         // collect all convertible ones to build the final `SearchArgument`.
         val convertibleFilters = for {
           filter <- filters
    -      _ <- buildSearchArgument(filter, SearchArgumentFactory.newBuilder())
    +      _ <- buildSearchArgument(dataTypeMap, filter, SearchArgumentFactory.newBuilder())
         } yield filter
     
         for {
           // Combines all convertible filters using `And` to produce a single conjunction
           conjunction <- convertibleFilters.reduceOption(And)
           // Then tries to build a single ORC `SearchArgument` for the conjunction predicate
    -      builder <- buildSearchArgument(conjunction, SearchArgumentFactory.newBuilder())
    +      builder <- buildSearchArgument(dataTypeMap, conjunction, SearchArgumentFactory.newBuilder())
         } yield builder.build()
       }
     
    -  private def buildSearchArgument(expression: Filter, builder: Builder): Option[Builder] = {
    +  private def buildSearchArgument(
    +      dataTypeMap: Map[String, DataType],
    +      expression: Filter,
    +      builder: Builder): Option[Builder] = {
         def newBuilder = SearchArgumentFactory.newBuilder()
     
    -    def isSearchableLiteral(value: Any): Boolean = value match {
    -      // These are types recognized by the `SearchArgumentImpl.BuilderImpl.boxLiteral()` method.
    -      case _: String | _: Long | _: Double | _: Byte | _: Short | _: Integer | _: Float => true
    -      case _: DateWritable | _: HiveDecimal | _: HiveChar | _: HiveVarchar => true
    +    def isSearchableType(dataType: DataType): Boolean = dataType match {
    +      // Only the values in the Spark types below can be recognized by
    +      // the `SearchArgumentImpl.BuilderImpl.boxLiteral()` method.
    +      case ByteType | ShortType | FloatType | DoubleType => true
    +      case IntegerType | LongType | StringType => true
    --- End diff --
    
    Note to myself: this should be okay because `CatalystTypeConverters.createToScalaConverter()` is called in `DataSourceStrategy` for the values. I checked all the cases and it seems there are no cases that they are converted to one of `DateWritable | _: HiveDecimal | _: HiveChar | _: HiveVarchar`.
    
    As the all values in `source.Filter` are converted by `DataType`, this should be okay. I check the test codes. `ParquetFilters` is also doing this in this way.
    
     


---
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: [SPARK-14962][SQL] Do not push down isnotnull/...

Posted by HyukjinKwon <gi...@git.apache.org>.
Github user HyukjinKwon commented on the pull request:

    https://github.com/apache/spark/pull/12777#issuecomment-215697713
  
    BTW, during doing this, I realised there is a unused classe and functions due to the change of `HadoopFsRelation`.
    
    - The class `OrcTableScan` is not used and It seems `OrcRelation.buildReader` is doing exactly the same thing. 
    - The functions `initializeLocalJobFunc`, `initializeDriverSideJobFunc` and `overrideMinSplitSize` in `ParquetRelation` are not also being used but I guess `overrideMinSplitSize()` is not handled in `ParquetRelation.buildReader`.
    
    Could I maybe test this like https://github.com/apache/spark/pull/8346 for `overrideMinSplitSize`  and remove those?


---
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: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215695460
  
    **[Test build #57330 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/57330/consoleFull)** for PR 12777 at commit [`0cbfbe4`](https://github.com/apache/spark/commit/0cbfbe4b495549e0456337b61c68e5596b97807d).


---
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: [SPARK-14962][SQL] Do not push down isnotnull/...

Posted by HyukjinKwon <gi...@git.apache.org>.
Github user HyukjinKwon commented on the pull request:

    https://github.com/apache/spark/pull/12777#issuecomment-217391825
  
    Hi @yhuai Would you mind taking a look for 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 pull request: [SPARK-14962][SQL] Do not push down isnotnull/...

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

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


---
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: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215708308
  
    **[Test build #57333 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/57333/consoleFull)** for PR 12777 at commit [`f51d04b`](https://github.com/apache/spark/commit/f51d04b4883f358d4f3dddce1767215275ffff7b).


---
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: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215741758
  
    Test PASSed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/57333/
    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 pull request: [SPARK-14962][SQL] Do not push down isnotnull/...

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

    https://github.com/apache/spark/pull/12777#issuecomment-215741463
  
    **[Test build #57333 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/57333/consoleFull)** for PR 12777 at commit [`f51d04b`](https://github.com/apache/spark/commit/f51d04b4883f358d4f3dddce1767215275ffff7b).
     * 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