You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by rxin <gi...@git.apache.org> on 2017/03/07 22:33:58 UTC

[GitHub] spark pull request #17196: [SPARK-19855][SQL] Create an internal FilePartiti...

GitHub user rxin opened a pull request:

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

    [SPARK-19855][SQL] Create an internal FilePartitionStrategy interface

    ## What changes were proposed in this pull request?
    The way we currently do file partitioning strategy is hard coded in FileSourceScanExec. This is not ideal for two reasons:
    
    1. It is difficult to unit test the default strategy. In order to test this, we need to do almost end-to-end tests by creating actual files on the file system.
    2. It is difficult to experiment with different partitioning strategies without adding a lot of if branches.
    
    This patch introduces an internal interface for this so we can make this pluggable for both better testing and experimentation.
    
    ## How was this patch tested?
    The change should be covered by integration test cases in FileSourceStrategySuite. A follow-up patch (tracked via SPARK-19856) will change the partitioning related test cases in FileSourceStrategySuite from integration tests into unit tests.

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

    $ git pull https://github.com/rxin/spark SPARK-19855

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

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

----


---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

    https://github.com/apache/spark/pull/17196
  
    **[Test build #74138 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/74138/testReport)** for PR 17196 at commit [`ad5c7d1`](https://github.com/apache/spark/commit/ad5c7d11f64ce86ec109031f3b861d6168e1c8da).
     * This patch **fails PySpark unit tests**.
     * This patch merges cleanly.
     * This patch adds the following public classes _(experimental)_:
      * `trait FilePartitionStrategy `
      * `class DefaultFilePartitionStrategy extends FilePartitionStrategy with Logging `


---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

    https://github.com/apache/spark/pull/17196
  
    **[Test build #74150 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/74150/testReport)** for PR 17196 at commit [`ad5c7d1`](https://github.com/apache/spark/commit/ad5c7d11f64ce86ec109031f3b861d6168e1c8da).
     * This patch **fails PySpark unit tests**.
     * This patch merges cleanly.
     * This patch adds the following public classes _(experimental)_:
      * `trait FilePartitionStrategy `
      * `class DefaultFilePartitionStrategy extends FilePartitionStrategy with Logging `


---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

    https://github.com/apache/spark/pull/17196
  
    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 #17196: [SPARK-19855][SQL] Create an internal FilePartiti...

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/17196#discussion_r104804221
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/FilePartitionStrategy.scala ---
    @@ -0,0 +1,156 @@
    +/*
    + * 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 scala.collection.mutable.ArrayBuffer
    +
    +import org.apache.hadoop.fs.{BlockLocation, FileStatus, LocatedFileStatus}
    +
    +import org.apache.spark.internal.Logging
    +import org.apache.spark.sql.SparkSession
    +import org.apache.spark.sql.catalyst.InternalRow
    +
    +/**
    + * An (internal) interface that takes in a list of files and partitions them for parallelization.
    + */
    +trait FilePartitionStrategy {
    +  /**
    +   * `input` is a list of input files, in the form of (partition column value, file status).
    +   *
    +   * The function should return a list of file blocks to read for each partition. The i-th position
    +   * indicates the list of file blocks to read for task i.
    +   */
    +  def partition(
    +      sparkSession: SparkSession,
    +      fileFormat: FileFormat,
    +      options: Map[String, String],
    +      input: Seq[(InternalRow, FileStatus)])
    +    : Seq[Seq[PartitionedFile]]
    +}
    +
    +
    +/**
    + * A default [[FilePartitionStrategy]] that binpacks files roughly into evenly sized partitions.
    + */
    +class DefaultFilePartitionStrategy extends FilePartitionStrategy with Logging {
    +  import DefaultFilePartitionStrategy._
    +
    +  override def partition(
    +      sparkSession: SparkSession,
    +      fileFormat: FileFormat,
    +      options: Map[String, String],
    +      input: Seq[(InternalRow, FileStatus)])
    +    : Seq[Seq[PartitionedFile]] = {
    +
    +    val defaultMaxSplitBytes = sparkSession.sessionState.conf.filesMaxPartitionBytes
    +    val openCostInBytes = sparkSession.sessionState.conf.filesOpenCostInBytes
    +    val defaultParallelism = sparkSession.sparkContext.defaultParallelism
    +    val totalBytes = input.map(_._2.getLen + openCostInBytes).sum
    +    val bytesPerCore = totalBytes / defaultParallelism
    +
    +    val maxSplitBytes = Math.min(defaultMaxSplitBytes, Math.max(openCostInBytes, bytesPerCore))
    +    logInfo(s"Planning scan with bin packing, max size: $maxSplitBytes bytes, " +
    +      s"open cost is considered as scanning $openCostInBytes bytes.")
    +
    +    val splitFiles: Array[PartitionedFile] = input.flatMap { case (partitionValues, file) =>
    +      val blockLocations = getBlockLocations(file)
    +      if (fileFormat.isSplitable(sparkSession, options, file.getPath)) {
    +        (0L until file.getLen by maxSplitBytes).map { offset =>
    +          val remaining = file.getLen - offset
    +          val size = if (remaining > maxSplitBytes) maxSplitBytes else remaining
    +          val hosts = getBlockHosts(blockLocations, offset, size)
    +          PartitionedFile(partitionValues, file.getPath.toUri.toString, offset, size, hosts)
    +        }
    +      } else {
    +        val hosts = getBlockHosts(blockLocations, 0, file.getLen)
    +        Seq(PartitionedFile(partitionValues, file.getPath.toUri.toString, 0, file.getLen, hosts))
    +      }
    +    }.toArray.sortBy(_.length)(implicitly[Ordering[Long]].reverse)
    +
    +    val partitions = new ArrayBuffer[Seq[PartitionedFile]]
    +    val currentFiles = new ArrayBuffer[PartitionedFile]
    +    var currentSize = 0L
    +
    +    /** Close the current partition and move to the next. */
    +    def closePartition(): Unit = {
    +      if (currentFiles.nonEmpty) {
    +        val newPartition = currentFiles.toArray.toSeq // Copy to a new Array.
    +        partitions += newPartition
    +      }
    +      currentFiles.clear()
    +      currentSize = 0
    +    }
    +
    +    // Assign files to partitions using "First Fit Decreasing" (FFD)
    +    splitFiles.foreach { file =>
    +      if (currentSize + file.length > maxSplitBytes) {
    +        closePartition()
    +      }
    +      // Add the given file to the current partition.
    +      currentSize += file.length + openCostInBytes
    +      currentFiles += file
    +    }
    +    closePartition()
    +
    +    partitions
    +  }
    +}
    +
    +
    +object DefaultFilePartitionStrategy {
    --- End diff --
    
    shall we just move functions here to `class DefaultFilePartitionStrategy`?


---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

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


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

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


[GitHub] spark issue #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

    https://github.com/apache/spark/pull/17196
  
    Test FAILed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/74150/
    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 #17196: [SPARK-19855][SQL] Create an internal FilePartiti...

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

    https://github.com/apache/spark/pull/17196#discussion_r104804384
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/FilePartitionStrategy.scala ---
    @@ -0,0 +1,156 @@
    +/*
    + * 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 scala.collection.mutable.ArrayBuffer
    +
    +import org.apache.hadoop.fs.{BlockLocation, FileStatus, LocatedFileStatus}
    +
    +import org.apache.spark.internal.Logging
    +import org.apache.spark.sql.SparkSession
    +import org.apache.spark.sql.catalyst.InternalRow
    +
    +/**
    + * An (internal) interface that takes in a list of files and partitions them for parallelization.
    + */
    +trait FilePartitionStrategy {
    +  /**
    +   * `input` is a list of input files, in the form of (partition column value, file status).
    +   *
    +   * The function should return a list of file blocks to read for each partition. The i-th position
    +   * indicates the list of file blocks to read for task i.
    +   */
    +  def partition(
    +      sparkSession: SparkSession,
    +      fileFormat: FileFormat,
    +      options: Map[String, String],
    +      input: Seq[(InternalRow, FileStatus)])
    +    : Seq[Seq[PartitionedFile]]
    +}
    +
    +
    +/**
    + * A default [[FilePartitionStrategy]] that binpacks files roughly into evenly sized partitions.
    + */
    +class DefaultFilePartitionStrategy extends FilePartitionStrategy with Logging {
    +  import DefaultFilePartitionStrategy._
    +
    +  override def partition(
    +      sparkSession: SparkSession,
    +      fileFormat: FileFormat,
    +      options: Map[String, String],
    +      input: Seq[(InternalRow, FileStatus)])
    +    : Seq[Seq[PartitionedFile]] = {
    +
    +    val defaultMaxSplitBytes = sparkSession.sessionState.conf.filesMaxPartitionBytes
    +    val openCostInBytes = sparkSession.sessionState.conf.filesOpenCostInBytes
    +    val defaultParallelism = sparkSession.sparkContext.defaultParallelism
    +    val totalBytes = input.map(_._2.getLen + openCostInBytes).sum
    +    val bytesPerCore = totalBytes / defaultParallelism
    +
    +    val maxSplitBytes = Math.min(defaultMaxSplitBytes, Math.max(openCostInBytes, bytesPerCore))
    +    logInfo(s"Planning scan with bin packing, max size: $maxSplitBytes bytes, " +
    +      s"open cost is considered as scanning $openCostInBytes bytes.")
    +
    +    val splitFiles: Array[PartitionedFile] = input.flatMap { case (partitionValues, file) =>
    +      val blockLocations = getBlockLocations(file)
    +      if (fileFormat.isSplitable(sparkSession, options, file.getPath)) {
    +        (0L until file.getLen by maxSplitBytes).map { offset =>
    +          val remaining = file.getLen - offset
    +          val size = if (remaining > maxSplitBytes) maxSplitBytes else remaining
    +          val hosts = getBlockHosts(blockLocations, offset, size)
    +          PartitionedFile(partitionValues, file.getPath.toUri.toString, offset, size, hosts)
    +        }
    +      } else {
    +        val hosts = getBlockHosts(blockLocations, 0, file.getLen)
    +        Seq(PartitionedFile(partitionValues, file.getPath.toUri.toString, 0, file.getLen, hosts))
    +      }
    +    }.toArray.sortBy(_.length)(implicitly[Ordering[Long]].reverse)
    +
    +    val partitions = new ArrayBuffer[Seq[PartitionedFile]]
    +    val currentFiles = new ArrayBuffer[PartitionedFile]
    +    var currentSize = 0L
    +
    +    /** Close the current partition and move to the next. */
    +    def closePartition(): Unit = {
    +      if (currentFiles.nonEmpty) {
    +        val newPartition = currentFiles.toArray.toSeq // Copy to a new Array.
    +        partitions += newPartition
    +      }
    +      currentFiles.clear()
    +      currentSize = 0
    +    }
    +
    +    // Assign files to partitions using "First Fit Decreasing" (FFD)
    +    splitFiles.foreach { file =>
    +      if (currentSize + file.length > maxSplitBytes) {
    +        closePartition()
    +      }
    +      // Add the given file to the current partition.
    +      currentSize += file.length + openCostInBytes
    +      currentFiles += file
    +    }
    +    closePartition()
    +
    +    partitions
    +  }
    +}
    +
    +
    +object DefaultFilePartitionStrategy {
    --- End diff --
    
    the other class still references 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 #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

    https://github.com/apache/spark/pull/17196
  
    **[Test build #3598 has started](https://amplab.cs.berkeley.edu/jenkins/job/NewSparkPullRequestBuilder/3598/testReport)** for PR 17196 at commit [`ad5c7d1`](https://github.com/apache/spark/commit/ad5c7d11f64ce86ec109031f3b861d6168e1c8da).


---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

    https://github.com/apache/spark/pull/17196
  
    **[Test build #3598 has finished](https://amplab.cs.berkeley.edu/jenkins/job/NewSparkPullRequestBuilder/3598/testReport)** for PR 17196 at commit [`ad5c7d1`](https://github.com/apache/spark/commit/ad5c7d11f64ce86ec109031f3b861d6168e1c8da).
     * This patch passes all tests.
     * This patch merges cleanly.
     * This patch adds the following public classes _(experimental)_:
      * `trait FilePartitionStrategy `
      * `class DefaultFilePartitionStrategy extends FilePartitionStrategy with Logging `


---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartiti...

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/17196#discussion_r104814151
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/execution/DataSourceScanExec.scala ---
    @@ -357,7 +355,8 @@ case class FileSourceScanExec(
         val bucketed =
           selectedPartitions.flatMap { p =>
             p.files.map { f =>
    -          val hosts = getBlockHosts(getBlockLocations(f), 0, f.getLen)
    +          val hosts = DefaultFilePartitionStrategy.getBlockHosts(
    +            DefaultFilePartitionStrategy.getBlockLocations(f), 0, f.getLen)
    --- End diff --
    
    nvm, we still need it inside `DefaultFilePartitionStrategy`


---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartiti...

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

    https://github.com/apache/spark/pull/17196#discussion_r104798525
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/execution/datasources/FilePartitionStrategy.scala ---
    @@ -0,0 +1,156 @@
    +/*
    + * 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 scala.collection.mutable.ArrayBuffer
    +
    +import org.apache.hadoop.fs.{BlockLocation, FileStatus, LocatedFileStatus}
    +
    +import org.apache.spark.internal.Logging
    +import org.apache.spark.sql.SparkSession
    +import org.apache.spark.sql.catalyst.InternalRow
    +
    +/**
    + * An (internal) interface that takes in a list of files and partitions them for parallelization.
    + */
    +trait FilePartitionStrategy {
    --- End diff --
    
    the majority of this file is just moving code from FileSourceStrategy over.



---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

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


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

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


[GitHub] spark issue #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

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


---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

    https://github.com/apache/spark/pull/17196
  
    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 #17196: [SPARK-19855][SQL] Create an internal FilePartiti...

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/17196#discussion_r104804778
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/execution/DataSourceScanExec.scala ---
    @@ -357,7 +355,8 @@ case class FileSourceScanExec(
         val bucketed =
           selectedPartitions.flatMap { p =>
             p.files.map { f =>
    -          val hosts = getBlockHosts(getBlockLocations(f), 0, f.getLen)
    +          val hosts = DefaultFilePartitionStrategy.getBlockHosts(
    +            DefaultFilePartitionStrategy.getBlockLocations(f), 0, f.getLen)
    --- End diff --
    
    since this logic is general across all partitioning strategy, shall we leave `getBlockHosts` method in this class?


---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartiti...

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

    https://github.com/apache/spark/pull/17196#discussion_r104817903
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/execution/DataSourceScanExec.scala ---
    @@ -357,7 +355,8 @@ case class FileSourceScanExec(
         val bucketed =
           selectedPartitions.flatMap { p =>
             p.files.map { f =>
    -          val hosts = getBlockHosts(getBlockLocations(f), 0, f.getLen)
    +          val hosts = DefaultFilePartitionStrategy.getBlockHosts(
    +            DefaultFilePartitionStrategy.getBlockLocations(f), 0, f.getLen)
    --- End diff --
    
    How about using the object name `FilePartitionStrategy` instead of `DefaultFilePartitionStrategy` if these methods are general across all partitioning strategy?


---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

    https://github.com/apache/spark/pull/17196
  
    Test FAILed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/74138/
    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 #17196: [SPARK-19855][SQL] Create an internal FilePartiti...

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

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


---
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 #17196: [SPARK-19855][SQL] Create an internal FilePartitionStrat...

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

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


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