spark-reviews mailing list archives

Site index · List index
Message view « Date » · « Thread »
Top « Date » · « Thread »
From andrewor14 <...@git.apache.org>
Subject [GitHub] spark pull request: SPARK-2045 Sort-based shuffle
Date Wed, 30 Jul 2014 00:52:16 GMT
Github user andrewor14 commented on a diff in the pull request:

    https://github.com/apache/spark/pull/1499#discussion_r15562065
  
    --- Diff: core/src/main/scala/org/apache/spark/util/collection/ExternalSorter.scala ---
    @@ -0,0 +1,667 @@
    +/*
    + * 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.util.collection
    +
    +import java.io._
    +import java.util.Comparator
    +
    +import scala.collection.mutable.ArrayBuffer
    +import scala.collection.mutable
    +
    +import com.google.common.io.ByteStreams
    +
    +import org.apache.spark.{Aggregator, SparkEnv, Logging, Partitioner}
    +import org.apache.spark.serializer.Serializer
    +import org.apache.spark.storage.BlockId
    +
    +/**
    + * Sorts and potentially merges a number of key-value pairs of type (K, V) to produce
key-combiner
    + * pairs of type (K, C). Uses a Partitioner to first group the keys into partitions,
and then
    + * optionally sorts keys within each partition using a custom Comparator. Can output
a single
    + * partitioned file with a different byte range for each partition, suitable for shuffle
fetches.
    + *
    + * If combining is disabled, the type C must equal V -- we'll cast the objects at the
end.
    + *
    + * @param aggregator optional Aggregator with combine functions to use for merging data
    + * @param partitioner optional Partitioner; if given, sort by partition ID and then key
    + * @param ordering optional Ordering to sort keys within each partition; should be a
total ordering
    + * @param serializer serializer to use when spilling to disk
    + *
    + * Note that if an Ordering is given, we'll always sort using it, so only provide it
if you really
    + * want the output keys to be sorted. In a map task without map-side combine for example,
you
    + * probably want to pass None as the ordering to avoid extra sorting. On the other hand,
if you do
    + * want to do combining, having an Ordering is more efficient than not having it.
    + *
    + * At a high level, this class works as follows:
    + *
    + * - We repeatedly fill up buffers of in-memory data, using either a SizeTrackingAppendOnlyMap
if
    + *   we want to combine by key, or an simple SizeTrackingBuffer if we don't. Inside these
buffers,
    + *   we sort elements of type ((Int, K), C) where the Int is the partition ID. This is
done to
    + *   avoid calling the partitioner multiple times on the same key (e.g. for RangePartitioner).
    + *
    + * - When each buffer reaches our memory limit, we spill it to a file. This file is sorted
first
    + *   by partition ID and possibly second by key or by hash code of the key, if we want
to do
    + *   aggregation. For each file, we track how many objects were in each partition in
memory, so we
    + *   don't have to write out the partition ID for every element.
    + *
    + * - When the user requests an iterator, the spilled files are merged, along with any
remaining
    + *   in-memory data, using the same sort order defined above (unless both sorting and
aggregation
    + *   are disabled). If we need to aggregate by key, we either use a total ordering from
the
    + *   ordering parameter, or read the keys with the same hash code and compare them with
each other
    + *   for equality to merge values.
    + *
    + * - Users are expected to call stop() at the end to delete all the intermediate files.
    + */
    +private[spark] class ExternalSorter[K, V, C](
    +    aggregator: Option[Aggregator[K, V, C]] = None,
    +    partitioner: Option[Partitioner] = None,
    +    ordering: Option[Ordering[K]] = None,
    +    serializer: Option[Serializer] = None) extends Logging {
    +
    +  private val numPartitions = partitioner.map(_.numPartitions).getOrElse(1)
    +  private val shouldPartition = numPartitions > 1
    +
    +  private val blockManager = SparkEnv.get.blockManager
    +  private val diskBlockManager = blockManager.diskBlockManager
    +  private val ser = Serializer.getSerializer(serializer)
    +  private val serInstance = ser.newInstance()
    +
    +  private val conf = SparkEnv.get.conf
    +  private val spillingEnabled = conf.getBoolean("spark.shuffle.spill", true)
    +  private val fileBufferSize = conf.getInt("spark.shuffle.file.buffer.kb", 100) * 1024
    +  private val serializerBatchSize = conf.getLong("spark.shuffle.spill.batchSize", 10000)
    +
    +  private def getPartition(key: K): Int = {
    +    if (shouldPartition) partitioner.get.getPartition(key) else 0
    +  }
    +
    +  // Data structures to store in-memory objects before we spill. Depending on whether
we have an
    +  // Aggregator set, we either put objects into an AppendOnlyMap where we combine them,
or we
    +  // store them in an array buffer.
    +  var map = new SizeTrackingAppendOnlyMap[(Int, K), C]
    +  var buffer = new SizeTrackingPairBuffer[(Int, K), C]
    +
    +  // Number of pairs read from input since last spill; note that we count them even if
a value is
    +  // merged with a previous key in case we're doing something like groupBy where the
result grows
    +  private var elementsRead = 0L
    +
    +  // What threshold of elementsRead we start estimating map size at.
    +  private val trackMemoryThreshold = 1000
    +
    +  // Spilling statistics
    +  private var spillCount = 0
    +  private var _memoryBytesSpilled = 0L
    +  private var _diskBytesSpilled = 0L
    +
    +  // Collective memory threshold shared across all running tasks
    +  private val maxMemoryThreshold = {
    +    val memoryFraction = conf.getDouble("spark.shuffle.memoryFraction", 0.2)
    +    val safetyFraction = conf.getDouble("spark.shuffle.safetyFraction", 0.8)
    +    (Runtime.getRuntime.maxMemory * memoryFraction * safetyFraction).toLong
    +  }
    +
    +  // How much of the shared memory pool this collection has claimed
    +  private var myMemoryThreshold = 0L
    +
    +  // A comparator for keys K that orders them within a partition to allow partial aggregation.
    +  // Can be a partial ordering by hash code if a total ordering is not provided through
by the
    +  // user. (A partial ordering means that equal keys have comparator.compare(k, k) =
0, but some
    +  // non-equal keys also have this, so we need to do a later pass to find truly equal
keys).
    +  // Note that we ignore this if no aggregator is given.
    --- End diff --
    
    I don't think this is true. We also use this if `ordering` is defined and `aggregator`
is not.


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

Mime
View raw message