class SparkConf extends Cloneable with Logging with Serializable
Configuration for a Spark application. Used to set various Spark parameters as key-value pairs.
Most of the time, you would create a SparkConf object with new SparkConf(), which will load
values from any spark.* Java system properties set in your application as well. In this case,
parameters you set directly on the SparkConf object take priority over system properties.
For unit tests, you can also call new SparkConf(false) to skip loading external settings and
get the same configuration no matter what the system properties are.
All setter methods in this class support chaining. For example, you can write
new SparkConf().setMaster("local").setAppName("My app").
- Source
 - SparkConf.scala
 - Note
 Once a SparkConf object is passed to Spark, it is cloned and can no longer be modified by the user. Spark does not support modifying the configuration at runtime.
- Alphabetic
 - By Inheritance
 
- SparkConf
 - Serializable
 - Serializable
 - Logging
 - Cloneable
 - Cloneable
 - AnyRef
 - Any
 
- Hide All
 - Show All
 
- Public
 - All
 
Instance Constructors
Value Members
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        !=(arg0: Any): Boolean
      
      
      
- Definition Classes
 - AnyRef → Any
 
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        ##(): Int
      
      
      
- Definition Classes
 - AnyRef → Any
 
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        ==(arg0: Any): Boolean
      
      
      
- Definition Classes
 - AnyRef → Any
 
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        asInstanceOf[T0]: T0
      
      
      
- Definition Classes
 - Any
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        clone(): SparkConf
      
      
      
Copy this object
Copy this object
- Definition Classes
 - SparkConf → AnyRef
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        contains(key: String): Boolean
      
      
      
Does the configuration contain a given parameter?
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        eq(arg0: AnyRef): Boolean
      
      
      
- Definition Classes
 - AnyRef
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        equals(arg0: Any): Boolean
      
      
      
- Definition Classes
 - AnyRef → Any
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        finalize(): Unit
      
      
      
- Attributes
 - protected[lang]
 - Definition Classes
 - AnyRef
 - Annotations
 - @throws( classOf[java.lang.Throwable] )
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        get(key: String, defaultValue: String): String
      
      
      
Get a parameter, falling back to a default if not set
 - 
      
      
      
        
      
    
      
        
        def
      
      
        get(key: String): String
      
      
      
Get a parameter; throws a NoSuchElementException if it's not set
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getAll: Array[(String, String)]
      
      
      
Get all parameters as a list of pairs
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getAllWithPrefix(prefix: String): Array[(String, String)]
      
      
      
Get all parameters that start with
prefix - 
      
      
      
        
      
    
      
        
        def
      
      
        getAppId: String
      
      
      
Returns the Spark application id, valid in the Driver after TaskScheduler registration and from the start in the Executor.
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getAvroSchema: Map[Long, String]
      
      
      
Gets all the avro schemas in the configuration used in the generic Avro record serializer
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getBoolean(key: String, defaultValue: Boolean): Boolean
      
      
      
Get a parameter as a boolean, falling back to a default if not set
Get a parameter as a boolean, falling back to a default if not set
- Exceptions thrown
 IllegalArgumentExceptionIf the value cannot be interpreted as a boolean
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        getClass(): Class[_]
      
      
      
- Definition Classes
 - AnyRef → Any
 - Annotations
 - @native()
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getDouble(key: String, defaultValue: Double): Double
      
      
      
Get a parameter as a double, falling back to a default if not ste
Get a parameter as a double, falling back to a default if not ste
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as a double
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getExecutorEnv: Seq[(String, String)]
      
      
      
Get all executor environment variables set on this SparkConf
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getInt(key: String, defaultValue: Int): Int
      
      
      
Get a parameter as an integer, falling back to a default if not set
Get a parameter as an integer, falling back to a default if not set
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as an integer
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getLong(key: String, defaultValue: Long): Long
      
      
      
Get a parameter as a long, falling back to a default if not set
Get a parameter as a long, falling back to a default if not set
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as a long
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getOption(key: String): Option[String]
      
      
      
Get a parameter as an Option
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getSizeAsBytes(key: String, defaultValue: Long): Long
      
      
      
Get a size parameter as bytes, falling back to a default if not set.
Get a size parameter as bytes, falling back to a default if not set.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as bytes
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getSizeAsBytes(key: String, defaultValue: String): Long
      
      
      
Get a size parameter as bytes, falling back to a default if not set.
Get a size parameter as bytes, falling back to a default if not set. If no suffix is provided then bytes are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as bytes
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getSizeAsBytes(key: String): Long
      
      
      
Get a size parameter as bytes; throws a NoSuchElementException if it's not set.
Get a size parameter as bytes; throws a NoSuchElementException if it's not set. If no suffix is provided then bytes are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as bytesjava.util.NoSuchElementExceptionIf the size parameter is not set
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getSizeAsGb(key: String, defaultValue: String): Long
      
      
      
Get a size parameter as Gibibytes, falling back to a default if not set.
Get a size parameter as Gibibytes, falling back to a default if not set. If no suffix is provided then Gibibytes are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as Gibibytes
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getSizeAsGb(key: String): Long
      
      
      
Get a size parameter as Gibibytes; throws a NoSuchElementException if it's not set.
Get a size parameter as Gibibytes; throws a NoSuchElementException if it's not set. If no suffix is provided then Gibibytes are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as Gibibytesjava.util.NoSuchElementExceptionIf the size parameter is not set
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getSizeAsKb(key: String, defaultValue: String): Long
      
      
      
Get a size parameter as Kibibytes, falling back to a default if not set.
Get a size parameter as Kibibytes, falling back to a default if not set. If no suffix is provided then Kibibytes are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as Kibibytes
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getSizeAsKb(key: String): Long
      
      
      
Get a size parameter as Kibibytes; throws a NoSuchElementException if it's not set.
Get a size parameter as Kibibytes; throws a NoSuchElementException if it's not set. If no suffix is provided then Kibibytes are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as Kibibytesjava.util.NoSuchElementExceptionIf the size parameter is not set
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getSizeAsMb(key: String, defaultValue: String): Long
      
      
      
Get a size parameter as Mebibytes, falling back to a default if not set.
Get a size parameter as Mebibytes, falling back to a default if not set. If no suffix is provided then Mebibytes are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as Mebibytes
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getSizeAsMb(key: String): Long
      
      
      
Get a size parameter as Mebibytes; throws a NoSuchElementException if it's not set.
Get a size parameter as Mebibytes; throws a NoSuchElementException if it's not set. If no suffix is provided then Mebibytes are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as Mebibytesjava.util.NoSuchElementExceptionIf the size parameter is not set
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getTimeAsMs(key: String, defaultValue: String): Long
      
      
      
Get a time parameter as milliseconds, falling back to a default if not set.
Get a time parameter as milliseconds, falling back to a default if not set. If no suffix is provided then milliseconds are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as milliseconds
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getTimeAsMs(key: String): Long
      
      
      
Get a time parameter as milliseconds; throws a NoSuchElementException if it's not set.
Get a time parameter as milliseconds; throws a NoSuchElementException if it's not set. If no suffix is provided then milliseconds are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as millisecondsjava.util.NoSuchElementExceptionIf the time parameter is not set
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getTimeAsSeconds(key: String, defaultValue: String): Long
      
      
      
Get a time parameter as seconds, falling back to a default if not set.
Get a time parameter as seconds, falling back to a default if not set. If no suffix is provided then seconds are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as seconds
 - 
      
      
      
        
      
    
      
        
        def
      
      
        getTimeAsSeconds(key: String): Long
      
      
      
Get a time parameter as seconds; throws a NoSuchElementException if it's not set.
Get a time parameter as seconds; throws a NoSuchElementException if it's not set. If no suffix is provided then seconds are assumed.
- Exceptions thrown
 NumberFormatExceptionIf the value cannot be interpreted as secondsjava.util.NoSuchElementExceptionIf the time parameter is not set
 - 
      
      
      
        
      
    
      
        
        def
      
      
        hashCode(): Int
      
      
      
- Definition Classes
 - AnyRef → Any
 - Annotations
 - @native()
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        initializeLogIfNecessary(isInterpreter: Boolean, silent: Boolean = false): Boolean
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        initializeLogIfNecessary(isInterpreter: Boolean): Unit
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        isInstanceOf[T0]: Boolean
      
      
      
- Definition Classes
 - Any
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        isTraceEnabled(): Boolean
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        log: Logger
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        logDebug(msg: ⇒ String, throwable: Throwable): Unit
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        logDebug(msg: ⇒ String): Unit
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        logError(msg: ⇒ String, throwable: Throwable): Unit
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        logError(msg: ⇒ String): Unit
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        logInfo(msg: ⇒ String, throwable: Throwable): Unit
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        logInfo(msg: ⇒ String): Unit
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        logName: String
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        logTrace(msg: ⇒ String, throwable: Throwable): Unit
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        logTrace(msg: ⇒ String): Unit
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        logWarning(msg: ⇒ String, throwable: Throwable): Unit
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        logWarning(msg: ⇒ String): Unit
      
      
      
- Attributes
 - protected
 - Definition Classes
 - Logging
 
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        ne(arg0: AnyRef): Boolean
      
      
      
- Definition Classes
 - AnyRef
 
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        notify(): Unit
      
      
      
- Definition Classes
 - AnyRef
 - Annotations
 - @native()
 
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        notifyAll(): Unit
      
      
      
- Definition Classes
 - AnyRef
 - Annotations
 - @native()
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        registerAvroSchemas(schemas: Schema*): SparkConf
      
      
      
Use Kryo serialization and register the given set of Avro schemas so that the generic record serializer can decrease network IO
 - 
      
      
      
        
      
    
      
        
        def
      
      
        registerKryoClasses(classes: Array[Class[_]]): SparkConf
      
      
      
Use Kryo serialization and register the given set of classes with Kryo.
Use Kryo serialization and register the given set of classes with Kryo. If called multiple times, this will append the classes from all calls together.
 - 
      
      
      
        
      
    
      
        
        def
      
      
        remove(key: String): SparkConf
      
      
      
Remove a parameter from the configuration
 - 
      
      
      
        
      
    
      
        
        def
      
      
        set(key: String, value: String): SparkConf
      
      
      
Set a configuration variable.
 - 
      
      
      
        
      
    
      
        
        def
      
      
        setAll(settings: Iterable[(String, String)]): SparkConf
      
      
      
Set multiple parameters together
 - 
      
      
      
        
      
    
      
        
        def
      
      
        setAppName(name: String): SparkConf
      
      
      
Set a name for your application.
Set a name for your application. Shown in the Spark web UI.
 - 
      
      
      
        
      
    
      
        
        def
      
      
        setExecutorEnv(variables: Array[(String, String)]): SparkConf
      
      
      
Set multiple environment variables to be used when launching executors.
Set multiple environment variables to be used when launching executors. (Java-friendly version.)
 - 
      
      
      
        
      
    
      
        
        def
      
      
        setExecutorEnv(variables: Seq[(String, String)]): SparkConf
      
      
      
Set multiple environment variables to be used when launching executors.
Set multiple environment variables to be used when launching executors. These variables are stored as properties of the form spark.executorEnv.VAR_NAME (for example spark.executorEnv.PATH) but this method makes them easier to set.
 - 
      
      
      
        
      
    
      
        
        def
      
      
        setExecutorEnv(variable: String, value: String): SparkConf
      
      
      
Set an environment variable to be used when launching executors for this application.
Set an environment variable to be used when launching executors for this application. These variables are stored as properties of the form spark.executorEnv.VAR_NAME (for example spark.executorEnv.PATH) but this method makes them easier to set.
 - 
      
      
      
        
      
    
      
        
        def
      
      
        setIfMissing(key: String, value: String): SparkConf
      
      
      
Set a parameter if it isn't already configured
 - 
      
      
      
        
      
    
      
        
        def
      
      
        setJars(jars: Array[String]): SparkConf
      
      
      
Set JAR files to distribute to the cluster.
Set JAR files to distribute to the cluster. (Java-friendly version.)
 - 
      
      
      
        
      
    
      
        
        def
      
      
        setJars(jars: Seq[String]): SparkConf
      
      
      
Set JAR files to distribute to the cluster.
 - 
      
      
      
        
      
    
      
        
        def
      
      
        setMaster(master: String): SparkConf
      
      
      
The master URL to connect to, such as "local" to run locally with one thread, "local[4]" to run locally with 4 cores, or "spark://master:7077" to run on a Spark standalone cluster.
 - 
      
      
      
        
      
    
      
        
        def
      
      
        setSparkHome(home: String): SparkConf
      
      
      
Set the location where Spark is installed on worker nodes.
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        synchronized[T0](arg0: ⇒ T0): T0
      
      
      
- Definition Classes
 - AnyRef
 
 - 
      
      
      
        
      
    
      
        
        def
      
      
        toDebugString: String
      
      
      
Return a string listing all keys and values, one per line.
Return a string listing all keys and values, one per line. This is useful to print the configuration out for debugging.
 - 
      
      
      
        
      
    
      
        
        def
      
      
        toString(): String
      
      
      
- Definition Classes
 - AnyRef → Any
 
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        wait(): Unit
      
      
      
- Definition Classes
 - AnyRef
 - Annotations
 - @throws( ... )
 
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        wait(arg0: Long, arg1: Int): Unit
      
      
      
- Definition Classes
 - AnyRef
 - Annotations
 - @throws( ... )
 
 - 
      
      
      
        
      
    
      
        final 
        def
      
      
        wait(arg0: Long): Unit
      
      
      
- Definition Classes
 - AnyRef
 - Annotations
 - @throws( ... ) @native()
 
 
Deprecated Value Members
- 
      
      
      
        
      
    
      
        
        def
      
      
        setAll(settings: Traversable[(String, String)]): SparkConf
      
      
      
Set multiple parameters together
Set multiple parameters together
- Annotations
 - @deprecated
 - Deprecated
 (Since version 3.0.0) Use setAll(Iterable) instead