org.apache.spark

SparkContext

class SparkContext extends Logging with ExecutorAllocationClient

Main entry point for Spark functionality. A SparkContext represents the connection to a Spark cluster, and can be used to create RDDs, accumulators and broadcast variables on that cluster.

Only one SparkContext may be active per JVM. You must stop() the active SparkContext before creating a new one. This limitation may eventually be removed; see SPARK-2243 for more details.

Linear Supertypes
ExecutorAllocationClient, Logging, AnyRef, Any
Ordering
  1. Alphabetic
  2. By inheritance
Inherited
  1. SparkContext
  2. ExecutorAllocationClient
  3. Logging
  4. AnyRef
  5. Any
  1. Hide All
  2. Show all
Learn more about member selection
Visibility
  1. Public
  2. All

Instance Constructors

  1. new SparkContext(master: String, appName: String, sparkHome: String = null, jars: Seq[String] = Nil, environment: Map[String, String] = Map(), preferredNodeLocationData: Map[String, Set[SplitInfo]] = Map())

    Alternative constructor that allows setting common Spark properties directly

    Alternative constructor that allows setting common Spark properties directly

    master

    Cluster URL to connect to (e.g. mesos://host:port, spark://host:port, local[4]).

    appName

    A name for your application, to display on the cluster web UI.

    sparkHome

    Location where Spark is installed on cluster nodes.

    jars

    Collection of JARs to send to the cluster. These can be paths on the local file system or HDFS, HTTP, HTTPS, or FTP URLs.

    environment

    Environment variables to set on worker nodes.

  2. new SparkContext(master: String, appName: String, conf: SparkConf)

    Alternative constructor that allows setting common Spark properties directly

    Alternative constructor that allows setting common Spark properties directly

    master

    Cluster URL to connect to (e.g. mesos://host:port, spark://host:port, local[4]).

    appName

    A name for your application, to display on the cluster web UI

    conf

    a org.apache.spark.SparkConf object specifying other Spark parameters

  3. new SparkContext(config: SparkConf, preferredNodeLocationData: Map[String, Set[SplitInfo]])

    :: DeveloperApi :: Alternative constructor for setting preferred locations where Spark will create executors.

    :: DeveloperApi :: Alternative constructor for setting preferred locations where Spark will create executors.

    preferredNodeLocationData

    used in YARN mode to select nodes to launch containers on. Can be generated using org.apache.spark.scheduler.InputFormatInfo.computePreferredLocations from a list of input files or InputFormats for the application.

    Annotations
    @DeveloperApi()
  4. new SparkContext()

    Create a SparkContext that loads settings from system properties (for instance, when launching with .

    Create a SparkContext that loads settings from system properties (for instance, when launching with ./bin/spark-submit).

  5. new SparkContext(config: SparkConf)

    config

    a Spark Config object describing the application configuration. Any settings in this config overrides the default configs as well as system properties.

Value Members

  1. final def !=(arg0: AnyRef): Boolean

    Definition Classes
    AnyRef
  2. final def !=(arg0: Any): Boolean

    Definition Classes
    Any
  3. final def ##(): Int

    Definition Classes
    AnyRef → Any
  4. final def ==(arg0: AnyRef): Boolean

    Definition Classes
    AnyRef
  5. final def ==(arg0: Any): Boolean

    Definition Classes
    Any
  6. def accumulable[R, T](initialValue: R, name: String)(implicit param: AccumulableParam[R, T]): Accumulable[R, T]

    Create an org.apache.spark.Accumulable shared variable, with a name for display in the Spark UI.

    Create an org.apache.spark.Accumulable shared variable, with a name for display in the Spark UI. Tasks can add values to the accumuable using the += operator. Only the driver can access the accumuable's value.

    R

    accumulator result type

    T

    type that can be added to the accumulator

  7. def accumulable[R, T](initialValue: R)(implicit param: AccumulableParam[R, T]): Accumulable[R, T]

    Create an org.apache.spark.Accumulable shared variable, to which tasks can add values with +=.

    Create an org.apache.spark.Accumulable shared variable, to which tasks can add values with +=. Only the driver can access the accumuable's value.

    R

    accumulator result type

    T

    type that can be added to the accumulator

  8. def accumulableCollection[R, T](initialValue: R)(implicit arg0: (R) ⇒ Growable[T] with TraversableOnce[T] with Serializable, arg1: ClassTag[R]): Accumulable[R, T]

    Create an accumulator from a "mutable collection" type.

    Create an accumulator from a "mutable collection" type.

    Growable and TraversableOnce are the standard APIs that guarantee += and ++=, implemented by standard mutable collections. So you can use this with mutable Map, Set, etc.

  9. def accumulator[T](initialValue: T, name: String)(implicit param: AccumulatorParam[T]): Accumulator[T]

    Create an org.apache.spark.Accumulator variable of a given type, with a name for display in the Spark UI.

    Create an org.apache.spark.Accumulator variable of a given type, with a name for display in the Spark UI. Tasks can "add" values to the accumulator using the += method. Only the driver can access the accumulator's value.

  10. def accumulator[T](initialValue: T)(implicit param: AccumulatorParam[T]): Accumulator[T]

    Create an org.apache.spark.Accumulator variable of a given type, which tasks can "add" values to using the += method.

    Create an org.apache.spark.Accumulator variable of a given type, which tasks can "add" values to using the += method. Only the driver can access the accumulator's value.

  11. def addFile(path: String): Unit

    Add a file to be downloaded with this Spark job on every node.

    Add a file to be downloaded with this Spark job on every node. The path passed can be either a local file, a file in HDFS (or other Hadoop-supported filesystems), or an HTTP, HTTPS or FTP URI. To access the file in Spark jobs, use SparkFiles.get(fileName) to find its download location.

  12. def addJar(path: String): Unit

    Adds a JAR dependency for all tasks to be executed on this SparkContext in the future.

    Adds a JAR dependency for all tasks to be executed on this SparkContext in the future. The path passed can be either a local file, a file in HDFS (or other Hadoop-supported filesystems), an HTTP, HTTPS or FTP URI, or local:/path for a file on every worker node.

  13. def addSparkListener(listener: SparkListener): Unit

    :: DeveloperApi :: Register a listener to receive up-calls from events that happen during execution.

    :: DeveloperApi :: Register a listener to receive up-calls from events that happen during execution.

    Annotations
    @DeveloperApi()
  14. val appName: String

  15. val applicationId: String

  16. final def asInstanceOf[T0]: T0

    Definition Classes
    Any
  17. def binaryFiles(path: String, minPartitions: Int = defaultMinPartitions): RDD[(String, PortableDataStream)]

    :: Experimental ::

    :: Experimental ::

    Get an RDD for a Hadoop-readable dataset as PortableDataStream for each file (useful for binary data)

    For example, if you have the following files:

    hdfs://a-hdfs-path/part-00000
    hdfs://a-hdfs-path/part-00001
    ...
    hdfs://a-hdfs-path/part-nnnnn

    Do val rdd = sparkContext.dataStreamFiles("hdfs://a-hdfs-path"),

    then rdd contains

    (a-hdfs-path/part-00000, its content)
    (a-hdfs-path/part-00001, its content)
    ...
    (a-hdfs-path/part-nnnnn, its content)
    minPartitions

    A suggestion value of the minimal splitting number for input data.

    Annotations
    @Experimental()
    Note

    Small files are preferred; very large files may cause bad performance.

  18. def binaryRecords(path: String, recordLength: Int, conf: Configuration = hadoopConfiguration): RDD[Array[Byte]]

    :: Experimental ::

    :: Experimental ::

    Load data from a flat binary file, assuming the length of each record is constant.

    path

    Directory to the input data files

    recordLength

    The length at which to split the records

    returns

    An RDD of data with values, represented as byte arrays

    Annotations
    @Experimental()
  19. def broadcast[T](value: T)(implicit arg0: ClassTag[T]): Broadcast[T]

    Broadcast a read-only variable to the cluster, returning a org.apache.spark.broadcast.Broadcast object for reading it in distributed functions.

    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.

  20. def cancelAllJobs(): Unit

    Cancel all jobs that have been scheduled or are running.

  21. def cancelJobGroup(groupId: String): Unit

    Cancel active jobs for the specified group.

    Cancel active jobs for the specified group. See org.apache.spark.SparkContext.setJobGroup for more information.

  22. def checkpointFile[T](path: String)(implicit arg0: ClassTag[T]): RDD[T]

    Attributes
    protected[org.apache.spark]
  23. def clearCallSite(): Unit

    Clear the thread-local property for overriding the call sites of actions and RDDs.

  24. def clearJobGroup(): Unit

    Clear the current thread's job group ID and its description.

  25. def clone(): AnyRef

    Attributes
    protected[java.lang]
    Definition Classes
    AnyRef
    Annotations
    @throws( ... )
  26. def defaultMinPartitions: Int

    Default min number of partitions for Hadoop RDDs when not given by user

  27. def defaultParallelism: Int

    Default level of parallelism to use when not given by user (e.

    Default level of parallelism to use when not given by user (e.g. parallelize and makeRDD).

  28. def emptyRDD[T](implicit arg0: ClassTag[T]): EmptyRDD[T]

    Get an RDD that has no partitions or elements.

  29. final def eq(arg0: AnyRef): Boolean

    Definition Classes
    AnyRef
  30. def equals(arg0: Any): Boolean

    Definition Classes
    AnyRef → Any
  31. val files: Seq[String]

  32. def finalize(): Unit

    Attributes
    protected[java.lang]
    Definition Classes
    AnyRef
    Annotations
    @throws( classOf[java.lang.Throwable] )
  33. def getAllPools: Seq[Schedulable]

    :: DeveloperApi :: Return pools for fair scheduler

    :: DeveloperApi :: Return pools for fair scheduler

    Annotations
    @DeveloperApi()
  34. def getCheckpointDir: Option[String]

  35. final def getClass(): Class[_]

    Definition Classes
    AnyRef → Any
  36. def getConf: SparkConf

    Return a copy of this SparkContext's configuration.

    Return a copy of this SparkContext's configuration. The configuration cannot be changed at runtime.

  37. def getExecutorMemoryStatus: Map[String, (Long, Long)]

    Return a map from the slave to the max memory available for caching and the remaining memory available for caching.

  38. def getExecutorStorageStatus: Array[StorageStatus]

    :: DeveloperApi :: Return information about blocks stored in all of the slaves

    :: DeveloperApi :: Return information about blocks stored in all of the slaves

    Annotations
    @DeveloperApi()
  39. def getLocalProperty(key: String): String

    Get a local property set in this thread, or null if it is missing.

    Get a local property set in this thread, or null if it is missing. See org.apache.spark.SparkContext.setLocalProperty.

  40. def getPersistentRDDs: Map[Int, RDD[_]]

    Returns an immutable map of RDDs that have marked themselves as persistent via cache() call.

    Returns an immutable map of RDDs that have marked themselves as persistent via cache() call. Note that this does not necessarily mean the caching or computation was successful.

  41. def getPoolForName(pool: String): Option[Schedulable]

    :: DeveloperApi :: Return the pool associated with the given name, if one exists

    :: DeveloperApi :: Return the pool associated with the given name, if one exists

    Annotations
    @DeveloperApi()
  42. def getRDDStorageInfo: Array[RDDInfo]

    :: DeveloperApi :: Return information about what RDDs are cached, if they are in mem or on disk, how much space they take, etc.

    :: DeveloperApi :: Return information about what RDDs are cached, if they are in mem or on disk, how much space they take, etc.

    Annotations
    @DeveloperApi()
  43. def getSchedulingMode: SchedulingMode

    Return current scheduling mode

  44. val hadoopConfiguration: Configuration

    A default Hadoop Configuration for the Hadoop code (e.

    A default Hadoop Configuration for the Hadoop code (e.g. file systems) that we reuse.

  45. def hadoopFile[K, V, F <: InputFormat[K, V]](path: String)(implicit km: ClassTag[K], vm: ClassTag[V], fm: ClassTag[F]): RDD[(K, V)]

    Smarter version of hadoopFile() that uses class tags to figure out the classes of keys, values and the InputFormat so that users don't need to pass them directly.

    Smarter version of hadoopFile() that uses class tags to figure out the classes of keys, values and the InputFormat so that users don't need to pass them directly. Instead, callers can just write, for example,

    val file = sparkContext.hadoopFile[LongWritable, Text, TextInputFormat](path)

    Note: Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD will create many references to the same object. If you plan to directly cache Hadoop writable objects, you should first copy them using a map function.

  46. def hadoopFile[K, V, F <: InputFormat[K, V]](path: String, minPartitions: Int)(implicit km: ClassTag[K], vm: ClassTag[V], fm: ClassTag[F]): RDD[(K, V)]

    Smarter version of hadoopFile() that uses class tags to figure out the classes of keys, values and the InputFormat so that users don't need to pass them directly.

    Smarter version of hadoopFile() that uses class tags to figure out the classes of keys, values and the InputFormat so that users don't need to pass them directly. Instead, callers can just write, for example,

    val file = sparkContext.hadoopFile[LongWritable, Text, TextInputFormat](path, minPartitions)

    Note: Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD will create many references to the same object. If you plan to directly cache Hadoop writable objects, you should first copy them using a map function.

  47. def hadoopFile[K, V](path: String, inputFormatClass: Class[_ <: InputFormat[K, V]], keyClass: Class[K], valueClass: Class[V], minPartitions: Int = defaultMinPartitions): RDD[(K, V)]

    Get an RDD for a Hadoop file with an arbitrary InputFormat

    Get an RDD for a Hadoop file with an arbitrary InputFormat

    Note: Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD will create many references to the same object. If you plan to directly cache Hadoop writable objects, you should first copy them using a map function.

  48. def hadoopRDD[K, V](conf: JobConf, inputFormatClass: Class[_ <: InputFormat[K, V]], keyClass: Class[K], valueClass: Class[V], minPartitions: Int = defaultMinPartitions): RDD[(K, V)]

    Get an RDD for a Hadoop-readable dataset from a Hadoop JobConf given its InputFormat and other necessary info (e.

    Get an RDD for a Hadoop-readable dataset from a Hadoop JobConf given its InputFormat and other necessary info (e.g. file name for a filesystem-based dataset, table name for HyperTable), using the older MapReduce API (org.apache.hadoop.mapred).

    conf

    JobConf for setting up the dataset

    inputFormatClass

    Class of the InputFormat

    keyClass

    Class of the keys

    valueClass

    Class of the values

    minPartitions

    Minimum number of Hadoop Splits to generate.

    Note: Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD will create many references to the same object. If you plan to directly cache Hadoop writable objects, you should first copy them using a map function.

  49. def hashCode(): Int

    Definition Classes
    AnyRef → Any
  50. final def isInstanceOf[T0]: Boolean

    Definition Classes
    Any
  51. val isLocal: Boolean

  52. def isTraceEnabled(): Boolean

    Attributes
    protected
    Definition Classes
    Logging
  53. val jars: Seq[String]

  54. def killExecutor(executorId: String): Boolean

    :: DeveloperApi :: Request that cluster manager the kill the specified executor.

    :: DeveloperApi :: Request that cluster manager the kill the specified executor. This is currently only supported in Yarn mode. Return whether the request is received.

    Definition Classes
    SparkContext → ExecutorAllocationClient
    Annotations
    @DeveloperApi()
  55. def killExecutors(executorIds: Seq[String]): Boolean

    :: DeveloperApi :: Request that the cluster manager kill the specified executors.

    :: DeveloperApi :: Request that the cluster manager kill the specified executors. This is currently only supported in Yarn mode. Return whether the request is received.

    Definition Classes
    SparkContext → ExecutorAllocationClient
    Annotations
    @DeveloperApi()
  56. def log: Logger

    Attributes
    protected
    Definition Classes
    Logging
  57. def logDebug(msg: ⇒ String, throwable: Throwable): Unit

    Attributes
    protected
    Definition Classes
    Logging
  58. def logDebug(msg: ⇒ String): Unit

    Attributes
    protected
    Definition Classes
    Logging
  59. def logError(msg: ⇒ String, throwable: Throwable): Unit

    Attributes
    protected
    Definition Classes
    Logging
  60. def logError(msg: ⇒ String): Unit

    Attributes
    protected
    Definition Classes
    Logging
  61. def logInfo(msg: ⇒ String, throwable: Throwable): Unit

    Attributes
    protected
    Definition Classes
    Logging
  62. def logInfo(msg: ⇒ String): Unit

    Attributes
    protected
    Definition Classes
    Logging
  63. def logName: String

    Attributes
    protected
    Definition Classes
    Logging
  64. def logTrace(msg: ⇒ String, throwable: Throwable): Unit

    Attributes
    protected
    Definition Classes
    Logging
  65. def logTrace(msg: ⇒ String): Unit

    Attributes
    protected
    Definition Classes
    Logging
  66. def logWarning(msg: ⇒ String, throwable: Throwable): Unit

    Attributes
    protected
    Definition Classes
    Logging
  67. def logWarning(msg: ⇒ String): Unit

    Attributes
    protected
    Definition Classes
    Logging
  68. def makeRDD[T](seq: Seq[(T, Seq[String])])(implicit arg0: ClassTag[T]): RDD[T]

    Distribute a local Scala collection to form an RDD, with one or more location preferences (hostnames of Spark nodes) for each object.

    Distribute a local Scala collection to form an RDD, with one or more location preferences (hostnames of Spark nodes) for each object. Create a new partition for each collection item.

  69. def makeRDD[T](seq: Seq[T], numSlices: Int = defaultParallelism)(implicit arg0: ClassTag[T]): RDD[T]

    Distribute a local Scala collection to form an RDD.

    Distribute a local Scala collection to form an RDD.

    This method is identical to parallelize.

  70. val master: String

  71. val metricsSystem: MetricsSystem

  72. final def ne(arg0: AnyRef): Boolean

    Definition Classes
    AnyRef
  73. def newAPIHadoopFile[K, V, F <: InputFormat[K, V]](path: String, fClass: Class[F], kClass: Class[K], vClass: Class[V], conf: Configuration = hadoopConfiguration): RDD[(K, V)]

    Get an RDD for a given Hadoop file with an arbitrary new API InputFormat and extra configuration options to pass to the input format.

    Get an RDD for a given Hadoop file with an arbitrary new API InputFormat and extra configuration options to pass to the input format.

    Note: Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD will create many references to the same object. If you plan to directly cache Hadoop writable objects, you should first copy them using a map function.

  74. def newAPIHadoopFile[K, V, F <: InputFormat[K, V]](path: String)(implicit km: ClassTag[K], vm: ClassTag[V], fm: ClassTag[F]): RDD[(K, V)]

    Get an RDD for a Hadoop file with an arbitrary new API InputFormat.

  75. def newAPIHadoopRDD[K, V, F <: InputFormat[K, V]](conf: Configuration = hadoopConfiguration, fClass: Class[F], kClass: Class[K], vClass: Class[V]): RDD[(K, V)]

    Get an RDD for a given Hadoop file with an arbitrary new API InputFormat and extra configuration options to pass to the input format.

    Get an RDD for a given Hadoop file with an arbitrary new API InputFormat and extra configuration options to pass to the input format.

    Note: Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD will create many references to the same object. If you plan to directly cache Hadoop writable objects, you should first copy them using a map function.

  76. final def notify(): Unit

    Definition Classes
    AnyRef
  77. final def notifyAll(): Unit

    Definition Classes
    AnyRef
  78. def objectFile[T](path: String, minPartitions: Int = defaultMinPartitions)(implicit arg0: ClassTag[T]): RDD[T]

    Load an RDD saved as a SequenceFile containing serialized objects, with NullWritable keys and BytesWritable values that contain a serialized partition.

    Load an RDD saved as a SequenceFile containing serialized objects, with NullWritable keys and BytesWritable values that contain a serialized partition. This is still an experimental storage format and may not be supported exactly as is in future Spark releases. It will also be pretty slow if you use the default serializer (Java serialization), though the nice thing about it is that there's very little effort required to save arbitrary objects.

  79. def parallelize[T](seq: Seq[T], numSlices: Int = defaultParallelism)(implicit arg0: ClassTag[T]): RDD[T]

    Distribute a local Scala collection to form an RDD.

    Distribute a local Scala collection to form an RDD.

    Note

    Parallelize acts lazily. If seq is a mutable collection and is altered after the call to parallelize and before the first action on the RDD, the resultant RDD will reflect the modified collection. Pass a copy of the argument to avoid this.

  80. def requestExecutors(numAdditionalExecutors: Int): Boolean

    :: DeveloperApi :: Request an additional number of executors from the cluster manager.

    :: DeveloperApi :: Request an additional number of executors from the cluster manager. This is currently only supported in Yarn mode. Return whether the request is received.

    Definition Classes
    SparkContext → ExecutorAllocationClient
    Annotations
    @DeveloperApi()
  81. def runApproximateJob[T, U, R](rdd: RDD[T], func: (TaskContext, Iterator[T]) ⇒ U, evaluator: ApproximateEvaluator[U, R], timeout: Long): PartialResult[R]

    :: DeveloperApi :: Run a job that can return approximate results.

    :: DeveloperApi :: Run a job that can return approximate results.

    Annotations
    @DeveloperApi()
  82. def runJob[T, U](rdd: RDD[T], processPartition: (Iterator[T]) ⇒ U, resultHandler: (Int, U) ⇒ Unit)(implicit arg0: ClassTag[U]): Unit

    Run a job on all partitions in an RDD and pass the results to a handler function.

  83. def runJob[T, U](rdd: RDD[T], processPartition: (TaskContext, Iterator[T]) ⇒ U, resultHandler: (Int, U) ⇒ Unit)(implicit arg0: ClassTag[U]): Unit

    Run a job on all partitions in an RDD and pass the results to a handler function.

  84. def runJob[T, U](rdd: RDD[T], func: (Iterator[T]) ⇒ U)(implicit arg0: ClassTag[U]): Array[U]

    Run a job on all partitions in an RDD and return the results in an array.

  85. def runJob[T, U](rdd: RDD[T], func: (TaskContext, Iterator[T]) ⇒ U)(implicit arg0: ClassTag[U]): Array[U]

    Run a job on all partitions in an RDD and return the results in an array.

  86. def runJob[T, U](rdd: RDD[T], func: (Iterator[T]) ⇒ U, partitions: Seq[Int], allowLocal: Boolean)(implicit arg0: ClassTag[U]): Array[U]

    Run a job on a given set of partitions of an RDD, but take a function of type Iterator[T] => U instead of (TaskContext, Iterator[T]) => U.

  87. def runJob[T, U](rdd: RDD[T], func: (TaskContext, Iterator[T]) ⇒ U, partitions: Seq[Int], allowLocal: Boolean)(implicit arg0: ClassTag[U]): Array[U]

    Run a function on a given set of partitions in an RDD and return the results as an array.

    Run a function on a given set of partitions in an RDD and return the results as an array. The allowLocal flag specifies whether the scheduler can run the computation on the driver rather than shipping it out to the cluster, for short actions like first().

  88. def runJob[T, U](rdd: RDD[T], func: (TaskContext, Iterator[T]) ⇒ U, partitions: Seq[Int], allowLocal: Boolean, resultHandler: (Int, U) ⇒ Unit)(implicit arg0: ClassTag[U]): Unit

    Run a function on a given set of partitions in an RDD and pass the results to the given handler function.

    Run a function on a given set of partitions in an RDD and pass the results to the given handler function. This is the main entry point for all actions in Spark. The allowLocal flag specifies whether the scheduler can run the computation on the driver rather than shipping it out to the cluster, for short actions like first().

  89. def sequenceFile[K, V](path: String, minPartitions: Int = defaultMinPartitions)(implicit km: ClassTag[K], vm: ClassTag[V], kcf: () ⇒ WritableConverter[K], vcf: () ⇒ WritableConverter[V]): RDD[(K, V)]

    Version of sequenceFile() for types implicitly convertible to Writables through a WritableConverter.

    Version of sequenceFile() for types implicitly convertible to Writables through a WritableConverter. For example, to access a SequenceFile where the keys are Text and the values are IntWritable, you could simply write

    sparkContext.sequenceFile[String, Int](path, ...)

    WritableConverters are provided in a somewhat strange way (by an implicit function) to support both subclasses of Writable and types for which we define a converter (e.g. Int to IntWritable). The most natural thing would've been to have implicit objects for the converters, but then we couldn't have an object for every subclass of Writable (you can't have a parameterized singleton object). We use functions instead to create a new converter for the appropriate type. In addition, we pass the converter a ClassTag of its type to allow it to figure out the Writable class to use in the subclass case.

    Note: Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD will create many references to the same object. If you plan to directly cache Hadoop writable objects, you should first copy them using a map function.

  90. def sequenceFile[K, V](path: String, keyClass: Class[K], valueClass: Class[V]): RDD[(K, V)]

    Get an RDD for a Hadoop SequenceFile with given key and value types.

    Get an RDD for a Hadoop SequenceFile with given key and value types.

    Note: Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD will create many references to the same object. If you plan to directly cache Hadoop writable objects, you should first copy them using a map function.

  91. def sequenceFile[K, V](path: String, keyClass: Class[K], valueClass: Class[V], minPartitions: Int): RDD[(K, V)]

    Get an RDD for a Hadoop SequenceFile with given key and value types.

    Get an RDD for a Hadoop SequenceFile with given key and value types.

    Note: Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD will create many references to the same object. If you plan to directly cache Hadoop writable objects, you should first copy them using a map function.

  92. def setCallSite(shortCallSite: String): Unit

    Set the thread-local property for overriding the call sites of actions and RDDs.

  93. def setCheckpointDir(directory: String): Unit

    Set the directory under which RDDs are going to be checkpointed.

    Set the directory under which RDDs are going to be checkpointed. The directory must be a HDFS path if running on a cluster.

  94. def setJobGroup(groupId: String, description: String, interruptOnCancel: Boolean = false): Unit

    Assigns a group ID to all the jobs started by this thread until the group ID is set to a different value or cleared.

    Assigns a group ID to all the jobs started by this thread until the group ID is set to a different value or cleared.

    Often, a unit of execution in an application consists of multiple Spark actions or jobs. Application programmers can use this method to group all those jobs together and give a group description. Once set, the Spark web UI will associate such jobs with this group.

    The application can also use org.apache.spark.SparkContext.cancelJobGroup to cancel all running jobs in this group. For example,

    // In the main thread:
    sc.setJobGroup("some_job_to_cancel", "some job description")
    sc.parallelize(1 to 10000, 2).map { i => Thread.sleep(10); i }.count()
    
    // In a separate thread:
    sc.cancelJobGroup("some_job_to_cancel")

    If interruptOnCancel is set to true for the job group, then job cancellation will result in Thread.interrupt() being called on the job's executor threads. This is useful to help ensure that the tasks are actually stopped in a timely manner, but is off by default due to HDFS-1208, where HDFS may respond to Thread.interrupt() by marking nodes as dead.

  95. def setLocalProperty(key: String, value: String): Unit

    Set a local property that affects jobs submitted from this thread, such as the Spark fair scheduler pool.

  96. val sparkUser: String

  97. val startTime: Long

  98. val statusTracker: SparkStatusTracker

  99. def stop(): Unit

    Shut down the SparkContext.

  100. def submitJob[T, U, R](rdd: RDD[T], processPartition: (Iterator[T]) ⇒ U, partitions: Seq[Int], resultHandler: (Int, U) ⇒ Unit, resultFunc: ⇒ R): SimpleFutureAction[R]

    :: Experimental :: Submit a job for execution and return a FutureJob holding the result.

    :: Experimental :: Submit a job for execution and return a FutureJob holding the result.

    Annotations
    @Experimental()
  101. final def synchronized[T0](arg0: ⇒ T0): T0

    Definition Classes
    AnyRef
  102. val tachyonFolderName: String

  103. def textFile(path: String, minPartitions: Int = defaultMinPartitions): RDD[String]

    Read a text file from HDFS, a local file system (available on all nodes), or any Hadoop-supported file system URI, and return it as an RDD of Strings.

  104. def toString(): String

    Definition Classes
    AnyRef → Any
  105. def union[T](first: RDD[T], rest: RDD[T]*)(implicit arg0: ClassTag[T]): RDD[T]

    Build the union of a list of RDDs passed as variable-length arguments.

  106. def union[T](rdds: Seq[RDD[T]])(implicit arg0: ClassTag[T]): RDD[T]

    Build the union of a list of RDDs.

  107. def version: String

    The version of Spark on which this application is running.

  108. final def wait(): Unit

    Definition Classes
    AnyRef
    Annotations
    @throws( ... )
  109. final def wait(arg0: Long, arg1: Int): Unit

    Definition Classes
    AnyRef
    Annotations
    @throws( ... )
  110. final def wait(arg0: Long): Unit

    Definition Classes
    AnyRef
    Annotations
    @throws( ... )
  111. def wholeTextFiles(path: String, minPartitions: Int = defaultMinPartitions): RDD[(String, String)]

    Read a directory of text files from HDFS, a local file system (available on all nodes), or any Hadoop-supported file system URI.

    Read a directory of text files from HDFS, a local file system (available on all nodes), or any Hadoop-supported file system URI. Each file is read as a single record and returned in a key-value pair, where the key is the path of each file, the value is the content of each file.

    For example, if you have the following files:

    hdfs://a-hdfs-path/part-00000
    hdfs://a-hdfs-path/part-00001
    ...
    hdfs://a-hdfs-path/part-nnnnn

    Do val rdd = sparkContext.wholeTextFile("hdfs://a-hdfs-path"),

    then rdd contains

    (a-hdfs-path/part-00000, its content)
    (a-hdfs-path/part-00001, its content)
    ...
    (a-hdfs-path/part-nnnnn, its content)
    minPartitions

    A suggestion value of the minimal splitting number for input data.

    Note

    Small files are preferred, large file is also allowable, but may cause bad performance.

Deprecated Value Members

  1. def clearFiles(): Unit

    Clear the job's list of files added by addFile so that they do not get downloaded to any new nodes.

    Clear the job's list of files added by addFile so that they do not get downloaded to any new nodes.

    Annotations
    @deprecated
    Deprecated

    (Since version 1.0.0) adding files no longer creates local copies that need to be deleted

  2. def clearJars(): Unit

    Clear the job's list of JARs added by addJar so that they do not get downloaded to any new nodes.

    Clear the job's list of JARs added by addJar so that they do not get downloaded to any new nodes.

    Annotations
    @deprecated
    Deprecated

    (Since version 1.0.0) adding jars no longer creates local copies that need to be deleted

  3. def defaultMinSplits: Int

    Default min number of partitions for Hadoop RDDs when not given by user

    Default min number of partitions for Hadoop RDDs when not given by user

    Annotations
    @deprecated
    Deprecated

    (Since version 1.0.0) use defaultMinPartitions

  4. def initLocalProperties(): Unit

    Annotations
    @deprecated
    Deprecated

    (Since version 1.0.0) Properties no longer need to be explicitly initialized.

  5. def setJobDescription(value: String): Unit

    Set a human readable description of the current job.

    Set a human readable description of the current job.

    Annotations
    @deprecated
    Deprecated

    (Since version 0.8.1) use setJobGroup

Inherited from ExecutorAllocationClient

Inherited from Logging

Inherited from AnyRef

Inherited from Any

Ungrouped