You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@spark.apache.org by yh...@apache.org on 2016/02/26 08:10:44 UTC

spark git commit: [SPARK-13487][SQL] User-facing RuntimeConfig interface

Repository: spark
Updated Branches:
  refs/heads/master 8afe49141 -> 26ac60806


[SPARK-13487][SQL] User-facing RuntimeConfig interface

## What changes were proposed in this pull request?
This patch creates the public API for runtime configuration and an implementation for it. The public runtime configuration includes configs for existing SQL, as well as Hadoop Configuration.

This new interface is currently dead code. It will be added to SQLContext and a session entry point to Spark when we add that.

## How was this patch tested?
a new unit test suite

Author: Reynold Xin <rx...@databricks.com>

Closes #11378 from rxin/SPARK-13487.


Project: http://git-wip-us.apache.org/repos/asf/spark/repo
Commit: http://git-wip-us.apache.org/repos/asf/spark/commit/26ac6080
Tree: http://git-wip-us.apache.org/repos/asf/spark/tree/26ac6080
Diff: http://git-wip-us.apache.org/repos/asf/spark/diff/26ac6080

Branch: refs/heads/master
Commit: 26ac60806cc23527ea8a75986c1eab83d312a15d
Parents: 8afe491
Author: Reynold Xin <rx...@databricks.com>
Authored: Thu Feb 25 23:10:40 2016 -0800
Committer: Yin Huai <yh...@databricks.com>
Committed: Thu Feb 25 23:10:40 2016 -0800

----------------------------------------------------------------------
 .../org/apache/spark/sql/RuntimeConfig.scala    | 100 +++++++++++++++++++
 .../spark/sql/internal/RuntimeConfigImpl.scala  |  73 ++++++++++++++
 .../org/apache/spark/sql/internal/SQLConf.scala |   3 +-
 .../spark/sql/internal/RuntimeConfigSuite.scala |  86 ++++++++++++++++
 4 files changed, 261 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/spark/blob/26ac6080/sql/core/src/main/scala/org/apache/spark/sql/RuntimeConfig.scala
----------------------------------------------------------------------
diff --git a/sql/core/src/main/scala/org/apache/spark/sql/RuntimeConfig.scala b/sql/core/src/main/scala/org/apache/spark/sql/RuntimeConfig.scala
new file mode 100644
index 0000000..e90a042
--- /dev/null
+++ b/sql/core/src/main/scala/org/apache/spark/sql/RuntimeConfig.scala
@@ -0,0 +1,100 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.spark.sql
+
+/**
+ * Runtime configuration interface for Spark. To access this, use `SparkSession.conf`.
+ *
+ * @since 2.0.0
+ */
+abstract class RuntimeConfig {
+
+  /**
+   * Sets the given Spark runtime configuration property.
+   *
+   * @since 2.0.0
+   */
+  def set(key: String, value: String): RuntimeConfig
+
+  /**
+   * Sets the given Spark runtime configuration property.
+   *
+   * @since 2.0.0
+   */
+  def set(key: String, value: Boolean): RuntimeConfig
+
+  /**
+   * Sets the given Spark runtime configuration property.
+   *
+   * @since 2.0.0
+   */
+  def set(key: String, value: Long): RuntimeConfig
+
+  /**
+   * Returns the value of Spark runtime configuration property for the given key.
+   *
+   * @throws NoSuchElementException if the key is not set and does not have a default value
+   * @since 2.0.0
+   */
+  @throws[NoSuchElementException]("if the key is not set")
+  def get(key: String): String
+
+  /**
+   * Returns the value of Spark runtime configuration property for the given key.
+   *
+   * @since 2.0.0
+   */
+  def getOption(key: String): Option[String]
+
+  /**
+   * Resets the configuration property for the given key.
+   *
+   * @since 2.0.0
+   */
+  def unset(key: String): Unit
+
+  /**
+   * Sets the given Hadoop configuration property. This is passed directly to Hadoop during I/O.
+   *
+   * @since 2.0.0
+   */
+  def setHadoop(key: String, value: String): RuntimeConfig
+
+  /**
+   * Returns the value of the Hadoop configuration property.
+   *
+   * @throws NoSuchElementException if the key is not set
+   * @since 2.0.0
+   */
+  @throws[NoSuchElementException]("if the key is not set")
+  def getHadoop(key: String): String
+
+  /**
+   * Returns the value of the Hadoop configuration property.
+   *
+   * @since 2.0.0
+   */
+  def getHadoopOption(key: String): Option[String]
+
+  /**
+   * Resets the Hadoop configuration property for the given key.
+   *
+   * @since 2.0.0
+   */
+  def unsetHadoop(key: String): Unit
+}

http://git-wip-us.apache.org/repos/asf/spark/blob/26ac6080/sql/core/src/main/scala/org/apache/spark/sql/internal/RuntimeConfigImpl.scala
----------------------------------------------------------------------
diff --git a/sql/core/src/main/scala/org/apache/spark/sql/internal/RuntimeConfigImpl.scala b/sql/core/src/main/scala/org/apache/spark/sql/internal/RuntimeConfigImpl.scala
new file mode 100644
index 0000000..058df1e
--- /dev/null
+++ b/sql/core/src/main/scala/org/apache/spark/sql/internal/RuntimeConfigImpl.scala
@@ -0,0 +1,73 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.spark.sql.internal
+
+import org.apache.spark.sql.RuntimeConfig
+
+/**
+ * Implementation for [[RuntimeConfig]].
+ */
+class RuntimeConfigImpl extends RuntimeConfig {
+
+  private val conf = new SQLConf
+
+  private val hadoopConf = java.util.Collections.synchronizedMap(
+    new java.util.HashMap[String, String]())
+
+  override def set(key: String, value: String): RuntimeConfig = {
+    conf.setConfString(key, value)
+    this
+  }
+
+  override def set(key: String, value: Boolean): RuntimeConfig = set(key, value.toString)
+
+  override def set(key: String, value: Long): RuntimeConfig = set(key, value.toString)
+
+  @throws[NoSuchElementException]("if the key is not set")
+  override def get(key: String): String = conf.getConfString(key)
+
+  override def getOption(key: String): Option[String] = {
+    try Option(get(key)) catch {
+      case _: NoSuchElementException => None
+    }
+  }
+
+  override def unset(key: String): Unit = conf.unsetConf(key)
+
+  override def setHadoop(key: String, value: String): RuntimeConfig = {
+    hadoopConf.put(key, value)
+    this
+  }
+
+  @throws[NoSuchElementException]("if the key is not set")
+  override def getHadoop(key: String): String = hadoopConf.synchronized {
+    if (hadoopConf.containsKey(key)) {
+      hadoopConf.get(key)
+    } else {
+      throw new NoSuchElementException(key)
+    }
+  }
+
+  override def getHadoopOption(key: String): Option[String] = {
+    try Option(getHadoop(key)) catch {
+      case _: NoSuchElementException => None
+    }
+  }
+
+  override def unsetHadoop(key: String): Unit = hadoopConf.remove(key)
+}

http://git-wip-us.apache.org/repos/asf/spark/blob/26ac6080/sql/core/src/main/scala/org/apache/spark/sql/internal/SQLConf.scala
----------------------------------------------------------------------
diff --git a/sql/core/src/main/scala/org/apache/spark/sql/internal/SQLConf.scala b/sql/core/src/main/scala/org/apache/spark/sql/internal/SQLConf.scala
index c1e3f38..9a50ef7 100644
--- a/sql/core/src/main/scala/org/apache/spark/sql/internal/SQLConf.scala
+++ b/sql/core/src/main/scala/org/apache/spark/sql/internal/SQLConf.scala
@@ -17,7 +17,7 @@
 
 package org.apache.spark.sql.internal
 
-import java.util.Properties
+import java.util.{NoSuchElementException, Properties}
 
 import scala.collection.JavaConverters._
 import scala.collection.immutable
@@ -649,6 +649,7 @@ class SQLConf extends Serializable with CatalystConf with ParserConf with Loggin
   }
 
   /** Return the value of Spark SQL configuration property for the given key. */
+  @throws[NoSuchElementException]("if key is not set")
   def getConfString(key: String): String = {
     Option(settings.get(key)).
       orElse {

http://git-wip-us.apache.org/repos/asf/spark/blob/26ac6080/sql/core/src/test/scala/org/apache/spark/sql/internal/RuntimeConfigSuite.scala
----------------------------------------------------------------------
diff --git a/sql/core/src/test/scala/org/apache/spark/sql/internal/RuntimeConfigSuite.scala b/sql/core/src/test/scala/org/apache/spark/sql/internal/RuntimeConfigSuite.scala
new file mode 100644
index 0000000..f809e01
--- /dev/null
+++ b/sql/core/src/test/scala/org/apache/spark/sql/internal/RuntimeConfigSuite.scala
@@ -0,0 +1,86 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.spark.sql.internal
+
+import org.apache.spark.SparkFunSuite
+import org.apache.spark.sql.RuntimeConfig
+
+class RuntimeConfigSuite extends SparkFunSuite {
+
+  private def newConf(): RuntimeConfig = new RuntimeConfigImpl
+
+  test("set and get") {
+    val conf = newConf()
+    conf
+      .set("k1", "v1")
+      .set("k2", 2)
+      .set("k3", value = false)
+
+    assert(conf.get("k1") == "v1")
+    assert(conf.get("k2") == "2")
+    assert(conf.get("k3") == "false")
+
+    intercept[NoSuchElementException] {
+      conf.get("notset")
+    }
+  }
+
+  test("getOption") {
+    val conf = newConf().set("k1", "v1")
+    assert(conf.getOption("k1") == Some("v1"))
+    assert(conf.getOption("notset") == None)
+  }
+
+  test("unset") {
+    val conf = newConf().set("k1", "v1")
+    assert(conf.get("k1") == "v1")
+    conf.unset("k1")
+    intercept[NoSuchElementException] {
+      conf.get("k1")
+    }
+  }
+
+  test("set and get hadoop configuration") {
+    val conf = newConf()
+    conf
+      .setHadoop("k1", "v1")
+      .setHadoop("k2", "v2")
+
+    assert(conf.getHadoop("k1") == "v1")
+    assert(conf.getHadoop("k2") == "v2")
+
+    intercept[NoSuchElementException] {
+      conf.get("notset")
+    }
+  }
+
+  test("getHadoopOption") {
+    val conf = newConf().setHadoop("k1", "v1")
+    assert(conf.getHadoopOption("k1") == Some("v1"))
+    assert(conf.getHadoopOption("notset") == None)
+  }
+
+  test("unsetHadoop") {
+    val conf = newConf().setHadoop("k1", "v1")
+    assert(conf.getHadoop("k1") == "v1")
+    conf.unsetHadoop("k1")
+    intercept[NoSuchElementException] {
+      conf.getHadoop("k1")
+    }
+  }
+}


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@spark.apache.org
For additional commands, e-mail: commits-help@spark.apache.org