1717
1818package org .apache .spark
1919
20- import scala .concurrent ._
21- import scala .concurrent .duration .Duration
22- import scala .util .Try
20+ import java .util .concurrent .TimeUnit
2321
24- import org .apache .spark .annotation . Experimental
22+ import org .apache .spark .api . java . JavaFutureAction
2523import org .apache .spark .rdd .RDD
2624import org .apache .spark .scheduler .{JobFailed , JobSucceeded , JobWaiter }
2725
26+ import scala .concurrent ._
27+ import scala .concurrent .duration .Duration
28+ import scala .util .{Failure , Try }
29+
2830/**
29- * :: Experimental ::
3031 * A future for the result of an action to support cancellation. This is an extension of the
3132 * Scala Future interface to support cancellation.
3233 */
33- @ Experimental
3434trait FutureAction [T ] extends Future [T ] {
3535 // Note that we redefine methods of the Future trait here explicitly so we can specify a different
3636 // documentation (with reference to the word "action").
@@ -69,6 +69,11 @@ trait FutureAction[T] extends Future[T] {
6969 */
7070 override def isCompleted : Boolean
7171
72+ /**
73+ * Returns whether the action has been cancelled.
74+ */
75+ def isCancelled : Boolean
76+
7277 /**
7378 * The value of this Future.
7479 *
@@ -96,15 +101,16 @@ trait FutureAction[T] extends Future[T] {
96101
97102
98103/**
99- * :: Experimental ::
100104 * A [[FutureAction ]] holding the result of an action that triggers a single job. Examples include
101105 * count, collect, reduce.
102106 */
103- @ Experimental
104107class SimpleFutureAction [T ] private [spark](jobWaiter : JobWaiter [_], resultFunc : => T )
105108 extends FutureAction [T ] {
106109
110+ @ volatile private var _cancelled : Boolean = false
111+
107112 override def cancel () {
113+ _cancelled = true
108114 jobWaiter.cancel()
109115 }
110116
@@ -143,6 +149,8 @@ class SimpleFutureAction[T] private[spark](jobWaiter: JobWaiter[_], resultFunc:
143149 }
144150
145151 override def isCompleted : Boolean = jobWaiter.jobFinished
152+
153+ override def isCancelled : Boolean = _cancelled
146154
147155 override def value : Option [Try [T ]] = {
148156 if (jobWaiter.jobFinished) {
@@ -164,12 +172,10 @@ class SimpleFutureAction[T] private[spark](jobWaiter: JobWaiter[_], resultFunc:
164172
165173
166174/**
167- * :: Experimental ::
168175 * A [[FutureAction ]] for actions that could trigger multiple Spark jobs. Examples include take,
169176 * takeSample. Cancellation works by setting the cancelled flag to true and interrupting the
170177 * action thread if it is being blocked by a job.
171178 */
172- @ Experimental
173179class ComplexFutureAction [T ] extends FutureAction [T ] {
174180
175181 // Pointer to the thread that is executing the action. It is set when the action is run.
@@ -222,7 +228,7 @@ class ComplexFutureAction[T] extends FutureAction[T] {
222228 // If the action hasn't been cancelled yet, submit the job. The check and the submitJob
223229 // command need to be in an atomic block.
224230 val job = this .synchronized {
225- if (! cancelled ) {
231+ if (! isCancelled ) {
226232 rdd.context.submitJob(rdd, processPartition, partitions, resultHandler, resultFunc)
227233 } else {
228234 throw new SparkException (" Action has been cancelled" )
@@ -243,10 +249,7 @@ class ComplexFutureAction[T] extends FutureAction[T] {
243249 }
244250 }
245251
246- /**
247- * Returns whether the promise has been cancelled.
248- */
249- def cancelled : Boolean = _cancelled
252+ override def isCancelled : Boolean = _cancelled
250253
251254 @ throws(classOf [InterruptedException ])
252255 @ throws(classOf [scala.concurrent.TimeoutException ])
@@ -271,3 +274,55 @@ class ComplexFutureAction[T] extends FutureAction[T] {
271274 def jobIds = jobs
272275
273276}
277+
278+ private [spark]
279+ class JavaFutureActionWrapper [S , T ](futureAction : FutureAction [S ], converter : S => T )
280+ extends JavaFutureAction [T ] {
281+
282+ import scala .collection .JavaConverters ._
283+
284+ override def isCancelled : Boolean = futureAction.isCancelled
285+
286+ override def isDone : Boolean = {
287+ // According to java.util.Future's Javadoc, this returns True if the task was completed,
288+ // whether that completion was due to succesful execution, an exception, or a cancellation.
289+ futureAction.isCancelled || futureAction.isCompleted
290+ }
291+
292+ override def jobIds (): java.util.List [java.lang.Integer ] = {
293+ new java.util.ArrayList (futureAction.jobIds.map(x => new Integer (x)).asJava)
294+ }
295+
296+ private def getImpl (timeout : Duration ): T = {
297+ // This will throw TimeoutException on timeout:
298+ Await .ready(futureAction, timeout)
299+ futureAction.value.get match {
300+ case scala.util.Success (value) => converter(value)
301+ case Failure (exception) =>
302+ if (isCancelled) {
303+ throw new CancellationException (" Job cancelled: ${exception.message}" );
304+ } else {
305+ // java.util.Future.get() wraps exceptions in ExecutionException
306+ throw new ExecutionException (" Exception thrown by job: " , exception)
307+ }
308+ }
309+ }
310+
311+ override def get (): T = getImpl(Duration .Inf )
312+
313+ override def get (timeout : Long , unit : TimeUnit ): T =
314+ getImpl(Duration .fromNanos(unit.toNanos(timeout)))
315+
316+ override def cancel (mayInterruptIfRunning : Boolean ): Boolean = {
317+ if (isDone) {
318+ // According to java.util.Future's Javadoc, this should return false if the task is completed.
319+ false
320+ } else {
321+ // We're limited in terms of the semantics we can provide here; our cancellation is
322+ // asynchronous and doesn't provide a mechanism to not cancel if the job is running.
323+ futureAction.cancel()
324+ true
325+ }
326+ }
327+
328+ }
0 commit comments