Skip to content

Commit

Permalink
Rename SortShuffleSorter to SortShuffleFileWriter
Browse files Browse the repository at this point in the history
This is a better name, since one of the implementations never sorts.
  • Loading branch information
JoshRosen committed May 26, 2015
1 parent 0d9848c commit 25b964f
Show file tree
Hide file tree
Showing 4 changed files with 8 additions and 5 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -42,7 +42,7 @@ private[spark] class BypassMergeSortShuffleWriter[K, V](
partitioner: Partitioner,
writeMetrics: ShuffleWriteMetrics,
serializer: Serializer)
extends Logging with SortShuffleSorter[K, V] {
extends Logging with SortShuffleFileWriter[K, V] {

private[this] val numPartitions = partitioner.numPartitions

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -22,7 +22,10 @@ import java.io.{IOException, File}
import org.apache.spark.TaskContext
import org.apache.spark.storage.BlockId

private[spark] trait SortShuffleSorter[K, V] {
/**
* Interface for objects that [[SortShuffleWriter]] uses to write its output files.
*/
private[spark] trait SortShuffleFileWriter[K, V] {

@throws[IOException]
def insertAll(records: Iterator[_ <: Product2[K, V]]): Unit
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -36,7 +36,7 @@ private[spark] class SortShuffleWriter[K, V, C](

private val blockManager = SparkEnv.get.blockManager

private var sorter: SortShuffleSorter[K, V] = null
private var sorter: SortShuffleFileWriter[K, V] = null

// Are we in the process of stopping? Because map tasks can call stop() with success = true
// and then call stop() with success = false if they get an exception, we want to make sure
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -29,7 +29,7 @@ import com.google.common.io.ByteStreams
import org.apache.spark._
import org.apache.spark.serializer._
import org.apache.spark.executor.ShuffleWriteMetrics
import org.apache.spark.shuffle.sort.{SortShuffleWriter, SortShuffleSorter}
import org.apache.spark.shuffle.sort.{SortShuffleFileWriter, SortShuffleWriter}
import org.apache.spark.storage.{BlockId, BlockObjectWriter}

/**
Expand Down Expand Up @@ -94,7 +94,7 @@ private[spark] class ExternalSorter[K, V, C](
serializer: Option[Serializer] = None)
extends Logging
with Spillable[WritablePartitionedPairCollection[K, C]]
with SortShuffleSorter[K, V] {
with SortShuffleFileWriter[K, V] {

private val conf = SparkEnv.get.conf

Expand Down

0 comments on commit 25b964f

Please sign in to comment.