Skip to content

Commit

Permalink
[SPARK-24337][Core] Improve error messages for Spark conf values
Browse files Browse the repository at this point in the history
  • Loading branch information
PenguinToast committed May 29, 2018
1 parent 900bc1f commit f198f28
Show file tree
Hide file tree
Showing 2 changed files with 95 additions and 22 deletions.
81 changes: 60 additions & 21 deletions core/src/main/scala/org/apache/spark/SparkConf.scala
Original file line number Diff line number Diff line change
Expand Up @@ -265,108 +265,121 @@ class SparkConf(loadDefaults: Boolean) extends Cloneable with Logging with Seria
* Get a time parameter as seconds; throws a NoSuchElementException if it's not set. If no
* suffix is provided then seconds are assumed.
* @throws java.util.NoSuchElementException If the time parameter is not set
* @throws IllegalArgumentException If the value can't be interpreted as seconds
*/
def getTimeAsSeconds(key: String): Long = {
def getTimeAsSeconds(key: String): Long = catchIllegalArgument(key) {
Utils.timeStringAsSeconds(get(key))
}

/**
* Get a time parameter as seconds, falling back to a default if not set. If no
* suffix is provided then seconds are assumed.
* @throws IllegalArgumentException If the value can't be interpreted as seconds
*/
def getTimeAsSeconds(key: String, defaultValue: String): Long = {
def getTimeAsSeconds(key: String, defaultValue: String): Long = catchIllegalArgument(key) {
Utils.timeStringAsSeconds(get(key, defaultValue))
}

/**
* Get a time parameter as milliseconds; throws a NoSuchElementException if it's not set. If no
* suffix is provided then milliseconds are assumed.
* @throws java.util.NoSuchElementException If the time parameter is not set
* @throws IllegalArgumentException If the value can't be interpreted as milliseconds
*/
def getTimeAsMs(key: String): Long = {
def getTimeAsMs(key: String): Long = catchIllegalArgument(key) {
Utils.timeStringAsMs(get(key))
}

/**
* Get a time parameter as milliseconds, falling back to a default if not set. If no
* suffix is provided then milliseconds are assumed.
* @throws IllegalArgumentException If the value can't be interpreted as milliseconds
*/
def getTimeAsMs(key: String, defaultValue: String): Long = {
def getTimeAsMs(key: String, defaultValue: String): Long = catchIllegalArgument(key) {
Utils.timeStringAsMs(get(key, defaultValue))
}

/**
* Get a size parameter as bytes; throws a NoSuchElementException if it's not set. If no
* suffix is provided then bytes are assumed.
* @throws java.util.NoSuchElementException If the size parameter is not set
* @throws IllegalArgumentException If the value can't be interpreted as bytes
*/
def getSizeAsBytes(key: String): Long = {
def getSizeAsBytes(key: String): Long = catchIllegalArgument(key) {
Utils.byteStringAsBytes(get(key))
}

/**
* Get a size parameter as bytes, falling back to a default if not set. If no
* suffix is provided then bytes are assumed.
* @throws IllegalArgumentException If the value can't be interpreted as bytes
*/
def getSizeAsBytes(key: String, defaultValue: String): Long = {
def getSizeAsBytes(key: String, defaultValue: String): Long = catchIllegalArgument(key) {
Utils.byteStringAsBytes(get(key, defaultValue))
}

/**
* Get a size parameter as bytes, falling back to a default if not set.
* @throws IllegalArgumentException If the value can't be interpreted as bytes
*/
def getSizeAsBytes(key: String, defaultValue: Long): Long = {
def getSizeAsBytes(key: String, defaultValue: Long): Long = catchIllegalArgument(key) {
Utils.byteStringAsBytes(get(key, defaultValue + "B"))
}

/**
* Get a size parameter as Kibibytes; throws a NoSuchElementException if it's not set. If no
* suffix is provided then Kibibytes are assumed.
* @throws java.util.NoSuchElementException If the size parameter is not set
* @throws IllegalArgumentException If the value can't be interpreted as Kibibytes
*/
def getSizeAsKb(key: String): Long = {
def getSizeAsKb(key: String): Long = catchIllegalArgument(key) {
Utils.byteStringAsKb(get(key))
}

/**
* Get a size parameter as Kibibytes, falling back to a default if not set. If no
* suffix is provided then Kibibytes are assumed.
* @throws IllegalArgumentException If the value can't be interpreted as Kibibytes
*/
def getSizeAsKb(key: String, defaultValue: String): Long = {
def getSizeAsKb(key: String, defaultValue: String): Long = catchIllegalArgument(key) {
Utils.byteStringAsKb(get(key, defaultValue))
}

/**
* Get a size parameter as Mebibytes; throws a NoSuchElementException if it's not set. If no
* suffix is provided then Mebibytes are assumed.
* @throws java.util.NoSuchElementException If the size parameter is not set
* @throws IllegalArgumentException If the value can't be interpreted as Mebibytes
*/
def getSizeAsMb(key: String): Long = {
def getSizeAsMb(key: String): Long = catchIllegalArgument(key) {
Utils.byteStringAsMb(get(key))
}

/**
* Get a size parameter as Mebibytes, falling back to a default if not set. If no
* suffix is provided then Mebibytes are assumed.
* @throws IllegalArgumentException If the value can't be interpreted as Mebibytes
*/
def getSizeAsMb(key: String, defaultValue: String): Long = {
def getSizeAsMb(key: String, defaultValue: String): Long = catchIllegalArgument(key) {
Utils.byteStringAsMb(get(key, defaultValue))
}

/**
* Get a size parameter as Gibibytes; throws a NoSuchElementException if it's not set. If no
* suffix is provided then Gibibytes are assumed.
* @throws java.util.NoSuchElementException If the size parameter is not set
* @throws IllegalArgumentException If the value can't be interpreted as Gibibytes
*/
def getSizeAsGb(key: String): Long = {
def getSizeAsGb(key: String): Long = catchIllegalArgument(key) {
Utils.byteStringAsGb(get(key))
}

/**
* Get a size parameter as Gibibytes, falling back to a default if not set. If no
* suffix is provided then Gibibytes are assumed.
* @throws IllegalArgumentException If the value can't be interpreted as Gibibytes
*/
def getSizeAsGb(key: String, defaultValue: String): Long = {
def getSizeAsGb(key: String, defaultValue: String): Long = catchIllegalArgument(key) {
Utils.byteStringAsGb(get(key, defaultValue))
}

Expand Down Expand Up @@ -394,23 +407,35 @@ class SparkConf(loadDefaults: Boolean) extends Cloneable with Logging with Seria
}


/** Get a parameter as an integer, falling back to a default if not set */
def getInt(key: String, defaultValue: Int): Int = {
/**
* Get a parameter as an integer, falling back to a default if not set
* @throws IllegalArgumentException If the value can't be interpreted as an integer
*/
def getInt(key: String, defaultValue: Int): Int = catchIllegalArgument(key) {
getOption(key).map(_.toInt).getOrElse(defaultValue)
}

/** Get a parameter as a long, falling back to a default if not set */
def getLong(key: String, defaultValue: Long): Long = {
/**
* Get a parameter as a long, falling back to a default if not set
* @throws IllegalArgumentException If the value can't be interpreted as an long
*/
def getLong(key: String, defaultValue: Long): Long = catchIllegalArgument(key) {
getOption(key).map(_.toLong).getOrElse(defaultValue)
}

/** Get a parameter as a double, falling back to a default if not set */
def getDouble(key: String, defaultValue: Double): Double = {
/**
* Get a parameter as a double, falling back to a default if not ste
* @throws IllegalArgumentException If the value can't be interpreted as an double
*/
def getDouble(key: String, defaultValue: Double): Double = catchIllegalArgument(key) {
getOption(key).map(_.toDouble).getOrElse(defaultValue)
}

/** Get a parameter as a boolean, falling back to a default if not set */
def getBoolean(key: String, defaultValue: Boolean): Boolean = {
/**
* Get a parameter as a boolean, falling back to a default if not set
* @throws IllegalArgumentException If the value can't be interpreted as an boolean
*/
def getBoolean(key: String, defaultValue: Boolean): Boolean = catchIllegalArgument(key) {
getOption(key).map(_.toBoolean).getOrElse(defaultValue)
}

Expand Down Expand Up @@ -448,6 +473,20 @@ class SparkConf(loadDefaults: Boolean) extends Cloneable with Logging with Seria
*/
private[spark] def getenv(name: String): String = System.getenv(name)

/**
* Wrapper method for get*() methods which require some specific value format. This catches
* any [[NumberFormatException]] or [[IllegalArgumentException]] and re-raises it with the
* incorrectly configured key in the exception message.
*/
private def catchIllegalArgument[T](key: String)(getValue: => T): T = {
try {
getValue
} catch {
case e @ (_ : NumberFormatException | _ : IllegalArgumentException) =>
throw new IllegalArgumentException(s"Illegal value for config key $key", e)
}
}

/**
* Checks for illegal or deprecated config settings. Throws an exception for the former. Not
* idempotent - may mutate this conf object to convert deprecated settings to supported ones.
Expand Down
36 changes: 35 additions & 1 deletion core/src/test/scala/org/apache/spark/SparkConfSuite.scala
Original file line number Diff line number Diff line change
Expand Up @@ -25,14 +25,16 @@ import scala.language.postfixOps
import scala.util.{Random, Try}

import com.esotericsoftware.kryo.Kryo
import org.scalatest.Matchers

import org.apache.spark.deploy.history.config._
import org.apache.spark.internal.config._
import org.apache.spark.network.util.ByteUnit
import org.apache.spark.serializer.{JavaSerializer, KryoRegistrator, KryoSerializer}
import org.apache.spark.util.{ResetSystemProperties, RpcUtils}

class SparkConfSuite extends SparkFunSuite with LocalSparkContext with ResetSystemProperties {
class SparkConfSuite extends SparkFunSuite with LocalSparkContext with ResetSystemProperties with
Matchers {
test("Test byteString conversion") {
val conf = new SparkConf()
// Simply exercise the API, we don't need a complete conversion test since that's handled in
Expand Down Expand Up @@ -339,6 +341,38 @@ class SparkConfSuite extends SparkFunSuite with LocalSparkContext with ResetSyst
}
}

val defaultIllegalValue = "SomeIllegalValue"
val illegalArgumentTests : Map[String, (SparkConf, String) => Any] = Map(
"getTimeAsSeconds" -> (_.getTimeAsSeconds(_)),
"getTimeAsSeconds with default" -> (_.getTimeAsSeconds(_, defaultIllegalValue)),
"getTimeAsMs" -> (_.getTimeAsMs(_)),
"getTimeAsMs with default" -> (_.getTimeAsMs(_, defaultIllegalValue)),
"getSizeAsBytes" -> (_.getSizeAsBytes(_)),
"getSizeAsBytes with default string" -> (_.getSizeAsBytes(_, defaultIllegalValue)),
"getSizeAsBytes with default long" -> (_.getSizeAsBytes(_, 0L)),
"getSizeAsKb" -> (_.getSizeAsKb(_)),
"getSizeAsKb with default" -> (_.getSizeAsKb(_, defaultIllegalValue)),
"getSizeAsMb" -> (_.getSizeAsMb(_)),
"getSizeAsMb with default" -> (_.getSizeAsMb(_, defaultIllegalValue)),
"getSizeAsGb" -> (_.getSizeAsGb(_)),
"getSizeAsGb with default" -> (_.getSizeAsGb(_, defaultIllegalValue)),
"getInt" -> (_.getInt(_, 0)),
"getLong" -> (_.getLong(_, 0L)),
"getDouble" -> (_.getDouble(_, 0.0)),
"getBoolean" -> (_.getBoolean(_, false))
)

illegalArgumentTests.foreach { case (name, getValue) =>
test(s"SPARK-24337: $name throws an useful error message with key name") {
val key = "SomeKey"
val conf = new SparkConf()
conf.set(key, "SomeInvalidValue")
val thrown = the [IllegalArgumentException] thrownBy {
getValue(conf, key)
}
thrown.getMessage should include (key)
}
}
}

class Class1 {}
Expand Down

0 comments on commit f198f28

Please sign in to comment.