Skip to content

Commit

Permalink
[SPARK-12317][SQL] Support units (m,k,g) in SQLConf
Browse files Browse the repository at this point in the history
This PR is continue from previous closed PR 10314.

In this PR, SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE will be taken memory string conventions as input.

For example, the user can now specify 10g for SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE in SQLConf file.

marmbrus srowen : Can you help review this code changes ? Thanks.

Author: Kevin Yu <[email protected]>

Closes #10629 from kevinyu98/spark-12317.
  • Loading branch information
kevinyu98 authored and rxin committed Jan 8, 2016
1 parent 28e0e50 commit 5028a00
Show file tree
Hide file tree
Showing 2 changed files with 60 additions and 1 deletion.
22 changes: 21 additions & 1 deletion sql/core/src/main/scala/org/apache/spark/sql/SQLConf.scala
Original file line number Diff line number Diff line change
Expand Up @@ -26,6 +26,7 @@ import org.apache.parquet.hadoop.ParquetOutputCommitter

import org.apache.spark.sql.catalyst.CatalystConf
import org.apache.spark.sql.catalyst.parser.ParserConf
import org.apache.spark.util.Utils

////////////////////////////////////////////////////////////////////////////////////////////////////
// This file defines the configuration options for Spark SQL.
Expand Down Expand Up @@ -115,6 +116,25 @@ private[spark] object SQLConf {
}
}, _.toString, doc, isPublic)

def longMemConf(
key: String,
defaultValue: Option[Long] = None,
doc: String = "",
isPublic: Boolean = true): SQLConfEntry[Long] =
SQLConfEntry(key, defaultValue, { v =>
try {
v.toLong
} catch {
case _: NumberFormatException =>
try {
Utils.byteStringAsBytes(v)
} catch {
case _: NumberFormatException =>
throw new IllegalArgumentException(s"$key should be long, but was $v")
}
}
}, _.toString, doc, isPublic)

def doubleConf(
key: String,
defaultValue: Option[Double] = None,
Expand Down Expand Up @@ -235,7 +255,7 @@ private[spark] object SQLConf {
doc = "The default number of partitions to use when shuffling data for joins or aggregations.")

val SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE =
longConf("spark.sql.adaptive.shuffle.targetPostShuffleInputSize",
longMemConf("spark.sql.adaptive.shuffle.targetPostShuffleInputSize",
defaultValue = Some(64 * 1024 * 1024),
doc = "The target post-shuffle input size in bytes of a task.")

Expand Down
39 changes: 39 additions & 0 deletions sql/core/src/test/scala/org/apache/spark/sql/SQLConfSuite.scala
Original file line number Diff line number Diff line change
Expand Up @@ -92,4 +92,43 @@ class SQLConfSuite extends QueryTest with SharedSQLContext {
}
assert(e.getMessage === s"${SQLConf.CASE_SENSITIVE.key} should be boolean, but was 10")
}

test("Test SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE's method") {
sqlContext.conf.clear()

sqlContext.setConf(SQLConf.SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE.key, "100")
assert(sqlContext.conf.targetPostShuffleInputSize === 100)

sqlContext.setConf(SQLConf.SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE.key, "1k")
assert(sqlContext.conf.targetPostShuffleInputSize === 1024)

sqlContext.setConf(SQLConf.SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE.key, "1M")
assert(sqlContext.conf.targetPostShuffleInputSize === 1048576)

sqlContext.setConf(SQLConf.SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE.key, "1g")
assert(sqlContext.conf.targetPostShuffleInputSize === 1073741824)

sqlContext.setConf(SQLConf.SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE.key, "-1")
assert(sqlContext.conf.targetPostShuffleInputSize === -1)

// Test overflow exception
intercept[IllegalArgumentException] {
// This value exceeds Long.MaxValue
// Utils.byteStringAsBytes("90000000000g")
sqlContext.setConf(SQLConf.SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE.key, "90000000000g")
}

intercept[IllegalArgumentException] {
// This value less than Int.MinValue
// Utils.byteStringAsBytes("-90000000000g")
sqlContext.setConf(SQLConf.SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE.key, "-90000000000g")
}
// Test invalid input
intercept[IllegalArgumentException] {
// This value exceeds Long.MaxValue
// Utils.byteStringAsBytes("-1g")
sqlContext.setConf(SQLConf.SHUFFLE_TARGET_POSTSHUFFLE_INPUT_SIZE.key, "-1g")
}
sqlContext.conf.clear()
}
}

0 comments on commit 5028a00

Please sign in to comment.