You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by cloud-fan <gi...@git.apache.org> on 2017/02/16 19:43:03 UTC

[GitHub] spark pull request #16962: [SPARK-18120 ][SQL] Call QueryExecutionListener c...

GitHub user cloud-fan opened a pull request:

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

    [SPARK-18120 ][SQL] Call QueryExecutionListener callback methods for DataFrameWriter methods

    ## What changes were proposed in this pull request?
    
    We only notify `QueryExecutionListener` for several `Dataset` operations, e.g. collect, take, etc. We should also do the notification for `DataFrameWriter` operations.
    
    ## How was this patch tested?
    
    new regression test

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

    $ git pull https://github.com/cloud-fan/spark insert

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

    https://github.com/apache/spark/pull/16962.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 #16962
    
----
commit ce0e126e606c6fe82185f59759ab51aa036ab8f9
Author: Wenchen Fan <we...@databricks.com>
Date:   2017-02-16T19:40:32Z

    Call QueryExecutionListener callback methods for DataFrameWriter methods

----


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    **[Test build #73012 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/73012/testReport)** for PR 16962 at commit [`c0dcd24`](https://github.com/apache/spark/commit/c0dcd2491be3a980e7992bb92f94e5e0927cd646).
     * 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 #16962: [SPARK-18120][SPARK-19557][SQL] Call QueryExecuti...

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

    https://github.com/apache/spark/pull/16962#discussion_r101724210
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/DataFrameWriter.scala ---
    @@ -573,6 +575,21 @@ final class DataFrameWriter[T] private[sql](ds: Dataset[T]) {
         format("csv").save(path)
       }
     
    +  private def runCommand(session: SparkSession, name: String)(command: LogicalPlan): Unit = {
    --- End diff --
    
    Why don't we use `SQLExecution` instead of this?


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    **[Test build #73009 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/73009/testReport)** for PR 16962 at commit [`ce0e126`](https://github.com/apache/spark/commit/ce0e126e606c6fe82185f59759ab51aa036ab8f9).
     * This patch passes all tests.
     * This patch merges cleanly.
     * This patch adds the following public classes _(experimental)_:
      * `case class SaveIntoDataSourceCommand(`


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecut...

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

    https://github.com/apache/spark/pull/16962#discussion_r101656725
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/SaveIntoDataSourceCommand.scala ---
    @@ -0,0 +1,52 @@
    +/*
    + * 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.execution.datasources
    +
    +import org.apache.spark.sql.{Dataset, Row, SaveMode, SparkSession}
    +import org.apache.spark.sql.catalyst.plans.QueryPlan
    +import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan
    +import org.apache.spark.sql.execution.command.RunnableCommand
    +
    +/**
    + * Saves the results of `query` in to a data source.
    + *
    + * Note that this command is different from [[InsertIntoDataSourceCommand]]. This command will call
    + * `CreatableRelationProvider.createRelation` to write out the data, while
    + * [[InsertIntoDataSourceCommand]] calls `InsertableRelation.insert`. Ideally these 2 data source
    + * interfaces should do the same thing, but as we've already published these 2 interfaces and the
    + * implementations may have different logic, we have to keep these 2 different commands.
    + */
    +case class SaveIntoDataSourceCommand(
    +    query: LogicalPlan,
    +    provider: String,
    +    partitionColumns: Seq[String],
    +    options: Map[String, String],
    +    mode: SaveMode) extends RunnableCommand {
    +
    +  override protected def innerChildren: Seq[QueryPlan[_]] = Seq(query)
    +
    +  override def run(sparkSession: SparkSession): Seq[Row] = {
    +    DataSource(
    +      sparkSession,
    +      className = provider,
    +      partitionColumns = partitionColumns,
    +      options = options).write(mode, Dataset.ofRows(sparkSession, query))
    +
    --- End diff --
    
    we don't have the `LogicalRelation` to be used as cache key.


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecut...

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

    https://github.com/apache/spark/pull/16962#discussion_r101655346
  
    --- Diff: sql/core/src/test/scala/org/apache/spark/sql/util/DataFrameCallbackSuite.scala ---
    @@ -159,4 +161,56 @@ class DataFrameCallbackSuite extends QueryTest with SharedSQLContext {
     
         spark.listenerManager.unregister(listener)
       }
    +
    +  test("execute callback functions for DataFrameWriter") {
    +    val commands = ArrayBuffer.empty[(String, LogicalPlan)]
    +    val exceptions = ArrayBuffer.empty[(String, Exception)]
    +    val listener = new QueryExecutionListener {
    +      // Only test successful case here, so no need to implement `onFailure`
    --- End diff --
    
    invalid comment?


---
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 #16962: [SPARK-18120][SPARK-19557][SQL] Call QueryExecutionListe...

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

    https://github.com/apache/spark/pull/16962
  
    thanks for the review, merging to 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 issue #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    **[Test build #73012 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/73012/testReport)** for PR 16962 at commit [`c0dcd24`](https://github.com/apache/spark/commit/c0dcd2491be3a980e7992bb92f94e5e0927cd646).


---
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 #16962: [SPARK-18120 ][SQL] Call QueryExecutionListener c...

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

    https://github.com/apache/spark/pull/16962#discussion_r101610336
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/DataFrameWriter.scala ---
    @@ -211,13 +211,15 @@ final class DataFrameWriter[T] private[sql](ds: Dataset[T]) {
         }
     
         assertNotBucketed("save")
    -    val dataSource = DataSource(
    -      df.sparkSession,
    -      className = source,
    -      partitionColumns = partitioningColumns.getOrElse(Nil),
    -      options = extraOptions.toMap)
     
    -    dataSource.write(mode, df)
    +    runCommand(df.sparkSession, "save") {
    +      SaveIntoDataSourceCommand(
    --- End diff --
    
    Does this also cover SPARK-19557? If so, might as well mention that in the PR (or close the bug as a duplicate or related or something).


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    LGTM


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    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 #16962: [SPARK-18120][SPARK-19557][SQL] Call QueryExecuti...

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

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


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    Test PASSed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/73023/
    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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecut...

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

    https://github.com/apache/spark/pull/16962#discussion_r101622507
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/DataFrameWriter.scala ---
    @@ -573,6 +575,24 @@ final class DataFrameWriter[T] private[sql](ds: Dataset[T]) {
         format("csv").save(path)
       }
     
    +  private def runCommand(session: SparkSession, name: String)(command: LogicalPlan): Unit = {
    +    val qe = session.sessionState.executePlan(command)
    +    try {
    +      qe.executedPlan.foreach { plan =>
    +        plan.resetMetrics()
    --- End diff --
    
    just realized that, in this code path, we pass in a logical plan and will always get a new physical plan, so we don't need to reset metrics here, let me remove 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 #16962: [SPARK-18120][SPARK-19557][SQL] Call QueryExecuti...

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

    https://github.com/apache/spark/pull/16962#discussion_r101813229
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/DataFrameWriter.scala ---
    @@ -573,6 +575,21 @@ final class DataFrameWriter[T] private[sql](ds: Dataset[T]) {
         format("csv").save(path)
       }
     
    +  private def runCommand(session: SparkSession, name: String)(command: LogicalPlan): Unit = {
    --- End diff --
    
    the problem is that, in some commands, like `InsertIntoHiveTable`, we already use `SQLExecution.newExecution`, so we can't use it again to wrap these commands.
    
    In the future we should figure out a central place to put `SQLExecution.newExecution`


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    **[Test build #73023 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/73023/testReport)** for PR 16962 at commit [`d35fac3`](https://github.com/apache/spark/commit/d35fac34247d6d6f2593b5284bbaba906a8dd033).


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecut...

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

    https://github.com/apache/spark/pull/16962#discussion_r101653599
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/SaveIntoDataSourceCommand.scala ---
    @@ -0,0 +1,52 @@
    +/*
    + * 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.execution.datasources
    +
    +import org.apache.spark.sql.{Dataset, Row, SaveMode, SparkSession}
    +import org.apache.spark.sql.catalyst.plans.QueryPlan
    +import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan
    +import org.apache.spark.sql.execution.command.RunnableCommand
    +
    +/**
    + * Saves the results of `query` in to a data source.
    + *
    + * Note that this command is different from [[InsertIntoDataSourceCommand]]. This command will call
    + * `CreatableRelationProvider.createRelation` to write out the data, while
    + * [[InsertIntoDataSourceCommand]] calls `InsertableRelation.insert`. Ideally these 2 data source
    + * interfaces should do the same thing, but as we've already published these 2 interfaces and the
    + * implementations may have different logic, we have to keep these 2 different commands.
    + */
    +case class SaveIntoDataSourceCommand(
    +    query: LogicalPlan,
    +    provider: String,
    +    partitionColumns: Seq[String],
    +    options: Map[String, String],
    +    mode: SaveMode) extends RunnableCommand {
    +
    +  override protected def innerChildren: Seq[QueryPlan[_]] = Seq(query)
    +
    +  override def run(sparkSession: SparkSession): Seq[Row] = {
    +    DataSource(
    +      sparkSession,
    +      className = provider,
    +      partitionColumns = partitionColumns,
    +      options = options).write(mode, Dataset.ofRows(sparkSession, query))
    +
    --- End diff --
    
    Do we need to  Invalidate the cache to be consistent with `InsertIntoDataSourceCommand`?
    ```
        sparkSession.sharedState.cacheManager.invalidateCache(query)
    ```


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    **[Test build #73023 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/73023/testReport)** for PR 16962 at commit [`d35fac3`](https://github.com/apache/spark/commit/d35fac34247d6d6f2593b5284bbaba906a8dd033).
     * 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 #16962: [SPARK-18120 ][SQL] Call QueryExecutionListener callback...

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

    https://github.com/apache/spark/pull/16962
  
    cc @salilsurendran @vanzin @gatorsmile @sameeragarwal 


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    LGTM except three minor comments.


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecut...

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

    https://github.com/apache/spark/pull/16962#discussion_r101666288
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/SaveIntoDataSourceCommand.scala ---
    @@ -0,0 +1,52 @@
    +/*
    + * 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.execution.datasources
    +
    +import org.apache.spark.sql.{Dataset, Row, SaveMode, SparkSession}
    +import org.apache.spark.sql.catalyst.plans.QueryPlan
    +import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan
    +import org.apache.spark.sql.execution.command.RunnableCommand
    +
    +/**
    + * Saves the results of `query` in to a data source.
    + *
    + * Note that this command is different from [[InsertIntoDataSourceCommand]]. This command will call
    + * `CreatableRelationProvider.createRelation` to write out the data, while
    + * [[InsertIntoDataSourceCommand]] calls `InsertableRelation.insert`. Ideally these 2 data source
    + * interfaces should do the same thing, but as we've already published these 2 interfaces and the
    + * implementations may have different logic, we have to keep these 2 different commands.
    + */
    +case class SaveIntoDataSourceCommand(
    +    query: LogicalPlan,
    +    provider: String,
    +    partitionColumns: Seq[String],
    +    options: Map[String, String],
    +    mode: SaveMode) extends RunnableCommand {
    +
    +  override protected def innerChildren: Seq[QueryPlan[_]] = Seq(query)
    +
    +  override def run(sparkSession: SparkSession): Seq[Row] = {
    +    DataSource(
    +      sparkSession,
    +      className = provider,
    +      partitionColumns = partitionColumns,
    +      options = options).write(mode, Dataset.ofRows(sparkSession, query))
    +
    --- End diff --
    
    : )


---
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 #16962: [SPARK-18120 ][SQL] Call QueryExecutionListener callback...

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

    https://github.com/apache/spark/pull/16962
  
    **[Test build #73009 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/73009/testReport)** for PR 16962 at commit [`ce0e126`](https://github.com/apache/spark/commit/ce0e126e606c6fe82185f59759ab51aa036ab8f9).


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    Test PASSed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/73012/
    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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    Test PASSed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/73009/
    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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecut...

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

    https://github.com/apache/spark/pull/16962#discussion_r101655120
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/DataFrameWriter.scala ---
    @@ -573,6 +575,21 @@ final class DataFrameWriter[T] private[sql](ds: Dataset[T]) {
         format("csv").save(path)
       }
     
    +  private def runCommand(session: SparkSession, name: String)(command: LogicalPlan): Unit = {
    --- End diff --
    
    Add a function description like?
    
      /**
       * Wrap a DataFrameWriter action to track the QueryExecution and time cost, then report to the
       * user-registered callback functions.
       */


---
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 #16962: [SPARK-18120 ][SPARK-19557][SQL] Call QueryExecutionList...

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

    https://github.com/apache/spark/pull/16962
  
    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