diff --git a/kubernetes/core/src/main/scala/org/apache/spark/deploy/kubernetes/Client.scala b/kubernetes/core/src/main/scala/org/apache/spark/deploy/kubernetes/Client.scala
index 21c83dbf40e21..721bc6a2be455 100644
--- a/kubernetes/core/src/main/scala/org/apache/spark/deploy/kubernetes/Client.scala
+++ b/kubernetes/core/src/main/scala/org/apache/spark/deploy/kubernetes/Client.scala
@@ -35,7 +35,7 @@ import scala.concurrent.duration.DurationInt
import scala.util.Success
import org.apache.spark.{SPARK_VERSION, SparkConf}
-import org.apache.spark.deploy.rest.{AppResource, KubernetesCreateSubmissionRequest, RemoteAppResource, UploadedAppResource}
+import org.apache.spark.deploy.rest.{AppResource, KubernetesCreateSubmissionRequest, RemoteAppResource, TarGzippedData, UploadedAppResource}
import org.apache.spark.deploy.rest.kubernetes._
import org.apache.spark.internal.Logging
import org.apache.spark.util.Utils
@@ -282,8 +282,8 @@ private[spark] class Client(
case other => RemoteAppResource(other)
}
- val uploadDriverExtraClasspathBase64Contents = getFileContents(uploadedDriverExtraClasspath)
- val uploadJarsBase64Contents = getFileContents(uploadedJars)
+ val uploadDriverExtraClasspathBase64Contents = compressJars(uploadedDriverExtraClasspath)
+ val uploadJarsBase64Contents = compressJars(uploadedJars)
KubernetesCreateSubmissionRequest(
appResource = resolvedAppResource,
mainClass = mainClass,
@@ -294,19 +294,10 @@ private[spark] class Client(
uploadedJarsBase64Contents = uploadJarsBase64Contents)
}
- def getFileContents(maybeFilePaths: Option[String]): Array[(String, String)] = {
+ def compressJars(maybeFilePaths: Option[String]): Option[TarGzippedData] = {
maybeFilePaths
- .map(_.split(",").map(filePath => {
- val fileToUpload = new File(filePath)
- if (!fileToUpload.isFile) {
- throw new IllegalStateException("Provided file to upload for driver extra classpath" +
- s" does not exist or is not a file: $filePath")
- } else {
- val fileBytes = Files.toByteArray(fileToUpload)
- val fileBase64 = Base64.encodeBase64String(fileBytes)
- (fileToUpload.getName, fileBase64)
- }
- })).getOrElse(Array.empty[(String, String)])
+ .map(_.split(","))
+ .map(CompressionUtils.createTarGzip(_))
}
private def getDriverLauncherService(
diff --git a/kubernetes/core/src/main/scala/org/apache/spark/deploy/rest/KubernetesRestProtocolMessages.scala b/kubernetes/core/src/main/scala/org/apache/spark/deploy/rest/KubernetesRestProtocolMessages.scala
index 4b7bb66083f29..6da1a848b25e7 100644
--- a/kubernetes/core/src/main/scala/org/apache/spark/deploy/rest/KubernetesRestProtocolMessages.scala
+++ b/kubernetes/core/src/main/scala/org/apache/spark/deploy/rest/KubernetesRestProtocolMessages.scala
@@ -27,14 +27,19 @@ case class KubernetesCreateSubmissionRequest(
val appArgs: Array[String],
val sparkProperties: Map[String, String],
val secret: String,
- val uploadedDriverExtraClasspathBase64Contents: Array[(String, String)]
- = Array.empty[(String, String)],
- val uploadedJarsBase64Contents: Array[(String, String)]
- = Array.empty[(String, String)]) extends SubmitRestProtocolRequest {
+ val uploadedDriverExtraClasspathBase64Contents: Option[TarGzippedData],
+ val uploadedJarsBase64Contents: Option[TarGzippedData]) extends SubmitRestProtocolRequest {
message = "create"
clientSparkVersion = SPARK_VERSION
}
+case class TarGzippedData(
+ val dataBase64: String,
+ val blockSize: Int = 10240,
+ val recordSize: Int = 512,
+ val encoding: String
+)
+
@JsonTypeInfo(
use = JsonTypeInfo.Id.NAME,
include = JsonTypeInfo.As.PROPERTY,
diff --git a/kubernetes/core/src/main/scala/org/apache/spark/deploy/rest/kubernetes/CompressionUtils.scala b/kubernetes/core/src/main/scala/org/apache/spark/deploy/rest/kubernetes/CompressionUtils.scala
new file mode 100644
index 0000000000000..cbbaa3a789693
--- /dev/null
+++ b/kubernetes/core/src/main/scala/org/apache/spark/deploy/rest/kubernetes/CompressionUtils.scala
@@ -0,0 +1,113 @@
+/*
+ * 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.deploy.rest.kubernetes
+
+import java.io.{ByteArrayInputStream, File, FileInputStream, FileOutputStream}
+import java.util.zip.{GZIPInputStream, GZIPOutputStream}
+
+import com.google.common.io.Files
+import org.apache.commons.codec.binary.Base64
+import org.apache.commons.compress.archivers.tar.{TarArchiveEntry, TarArchiveInputStream, TarArchiveOutputStream}
+import org.apache.commons.compress.utils.CharsetNames
+import org.apache.commons.io.IOUtils
+import scala.collection.mutable
+
+import org.apache.spark.deploy.rest.TarGzippedData
+import org.apache.spark.util.{ByteBufferOutputStream, Utils}
+
+private[spark] object CompressionUtils {
+ private val BLOCK_SIZE = 10240
+ private val RECORD_SIZE = 512
+ private val ENCODING = CharsetNames.UTF_8
+
+ def createTarGzip(paths: Iterable[String]): TarGzippedData = {
+ val compressedBytesStream = Utils.tryWithResource(new ByteBufferOutputStream()) { raw =>
+ Utils.tryWithResource(new GZIPOutputStream(raw)) { gzipping =>
+ Utils.tryWithResource(new TarArchiveOutputStream(
+ gzipping,
+ BLOCK_SIZE,
+ RECORD_SIZE,
+ ENCODING)) { tarStream =>
+ val usedFileNames = mutable.HashSet.empty[String]
+ for (path <- paths) {
+ val file = new File(path)
+ if (!file.isFile) {
+ throw new IllegalArgumentException(s"Cannot add $path to tarball; either does" +
+ s" not exist or is a directory.")
+ }
+ var resolvedFileName = file.getName
+ val extension = Files.getFileExtension(file.getName)
+ val nameWithoutExtension = Files.getNameWithoutExtension(file.getName)
+ var deduplicationCounter = 1
+ while (usedFileNames.contains(resolvedFileName)) {
+ resolvedFileName = s"$nameWithoutExtension-$deduplicationCounter.$extension"
+ deduplicationCounter += 1
+ }
+ usedFileNames += resolvedFileName
+ val tarEntry = new TarArchiveEntry(file, resolvedFileName)
+ tarStream.putArchiveEntry(tarEntry)
+ Utils.tryWithResource(new FileInputStream(file)) { fileInput =>
+ IOUtils.copy(fileInput, tarStream)
+ }
+ tarStream.closeArchiveEntry()
+ }
+ }
+ }
+ raw
+ }
+ val compressedAsBase64 = Base64.encodeBase64String(compressedBytesStream.toByteBuffer.array)
+ TarGzippedData(
+ dataBase64 = compressedAsBase64,
+ blockSize = BLOCK_SIZE,
+ recordSize = RECORD_SIZE,
+ encoding = ENCODING
+ )
+ }
+
+ def unpackAndWriteCompressedFiles(
+ compressedData: TarGzippedData,
+ rootOutputDir: File): Seq[String] = {
+ val paths = mutable.Buffer.empty[String]
+ val compressedBytes = Base64.decodeBase64(compressedData.dataBase64)
+ if (!rootOutputDir.exists) {
+ rootOutputDir.mkdir
+ } else if (rootOutputDir.isFile) {
+ throw new IllegalArgumentException(s"Root dir for writing decompressed files: " +
+ s"${rootOutputDir.getAbsolutePath} exists and is not a directory.")
+ }
+ Utils.tryWithResource(new ByteArrayInputStream(compressedBytes)) { compressedBytesStream =>
+ Utils.tryWithResource(new GZIPInputStream(compressedBytesStream)) { gzipped =>
+ Utils.tryWithResource(new TarArchiveInputStream(
+ gzipped,
+ compressedData.blockSize,
+ compressedData.recordSize,
+ compressedData.encoding)) { tarInputStream =>
+ var nextTarEntry = tarInputStream.getNextTarEntry
+ while (nextTarEntry != null) {
+ val outputFile = new File(rootOutputDir, nextTarEntry.getName)
+ Utils.tryWithResource(new FileOutputStream(outputFile)) { fileOutputStream =>
+ IOUtils.copy(tarInputStream, fileOutputStream)
+ }
+ paths += outputFile.getAbsolutePath
+ nextTarEntry = tarInputStream.getNextTarEntry
+ }
+ }
+ }
+ }
+ paths.toSeq
+ }
+}
diff --git a/kubernetes/core/src/main/scala/org/apache/spark/deploy/rest/kubernetes/KubernetesSparkRestServer.scala b/kubernetes/core/src/main/scala/org/apache/spark/deploy/rest/kubernetes/KubernetesSparkRestServer.scala
index 0a2e8176394ab..2ca3d4a8c0656 100644
--- a/kubernetes/core/src/main/scala/org/apache/spark/deploy/rest/kubernetes/KubernetesSparkRestServer.scala
+++ b/kubernetes/core/src/main/scala/org/apache/spark/deploy/rest/kubernetes/KubernetesSparkRestServer.scala
@@ -217,30 +217,11 @@ private[spark] class KubernetesSparkRestServer(
}
private def writeBase64ContentsToFiles(
- filesBase64Contents: Array[(String, String)],
+ maybeCompressedFiles: Option[TarGzippedData],
rootDir: File): Seq[String] = {
- val resolvedFileNames = new scala.collection.mutable.HashSet[String]
- val resolvedFilePaths = new ArrayBuffer[String]
- for (file <- filesBase64Contents) {
- var currentFileName = file._1
- var deduplicationCounter = 1
- while (resolvedFileNames.contains(currentFileName)) {
- // Prepend the deduplication counter so as to not mess with the extension
- currentFileName = s"$deduplicationCounter-$currentFileName"
- deduplicationCounter += 1
- }
- val resolvedFile = new File(rootDir, currentFileName)
- val resolvedFilePath = resolvedFile.getAbsolutePath
- if (resolvedFile.createNewFile()) {
- val fileContents = Base64.decodeBase64(file._2)
- Files.write(fileContents, resolvedFile)
- } else {
- throw new IllegalStateException(s"Could not write jar file to $resolvedFilePath")
- }
- resolvedFileNames += currentFileName
- resolvedFilePaths += resolvedFilePath
- }
- resolvedFilePaths.toSeq
+ maybeCompressedFiles.map { compressedFiles =>
+ CompressionUtils.unpackAndWriteCompressedFiles(compressedFiles, rootDir)
+ }.getOrElse(Seq.empty[String])
}
}
diff --git a/kubernetes/integration-tests-spark-jobs-helpers/pom.xml b/kubernetes/integration-tests-spark-jobs-helpers/pom.xml
new file mode 100644
index 0000000000000..742e299fc0884
--- /dev/null
+++ b/kubernetes/integration-tests-spark-jobs-helpers/pom.xml
@@ -0,0 +1,33 @@
+
+
+
+ 4.0.0
+
+ org.apache.spark
+ spark-parent_2.11
+ 2.1.0-SNAPSHOT
+ ../../pom.xml
+
+
+ spark-kubernetes-integration-tests-spark-jobs-helpers_2.11
+ jar
+ Spark Project Kubernetes Integration Tests Spark Jobs Helpers
+
+
+
+
diff --git a/kubernetes/integration-tests-spark-jobs-helpers/src/main/java/org/apache/spark/deploy/kubernetes/integrationtest/PiHelper.java b/kubernetes/integration-tests-spark-jobs-helpers/src/main/java/org/apache/spark/deploy/kubernetes/integrationtest/PiHelper.java
new file mode 100644
index 0000000000000..a9775b0d7fba8
--- /dev/null
+++ b/kubernetes/integration-tests-spark-jobs-helpers/src/main/java/org/apache/spark/deploy/kubernetes/integrationtest/PiHelper.java
@@ -0,0 +1,29 @@
+/*
+ * 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.deploy.kubernetes.integrationtest;
+
+public class PiHelper {
+ public static int helpPi() {
+ double x = Math.random() * 2 - 1;
+ double y = Math.random() * 2 - 1;
+ if (x*x + y*y < 1) {
+ return 1;
+ } else {
+ return 0;
+ }
+ }
+}
diff --git a/kubernetes/integration-tests-spark-jobs/pom.xml b/kubernetes/integration-tests-spark-jobs/pom.xml
index 17f1c4906214f..8cb8a54043ae7 100644
--- a/kubernetes/integration-tests-spark-jobs/pom.xml
+++ b/kubernetes/integration-tests-spark-jobs/pom.xml
@@ -29,6 +29,12 @@
Spark Project Kubernetes Integration Tests Spark Jobs
+
+ org.apache.spark
+ spark-kubernetes-integration-tests-spark-jobs-helpers_${scala.binary.version}
+ ${project.version}
+ provided
+
org.apache.spark
spark-core_${scala.binary.version}
diff --git a/kubernetes/integration-tests-spark-jobs/src/main/scala/org/apache/spark/deploy/kubernetes/integrationtest/jobs/SparkPiWithInfiniteWait.scala b/kubernetes/integration-tests-spark-jobs/src/main/scala/org/apache/spark/deploy/kubernetes/integrationtest/jobs/SparkPiWithInfiniteWait.scala
index 6e4660b771305..d3372749f999e 100644
--- a/kubernetes/integration-tests-spark-jobs/src/main/scala/org/apache/spark/deploy/kubernetes/integrationtest/jobs/SparkPiWithInfiniteWait.scala
+++ b/kubernetes/integration-tests-spark-jobs/src/main/scala/org/apache/spark/deploy/kubernetes/integrationtest/jobs/SparkPiWithInfiniteWait.scala
@@ -16,8 +16,7 @@
*/
package org.apache.spark.deploy.kubernetes.integrationtest.jobs
-import scala.math.random
-
+import org.apache.spark.deploy.kubernetes.integrationtest.PiHelper
import org.apache.spark.sql.SparkSession
// Equivalent to SparkPi except does not stop the Spark Context
@@ -32,10 +31,8 @@ private[spark] object SparkPiWithInfiniteWait {
.getOrCreate()
val slices = if (args.length > 0) args(0).toInt else 10
val n = math.min(100000L * slices, Int.MaxValue).toInt // avoid overflow
- val count = spark.sparkContext.parallelize(1 until n, slices).map { i =>
- val x = random * 2 - 1
- val y = random * 2 - 1
- if (x*x + y*y < 1) 1 else 0
+ val count = spark.sparkContext.parallelize(1 until n, slices).map { _ =>
+ PiHelper.helpPi()
}.reduce(_ + _)
// scalastyle:off println
println("Pi is roughly " + 4.0 * count / (n - 1))
diff --git a/kubernetes/integration-tests/pom.xml b/kubernetes/integration-tests/pom.xml
index 0568cb1e21826..1056134c36208 100644
--- a/kubernetes/integration-tests/pom.xml
+++ b/kubernetes/integration-tests/pom.xml
@@ -48,6 +48,12 @@
${project.version}
test
+
+ org.apache.spark
+ spark-kubernetes-integration-tests-spark-jobs-helpers_${scala.binary.version}
+ ${project.version}
+ test
+
org.apache.spark
spark-docker-minimal-bundle_${scala.binary.version}
@@ -123,6 +129,13 @@
jar
${project.build.directory}/integration-tests-spark-jobs
+
+ org.apache.spark
+ spark-kubernetes-integration-tests-spark-jobs-helpers_${scala.binary.version}
+ ${project.version}
+ jar
+ ${project.build.directory}/integration-tests-spark-jobs-helpers
+
diff --git a/kubernetes/integration-tests/src/test/scala/org/apache/spark/deploy/kubernetes/integrationtest/KubernetesSuite.scala b/kubernetes/integration-tests/src/test/scala/org/apache/spark/deploy/kubernetes/integrationtest/KubernetesSuite.scala
index 902631b874539..90765e35d5662 100644
--- a/kubernetes/integration-tests/src/test/scala/org/apache/spark/deploy/kubernetes/integrationtest/KubernetesSuite.scala
+++ b/kubernetes/integration-tests/src/test/scala/org/apache/spark/deploy/kubernetes/integrationtest/KubernetesSuite.scala
@@ -41,6 +41,11 @@ private[spark] class KubernetesSuite extends SparkFunSuite with BeforeAndAfter {
.listFiles()(0)
.getAbsolutePath
+ private val HELPER_JAR = Paths.get("target", "integration-tests-spark-jobs-helpers")
+ .toFile
+ .listFiles()(0)
+ .getAbsolutePath
+
private val TIMEOUT = PatienceConfiguration.Timeout(Span(2, Minutes))
private val INTERVAL = PatienceConfiguration.Interval(Span(2, Seconds))
private val MAIN_CLASS = "org.apache.spark.deploy.kubernetes" +
@@ -118,6 +123,7 @@ private[spark] class KubernetesSuite extends SparkFunSuite with BeforeAndAfter {
.set("spark.kubernetes.namespace", NAMESPACE)
.set("spark.kubernetes.driver.docker.image", "spark-driver:latest")
.set("spark.kubernetes.executor.docker.image", "spark-executor:latest")
+ .set("spark.kubernetes.driver.uploads.jars", HELPER_JAR)
.set("spark.executor.memory", "500m")
.set("spark.executor.cores", "1")
.set("spark.executors.instances", "1")
@@ -144,6 +150,7 @@ private[spark] class KubernetesSuite extends SparkFunSuite with BeforeAndAfter {
"--executor-memory", "512m",
"--executor-cores", "1",
"--num-executors", "1",
+ "--upload-jars", HELPER_JAR,
"--class", MAIN_CLASS,
"--conf", s"spark.kubernetes.submit.caCertFile=${clientConfig.getCaCertFile}",
"--conf", s"spark.kubernetes.submit.clientKeyFile=${clientConfig.getClientKeyFile}",
diff --git a/pom.xml b/pom.xml
index 78e2730d8f76c..f4cbfb57db9b3 100644
--- a/pom.xml
+++ b/pom.xml
@@ -2638,6 +2638,7 @@
kubernetes/docker-minimal-bundle
kubernetes/integration-tests
kubernetes/integration-tests-spark-jobs
+ kubernetes/integration-tests-spark-jobs-helpers