Skip to content
Snippets Groups Projects
Commit 72eed2b9 authored by Tathagata Das's avatar Tathagata Das
Browse files

Converted CheckpointState in RDDCheckpointData to use scala Enumeration.

parent 8e74fac2
No related branches found
No related tags found
No related merge requests found
...@@ -5,45 +5,41 @@ import rdd.CoalescedRDD ...@@ -5,45 +5,41 @@ import rdd.CoalescedRDD
import scheduler.{ResultTask, ShuffleMapTask} import scheduler.{ResultTask, ShuffleMapTask}
/** /**
* This class contains all the information of the regarding RDD checkpointing. * Enumeration to manage state transitions of an RDD through checkpointing
* [ Initialized --> marked for checkpointing --> checkpointing in progress --> checkpointed ]
*/ */
private[spark] object CheckpointState extends Enumeration {
type CheckpointState = Value
val Initialized, MarkedForCheckpoint, CheckpointingInProgress, Checkpointed = Value
}
/**
* This class contains all the information of the regarding RDD checkpointing.
*/
private[spark] class RDDCheckpointData[T: ClassManifest](rdd: RDD[T]) private[spark] class RDDCheckpointData[T: ClassManifest](rdd: RDD[T])
extends Logging with Serializable { extends Logging with Serializable {
/** import CheckpointState._
* This class manages the state transition of an RDD through checkpointing
* [ Not checkpointed --> marked for checkpointing --> checkpointing in progress --> checkpointed ]
*/
class CheckpointState extends Serializable {
var state = 0
def mark() { if (state == 0) state = 1 } var cpState = Initialized
def start() { assert(state == 1); state = 2 }
def finish() { assert(state == 2); state = 3 }
def isMarked() = { state == 1 }
def isInProgress = { state == 2 }
def isCheckpointed = { state == 3 }
}
val cpState = new CheckpointState()
@transient var cpFile: Option[String] = None @transient var cpFile: Option[String] = None
@transient var cpRDD: Option[RDD[T]] = None @transient var cpRDD: Option[RDD[T]] = None
@transient var cpRDDSplits: Seq[Split] = Nil @transient var cpRDDSplits: Seq[Split] = Nil
// Mark the RDD for checkpointing // Mark the RDD for checkpointing
def markForCheckpoint() = { def markForCheckpoint() {
RDDCheckpointData.synchronized { cpState.mark() } RDDCheckpointData.synchronized {
if (cpState == Initialized) cpState = MarkedForCheckpoint
}
} }
// Is the RDD already checkpointed // Is the RDD already checkpointed
def isCheckpointed() = { def isCheckpointed(): Boolean = {
RDDCheckpointData.synchronized { cpState.isCheckpointed } RDDCheckpointData.synchronized { cpState == Checkpointed }
} }
// Get the file to which this RDD was checkpointed to as a Option // Get the file to which this RDD was checkpointed to as an Option
def getCheckpointFile() = { def getCheckpointFile(): Option[String] = {
RDDCheckpointData.synchronized { cpFile } RDDCheckpointData.synchronized { cpFile }
} }
...@@ -52,8 +48,8 @@ extends Logging with Serializable { ...@@ -52,8 +48,8 @@ extends Logging with Serializable {
// If it is marked for checkpointing AND checkpointing is not already in progress, // If it is marked for checkpointing AND checkpointing is not already in progress,
// then set it to be in progress, else return // then set it to be in progress, else return
RDDCheckpointData.synchronized { RDDCheckpointData.synchronized {
if (cpState.isMarked && !cpState.isInProgress) { if (cpState == MarkedForCheckpoint) {
cpState.start() cpState = CheckpointingInProgress
} else { } else {
return return
} }
...@@ -87,7 +83,7 @@ extends Logging with Serializable { ...@@ -87,7 +83,7 @@ extends Logging with Serializable {
cpRDD = Some(newRDD) cpRDD = Some(newRDD)
cpRDDSplits = newRDD.splits cpRDDSplits = newRDD.splits
rdd.changeDependencies(newRDD) rdd.changeDependencies(newRDD)
cpState.finish() cpState = Checkpointed
RDDCheckpointData.checkpointCompleted() RDDCheckpointData.checkpointCompleted()
logInfo("Done checkpointing RDD " + rdd.id + ", new parent is RDD " + newRDD.id) logInfo("Done checkpointing RDD " + rdd.id + ", new parent is RDD " + newRDD.id)
} }
......
0% Loading or .
You are about to add 0 people to the discussion. Proceed with caution.
Finish editing this message first!
Please register or to comment