org.apache.spark.api.java

JavaSparkContext

class JavaSparkContext extends JavaSparkContextVarargsWorkaround with Closeable

A Java-friendly version of org.apache.spark.SparkContext that returns org.apache.spark.api.java.JavaRDDs and works with Java collections instead of Scala ones.

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.

Source
JavaSparkContext.scala
Linear Supertypes
Closeable, AutoCloseable, JavaSparkContextVarargsWorkaround, AnyRef, Any
Ordering
  1. Alphabetic
  2. By inheritance
Inherited
  1. JavaSparkContext
  2. Closeable
  3. AutoCloseable
  4. JavaSparkContextVarargsWorkaround
  5. AnyRef
  6. Any
  1. Hide All
  2. Show all
Learn more about member selection
Visibility
  1. Public
  2. All

Instance Constructors

  1. new JavaSparkContext(master: String, appName: String, sparkHome: String, jars: Array[String], 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

    The SPARK_HOME directory on the slave 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 JavaSparkContext(master: String, appName: String, sparkHome: String, jars: Array[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

    The SPARK_HOME directory on the slave 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.

  3. new JavaSparkContext(master: String, appName: String, sparkHome: String, jarFile: 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

    The SPARK_HOME directory on the slave nodes

    jarFile

    JAR file to send to the cluster. This can be a path on the local file system or an HDFS, HTTP, HTTPS, or FTP URL.

  4. new JavaSparkContext(master: String, appName: String, conf: SparkConf)

    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

  5. new JavaSparkContext(master: String, appName: 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

  6. new JavaSparkContext(conf: SparkConf)

    conf

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

  7. new JavaSparkContext()

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

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

  8. new JavaSparkContext(sc: SparkContext)

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, name: String, param: AccumulableParam[T, R]): Accumulable[T, R]

    Create an org.apache.spark.Accumulable shared variable of the given type, to which tasks can "add" values with add.

    Create an org.apache.spark.Accumulable shared variable of the given type, to which tasks can "add" values with add. Only the master can access the accumuable's value.

    This version supports naming the accumulator for display in Spark's web UI.

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

    Create an org.apache.spark.Accumulable shared variable of the given type, to which tasks can "add" values with add.

    Create an org.apache.spark.Accumulable shared variable of the given type, to which tasks can "add" values with add. Only the master can access the accumuable's value.

  8. def accumulator[T](initialValue: T, name: String, accumulatorParam: AccumulatorParam[T]): Accumulator[T]

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

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

    This version supports naming the accumulator for display in Spark's web UI.

  9. def accumulator[T](initialValue: T, accumulatorParam: AccumulatorParam[T]): Accumulator[T]

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

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

  10. def accumulator(initialValue: Double, name: String): Accumulator[Double]

    Create an org.apache.spark.Accumulator double variable, which tasks can "add" values to using the add method.

    Create an org.apache.spark.Accumulator double variable, which tasks can "add" values to using the add method. Only the master can access the accumulator's value.

    This version supports naming the accumulator for display in Spark's web UI.

  11. def accumulator(initialValue: Double): Accumulator[Double]

    Create an org.apache.spark.Accumulator double variable, which tasks can "add" values to using the add method.

    Create an org.apache.spark.Accumulator double variable, which tasks can "add" values to using the add method. Only the master can access the accumulator's value.

  12. def accumulator(initialValue: Int, name: String): Accumulator[Integer]

    Create an org.apache.spark.Accumulator integer variable, which tasks can "add" values to using the add method.

    Create an org.apache.spark.Accumulator integer variable, which tasks can "add" values to using the add method. Only the master can access the accumulator's value.

    This version supports naming the accumulator for display in Spark's web UI.

  13. def accumulator(initialValue: Int): Accumulator[Integer]

    Create an org.apache.spark.Accumulator integer variable, which tasks can "add" values to using the add method.

    Create an org.apache.spark.Accumulator integer variable, which tasks can "add" values to using the add method. Only the master can access the accumulator's value.

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

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

  16. def appName: String

  17. final def asInstanceOf[T0]: T0

    Definition Classes
    Any
  18. def binaryFiles(path: String): JavaPairRDD[String, PortableDataStream]

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

    Read a directory of binary files from HDFS, a local file system (available on all nodes), or any Hadoop-supported file system URI as a byte array. 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 JavaPairRDD<String, byte[]> 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)
    Note

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

  19. def binaryFiles(path: String, minPartitions: Int): JavaPairRDD[String, PortableDataStream]

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

    Read a directory of binary files from HDFS, a local file system (available on all nodes), or any Hadoop-supported file system URI as a byte array. 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 JavaPairRDD<String, byte[]> 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.

    Note

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

  20. def binaryRecords(path: String, recordLength: Int): JavaRDD[Array[Byte]]

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

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

    path

    Directory to the input data files

    returns

    An RDD of data with values, represented as byte arrays

  21. def broadcast[T](value: 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.

  22. def cancelAllJobs(): Unit

    Cancel all jobs that have been scheduled or are running.

  23. def cancelJobGroup(groupId: String): Unit

    Cancel active jobs for the specified group.

    Cancel active jobs for the specified group. See org.apache.spark.api.java.JavaSparkContext.setJobGroup for more information.

  24. def checkpointFile[T](path: String): JavaRDD[T]

    Attributes
    protected
  25. def clearCallSite(): Unit

    Pass-through to SparkContext.

    Pass-through to SparkContext.setCallSite. For API support only.

  26. def clearJobGroup(): Unit

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

  27. def clone(): AnyRef

    Attributes
    protected[java.lang]
    Definition Classes
    AnyRef
    Annotations
    @throws( ... )
  28. def close(): Unit

    Definition Classes
    JavaSparkContext → Closeable → AutoCloseable
  29. def defaultMinPartitions: Integer

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

  30. def defaultParallelism: Integer

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

  31. def doubleAccumulator(initialValue: Double, name: String): Accumulator[Double]

    Create an org.apache.spark.Accumulator double variable, which tasks can "add" values to using the add method.

    Create an org.apache.spark.Accumulator double variable, which tasks can "add" values to using the add method. Only the master can access the accumulator's value.

    This version supports naming the accumulator for display in Spark's web UI.

  32. def doubleAccumulator(initialValue: Double): Accumulator[Double]

    Create an org.apache.spark.Accumulator double variable, which tasks can "add" values to using the add method.

    Create an org.apache.spark.Accumulator double variable, which tasks can "add" values to using the add method. Only the master can access the accumulator's value.

  33. def emptyRDD[T]: JavaRDD[T]

    Get an RDD that has no partitions or elements.

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

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

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

    Attributes
    protected[java.lang]
    Definition Classes
    AnyRef
    Annotations
    @throws( classOf[java.lang.Throwable] )
  37. def getCheckpointDir: Optional[String]

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

    Definition Classes
    AnyRef → Any
  39. def getConf: SparkConf

    Return a copy of this JavaSparkContext's configuration.

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

  40. 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.api.java.JavaSparkContext.setLocalProperty.

  41. def getSparkHome(): Optional[String]

    Get Spark's home location from either a value set through the constructor, or the spark.

    Get Spark's home location from either a value set through the constructor, or the spark.home Java property, or the SPARK_HOME environment variable (in that order of preference). If neither of these is set, return None.

  42. def hadoopConfiguration(): Configuration

    Returns the Hadoop configuration used for the Hadoop code (e.

    Returns the Hadoop configuration used for the Hadoop code (e.g. file systems) we reuse.

    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.

  43. def hadoopFile[K, V, F <: InputFormat[K, V]](path: String, inputFormatClass: Class[F], keyClass: Class[K], valueClass: Class[V]): JavaPairRDD[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.

  44. def hadoopFile[K, V, F <: InputFormat[K, V]](path: String, inputFormatClass: Class[F], keyClass: Class[K], valueClass: Class[V], minPartitions: Int): JavaPairRDD[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.

  45. def hadoopRDD[K, V, F <: InputFormat[K, V]](conf: JobConf, inputFormatClass: Class[F], keyClass: Class[K], valueClass: Class[V]): JavaPairRDD[K, V]

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

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

    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

    Class of the InputFormat

    keyClass

    Class of the keys

    valueClass

    Class of the values

    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 hadoopRDD[K, V, F <: InputFormat[K, V]](conf: JobConf, inputFormatClass: Class[F], keyClass: Class[K], valueClass: Class[V], minPartitions: Int): JavaPairRDD[K, V]

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

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

    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

    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.

  47. def hashCode(): Int

    Definition Classes
    AnyRef → Any
  48. def intAccumulator(initialValue: Int, name: String): Accumulator[Integer]

    Create an org.apache.spark.Accumulator integer variable, which tasks can "add" values to using the add method.

    Create an org.apache.spark.Accumulator integer variable, which tasks can "add" values to using the add method. Only the master can access the accumulator's value.

    This version supports naming the accumulator for display in Spark's web UI.

  49. def intAccumulator(initialValue: Int): Accumulator[Integer]

    Create an org.apache.spark.Accumulator integer variable, which tasks can "add" values to using the add method.

    Create an org.apache.spark.Accumulator integer variable, which tasks can "add" values to using the add method. Only the master can access the accumulator's value.

  50. final def isInstanceOf[T0]: Boolean

    Definition Classes
    Any
  51. def isLocal: Boolean

  52. def jars: List[String]

  53. def master: String

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

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

  56. def newAPIHadoopRDD[K, V, F <: InputFormat[K, V]](conf: Configuration, fClass: Class[F], kClass: Class[K], vClass: Class[V]): JavaPairRDD[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.

    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

    Class of the InputFormat

    kClass

    Class of the keys

    vClass

    Class of the values

    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.

  57. final def notify(): Unit

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

    Definition Classes
    AnyRef
  59. def objectFile[T](path: String): JavaRDD[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.

  60. def objectFile[T](path: String, minPartitions: Int): JavaRDD[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.

  61. def parallelize[T](list: List[T]): JavaRDD[T]

    Distribute a local Scala collection to form an RDD.

  62. def parallelize[T](list: List[T], numSlices: Int): JavaRDD[T]

    Distribute a local Scala collection to form an RDD.

  63. def parallelizeDoubles(list: List[Double]): JavaDoubleRDD

    Distribute a local Scala collection to form an RDD.

  64. def parallelizeDoubles(list: List[Double], numSlices: Int): JavaDoubleRDD

    Distribute a local Scala collection to form an RDD.

  65. def parallelizePairs[K, V](list: List[(K, V)]): JavaPairRDD[K, V]

    Distribute a local Scala collection to form an RDD.

  66. def parallelizePairs[K, V](list: List[(K, V)], numSlices: Int): JavaPairRDD[K, V]

    Distribute a local Scala collection to form an RDD.

  67. val sc: SparkContext

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

    Get an RDD for a Hadoop SequenceFile.

    Get an RDD for a Hadoop SequenceFile.

    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.

  69. def sequenceFile[K, V](path: String, keyClass: Class[K], valueClass: Class[V], minPartitions: Int): JavaPairRDD[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.

  70. def setCallSite(site: String): Unit

    Pass-through to SparkContext.

    Pass-through to SparkContext.setCallSite. For API support only.

  71. def setCheckpointDir(dir: 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.

  72. def setJobGroup(groupId: String, description: String): 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.

    See also

    setJobGroup(groupId: String, description: String, interruptThread: Boolean). This method sets interruptOnCancel to false.

  73. def setJobGroup(groupId: String, description: String, interruptOnCancel: Boolean): 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.api.java.JavaSparkContext.cancelJobGroup to cancel all running jobs in this group. For example,

    // In the main thread:
    sc.setJobGroup("some_job_to_cancel", "some job description");
    rdd.map(...).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.

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

  75. def setLogLevel(logLevel: String): Unit

    Control our logLevel.

    Control our logLevel. This overrides any user-defined log settings.

    logLevel

    The desired log level as a string. Valid log levels include: ALL, DEBUG, ERROR, FATAL, INFO, OFF, TRACE, WARN

  76. def sparkUser: String

  77. def startTime: Long

  78. def statusTracker: JavaSparkStatusTracker

  79. def stop(): Unit

    Shut down the SparkContext.

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

    Definition Classes
    AnyRef
  81. def textFile(path: String, minPartitions: Int): JavaRDD[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.

  82. def textFile(path: String): JavaRDD[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.

  83. def toString(): String

    Definition Classes
    AnyRef → Any
  84. def union(first: JavaDoubleRDD, rest: List[JavaDoubleRDD]): JavaDoubleRDD

    Build the union of two or more RDDs.

    Build the union of two or more RDDs.

    Definition Classes
    JavaSparkContext → JavaSparkContextVarargsWorkaround
  85. def union[K, V](first: JavaPairRDD[K, V], rest: List[JavaPairRDD[K, V]]): JavaPairRDD[K, V]

    Build the union of two or more RDDs.

    Build the union of two or more RDDs.

    Definition Classes
    JavaSparkContext → JavaSparkContextVarargsWorkaround
  86. def union[T](first: JavaRDD[T], rest: List[JavaRDD[T]]): JavaRDD[T]

    Build the union of two or more RDDs.

    Build the union of two or more RDDs.

    Definition Classes
    JavaSparkContext → JavaSparkContextVarargsWorkaround
  87. final def union[K, V](rdds: <repeated...>[JavaPairRDD[K, V]]): JavaPairRDD[K, V]

    Definition Classes
    JavaSparkContextVarargsWorkaround
  88. def union(rdds: <repeated...>[JavaDoubleRDD]): JavaDoubleRDD

    Definition Classes
    JavaSparkContextVarargsWorkaround
  89. final def union[T](rdds: <repeated...>[JavaRDD[T]]): JavaRDD[T]

    Definition Classes
    JavaSparkContextVarargsWorkaround
  90. def version: String

    The version of Spark on which this application is running.

  91. final def wait(): Unit

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

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

    Definition Classes
    AnyRef
    Annotations
    @throws( ... )
  94. def wholeTextFiles(path: String): JavaPairRDD[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.

    See also

    wholeTextFiles(path: String, minPartitions: Int).

  95. def wholeTextFiles(path: String, minPartitions: Int): JavaPairRDD[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

    JavaPairRDD<String, String> rdd = sparkContext.wholeTextFiles("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: Integer

    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

Inherited from Closeable

Inherited from AutoCloseable

Inherited from JavaSparkContextVarargsWorkaround

Inherited from AnyRef

Inherited from Any

Ungrouped