Class SparkContext

Object
org.apache.spark.SparkContext
All Implemented Interfaces:
org.apache.spark.internal.Logging

public class SparkContext extends Object implements org.apache.spark.internal.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.

Note:
Only one SparkContext should be active per JVM. You must stop() the active SparkContext before creating a new one. param: config a Spark Config object describing the application configuration. Any settings in this config overrides the default configs as well as system properties.
  • Nested Class Summary

    Nested classes/interfaces inherited from interface org.apache.spark.internal.Logging

    org.apache.spark.internal.Logging.SparkShellLoggingFilter
  • Constructor Summary

    Constructors
    Constructor
    Description
    Create a SparkContext that loads settings from system properties (for instance, when launching with .
    SparkContext(String master, String appName, String sparkHome, scala.collection.Seq<String> jars, scala.collection.Map<String,String> environment)
    Alternative constructor that allows setting common Spark properties directly
    SparkContext(String master, String appName, SparkConf conf)
    Alternative constructor that allows setting common Spark properties directly
     
  • Method Summary

    Modifier and Type
    Method
    Description
    void
    :: Experimental :: Add an archive to be downloaded and unpacked with this Spark job on every node.
    void
    Add a file to be downloaded with this Spark job on every node.
    void
    addFile(String path, boolean recursive)
    Add a file to be downloaded with this Spark job on every node.
    void
    addJar(String path)
    Adds a JAR dependency for all tasks to be executed on this SparkContext in the future.
    void
    Add a tag to be assigned to all the jobs started by this thread.
    void
    :: DeveloperApi :: Register a listener to receive up-calls from events that happen during execution.
    scala.Option<String>
     
    A unique identifier for the Spark application.
     
    scala.collection.Seq<String>
     
    binaryFiles(String path, int minPartitions)
    Get an RDD for a Hadoop-readable dataset as PortableDataStream for each file (useful for binary data)
    RDD<byte[]>
    binaryRecords(String path, int recordLength, org.apache.hadoop.conf.Configuration conf)
    Load data from a flat binary file, assuming the length of each record is constant.
    <T> Broadcast<T>
    broadcast(T value, scala.reflect.ClassTag<T> evidence$9)
    Broadcast a read-only variable to the cluster, returning a Broadcast object for reading it in distributed functions.
    void
    Cancel all jobs that have been scheduled or are running.
    void
    cancelJob(int jobId)
    Cancel a given job if it's scheduled or running.
    void
    cancelJob(int jobId, String reason)
    Cancel a given job if it's scheduled or running.
    void
    Cancel active jobs for the specified group.
    void
    Cancel active jobs that have the specified tag.
    void
    cancelStage(int stageId)
    Cancel a given stage and all jobs associated with it.
    void
    cancelStage(int stageId, String reason)
    Cancel a given stage and all jobs associated with it.
    void
    Clear the thread-local property for overriding the call sites of actions and RDDs.
    void
    Clear the current thread's job group ID and its description.
    void
    Clear the current thread's job tags.
    Create and register a CollectionAccumulator, which starts with empty list and accumulates inputs by adding them into the list.
    Create and register a CollectionAccumulator, which starts with empty list and accumulates inputs by adding them into the list.
    int
    Default min number of partitions for Hadoop RDDs when not given by user Notice that we use math.min so the "defaultMinPartitions" cannot be higher than 2.
    int
    Default level of parallelism to use when not given by user (e.g. parallelize and makeRDD).
     
    Create and register a double accumulator, which starts with 0 and accumulates inputs by add.
    Create and register a double accumulator, which starts with 0 and accumulates inputs by add.
    <T> RDD<T>
    emptyRDD(scala.reflect.ClassTag<T> evidence$8)
    Get an RDD that has no partitions or elements.
    scala.collection.Seq<String>
     
    scala.collection.Seq<Schedulable>
    :: DeveloperApi :: Return pools for fair scheduler
    scala.Option<String>
     
    Return a copy of this SparkContext's configuration.
    scala.collection.Map<String,scala.Tuple2<Object,Object>>
    Return a map from the block manager to the max memory available for caching and the remaining memory available for caching.
    scala.collection.immutable.Set<String>
    Get the tags that are currently set to be assigned to all the jobs started by this thread.
    Get a local property set in this thread, or null if it is missing.
    This function may be used to get or instantiate a SparkContext and register it as a singleton object.
    This function may be used to get or instantiate a SparkContext and register it as a singleton object.
    scala.collection.Map<Object,RDD<?>>
    Returns an immutable map of RDDs that have marked themselves as persistent via cache() call.
    scala.Option<Schedulable>
    :: DeveloperApi :: Return the pool associated with the given name, if one exists
    :: DeveloperApi :: Return information about what RDDs are cached, if they are in mem or on disk, how much space they take, etc.
    scala.Enumeration.Value
    Return current scheduling mode
    org.apache.hadoop.conf.Configuration
    A default Hadoop Configuration for the Hadoop code (e.g. file systems) that we reuse.
    <K, V, F extends org.apache.hadoop.mapred.InputFormat<K, V>>
    RDD<scala.Tuple2<K,V>>
    hadoopFile(String path, int minPartitions, scala.reflect.ClassTag<K> km, scala.reflect.ClassTag<V> vm, scala.reflect.ClassTag<F> fm)
    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.
    <K, V> RDD<scala.Tuple2<K,V>>
    hadoopFile(String path, Class<? extends org.apache.hadoop.mapred.InputFormat<K,V>> inputFormatClass, Class<K> keyClass, Class<V> valueClass, int minPartitions)
    Get an RDD for a Hadoop file with an arbitrary InputFormat
    <K, V, F extends org.apache.hadoop.mapred.InputFormat<K, V>>
    RDD<scala.Tuple2<K,V>>
    hadoopFile(String path, scala.reflect.ClassTag<K> km, scala.reflect.ClassTag<V> vm, scala.reflect.ClassTag<F> fm)
    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.
    <K, V> RDD<scala.Tuple2<K,V>>
    hadoopRDD(org.apache.hadoop.mapred.JobConf conf, Class<? extends org.apache.hadoop.mapred.InputFormat<K,V>> inputFormatClass, Class<K> keyClass, Class<V> valueClass, int minPartitions)
    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).
    boolean
     
    boolean
     
    static scala.Option<String>
    jarOfClass(Class<?> cls)
    Find the JAR from which a given class was loaded, to make it easy for users to pass their JARs to SparkContext.
    static scala.Option<String>
    Find the JAR that contains the class of a particular object, to make it easy for users to pass their JARs to SparkContext.
    scala.collection.Seq<String>
     
    boolean
    killExecutor(String executorId)
    :: DeveloperApi :: Request that the cluster manager kill the specified executor.
    boolean
    killExecutors(scala.collection.Seq<String> executorIds)
    :: DeveloperApi :: Request that the cluster manager kill the specified executors.
    boolean
    killTaskAttempt(long taskId, boolean interruptThread, String reason)
    Kill and reschedule the given task attempt.
    scala.collection.Seq<String>
    :: Experimental :: Returns a list of archive paths that are added to resources.
    scala.collection.Seq<String>
    Returns a list of file paths that are added to resources.
    scala.collection.Seq<String>
    Returns a list of jar files that are added to resources.
    Create and register a long accumulator, which starts with 0 and accumulates inputs by add.
    Create and register a long accumulator, which starts with 0 and accumulates inputs by add.
    <T> RDD<T>
    makeRDD(scala.collection.Seq<scala.Tuple2<T,scala.collection.Seq<String>>> seq, scala.reflect.ClassTag<T> evidence$3)
    Distribute a local Scala collection to form an RDD, with one or more location preferences (hostnames of Spark nodes) for each object.
    <T> RDD<T>
    makeRDD(scala.collection.Seq<T> seq, int numSlices, scala.reflect.ClassTag<T> evidence$2)
    Distribute a local Scala collection to form an RDD.
     
    <K, V, F extends org.apache.hadoop.mapreduce.InputFormat<K, V>>
    RDD<scala.Tuple2<K,V>>
    newAPIHadoopFile(String path, Class<F> fClass, Class<K> kClass, Class<V> vClass, org.apache.hadoop.conf.Configuration conf)
    Get an RDD for a given Hadoop file with an arbitrary new API InputFormat and extra configuration options to pass to the input format.
    <K, V, F extends org.apache.hadoop.mapreduce.InputFormat<K, V>>
    RDD<scala.Tuple2<K,V>>
    newAPIHadoopFile(String path, scala.reflect.ClassTag<K> km, scala.reflect.ClassTag<V> vm, scala.reflect.ClassTag<F> fm)
    Smarter version of newApiHadoopFile that uses class tags to figure out the classes of keys, values and the org.apache.hadoop.mapreduce.InputFormat (new MapReduce API) so that user don't need to pass them directly.
    <K, V, F extends org.apache.hadoop.mapreduce.InputFormat<K, V>>
    RDD<scala.Tuple2<K,V>>
    newAPIHadoopRDD(org.apache.hadoop.conf.Configuration conf, Class<F> fClass, Class<K> kClass, Class<V> vClass)
    Get an RDD for a given Hadoop file with an arbitrary new API InputFormat and extra configuration options to pass to the input format.
    <T> RDD<T>
    objectFile(String path, int minPartitions, scala.reflect.ClassTag<T> evidence$4)
    Load an RDD saved as a SequenceFile containing serialized objects, with NullWritable keys and BytesWritable values that contain a serialized partition.
    static org.slf4j.Logger
     
    static void
     
    <T> RDD<T>
    parallelize(scala.collection.Seq<T> seq, int numSlices, scala.reflect.ClassTag<T> evidence$1)
    Distribute a local Scala collection to form an RDD.
    range(long start, long end, long step, int numSlices)
    Creates a new RDD[Long] containing elements from start to end(exclusive), increased by step every element.
    void
    Register the given accumulator.
    void
    register(AccumulatorV2<?,?> acc, String name)
    Register the given accumulator with given name.
    void
    Remove a tag previously added to be assigned to all the jobs started by this thread.
    void
    :: DeveloperApi :: Deregister the listener from Spark's listener bus.
    boolean
    requestExecutors(int numAdditionalExecutors)
    :: DeveloperApi :: Request an additional number of executors from the cluster manager.
    boolean
    requestTotalExecutors(int numExecutors, int localityAwareTasks, scala.collection.immutable.Map<String,Object> hostToLocalTaskCount)
    Update the cluster manager on our scheduling needs.
    scala.collection.Map<String,ResourceInformation>
     
    <T, U, R> PartialResult<R>
    runApproximateJob(RDD<T> rdd, scala.Function2<TaskContext,scala.collection.Iterator<T>,U> func, ApproximateEvaluator<U,R> evaluator, long timeout)
    :: DeveloperApi :: Run a job that can return approximate results.
    <T, U> Object
    runJob(RDD<T> rdd, scala.Function1<scala.collection.Iterator<T>,U> func, scala.collection.Seq<Object> partitions, scala.reflect.ClassTag<U> evidence$13)
    Run a function on a given set of partitions in an RDD and return the results as an array.
    <T, U> void
    runJob(RDD<T> rdd, scala.Function1<scala.collection.Iterator<T>,U> processPartition, scala.Function2<Object,U,scala.runtime.BoxedUnit> resultHandler, scala.reflect.ClassTag<U> evidence$17)
    Run a job on all partitions in an RDD and pass the results to a handler function.
    <T, U> Object
    runJob(RDD<T> rdd, scala.Function1<scala.collection.Iterator<T>,U> func, scala.reflect.ClassTag<U> evidence$15)
    Run a job on all partitions in an RDD and return the results in an array.
    <T, U> void
    runJob(RDD<T> rdd, scala.Function2<TaskContext,scala.collection.Iterator<T>,U> func, scala.collection.Seq<Object> partitions, scala.Function2<Object,U,scala.runtime.BoxedUnit> resultHandler, scala.reflect.ClassTag<U> evidence$11)
    Run a function on a given set of partitions in an RDD and pass the results to the given handler function.
    <T, U> Object
    runJob(RDD<T> rdd, scala.Function2<TaskContext,scala.collection.Iterator<T>,U> func, scala.collection.Seq<Object> partitions, scala.reflect.ClassTag<U> evidence$12)
    Run a function on a given set of partitions in an RDD and return the results as an array.
    <T, U> void
    runJob(RDD<T> rdd, scala.Function2<TaskContext,scala.collection.Iterator<T>,U> processPartition, scala.Function2<Object,U,scala.runtime.BoxedUnit> resultHandler, scala.reflect.ClassTag<U> evidence$16)
    Run a job on all partitions in an RDD and pass the results to a handler function.
    <T, U> Object
    runJob(RDD<T> rdd, scala.Function2<TaskContext,scala.collection.Iterator<T>,U> func, scala.reflect.ClassTag<U> evidence$14)
    Run a job on all partitions in an RDD and return the results in an array.
    <K, V> RDD<scala.Tuple2<K,V>>
    sequenceFile(String path, int minPartitions, scala.reflect.ClassTag<K> km, scala.reflect.ClassTag<V> vm, scala.Function0<org.apache.spark.WritableConverter<K>> kcf, scala.Function0<org.apache.spark.WritableConverter<V>> vcf)
    Version of sequenceFile() for types implicitly convertible to Writables through a WritableConverter.
    <K, V> RDD<scala.Tuple2<K,V>>
    sequenceFile(String path, Class<K> keyClass, Class<V> valueClass)
    Get an RDD for a Hadoop SequenceFile with given key and value types.
    <K, V> RDD<scala.Tuple2<K,V>>
    sequenceFile(String path, Class<K> keyClass, Class<V> valueClass, int minPartitions)
    Get an RDD for a Hadoop SequenceFile with given key and value types.
    void
    setCallSite(String shortCallSite)
    Set the thread-local property for overriding the call sites of actions and RDDs.
    void
    Set the directory under which RDDs are going to be checkpointed.
    void
    setInterruptOnCancel(boolean interruptOnCancel)
    Set the behavior of job cancellation from jobs started in this thread.
    void
    Set a human readable description of the current job.
    void
    setJobGroup(String groupId, String description, boolean interruptOnCancel)
    Assigns a group ID to all the jobs started by this thread until the group ID is set to a different value or cleared.
    void
    Set a local property that affects jobs submitted from this thread, such as the Spark fair scheduler pool.
    void
    setLogLevel(String logLevel)
    Control our logLevel.
     
    long
     
     
    void
    Shut down the SparkContext.
    void
    stop(int exitCode)
    Shut down the SparkContext with exit code that will passed to scheduler backend.
    <T, U, R> SimpleFutureAction<R>
    submitJob(RDD<T> rdd, scala.Function1<scala.collection.Iterator<T>,U> processPartition, scala.collection.Seq<Object> partitions, scala.Function2<Object,U,scala.runtime.BoxedUnit> resultHandler, scala.Function0<R> resultFunc)
    Submit a job for execution and return a FutureJob holding the result.
    textFile(String path, int minPartitions)
    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.
    scala.Option<String>
     
    <T> RDD<T>
    union(RDD<T> first, scala.collection.Seq<RDD<T>> rest, scala.reflect.ClassTag<T> evidence$7)
    Build the union of a list of RDDs passed as variable-length arguments.
    <T> RDD<T>
    union(scala.collection.Seq<RDD<T>> rdds, scala.reflect.ClassTag<T> evidence$6)
    Build the union of a list of RDDs.
    The version of Spark on which this application is running.
    RDD<scala.Tuple2<String,String>>
    wholeTextFiles(String path, int minPartitions)
    Read a directory of text files from HDFS, a local file system (available on all nodes), or any Hadoop-supported file system URI.

    Methods inherited from class java.lang.Object

    equals, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait

    Methods inherited from interface org.apache.spark.internal.Logging

    initializeForcefully, initializeLogIfNecessary, initializeLogIfNecessary, initializeLogIfNecessary$default$2, isTraceEnabled, log, logDebug, logDebug, logError, logError, logInfo, logInfo, logName, logTrace, logTrace, logWarning, logWarning, org$apache$spark$internal$Logging$$log_, org$apache$spark$internal$Logging$$log__$eq
  • Constructor Details

    • SparkContext

      public SparkContext(SparkConf config)
    • SparkContext

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

      public SparkContext(String master, String appName, SparkConf conf)
      Alternative constructor that allows setting common Spark properties directly

      Parameters:
      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 SparkConf object specifying other Spark parameters
    • SparkContext

      public SparkContext(String master, String appName, String sparkHome, scala.collection.Seq<String> jars, scala.collection.Map<String,String> environment)
      Alternative constructor that allows setting common Spark properties directly

      Parameters:
      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.
  • Method Details

    • getOrCreate

      public static SparkContext getOrCreate(SparkConf config)
      This function may be used to get or instantiate a SparkContext and register it as a singleton object. Because we can only have one active SparkContext per JVM, this is useful when applications may wish to share a SparkContext.

      Parameters:
      config - SparkConfig that will be used for initialisation of the SparkContext
      Returns:
      current SparkContext (or a new one if it wasn't created before the function call)
    • getOrCreate

      public static SparkContext getOrCreate()
      This function may be used to get or instantiate a SparkContext and register it as a singleton object. Because we can only have one active SparkContext per JVM, this is useful when applications may wish to share a SparkContext.

      This method allows not passing a SparkConf (useful if just retrieving).

      Returns:
      current SparkContext (or a new one if wasn't created before the function call)
    • jarOfClass

      public static scala.Option<String> jarOfClass(Class<?> cls)
      Find the JAR from which a given class was loaded, to make it easy for users to pass their JARs to SparkContext.

      Parameters:
      cls - class that should be inside of the jar
      Returns:
      jar that contains the Class, None if not found
    • jarOfObject

      public static scala.Option<String> jarOfObject(Object obj)
      Find the JAR that contains the class of a particular object, to make it easy for users to pass their JARs to SparkContext. In most cases you can call jarOfObject(this) in your driver program.

      Parameters:
      obj - reference to an instance which class should be inside of the jar
      Returns:
      jar that contains the class of the instance, None if not found
    • org$apache$spark$internal$Logging$$log_

      public static org.slf4j.Logger org$apache$spark$internal$Logging$$log_()
    • org$apache$spark$internal$Logging$$log__$eq

      public static void org$apache$spark$internal$Logging$$log__$eq(org.slf4j.Logger x$1)
    • startTime

      public long startTime()
    • getConf

      public SparkConf getConf()
      Return a copy of this SparkContext's configuration. The configuration ''cannot'' be changed at runtime.
      Returns:
      (undocumented)
    • resources

      public scala.collection.Map<String,ResourceInformation> resources()
    • jars

      public scala.collection.Seq<String> jars()
    • files

      public scala.collection.Seq<String> files()
    • archives

      public scala.collection.Seq<String> archives()
    • master

      public String master()
    • deployMode

      public String deployMode()
    • appName

      public String appName()
    • isLocal

      public boolean isLocal()
    • isStopped

      public boolean isStopped()
      Returns:
      true if context is stopped or in the midst of stopping.
    • statusTracker

      public SparkStatusTracker statusTracker()
    • uiWebUrl

      public scala.Option<String> uiWebUrl()
    • hadoopConfiguration

      public org.apache.hadoop.conf.Configuration hadoopConfiguration()
      A default Hadoop Configuration for the Hadoop code (e.g. file systems) that we reuse.

      Returns:
      (undocumented)
      Note:
      As it will be reused in all Hadoop RDDs, it's better not to modify it unless you plan to set some global configurations for all Hadoop RDDs.
    • sparkUser

      public String sparkUser()
    • applicationId

      public String applicationId()
      A unique identifier for the Spark application. Its format depends on the scheduler implementation. (i.e. in case of local spark app something like 'local-1433865536131' in case of YARN something like 'application_1433865536131_34483' in case of MESOS something like 'driver-20170926223339-0001' )
      Returns:
      (undocumented)
    • applicationAttemptId

      public scala.Option<String> applicationAttemptId()
    • setLogLevel

      public void setLogLevel(String logLevel)
      Control our logLevel. This overrides any user-defined log settings.
      Parameters:
      logLevel - The desired log level as a string. Valid log levels include: ALL, DEBUG, ERROR, FATAL, INFO, OFF, TRACE, WARN
    • setLocalProperty

      public void setLocalProperty(String key, String value)
      Set a local property that affects jobs submitted from this thread, such as the Spark fair scheduler pool. User-defined properties may also be set here. These properties are propagated through to worker tasks and can be accessed there via TaskContext.getLocalProperty(java.lang.String).

      These properties are inherited by child threads spawned from this thread. This may have unexpected consequences when working with thread pools. The standard java implementation of thread pools have worker threads spawn other worker threads. As a result, local properties may propagate unpredictably.

      Parameters:
      key - (undocumented)
      value - (undocumented)
    • getLocalProperty

      public String getLocalProperty(String key)
      Get a local property set in this thread, or null if it is missing. See org.apache.spark.SparkContext.setLocalProperty.
      Parameters:
      key - (undocumented)
      Returns:
      (undocumented)
    • setJobDescription

      public void setJobDescription(String value)
      Set a human readable description of the current job.
    • setJobGroup

      public void setJobGroup(String groupId, String description, boolean interruptOnCancel)
      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")
       

      Parameters:
      interruptOnCancel - If true, 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.
      groupId - (undocumented)
      description - (undocumented)
    • clearJobGroup

      public void clearJobGroup()
      Clear the current thread's job group ID and its description.
    • setInterruptOnCancel

      public void setInterruptOnCancel(boolean interruptOnCancel)
      Set the behavior of job cancellation from jobs started in this thread.

      Parameters:
      interruptOnCancel - If true, 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.

      Since:
      3.5.0
    • addJobTag

      public void addJobTag(String tag)
      Add a tag to be assigned to all the jobs started by this thread.

      Parameters:
      tag - The tag to be added. Cannot contain ',' (comma) character.

      Since:
      3.5.0
    • removeJobTag

      public void removeJobTag(String tag)
      Remove a tag previously added to be assigned to all the jobs started by this thread. Noop if such a tag was not added earlier.

      Parameters:
      tag - The tag to be removed. Cannot contain ',' (comma) character.

      Since:
      3.5.0
    • getJobTags

      public scala.collection.immutable.Set<String> getJobTags()
      Get the tags that are currently set to be assigned to all the jobs started by this thread.

      Returns:
      (undocumented)
      Since:
      3.5.0
    • clearJobTags

      public void clearJobTags()
      Clear the current thread's job tags.

      Since:
      3.5.0
    • parallelize

      public <T> RDD<T> parallelize(scala.collection.Seq<T> seq, int numSlices, scala.reflect.ClassTag<T> evidence$1)
      Distribute a local Scala collection to form an RDD.

      Parameters:
      seq - Scala collection to distribute
      numSlices - number of partitions to divide the collection into
      evidence$1 - (undocumented)
      Returns:
      RDD representing distributed collection
      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., avoid using parallelize(Seq()) to create an empty RDD. Consider emptyRDD for an RDD with no partitions, or parallelize(Seq[T]()) for an RDD of T with empty partitions.
    • range

      public RDD<Object> range(long start, long end, long step, int numSlices)
      Creates a new RDD[Long] containing elements from start to end(exclusive), increased by step every element.

      Parameters:
      start - the start value.
      end - the end value.
      step - the incremental step
      numSlices - number of partitions to divide the collection into
      Returns:
      RDD representing distributed range
      Note:
      if we need to cache this RDD, we should make sure each partition does not exceed limit.

    • makeRDD

      public <T> RDD<T> makeRDD(scala.collection.Seq<T> seq, int numSlices, scala.reflect.ClassTag<T> evidence$2)
      Distribute a local Scala collection to form an RDD.

      This method is identical to parallelize.

      Parameters:
      seq - Scala collection to distribute
      numSlices - number of partitions to divide the collection into
      evidence$2 - (undocumented)
      Returns:
      RDD representing distributed collection
    • makeRDD

      public <T> RDD<T> makeRDD(scala.collection.Seq<scala.Tuple2<T,scala.collection.Seq<String>>> seq, scala.reflect.ClassTag<T> evidence$3)
      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.
      Parameters:
      seq - list of tuples of data and location preferences (hostnames of Spark nodes)
      evidence$3 - (undocumented)
      Returns:
      RDD representing data partitioned according to location preferences
    • textFile

      public RDD<String> textFile(String path, int minPartitions)
      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. The text files must be encoded as UTF-8.

      Parameters:
      path - path to the text file on a supported file system
      minPartitions - suggested minimum number of partitions for the resulting RDD
      Returns:
      RDD of lines of the text file
    • wholeTextFiles

      public RDD<scala.Tuple2<String,String>> wholeTextFiles(String path, int minPartitions)
      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. The text files must be encoded as UTF-8.

      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)
       

      Parameters:
      path - Directory to the input data files, the path can be comma separated paths as the list of inputs.
      minPartitions - A suggestion value of the minimal splitting number for input data.
      Returns:
      RDD representing tuples of file path and the corresponding file content
      Note:
      Small files are preferred, large file is also allowable, but may cause bad performance., On some filesystems, .../path/&#42; can be a more efficient way to read all files in a directory rather than .../path/ or .../path, Partitioning is determined by data locality. This may result in too few partitions by default.

    • binaryFiles

      public RDD<scala.Tuple2<String,PortableDataStream>> binaryFiles(String path, int minPartitions)
      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.binaryFiles("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)
       

      Parameters:
      path - Directory to the input data files, the path can be comma separated paths as the list of inputs.
      minPartitions - A suggestion value of the minimal splitting number for input data.
      Returns:
      RDD representing tuples of file path and corresponding file content
      Note:
      Small files are preferred; very large files may cause bad performance., On some filesystems, .../path/&#42; can be a more efficient way to read all files in a directory rather than .../path/ or .../path, Partitioning is determined by data locality. This may result in too few partitions by default.

    • binaryRecords

      public RDD<byte[]> binaryRecords(String path, int recordLength, org.apache.hadoop.conf.Configuration conf)
      Load data from a flat binary file, assuming the length of each record is constant.

      Parameters:
      path - Directory to the input data files, the path can be comma separated paths as the list of inputs.
      recordLength - The length at which to split the records
      conf - Configuration for setting up the dataset.

      Returns:
      An RDD of data with values, represented as byte arrays
      Note:
      We ensure that the byte array for each record in the resulting RDD has the provided record length.

    • hadoopRDD

      public <K, V> RDD<scala.Tuple2<K,V>> hadoopRDD(org.apache.hadoop.mapred.JobConf conf, Class<? extends org.apache.hadoop.mapred.InputFormat<K,V>> inputFormatClass, Class<K> keyClass, Class<V> valueClass, int minPartitions)
      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).

      Parameters:
      conf - JobConf for setting up the dataset. Note: This will be put into a Broadcast. Therefore if you plan to reuse this conf to create multiple RDDs, you need to make sure you won't modify the conf. A safe approach is always creating a new conf for a new RDD.
      inputFormatClass - storage format of the data to be read
      keyClass - Class of the key associated with the inputFormatClass parameter
      valueClass - Class of the value associated with the inputFormatClass parameter
      minPartitions - Minimum number of Hadoop Splits to generate.
      Returns:
      RDD of tuples of key and corresponding value

      Note:
      Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD or directly passing it to an aggregation or shuffle operation will create many references to the same object. If you plan to directly cache, sort, or aggregate Hadoop writable objects, you should first copy them using a map function.
    • hadoopFile

      public <K, V> RDD<scala.Tuple2<K,V>> hadoopFile(String path, Class<? extends org.apache.hadoop.mapred.InputFormat<K,V>> inputFormatClass, Class<K> keyClass, Class<V> valueClass, int minPartitions)
      Get an RDD for a Hadoop file with an arbitrary InputFormat

      Parameters:
      path - directory to the input data files, the path can be comma separated paths as a list of inputs
      inputFormatClass - storage format of the data to be read
      keyClass - Class of the key associated with the inputFormatClass parameter
      valueClass - Class of the value associated with the inputFormatClass parameter
      minPartitions - suggested minimum number of partitions for the resulting RDD
      Returns:
      RDD of tuples of key and corresponding value
      Note:
      Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD or directly passing it to an aggregation or shuffle operation will create many references to the same object. If you plan to directly cache, sort, or aggregate Hadoop writable objects, you should first copy them using a map function.
    • hadoopFile

      public <K, V, F extends org.apache.hadoop.mapred.InputFormat<K, V>> RDD<scala.Tuple2<K,V>> hadoopFile(String path, int minPartitions, scala.reflect.ClassTag<K> km, scala.reflect.ClassTag<V> vm, scala.reflect.ClassTag<F> fm)
      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)
       

      Parameters:
      path - directory to the input data files, the path can be comma separated paths as a list of inputs
      minPartitions - suggested minimum number of partitions for the resulting RDD
      km - (undocumented)
      vm - (undocumented)
      fm - (undocumented)
      Returns:
      RDD of tuples of key and corresponding value
      Note:
      Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD or directly passing it to an aggregation or shuffle operation will create many references to the same object. If you plan to directly cache, sort, or aggregate Hadoop writable objects, you should first copy them using a map function.
    • hadoopFile

      public <K, V, F extends org.apache.hadoop.mapred.InputFormat<K, V>> RDD<scala.Tuple2<K,V>> hadoopFile(String path, scala.reflect.ClassTag<K> km, scala.reflect.ClassTag<V> vm, scala.reflect.ClassTag<F> fm)
      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)
       

      Parameters:
      path - directory to the input data files, the path can be comma separated paths as a list of inputs
      km - (undocumented)
      vm - (undocumented)
      fm - (undocumented)
      Returns:
      RDD of tuples of key and corresponding value
      Note:
      Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD or directly passing it to an aggregation or shuffle operation will create many references to the same object. If you plan to directly cache, sort, or aggregate Hadoop writable objects, you should first copy them using a map function.
    • newAPIHadoopFile

      public <K, V, F extends org.apache.hadoop.mapreduce.InputFormat<K, V>> RDD<scala.Tuple2<K,V>> newAPIHadoopFile(String path, scala.reflect.ClassTag<K> km, scala.reflect.ClassTag<V> vm, scala.reflect.ClassTag<F> fm)
      Smarter version of newApiHadoopFile that uses class tags to figure out the classes of keys, values and the org.apache.hadoop.mapreduce.InputFormat (new MapReduce API) so that user don't need to pass them directly. Instead, callers can just write, for example: val file = sparkContext.hadoopFile[LongWritable, Text, TextInputFormat](path)

      Parameters:
      path - directory to the input data files, the path can be comma separated paths as a list of inputs
      km - (undocumented)
      vm - (undocumented)
      fm - (undocumented)
      Returns:
      RDD of tuples of key and corresponding value
      Note:
      Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD or directly passing it to an aggregation or shuffle operation will create many references to the same object. If you plan to directly cache, sort, or aggregate Hadoop writable objects, you should first copy them using a map function.
    • newAPIHadoopFile

      public <K, V, F extends org.apache.hadoop.mapreduce.InputFormat<K, V>> RDD<scala.Tuple2<K,V>> newAPIHadoopFile(String path, Class<F> fClass, Class<K> kClass, Class<V> vClass, org.apache.hadoop.conf.Configuration conf)
      Get an RDD for a given Hadoop file with an arbitrary new API InputFormat and extra configuration options to pass to the input format.

      Parameters:
      path - directory to the input data files, the path can be comma separated paths as a list of inputs
      fClass - storage format of the data to be read
      kClass - Class of the key associated with the fClass parameter
      vClass - Class of the value associated with the fClass parameter
      conf - Hadoop configuration
      Returns:
      RDD of tuples of key and corresponding value
      Note:
      Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD or directly passing it to an aggregation or shuffle operation will create many references to the same object. If you plan to directly cache, sort, or aggregate Hadoop writable objects, you should first copy them using a map function.
    • newAPIHadoopRDD

      public <K, V, F extends org.apache.hadoop.mapreduce.InputFormat<K, V>> RDD<scala.Tuple2<K,V>> newAPIHadoopRDD(org.apache.hadoop.conf.Configuration conf, Class<F> fClass, Class<K> kClass, Class<V> vClass)
      Get an RDD for a given Hadoop file with an arbitrary new API InputFormat and extra configuration options to pass to the input format.

      Parameters:
      conf - Configuration for setting up the dataset. Note: This will be put into a Broadcast. Therefore if you plan to reuse this conf to create multiple RDDs, you need to make sure you won't modify the conf. A safe approach is always creating a new conf for a new RDD.
      fClass - storage format of the data to be read
      kClass - Class of the key associated with the fClass parameter
      vClass - Class of the value associated with the fClass parameter

      Returns:
      (undocumented)
      Note:
      Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD or directly passing it to an aggregation or shuffle operation will create many references to the same object. If you plan to directly cache, sort, or aggregate Hadoop writable objects, you should first copy them using a map function.
    • sequenceFile

      public <K, V> RDD<scala.Tuple2<K,V>> sequenceFile(String path, Class<K> keyClass, Class<V> valueClass, int minPartitions)
      Get an RDD for a Hadoop SequenceFile with given key and value types.

      Parameters:
      path - directory to the input data files, the path can be comma separated paths as a list of inputs
      keyClass - Class of the key associated with SequenceFileInputFormat
      valueClass - Class of the value associated with SequenceFileInputFormat
      minPartitions - suggested minimum number of partitions for the resulting RDD
      Returns:
      RDD of tuples of key and corresponding value
      Note:
      Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD or directly passing it to an aggregation or shuffle operation will create many references to the same object. If you plan to directly cache, sort, or aggregate Hadoop writable objects, you should first copy them using a map function.
    • sequenceFile

      public <K, V> RDD<scala.Tuple2<K,V>> sequenceFile(String path, Class<K> keyClass, Class<V> valueClass)
      Get an RDD for a Hadoop SequenceFile with given key and value types.

      Parameters:
      path - directory to the input data files, the path can be comma separated paths as a list of inputs
      keyClass - Class of the key associated with SequenceFileInputFormat
      valueClass - Class of the value associated with SequenceFileInputFormat
      Returns:
      RDD of tuples of key and corresponding value
      Note:
      Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD or directly passing it to an aggregation or shuffle operation will create many references to the same object. If you plan to directly cache, sort, or aggregate Hadoop writable objects, you should first copy them using a map function.
    • sequenceFile

      public <K, V> RDD<scala.Tuple2<K,V>> sequenceFile(String path, int minPartitions, scala.reflect.ClassTag<K> km, scala.reflect.ClassTag<V> vm, scala.Function0<org.apache.spark.WritableConverter<K>> kcf, scala.Function0<org.apache.spark.WritableConverter<V>> vcf)
      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.

      Parameters:
      path - directory to the input data files, the path can be comma separated paths as a list of inputs
      minPartitions - suggested minimum number of partitions for the resulting RDD
      km - (undocumented)
      vm - (undocumented)
      kcf - (undocumented)
      vcf - (undocumented)
      Returns:
      RDD of tuples of key and corresponding value
      Note:
      Because Hadoop's RecordReader class re-uses the same Writable object for each record, directly caching the returned RDD or directly passing it to an aggregation or shuffle operation will create many references to the same object. If you plan to directly cache, sort, or aggregate Hadoop writable objects, you should first copy them using a map function.
    • objectFile

      public <T> RDD<T> objectFile(String path, int minPartitions, scala.reflect.ClassTag<T> evidence$4)
      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.

      Parameters:
      path - directory to the input data files, the path can be comma separated paths as a list of inputs
      minPartitions - suggested minimum number of partitions for the resulting RDD
      evidence$4 - (undocumented)
      Returns:
      RDD representing deserialized data from the file(s)
    • union

      public <T> RDD<T> union(scala.collection.Seq<RDD<T>> rdds, scala.reflect.ClassTag<T> evidence$6)
      Build the union of a list of RDDs.
    • union

      public <T> RDD<T> union(RDD<T> first, scala.collection.Seq<RDD<T>> rest, scala.reflect.ClassTag<T> evidence$7)
      Build the union of a list of RDDs passed as variable-length arguments.
    • emptyRDD

      public <T> RDD<T> emptyRDD(scala.reflect.ClassTag<T> evidence$8)
      Get an RDD that has no partitions or elements.
    • register

      public void register(AccumulatorV2<?,?> acc)
      Register the given accumulator.

      Parameters:
      acc - (undocumented)
      Note:
      Accumulators must be registered before use, or it will throw exception.
    • register

      public void register(AccumulatorV2<?,?> acc, String name)
      Register the given accumulator with given name.

      Parameters:
      acc - (undocumented)
      name - (undocumented)
      Note:
      Accumulators must be registered before use, or it will throw exception.
    • longAccumulator

      public LongAccumulator longAccumulator()
      Create and register a long accumulator, which starts with 0 and accumulates inputs by add.
      Returns:
      (undocumented)
    • longAccumulator

      public LongAccumulator longAccumulator(String name)
      Create and register a long accumulator, which starts with 0 and accumulates inputs by add.
      Parameters:
      name - (undocumented)
      Returns:
      (undocumented)
    • doubleAccumulator

      public DoubleAccumulator doubleAccumulator()
      Create and register a double accumulator, which starts with 0 and accumulates inputs by add.
      Returns:
      (undocumented)
    • doubleAccumulator

      public DoubleAccumulator doubleAccumulator(String name)
      Create and register a double accumulator, which starts with 0 and accumulates inputs by add.
      Parameters:
      name - (undocumented)
      Returns:
      (undocumented)
    • collectionAccumulator

      public <T> CollectionAccumulator<T> collectionAccumulator()
      Create and register a CollectionAccumulator, which starts with empty list and accumulates inputs by adding them into the list.
      Returns:
      (undocumented)
    • collectionAccumulator

      public <T> CollectionAccumulator<T> collectionAccumulator(String name)
      Create and register a CollectionAccumulator, which starts with empty list and accumulates inputs by adding them into the list.
      Parameters:
      name - (undocumented)
      Returns:
      (undocumented)
    • broadcast

      public <T> Broadcast<T> broadcast(T value, scala.reflect.ClassTag<T> evidence$9)
      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 executor only once.

      Parameters:
      value - value to broadcast to the Spark nodes
      evidence$9 - (undocumented)
      Returns:
      Broadcast object, a read-only variable cached on each machine
    • addFile

      public void addFile(String path)
      Add a file to be downloaded with this Spark job on every node.

      If a file is added during execution, it will not be available until the next TaskSet starts.

      Parameters:
      path - 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.

      Note:
      A path can be added only once. Subsequent additions of the same path are ignored.
    • listFiles

      public scala.collection.Seq<String> listFiles()
      Returns a list of file paths that are added to resources.
      Returns:
      (undocumented)
    • addArchive

      public void addArchive(String path)
      :: Experimental :: Add an archive to be downloaded and unpacked with this Spark job on every node.

      If an archive is added during execution, it will not be available until the next TaskSet starts.

      Parameters:
      path - 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(paths-to-files) to find its download/unpacked location. The given path should be one of .zip, .tar, .tar.gz, .tgz and .jar.

      Since:
      3.1.0
      Note:
      A path can be added only once. Subsequent additions of the same path are ignored.

    • listArchives

      public scala.collection.Seq<String> listArchives()
      :: Experimental :: Returns a list of archive paths that are added to resources.

      Returns:
      (undocumented)
      Since:
      3.1.0
    • addFile

      public void addFile(String path, boolean recursive)
      Add a file to be downloaded with this Spark job on every node.

      If a file is added during execution, it will not be available until the next TaskSet starts.

      Parameters:
      path - 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.
      recursive - if true, a directory can be given in path. Currently directories are only supported for Hadoop-supported filesystems.

      Note:
      A path can be added only once. Subsequent additions of the same path are ignored.
    • addSparkListener

      public void addSparkListener(SparkListenerInterface listener)
      :: DeveloperApi :: Register a listener to receive up-calls from events that happen during execution.
      Parameters:
      listener - (undocumented)
    • removeSparkListener

      public void removeSparkListener(SparkListenerInterface listener)
      :: DeveloperApi :: Deregister the listener from Spark's listener bus.
      Parameters:
      listener - (undocumented)
    • requestTotalExecutors

      public boolean requestTotalExecutors(int numExecutors, int localityAwareTasks, scala.collection.immutable.Map<String,Object> hostToLocalTaskCount)
      Update the cluster manager on our scheduling needs. Three bits of information are included to help it make decisions. This applies to the default ResourceProfile.
      Parameters:
      numExecutors - The total number of executors we'd like to have. The cluster manager shouldn't kill any running executor to reach this number, but, if all existing executors were to die, this is the number of executors we'd want to be allocated.
      localityAwareTasks - The number of tasks in all active stages that have a locality preferences. This includes running, pending, and completed tasks.
      hostToLocalTaskCount - A map of hosts to the number of tasks from all active stages that would like to like to run on that host. This includes running, pending, and completed tasks.
      Returns:
      whether the request is acknowledged by the cluster manager.
    • requestExecutors

      public boolean requestExecutors(int numAdditionalExecutors)
      :: DeveloperApi :: Request an additional number of executors from the cluster manager.
      Parameters:
      numAdditionalExecutors - (undocumented)
      Returns:
      whether the request is received.
    • killExecutors

      public boolean killExecutors(scala.collection.Seq<String> executorIds)
      :: DeveloperApi :: Request that the cluster manager kill the specified executors.

      This is not supported when dynamic allocation is turned on.

      Parameters:
      executorIds - (undocumented)
      Returns:
      whether the request is received.
      Note:
      This is an indication to the cluster manager that the application wishes to adjust its resource usage downwards. If the application wishes to replace the executors it kills through this method with new ones, it should follow up explicitly with a call to {{SparkContext#requestExecutors}}.

    • killExecutor

      public boolean killExecutor(String executorId)
      :: DeveloperApi :: Request that the cluster manager kill the specified executor.

      Parameters:
      executorId - (undocumented)
      Returns:
      whether the request is received.
      Note:
      This is an indication to the cluster manager that the application wishes to adjust its resource usage downwards. If the application wishes to replace the executor it kills through this method with a new one, it should follow up explicitly with a call to {{SparkContext#requestExecutors}}.

    • version

      public String version()
      The version of Spark on which this application is running.
    • getExecutorMemoryStatus

      public scala.collection.Map<String,scala.Tuple2<Object,Object>> getExecutorMemoryStatus()
      Return a map from the block manager to the max memory available for caching and the remaining memory available for caching.
      Returns:
      (undocumented)
    • getRDDStorageInfo

      public RDDInfo[] getRDDStorageInfo()
      :: DeveloperApi :: Return information about what RDDs are cached, if they are in mem or on disk, how much space they take, etc.
      Returns:
      (undocumented)
    • getPersistentRDDs

      public scala.collection.Map<Object,RDD<?>> getPersistentRDDs()
      Returns an immutable map of RDDs that have marked themselves as persistent via cache() call.

      Returns:
      (undocumented)
      Note:
      This does not necessarily mean the caching or computation was successful.
    • getAllPools

      public scala.collection.Seq<Schedulable> getAllPools()
      :: DeveloperApi :: Return pools for fair scheduler
      Returns:
      (undocumented)
    • getPoolForName

      public scala.Option<Schedulable> getPoolForName(String pool)
      :: DeveloperApi :: Return the pool associated with the given name, if one exists
      Parameters:
      pool - (undocumented)
      Returns:
      (undocumented)
    • getSchedulingMode

      public scala.Enumeration.Value getSchedulingMode()
      Return current scheduling mode
      Returns:
      (undocumented)
    • addJar

      public void addJar(String path)
      Adds a JAR dependency for all tasks to be executed on this SparkContext in the future.

      If a jar is added during execution, it will not be available until the next TaskSet starts.

      Parameters:
      path - 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.

      Note:
      A path can be added only once. Subsequent additions of the same path are ignored.
    • listJars

      public scala.collection.Seq<String> listJars()
      Returns a list of jar files that are added to resources.
      Returns:
      (undocumented)
    • stop

      public void stop()
      Shut down the SparkContext.
    • stop

      public void stop(int exitCode)
      Shut down the SparkContext with exit code that will passed to scheduler backend. In client mode, client side may call SparkContext.stop() to clean up but exit with code not equal to 0. This behavior cause resource scheduler such as ApplicationMaster exit with success status but client side exited with failed status. Spark can call this method to stop SparkContext and pass client side correct exit code to scheduler backend. Then scheduler backend should send the exit code to corresponding resource scheduler to keep consistent.

      Parameters:
      exitCode - Specified exit code that will passed to scheduler backend in client mode.
    • setCallSite

      public void setCallSite(String shortCallSite)
      Set the thread-local property for overriding the call sites of actions and RDDs.
      Parameters:
      shortCallSite - (undocumented)
    • clearCallSite

      public void clearCallSite()
      Clear the thread-local property for overriding the call sites of actions and RDDs.
    • runJob

      public <T, U> void runJob(RDD<T> rdd, scala.Function2<TaskContext,scala.collection.Iterator<T>,U> func, scala.collection.Seq<Object> partitions, scala.Function2<Object,U,scala.runtime.BoxedUnit> resultHandler, scala.reflect.ClassTag<U> evidence$11)
      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.

      Parameters:
      rdd - target RDD to run tasks on
      func - a function to run on each partition of the RDD
      partitions - set of partitions to run on; some jobs may not want to compute on all partitions of the target RDD, e.g. for operations like first()
      resultHandler - callback to pass each result to
      evidence$11 - (undocumented)
    • runJob

      public <T, U> Object runJob(RDD<T> rdd, scala.Function2<TaskContext,scala.collection.Iterator<T>,U> func, scala.collection.Seq<Object> partitions, scala.reflect.ClassTag<U> evidence$12)
      Run a function on a given set of partitions in an RDD and return the results as an array. The function that is run against each partition additionally takes TaskContext argument.

      Parameters:
      rdd - target RDD to run tasks on
      func - a function to run on each partition of the RDD
      partitions - set of partitions to run on; some jobs may not want to compute on all partitions of the target RDD, e.g. for operations like first()
      evidence$12 - (undocumented)
      Returns:
      in-memory collection with a result of the job (each collection element will contain a result from one partition)
    • runJob

      public <T, U> Object runJob(RDD<T> rdd, scala.Function1<scala.collection.Iterator<T>,U> func, scala.collection.Seq<Object> partitions, scala.reflect.ClassTag<U> evidence$13)
      Run a function on a given set of partitions in an RDD and return the results as an array.

      Parameters:
      rdd - target RDD to run tasks on
      func - a function to run on each partition of the RDD
      partitions - set of partitions to run on; some jobs may not want to compute on all partitions of the target RDD, e.g. for operations like first()
      evidence$13 - (undocumented)
      Returns:
      in-memory collection with a result of the job (each collection element will contain a result from one partition)
    • runJob

      public <T, U> Object runJob(RDD<T> rdd, scala.Function2<TaskContext,scala.collection.Iterator<T>,U> func, scala.reflect.ClassTag<U> evidence$14)
      Run a job on all partitions in an RDD and return the results in an array. The function that is run against each partition additionally takes TaskContext argument.

      Parameters:
      rdd - target RDD to run tasks on
      func - a function to run on each partition of the RDD
      evidence$14 - (undocumented)
      Returns:
      in-memory collection with a result of the job (each collection element will contain a result from one partition)
    • runJob

      public <T, U> Object runJob(RDD<T> rdd, scala.Function1<scala.collection.Iterator<T>,U> func, scala.reflect.ClassTag<U> evidence$15)
      Run a job on all partitions in an RDD and return the results in an array.

      Parameters:
      rdd - target RDD to run tasks on
      func - a function to run on each partition of the RDD
      evidence$15 - (undocumented)
      Returns:
      in-memory collection with a result of the job (each collection element will contain a result from one partition)
    • runJob

      public <T, U> void runJob(RDD<T> rdd, scala.Function2<TaskContext,scala.collection.Iterator<T>,U> processPartition, scala.Function2<Object,U,scala.runtime.BoxedUnit> resultHandler, scala.reflect.ClassTag<U> evidence$16)
      Run a job on all partitions in an RDD and pass the results to a handler function. The function that is run against each partition additionally takes TaskContext argument.

      Parameters:
      rdd - target RDD to run tasks on
      processPartition - a function to run on each partition of the RDD
      resultHandler - callback to pass each result to
      evidence$16 - (undocumented)
    • runJob

      public <T, U> void runJob(RDD<T> rdd, scala.Function1<scala.collection.Iterator<T>,U> processPartition, scala.Function2<Object,U,scala.runtime.BoxedUnit> resultHandler, scala.reflect.ClassTag<U> evidence$17)
      Run a job on all partitions in an RDD and pass the results to a handler function.

      Parameters:
      rdd - target RDD to run tasks on
      processPartition - a function to run on each partition of the RDD
      resultHandler - callback to pass each result to
      evidence$17 - (undocumented)
    • runApproximateJob

      public <T, U, R> PartialResult<R> runApproximateJob(RDD<T> rdd, scala.Function2<TaskContext,scala.collection.Iterator<T>,U> func, ApproximateEvaluator<U,R> evaluator, long timeout)
      :: DeveloperApi :: Run a job that can return approximate results.

      Parameters:
      rdd - target RDD to run tasks on
      func - a function to run on each partition of the RDD
      evaluator - ApproximateEvaluator to receive the partial results
      timeout - maximum time to wait for the job, in milliseconds
      Returns:
      partial result (how partial depends on whether the job was finished before or after timeout)
    • submitJob

      public <T, U, R> SimpleFutureAction<R> submitJob(RDD<T> rdd, scala.Function1<scala.collection.Iterator<T>,U> processPartition, scala.collection.Seq<Object> partitions, scala.Function2<Object,U,scala.runtime.BoxedUnit> resultHandler, scala.Function0<R> resultFunc)
      Submit a job for execution and return a FutureJob holding the result.

      Parameters:
      rdd - target RDD to run tasks on
      processPartition - a function to run on each partition of the RDD
      partitions - set of partitions to run on; some jobs may not want to compute on all partitions of the target RDD, e.g. for operations like first()
      resultHandler - callback to pass each result to
      resultFunc - function to be executed when the result is ready
      Returns:
      (undocumented)
    • cancelJobGroup

      public void cancelJobGroup(String groupId)
      Cancel active jobs for the specified group. See org.apache.spark.SparkContext.setJobGroup for more information.
      Parameters:
      groupId - (undocumented)
    • cancelJobsWithTag

      public void cancelJobsWithTag(String tag)
      Cancel active jobs that have the specified tag. See org.apache.spark.SparkContext.addJobTag.

      Parameters:
      tag - The tag to be cancelled. Cannot contain ',' (comma) character.

      Since:
      3.5.0
    • cancelAllJobs

      public void cancelAllJobs()
      Cancel all jobs that have been scheduled or are running.
    • cancelJob

      public void cancelJob(int jobId, String reason)
      Cancel a given job if it's scheduled or running.

      Parameters:
      jobId - the job ID to cancel
      reason - optional reason for cancellation
      Note:
      Throws InterruptedException if the cancel message cannot be sent
    • cancelJob

      public void cancelJob(int jobId)
      Cancel a given job if it's scheduled or running.

      Parameters:
      jobId - the job ID to cancel
      Note:
      Throws InterruptedException if the cancel message cannot be sent
    • cancelStage

      public void cancelStage(int stageId, String reason)
      Cancel a given stage and all jobs associated with it.

      Parameters:
      stageId - the stage ID to cancel
      reason - reason for cancellation
      Note:
      Throws InterruptedException if the cancel message cannot be sent
    • cancelStage

      public void cancelStage(int stageId)
      Cancel a given stage and all jobs associated with it.

      Parameters:
      stageId - the stage ID to cancel
      Note:
      Throws InterruptedException if the cancel message cannot be sent
    • killTaskAttempt

      public boolean killTaskAttempt(long taskId, boolean interruptThread, String reason)
      Kill and reschedule the given task attempt. Task ids can be obtained from the Spark UI or through SparkListener.onTaskStart.

      Parameters:
      taskId - the task ID to kill. This id uniquely identifies the task attempt.
      interruptThread - whether to interrupt the thread running the task.
      reason - the reason for killing the task, which should be a short string. If a task is killed multiple times with different reasons, only one reason will be reported.

      Returns:
      Whether the task was successfully killed.
    • setCheckpointDir

      public void setCheckpointDir(String directory)
      Set the directory under which RDDs are going to be checkpointed.
      Parameters:
      directory - path to the directory where checkpoint files will be stored (must be HDFS path if running in cluster)
    • getCheckpointDir

      public scala.Option<String> getCheckpointDir()
    • defaultParallelism

      public int defaultParallelism()
      Default level of parallelism to use when not given by user (e.g. parallelize and makeRDD).
    • defaultMinPartitions

      public int defaultMinPartitions()
      Default min number of partitions for Hadoop RDDs when not given by user Notice that we use math.min so the "defaultMinPartitions" cannot be higher than 2. The reasons for this are discussed in https://github.com/mesos/spark/pull/718
      Returns:
      (undocumented)