[GitHub] spark pull request #15178: [SPARK-17556][SQL] Executor side broadcast for br...
Github user viirya closed the pull request at: https://github.com/apache/spark/pull/15178 --- - To unsubscribe, e-mail: reviews-unsubscr...@spark.apache.org For additional commands, e-mail: reviews-h...@spark.apache.org
[GitHub] spark pull request #15178: [SPARK-17556][SQL] Executor side broadcast for br...
Github user viirya commented on a diff in the pull request: https://github.com/apache/spark/pull/15178#discussion_r81085608 --- Diff: core/src/main/scala/org/apache/spark/SparkContext.scala --- @@ -1402,6 +1402,21 @@ class SparkContext(config: SparkConf) extends Logging with ExecutorAllocationCli } /** + * Broadcast a read-only variable to the cluster, returning a + * [[org.apache.spark.broadcast.Broadcast]] object for reading it in distributed functions. + * The variable will be sent to each cluster only once. + */ + def broadcastRDDOnExecutor[T: ClassTag, U: ClassTag]( --- End diff -- Yes. --- 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 infrastruct...@apache.org or file a JIRA ticket with INFRA. --- - To unsubscribe, e-mail: reviews-unsubscr...@spark.apache.org For additional commands, e-mail: reviews-h...@spark.apache.org
[GitHub] spark pull request #15178: [SPARK-17556][SQL] Executor side broadcast for br...
Github user viirya commented on a diff in the pull request: https://github.com/apache/spark/pull/15178#discussion_r81052440 --- Diff: core/src/main/scala/org/apache/spark/broadcast/TorrentExecutorBroadcast.scala --- @@ -0,0 +1,174 @@ +/* + * 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.broadcast + +import java.io.ObjectOutputStream + +import scala.collection.mutable.ArrayBuffer +import scala.reflect.ClassTag +import scala.util.Random + +import org.apache.spark._ +import org.apache.spark.internal.Logging +import org.apache.spark.storage.{BlockId, BlockResult, BroadcastBlockId, RDDBlockId, StorageLevel} +import org.apache.spark.util.Utils + +/** + * A BitTorrent-like implementation of [[org.apache.spark.broadcast.Broadcast]]. + * + * Different to [[TorrentBroadcast]], this implementation doesn't divide the object to broadcast. + * In contrast, this implementation performs broadcast on executor side for a RDD. So the results + * of the RDD does not need to collect first back to the driver before broadcasting. + * + * The mechanism is as follows: + * + * On each executor, the executor first attempts to fetch the object from its BlockManager. If + * it doesn not exist, it then uses remote fetches to fetch the blocks of the RDD from other + * executors if available. Once it gets the blocks, it puts the blocks in its own BlockManager, + * ready for other executors to fetch from. + * + * @tparam T The type of the element of RDD to be broadcasted. + * @tparam U The type of object transformed from the collection of elements of the RDD. + * + * @param numBlocks Total number of blocks this broadcast variable contains. + * @param rddId The id of the RDD to be broadcasted on executors. + * @param mode The [[BroadcastMode]] object used to transform the result of RDD to the object which + * will be stored in the [[BlockManager]]. + * @param id A unique identifier for the broadcast variable. + */ +private[spark] class TorrentExecutorBroadcast[T: ClassTag, U: ClassTag]( +numBlocks: Int, +rddId: Int, +mode: BroadcastMode[T], +id: Long) extends Broadcast[U](id) with Logging with Serializable { + + /** + * Value of the broadcast object on executors. This is reconstructed by [[readBroadcastBlock]], + * which builds this value by reading blocks from other executors. + */ + @transient private lazy val _value: U = readBroadcastBlock() + + private val broadcastId = BroadcastBlockId(id) + + override protected def getValue() = { +_value + } + + /** Fetch torrent blocks from other executors. */ + private def readBlocks(): Array[T] = { +// Fetch chunks of data. Note that all these chunks are stored in the BlockManager and reported +// to the driver, so other executors can pull these chunks from this executor as well. +val blocks = new Array[Array[T]](numBlocks) +val bm = SparkEnv.get.blockManager + +for (pid <- Random.shuffle(Seq.range(0, numBlocks))) { + val pieceId = RDDBlockId(rddId, pid) + // First try getLocalValues because there is a chance that previous attempts to fetch the + // broadcast blocks have already fetched some of the blocks. In that case, some blocks + // would be available locally (on this executor). + bm.getLocalValues(pieceId) match { +case Some(block: BlockResult) => + blocks(pid) = block.data.asInstanceOf[Iterator[T]].toArray +case None => + bm.get[T](pieceId) match { +case Some(b) => + val data = b.data.asInstanceOf[Iterator[T]].toArray + // We found the block from remote executors' BlockManager, so put the block + // in this executor's BlockManager. + if (!bm.putIterator(pieceId, data.toIterator, --- End diff -- One solution might be to
[GitHub] spark pull request #15178: [SPARK-17556][SQL] Executor side broadcast for br...
Github user holdenk commented on a diff in the pull request: https://github.com/apache/spark/pull/15178#discussion_r80945496 --- Diff: core/src/main/scala/org/apache/spark/broadcast/TorrentExecutorBroadcast.scala --- @@ -0,0 +1,174 @@ +/* + * 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.broadcast + +import java.io.ObjectOutputStream + +import scala.collection.mutable.ArrayBuffer +import scala.reflect.ClassTag +import scala.util.Random + +import org.apache.spark._ +import org.apache.spark.internal.Logging +import org.apache.spark.storage.{BlockId, BlockResult, BroadcastBlockId, RDDBlockId, StorageLevel} +import org.apache.spark.util.Utils + +/** + * A BitTorrent-like implementation of [[org.apache.spark.broadcast.Broadcast]]. + * + * Different to [[TorrentBroadcast]], this implementation doesn't divide the object to broadcast. + * In contrast, this implementation performs broadcast on executor side for a RDD. So the results + * of the RDD does not need to collect first back to the driver before broadcasting. + * + * The mechanism is as follows: + * + * On each executor, the executor first attempts to fetch the object from its BlockManager. If + * it doesn not exist, it then uses remote fetches to fetch the blocks of the RDD from other + * executors if available. Once it gets the blocks, it puts the blocks in its own BlockManager, + * ready for other executors to fetch from. + * + * @tparam T The type of the element of RDD to be broadcasted. + * @tparam U The type of object transformed from the collection of elements of the RDD. + * + * @param numBlocks Total number of blocks this broadcast variable contains. + * @param rddId The id of the RDD to be broadcasted on executors. + * @param mode The [[BroadcastMode]] object used to transform the result of RDD to the object which + * will be stored in the [[BlockManager]]. + * @param id A unique identifier for the broadcast variable. + */ +private[spark] class TorrentExecutorBroadcast[T: ClassTag, U: ClassTag]( +numBlocks: Int, +rddId: Int, +mode: BroadcastMode[T], +id: Long) extends Broadcast[U](id) with Logging with Serializable { + + /** + * Value of the broadcast object on executors. This is reconstructed by [[readBroadcastBlock]], + * which builds this value by reading blocks from other executors. + */ + @transient private lazy val _value: U = readBroadcastBlock() + + private val broadcastId = BroadcastBlockId(id) + + override protected def getValue() = { +_value + } + + /** Fetch torrent blocks from other executors. */ + private def readBlocks(): Array[T] = { +// Fetch chunks of data. Note that all these chunks are stored in the BlockManager and reported +// to the driver, so other executors can pull these chunks from this executor as well. +val blocks = new Array[Array[T]](numBlocks) +val bm = SparkEnv.get.blockManager + +for (pid <- Random.shuffle(Seq.range(0, numBlocks))) { + val pieceId = RDDBlockId(rddId, pid) + // First try getLocalValues because there is a chance that previous attempts to fetch the + // broadcast blocks have already fetched some of the blocks. In that case, some blocks + // would be available locally (on this executor). + bm.getLocalValues(pieceId) match { +case Some(block: BlockResult) => + blocks(pid) = block.data.asInstanceOf[Iterator[T]].toArray +case None => + bm.get[T](pieceId) match { +case Some(b) => + val data = b.data.asInstanceOf[Iterator[T]].toArray + // We found the block from remote executors' BlockManager, so put the block + // in this executor's BlockManager. + if (!bm.putIterator(pieceId, data.toIterator, --- End diff -- So were storing an RDD p
[GitHub] spark pull request #15178: [SPARK-17556][SQL] Executor side broadcast for br...
Github user holdenk commented on a diff in the pull request: https://github.com/apache/spark/pull/15178#discussion_r80939434 --- Diff: core/src/main/scala/org/apache/spark/SparkContext.scala --- @@ -1402,6 +1402,21 @@ class SparkContext(config: SparkConf) extends Logging with ExecutorAllocationCli } /** + * Broadcast a read-only variable to the cluster, returning a + * [[org.apache.spark.broadcast.Broadcast]] object for reading it in distributed functions. + * The variable will be sent to each cluster only once. + */ + def broadcastRDDOnExecutor[T: ClassTag, U: ClassTag]( --- End diff -- We might want to mark this as a developer API for now? --- 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 infrastruct...@apache.org or file a JIRA ticket with INFRA. --- - To unsubscribe, e-mail: reviews-unsubscr...@spark.apache.org For additional commands, e-mail: reviews-h...@spark.apache.org
[GitHub] spark pull request #15178: [SPARK-17556][SQL] Executor side broadcast for br...
Github user holdenk commented on a diff in the pull request: https://github.com/apache/spark/pull/15178#discussion_r80943501 --- Diff: core/src/main/scala/org/apache/spark/broadcast/TorrentExecutorBroadcast.scala --- @@ -0,0 +1,174 @@ +/* + * 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.broadcast + +import java.io.ObjectOutputStream + +import scala.collection.mutable.ArrayBuffer +import scala.reflect.ClassTag +import scala.util.Random + +import org.apache.spark._ +import org.apache.spark.internal.Logging +import org.apache.spark.storage.{BlockId, BlockResult, BroadcastBlockId, RDDBlockId, StorageLevel} +import org.apache.spark.util.Utils + +/** + * A BitTorrent-like implementation of [[org.apache.spark.broadcast.Broadcast]]. + * + * Different to [[TorrentBroadcast]], this implementation doesn't divide the object to broadcast. + * In contrast, this implementation performs broadcast on executor side for a RDD. So the results + * of the RDD does not need to collect first back to the driver before broadcasting. + * + * The mechanism is as follows: + * + * On each executor, the executor first attempts to fetch the object from its BlockManager. If + * it doesn not exist, it then uses remote fetches to fetch the blocks of the RDD from other + * executors if available. Once it gets the blocks, it puts the blocks in its own BlockManager, + * ready for other executors to fetch from. + * + * @tparam T The type of the element of RDD to be broadcasted. + * @tparam U The type of object transformed from the collection of elements of the RDD. + * + * @param numBlocks Total number of blocks this broadcast variable contains. + * @param rddId The id of the RDD to be broadcasted on executors. + * @param mode The [[BroadcastMode]] object used to transform the result of RDD to the object which + * will be stored in the [[BlockManager]]. + * @param id A unique identifier for the broadcast variable. + */ +private[spark] class TorrentExecutorBroadcast[T: ClassTag, U: ClassTag]( +numBlocks: Int, +rddId: Int, +mode: BroadcastMode[T], +id: Long) extends Broadcast[U](id) with Logging with Serializable { + + /** + * Value of the broadcast object on executors. This is reconstructed by [[readBroadcastBlock]], + * which builds this value by reading blocks from other executors. + */ + @transient private lazy val _value: U = readBroadcastBlock() + + private val broadcastId = BroadcastBlockId(id) + + override protected def getValue() = { +_value + } + + /** Fetch torrent blocks from other executors. */ + private def readBlocks(): Array[T] = { +// Fetch chunks of data. Note that all these chunks are stored in the BlockManager and reported +// to the driver, so other executors can pull these chunks from this executor as well. +val blocks = new Array[Array[T]](numBlocks) +val bm = SparkEnv.get.blockManager + +for (pid <- Random.shuffle(Seq.range(0, numBlocks))) { + val pieceId = RDDBlockId(rddId, pid) + // First try getLocalValues because there is a chance that previous attempts to fetch the + // broadcast blocks have already fetched some of the blocks. In that case, some blocks + // would be available locally (on this executor). + bm.getLocalValues(pieceId) match { +case Some(block: BlockResult) => + blocks(pid) = block.data.asInstanceOf[Iterator[T]].toArray +case None => + bm.get[T](pieceId) match { +case Some(b) => + val data = b.data.asInstanceOf[Iterator[T]].toArray + // We found the block from remote executors' BlockManager, so put the block + // in this executor's BlockManager. + if (!bm.putIterator(pieceId, data.toIterator, + StorageLevel.MEMORY_AND_DISK_S