You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by GitBox <gi...@apache.org> on 2022/02/16 18:27:05 UTC

[GitHub] [spark] mridulm commented on a change in pull request #35504: [SPARK-38194][YARN][MESOS][K8S] Make memory overhead factor configurable

mridulm commented on a change in pull request #35504:
URL: https://github.com/apache/spark/pull/35504#discussion_r808318065



##########
File path: core/src/main/scala/org/apache/spark/internal/config/package.scala
##########
@@ -105,6 +105,17 @@ package object config {
     .bytesConf(ByteUnit.MiB)
     .createOptional
 
+  private[spark] val DRIVER_MEMORY_OVERHEAD_FACTOR =
+    ConfigBuilder("spark.driver.memoryOverheadFactor")
+      .doc("The amount of non-heap memory to be allocated per driver in cluster mode, " +
+      "as a fraction of total driver memory. If memory overhead is specified directly, " +
+      "it takes precedence.")

Review comment:
       We can use the current description for `spark.kubernetes.memoryOverheadFactor` in `docs/running-on-kubernetes.md` as template (instead of non-heap memory).

##########
File path: core/src/main/scala/org/apache/spark/internal/config/package.scala
##########
@@ -105,6 +105,17 @@ package object config {
     .bytesConf(ByteUnit.MiB)
     .createOptional
 
+  private[spark] val DRIVER_MEMORY_OVERHEAD_FACTOR =
+    ConfigBuilder("spark.driver.memoryOverheadFactor")
+      .doc("The amount of non-heap memory to be allocated per driver in cluster mode, " +
+      "as a fraction of total driver memory. If memory overhead is specified directly, " +
+      "it takes precedence.")
+      .version("3.3.0")
+      .doubleConf
+      .checkValue(factor => factor >= 0 && factor < 1,
+        "Ensure that memory overhead is a double between 0 --> 1.0")

Review comment:
       `factor > 0` would should be sufficient, we can have cases where overhead is higher than jvm Xmx.

##########
File path: resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/k8s/Config.scala
##########
@@ -490,7 +490,7 @@ private[spark] object Config extends Logging {
       .doubleConf
       .checkValue(mem_overhead => mem_overhead >= 0,
         "Ensure that memory overhead is non-negative")
-      .createWithDefault(0.1)
+      .createOptional

Review comment:
       Keep default (see below).

##########
File path: resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/k8s/features/BasicExecutorFeatureStep.scala
##########
@@ -59,11 +59,13 @@ private[spark] class BasicExecutorFeatureStep(
   private val isDefaultProfile = resourceProfile.id == ResourceProfile.DEFAULT_RESOURCE_PROFILE_ID
   private val isPythonApp = kubernetesConf.get(APP_RESOURCE_TYPE) == Some(APP_RESOURCE_TYPE_PYTHON)
   private val disableConfigMap = kubernetesConf.get(KUBERNETES_EXECUTOR_DISABLE_CONFIGMAP)
+  private val memoryOverheadFactor = kubernetesConf.get(MEMORY_OVERHEAD_FACTOR)
+    .getOrElse(kubernetesConf.get(EXECUTOR_MEMORY_OVERHEAD_FACTOR))
 

Review comment:
       Same as in `BasicDriverFeatureStep` - change order of config query.

##########
File path: resource-managers/yarn/src/main/scala/org/apache/spark/deploy/yarn/Client.scala
##########
@@ -85,6 +84,8 @@ private[spark] class Client(
   private var appMaster: ApplicationMaster = _
   private var stagingDirPath: Path = _
 
+  private val amMemoryOverheadFactor = sparkConf.get(DRIVER_MEMORY_OVERHEAD_FACTOR)

Review comment:
       Do this only when in cluster mode - in client mode, AM is not the driver.

##########
File path: resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/k8s/features/BasicDriverFeatureStep.scala
##########
@@ -53,18 +53,20 @@ private[spark] class BasicDriverFeatureStep(conf: KubernetesDriverConf)
 
   // Memory settings
   private val driverMemoryMiB = conf.get(DRIVER_MEMORY)
+  private val memoryOverheadFactor = conf.get(MEMORY_OVERHEAD_FACTOR)
+    .getOrElse(conf.get(DRIVER_MEMORY_OVERHEAD_FACTOR))

Review comment:
       Use `DRIVER_MEMORY_OVERHEAD_FACTOR` and fallback to `MEMORY_OVERHEAD_FACTOR` when missing.
   We probably need to use `get(key, defaultValue)` method here though.




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



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