You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by gatorsmile <gi...@git.apache.org> on 2016/02/23 00:37:09 UTC

[GitHub] spark pull request: [SPARK-13723] [SQL] Comprehensive Verification...

GitHub user gatorsmile opened a pull request:

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

    [SPARK-13723] [SQL] Comprehensive Verification and Fixing of SQL Generation Support for Expressions

    #### What changes were proposed in this pull request?
    
    Ensure that all built-in expressions can be mapped to its SQL representation if there is one (e.g. ScalaUDF doesn't have a SQL representation). The function lists are from the expression list in `FunctionRegistry`.
    
    window functions, grouping sets functions (`cube`, `rollup`, `grouping`, `grouping_id`), generator functions (`explode` and `json_tuple`) are covered by separate JIRA and PRs. Thus, this PR does not cover them. Except these functions, all the built-in expressions are covered. For details, see the list in `ExpressionToSQLSuite`. 
    
    Fixed a few issues. For example, the `prettyName` of `approx_count_distinct` is not right. The `sql` of `hash` function is not right, since the `hash` function does not accept `seed`. 
    
    Additionally, also correct the order of expressions in `FunctionRegistry` so that people are easier to find which functions are missing. 
    
    cc @liancheng 
    
    #### How was the this patch tested?
    Added two test cases in LogicalPlanToSQLSuite for covering `not like` and `not in`. 
    
    Added a new test suite `ExpressionToSQLSuite` to cover the functions:
    
    1. misc non-aggregate functions + complex type creators + null expressions
    2. math functions
    3. aggregate functions
    4. string functions
    5. date time functions + calendar interval
    6. collection functions
    7. misc functions
    


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

    $ git pull https://github.com/gatorsmile/spark expressionToSQL

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

    https://github.com/apache/spark/pull/11314.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 #11314
    
----
commit 2fc93649b2e7294d79ab7c7c5c591b77d46d9c54
Author: gatorsmile <ga...@gmail.com>
Date:   2016-02-22T22:58:05Z

    Comprehensive SQL generation support for expressions

----


---
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-12723] [SQL] Comprehensive Verification...

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

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


---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187559335
  
    **[Test build #51722 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/51722/consoleFull)** for PR 11314 at commit [`1620378`](https://github.com/apache/spark/commit/1620378727920df24bf0950249e67dfc020f9be0).
     * 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 pull request: [SPARK-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187561335
  
    Thanks - merging in master.



---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187444465
  
    cc @liancheng 


---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187444069
  
    This is pretty cool.



---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#discussion_r53715863
  
    --- Diff: sql/hive/src/test/scala/org/apache/spark/sql/hive/ExpressionToSQLSuite.scala ---
    @@ -0,0 +1,271 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *    http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +
    +package org.apache.spark.sql.hive
    +
    +import scala.util.control.NonFatal
    +
    +import org.apache.spark.sql.functions._
    +import org.apache.spark.sql.test.SQLTestUtils
    +
    +class ExpressionToSQLSuite extends SQLBuilderTest with SQLTestUtils {
    --- End diff --
    
    sure, will do it. Thanks!


---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187443770
  
    Test FAILed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/51699/
    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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187516774
  
    **[Test build #51722 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/51722/consoleFull)** for PR 11314 at commit [`1620378`](https://github.com/apache/spark/commit/1620378727920df24bf0950249e67dfc020f9be0).


---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#discussion_r53715997
  
    --- Diff: sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/FunctionRegistry.scala ---
    @@ -125,13 +125,12 @@ object FunctionRegistry {
         expression[IsNull]("isnull"),
         expression[IsNotNull]("isnotnull"),
         expression[Least]("least"),
    +    expression[CreateNamedStruct]("named_struct"),
    +    expression[NaNvl]("nanvl"),
         expression[Coalesce]("nvl"),
         expression[Rand]("rand"),
         expression[Randn]("randn"),
         expression[CreateStruct]("struct"),
    -    expression[CreateNamedStruct]("named_struct"),
    --- End diff --
    
    ok, will do it.


---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#discussion_r53732236
  
    --- Diff: sql/hive/src/test/scala/org/apache/spark/sql/hive/ExpressionToSQLSuite.scala ---
    @@ -0,0 +1,271 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *    http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +
    +package org.apache.spark.sql.hive
    +
    +import scala.util.control.NonFatal
    +
    +import org.apache.spark.sql.functions._
    +import org.apache.spark.sql.test.SQLTestUtils
    +
    +class ExpressionToSQLSuite extends SQLBuilderTest with SQLTestUtils {
    --- End diff --
    
    Sorry, unable to move it to sql/core since this suite requires SQLBuilder and SQLBuilderTest. Both are in the hive package. : (


---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187541622
  
    Thanks! 


---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187560175
  
    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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187443767
  
    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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187560178
  
    Test PASSed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/51722/
    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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#discussion_r53715823
  
    --- Diff: sql/hive/src/test/scala/org/apache/spark/sql/hive/ExpressionToSQLSuite.scala ---
    @@ -0,0 +1,271 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *    http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +
    +package org.apache.spark.sql.hive
    +
    +import scala.util.control.NonFatal
    +
    +import org.apache.spark.sql.functions._
    +import org.apache.spark.sql.test.SQLTestUtils
    +
    +class ExpressionToSQLSuite extends SQLBuilderTest with SQLTestUtils {
    +  import testImplicits._
    +
    +  protected override def beforeAll(): Unit = {
    +    sql("DROP TABLE IF EXISTS t0")
    +    sql("DROP TABLE IF EXISTS t1")
    +    sql("DROP TABLE IF EXISTS t2")
    +
    +    val bytes = Array[Byte](1, 2, 3, 4)
    +    Seq((bytes, "AQIDBA==")).toDF("a", "b").write.saveAsTable("t0")
    +
    +    sqlContext
    +      .range(10)
    +      .select('id as 'key, concat(lit("val_"), 'id) as 'value)
    +      .write
    +      .saveAsTable("t1")
    +
    +    sqlContext.range(10).select('id as 'a, 'id as 'b, 'id as 'c, 'id as 'd).write.saveAsTable("t2")
    +  }
    +
    +  override protected def afterAll(): Unit = {
    +    sql("DROP TABLE IF EXISTS t0")
    +    sql("DROP TABLE IF EXISTS t1")
    +    sql("DROP TABLE IF EXISTS t2")
    +  }
    +
    +  private def checkHiveQl(hiveQl: String): Unit = {
    --- End diff --
    
    checkSqlGeneration
    



---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#discussion_r53715770
  
    --- Diff: sql/hive/src/test/scala/org/apache/spark/sql/hive/ExpressionToSQLSuite.scala ---
    @@ -0,0 +1,271 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *    http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +
    +package org.apache.spark.sql.hive
    +
    +import scala.util.control.NonFatal
    +
    +import org.apache.spark.sql.functions._
    +import org.apache.spark.sql.test.SQLTestUtils
    +
    +class ExpressionToSQLSuite extends SQLBuilderTest with SQLTestUtils {
    --- End diff --
    
    let's put this in sql/core rather than hive. otherwise we will need to move it again.



---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#discussion_r53757719
  
    --- Diff: sql/hive/src/test/scala/org/apache/spark/sql/hive/ExpressionToSQLSuite.scala ---
    @@ -0,0 +1,271 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *    http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +
    +package org.apache.spark.sql.hive
    +
    +import scala.util.control.NonFatal
    +
    +import org.apache.spark.sql.functions._
    +import org.apache.spark.sql.test.SQLTestUtils
    +
    +class ExpressionToSQLSuite extends SQLBuilderTest with SQLTestUtils {
    --- End diff --
    
    We will just move the entire suite over once we merge hive context and SQL
    context.
    
    On Tuesday, February 23, 2016, Sean Owen <no...@github.com> wrote:
    
    > In
    > sql/hive/src/test/scala/org/apache/spark/sql/hive/ExpressionToSQLSuite.scala
    > <https://github.com/apache/spark/pull/11314#discussion_r53756196>:
    >
    > > + *
    > > + * Unless required by applicable law or agreed to in writing, software
    > > + * distributed under the License is distributed on an "AS IS" BASIS,
    > > + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    > > + * See the License for the specific language governing permissions and
    > > + * limitations under the License.
    > > + */
    > > +
    > > +package org.apache.spark.sql.hive
    > > +
    > > +import scala.util.control.NonFatal
    > > +
    > > +import org.apache.spark.sql.functions._
    > > +import org.apache.spark.sql.test.SQLTestUtils
    > > +
    > > +class ExpressionToSQLSuite extends SQLBuilderTest with SQLTestUtils {
    >
    > Refactor the tests then? that much shouldn't dictate where you put
    > non-test code
    >
    > —
    > Reply to this email directly or view it on GitHub
    > <https://github.com/apache/spark/pull/11314/files#r53756196>.
    >



---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#discussion_r53715887
  
    --- Diff: sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/FunctionRegistry.scala ---
    @@ -125,13 +125,12 @@ object FunctionRegistry {
         expression[IsNull]("isnull"),
         expression[IsNotNull]("isnotnull"),
         expression[Least]("least"),
    +    expression[CreateNamedStruct]("named_struct"),
    +    expression[NaNvl]("nanvl"),
         expression[Coalesce]("nvl"),
         expression[Rand]("rand"),
         expression[Randn]("randn"),
         expression[CreateStruct]("struct"),
    -    expression[CreateNamedStruct]("named_struct"),
    --- End diff --
    
    I'd add a comment above saying
    
    "Whenever we add a new entry here, make sure we also update ExpressionToSQLSuite"


---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187444898
  
    ok to test


---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#discussion_r53756196
  
    --- Diff: sql/hive/src/test/scala/org/apache/spark/sql/hive/ExpressionToSQLSuite.scala ---
    @@ -0,0 +1,271 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *    http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +
    +package org.apache.spark.sql.hive
    +
    +import scala.util.control.NonFatal
    +
    +import org.apache.spark.sql.functions._
    +import org.apache.spark.sql.test.SQLTestUtils
    +
    +class ExpressionToSQLSuite extends SQLBuilderTest with SQLTestUtils {
    --- End diff --
    
    Refactor the tests then? that much shouldn't dictate where you put non-test code


---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#issuecomment-187530247
  
    LGTM pending tests


---
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-12723] [SQL] Comprehensive Verification...

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

    https://github.com/apache/spark/pull/11314#discussion_r53716023
  
    --- Diff: sql/hive/src/test/scala/org/apache/spark/sql/hive/ExpressionToSQLSuite.scala ---
    @@ -0,0 +1,271 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *    http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +
    +package org.apache.spark.sql.hive
    +
    +import scala.util.control.NonFatal
    +
    +import org.apache.spark.sql.functions._
    +import org.apache.spark.sql.test.SQLTestUtils
    +
    +class ExpressionToSQLSuite extends SQLBuilderTest with SQLTestUtils {
    +  import testImplicits._
    +
    +  protected override def beforeAll(): Unit = {
    +    sql("DROP TABLE IF EXISTS t0")
    +    sql("DROP TABLE IF EXISTS t1")
    +    sql("DROP TABLE IF EXISTS t2")
    +
    +    val bytes = Array[Byte](1, 2, 3, 4)
    +    Seq((bytes, "AQIDBA==")).toDF("a", "b").write.saveAsTable("t0")
    +
    +    sqlContext
    +      .range(10)
    +      .select('id as 'key, concat(lit("val_"), 'id) as 'value)
    +      .write
    +      .saveAsTable("t1")
    +
    +    sqlContext.range(10).select('id as 'a, 'id as 'b, 'id as 'c, 'id as 'd).write.saveAsTable("t2")
    +  }
    +
    +  override protected def afterAll(): Unit = {
    +    sql("DROP TABLE IF EXISTS t0")
    +    sql("DROP TABLE IF EXISTS t1")
    +    sql("DROP TABLE IF EXISTS t2")
    +  }
    +
    +  private def checkHiveQl(hiveQl: String): Unit = {
    --- End diff --
    
    ok, will rename it. 


---
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