spark

SparkContext

class SparkContext extends Logging

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.

Linear Supertypes
Logging, AnyRef, Any
Ordering
  1. Alphabetic
  2. By inheritance
Inherited
  1. Hide All
  2. Show all
  1. SparkContext
  2. Logging
  3. AnyRef
  4. Any
Visibility
  1. Public
  2. All

Instance Constructors

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

    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.

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[T, R](initialValue: T)(implicit param: AccumulableParam[T, R]): Accumulable[T, R]

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

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

    T

    accumulator type

    R

    type that can be added to the accumulator

  7. def accumulableCollection[R, T](initialValue: R)(implicit arg0: (R) ⇒ Growable[T] with TraversableOnce[T] with Serializable): 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.

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

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

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

  9. 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(path) to find its download location.

  10. 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), or an HTTP, HTTPS or FTP URI.

  11. val appName: String

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

  12. final def asInstanceOf[T0]: T0

    Definition Classes
    Any
  13. def broadcast[T](value: T): Broadcast[T]

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

    Broadcast a read-only variable to the cluster, returning a Broadcast object for reading it in distributed functions. The variable will be sent to each cluster only once.

  14. def checkpointFile[T](path: String)(implicit arg0: ClassManifest[T]): RDD[T]

    Attributes
    protected[spark]
  15. def clearFiles(): Unit

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

  16. def clearJars(): Unit

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

  17. def clone(): AnyRef

    Attributes
    protected[lang]
    Definition Classes
    AnyRef
    Annotations
    @throws()
  18. def defaultMinSplits: Int

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

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

  20. val environment: Map[String, String]

    Environment variables to set on worker nodes.

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

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

    Definition Classes
    AnyRef → Any
  23. def finalize(): Unit

    Attributes
    protected[lang]
    Definition Classes
    AnyRef
    Annotations
    @throws()
  24. final def getClass(): java.lang.Class[_]

    Definition Classes
    AnyRef → Any
  25. 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.

  26. def getExecutorStorageStatus: Array[StorageStatus]

    Return information about blocks stored in all of the slaves

  27. def getRDDStorageInfo: Array[RDDInfo]

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

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

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

    Smarter version of hadoopFile() that uses class manifests 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 manifests 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)
    
  30. def hadoopFile[K, V, F <: InputFormat[K, V]](path: String, minSplits: Int)(implicit km: ClassManifest[K], vm: ClassManifest[V], fm: ClassManifest[F]): RDD[(K, V)]

    Smarter version of hadoopFile() that uses class manifests 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 manifests 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, minSplits)
    
  31. def hadoopFile[K, V](path: String, inputFormatClass: Class[_ <: org.apache.hadoop.mapred.InputFormat[K,V]], keyClass: Class[K], valueClass: Class[V], minSplits: Int): RDD[(K, V)]

    Get an RDD for a Hadoop file with an arbitrary InputFormat

  32. def hadoopRDD[K, V](conf: JobConf, inputFormatClass: Class[_ <: org.apache.hadoop.mapred.InputFormat[K,V]], keyClass: Class[K], valueClass: Class[V], minSplits: Int): RDD[(K, V)]

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

    Get an RDD for a Hadoop-readable dataset from a Hadoop JobConf giving its InputFormat and any other necessary info (e.g. file name for a filesystem-based dataset, table name for HyperTable, etc).

  33. def hashCode(): Int

    Definition Classes
    AnyRef → Any
  34. def initLogging(): Unit

    Attributes
    protected
    Definition Classes
    Logging
  35. final def isInstanceOf[T0]: Boolean

    Definition Classes
    Any
  36. val jars: Seq[String]

    Collection of JARs to send to the cluster.

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

  37. def log: Logger

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

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

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

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

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

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

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

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

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

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

    Attributes
    protected
    Definition Classes
    Logging
  48. def makeRDD[T](seq: Seq[(T, Seq[String])])(implicit arg0: ClassManifest[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.

  49. def makeRDD[T](seq: Seq[T], numSlices: Int)(implicit arg0: ClassManifest[T]): RDD[T]

    Distribute a local Scala collection to form an RDD.

  50. val master: String

    Cluster URL to connect to (e.

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

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

    Definition Classes
    AnyRef
  52. def newAPIHadoopFile[K, V, F <: InputFormat[K, V]](path: String, fClass: Class[F], kClass: Class[K], vClass: Class[V], conf: Configuration): 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.

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

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

  54. def newAPIHadoopRDD[K, V, F <: InputFormat[K, V]](conf: Configuration, 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.

  55. final def notify(): Unit

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

    Definition Classes
    AnyRef
  57. def objectFile[T](path: String, minSplits: Int = defaultMinSplits)(implicit arg0: ClassManifest[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.

  58. def parallelize[T](seq: Seq[T], numSlices: Int)(implicit arg0: ClassManifest[T]): RDD[T]

    Distribute a local Scala collection to form an RDD.

  59. def runApproximateJob[T, U, R](rdd: RDD[T], func: (TaskContext, Iterator[T]) ⇒ U, evaluator: ApproximateEvaluator[U, R], timeout: Long): PartialResult[R]

    Run a job that can return approximate results.

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

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

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

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

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

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

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

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

  64. def runJob[T, U](rdd: RDD[T], func: (Iterator[T]) ⇒ U, partitions: Seq[Int], allowLocal: Boolean)(implicit arg0: ClassManifest[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.

  65. def runJob[T, U](rdd: RDD[T], func: (TaskContext, Iterator[T]) ⇒ U, partitions: Seq[Int], allowLocal: Boolean)(implicit arg0: ClassManifest[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().

  66. def runJob[T, U](rdd: RDD[T], func: (TaskContext, Iterator[T]) ⇒ U, partitions: Seq[Int], allowLocal: Boolean, resultHandler: (Int, U) ⇒ Unit)(implicit arg0: ClassManifest[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().

  67. def sequenceFile[K, V](path: String, minSplits: Int = defaultMinSplits)(implicit km: ClassManifest[K], vm: ClassManifest[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 ClassManifest of its type to allow it to figure out the Writable class to use in the subclass case.

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

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

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

  70. def setCheckpointDir(dir: String, useExisting: Boolean = false): 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. If the directory does not exist, it will be created. If the directory exists and useExisting is set to true, then the exisiting directory will be used. Otherwise an exception will be thrown to prevent accidental overriding of checkpoint files in the existing directory.

  71. val sparkHome: String

    Location where Spark is installed on cluster nodes.

  72. def stop(): Unit

    Shut down the SparkContext.

  73. final def synchronized[T0](arg0: ⇒ T0): T0

    Definition Classes
    AnyRef
  74. def textFile(path: String, minSplits: Int = defaultMinSplits): 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.

  75. def toString(): String

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

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

  77. def union[T](rdds: Seq[RDD[T]])(implicit arg0: ClassManifest[T]): RDD[T]

    Build the union of a list of RDDs.

  78. final def wait(): Unit

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

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

    Definition Classes
    AnyRef
    Annotations
    @throws()

Inherited from Logging

Inherited from AnyRef

Inherited from Any