Skip to content
This repository has been archived by the owner on Jan 9, 2020. It is now read-only.

Support using PEM files to configure SSL for driver submission #173

Merged
merged 5 commits into from
Mar 20, 2017

Conversation

mccheah
Copy link

@mccheah mccheah commented Mar 4, 2017

Closes #164. Note that this also changes some of the configuration keys in order to better clarify what context the certificate and key files are for (API server vs. driver submission server).

}
Utils.tryWithResource(new FileOutputStream(certPemFile)) { keyPemStream =>
Utils.tryWithResource(
new OutputStreamWriter(keyPemStream, Charsets.UTF_8)) { streamWriter =>
Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Missing an indent here, will fix on the next pass after review

@foxish
Copy link
Member

foxish commented Mar 4, 2017

This one can merge after the alpha?

@mccheah
Copy link
Author

mccheah commented Mar 4, 2017

Correct - this should merge to the mainline branch after we cut and freeze the release branch.

@mccheah
Copy link
Author

mccheah commented Mar 8, 2017

@robert3005 @ssuchter for review

Copy link

@robert3005 robert3005 left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Logic seems fine but the structure is off and makes understanding the flow really difficult.

val sslSecretsMap = mutable.HashMap[String, String]()
val storeBasedSslOptions = driverSubmitSslOptions.storeBasedSslOptions
val sslEnvs = mutable.Buffer[EnvVar]()

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Can you split setting on sslEnvs and sslSecretsMap. You can just concat the transformed options which is more in line with what scala does. Then you don't need mutable buffers

}

private def resolveLocalFile(file: Option[String],
fileType: String): (Boolean, Option[String]) = {

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

fileType seems unnecessary here

Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

It's required for the error message.

.map(Files.toString(_, Charsets.UTF_8))
val resolvedKeyStore = (parsedArguments.keyStoreFile, parsedArguments.keyPemFile) match {
case (None, Some(keyPemFile)) =>
parsedArguments.certPemFile match {

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

You're converting option to another option just to validate? This is map.orElse

Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Changed the structure a bit.

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Not a fan of matching on options. See http://blog.originate.com/blog/2014/06/15/idiomatic-scala-your-options-do-not-match/ (Spark codebase doesn't like folding on them). map(...).getOrElse(throw new SparkException).

}

private def parsePrivateKeyFromPemFile(keyPemFile: File): PrivateKey = {
Utils.tryWithResource(new FileInputStream(keyPemFile)) { keyPemStream =>

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Can you extract it to another function. This triply nested stream seems to be repeating

@robert3005
Copy link

Looks good now. I think like there's a ton of options though that would benefit refactoring but it doesn't seem like the right place to tackle that.

@mccheah
Copy link
Author

mccheah commented Mar 13, 2017

@robert3005 @foxish anything else needed for this to merge?

@foxish
Copy link
Member

foxish commented Mar 13, 2017

I'm waiting on cutting the alpha and moving to our new branch structure. If someone is ready to do the cherrypicks after we have our 2.1-kubernetes and 2.1-kubernetes-dev branches, we can merge this.

@ash211
Copy link

ash211 commented Mar 16, 2017

Please rebase onto branch-2.1-kubernetes and send this PR into that branch instead of k8s-support-alternate-incremental which is now deprecated.

@mccheah mccheah changed the base branch from k8s-support-alternate-incremental to branch-2.1-kubernetes March 16, 2017 20:35
@@ -345,7 +339,7 @@ private[spark] class Client(
private def configureOwnerReferences(
kubernetesClient: KubernetesClient,
submitServerSecret: Secret,
sslSecrets: Array[Secret],
sslSecrets: Option[Secret],
Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

rename to just sslSecret if there's only one now. Are we losing functionality by dropping from N to 1 secret?

Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Nope because we only create one secret in this code. It's one secret with multiple parts.

sslSecrets: Array[Secret],
sslPodVolume: Option[Volume],
sslPodVolumeMount: Option[VolumeMount],
sslSecrets: Option[Secret],
Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

singular

driverSubmitSslOptions.isKeyStoreLocalFile,
SUBMISSION_SSL_KEYSTORE_SECRET_NAME,
storeBasedSslOptions.keyStore,
"KeyStore")
Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

maybe make KeyStore a constant? Its casing is a bit different from https://github.com/apache-spark-on-k8s/spark/pull/173/files#diff-c0d91a23f31f682a15ce930d584f0e5cR225

it's just used in logging though, so maybe no big deal?

Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

The casing is intentionally different I believe because in this case the type is mentioned at the beginning of the sentence of the error message and in the other case it's in the middle, hence requiring different grammar considerations in the choice of the string literal =P

throw new SparkException(s"$secretType specified at ${file.getAbsolutePath} is not" +
s" a file or does not exist.")
}
val keyStoreBytes = Files.toByteArray(file)
Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

make sure file is closed?

Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yep it is

resolveLocalFile(maybeServerCertPem, "server cert PEM")
val (isLocalKeyPem, resolvedKeyPem) = resolveLocalFile(maybeKeyPem, "key PEM")
maybeTrustStore.foreach { trustStore =>
require(KubernetesFileUtils.isUriLocalFile(trustStore), s"Invalid trustStore URI" +
Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

no space between strings

private def validateSslOptions(parsedArguments: KubernetesSparkRestServerArguments): Unit = {
parsedArguments.keyStoreFile.foreach { _ =>
require(parsedArguments.keyPemFile.orElse(parsedArguments.certPemFile).isEmpty,
"Cannot provide both key/cert pem files and a keyStore file; select one or the other" +
Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

PEM

parsedArguments.keyStoreType)
})
}).getOrElse(throw new SparkException("When providing pem files, both the key and" +
" the certificate must be specified.")))
Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

mention in this message that these are for the listening port on the REST server

privateKey,
keyPassword.map(_.toCharArray).orNull,
Array(certificate))
val keyStoreOutputPath = Paths.get(s"keystore-${UUID.randomUUID()}.$resolvedKeyStoreType")
Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

this is in the current working directory of the rest service? safe to assume it's writable and has space?

Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Think this is normally fine but may depend on the Docker image setup. The safest bet is to attach an emptyDir volume and use that, but that's a bit of extra complexity to make the REST server aware of the writable directory.

Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

let's punt on the emptydir volume complexity until it's a problem


private def parseCertificateFromPemFile(certPemFile: File): X509Certificate = {
withPemParsedFromFile(certPemFile) { certPemParser =>
certPemParser.readObject() match {
Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

does this handle concatenated certs? meaning some PEM files have a series of -----BEGIN CERTIFICATE----- / -----END CERTIFICATE----- blocks with the whole cert chain -- e.g. root CA then intermediate CA then certificate.

Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Hm, another project indicates how to do this. We should try this and test accordingly: docker-java/docker-java@c49f598

Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Although once again I think meaningfully testing the processing of multiple certificates is going to be hard. That once again is best left to the unit level.

SparkSubmit.main(args)
}

test("Enable SSL on the driver submit server using PEM files") {
Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

difficult to add a test that verifies local:// URIs for PEM files? we'd have to bake them into the testing docker image?

Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yeah they would have to be added to the testing Docker image which isn't ideal. Perhaps possible since we build the testing Docker image through Java code right now though, so we could write the PEM and JKS files to the Docker build directory before building.

Copy link

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Do you think testing that use case directly is important enough? It's definitely one I think people (and us) will be using

Copy link
Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I can see that being left to the unit level which we don't have coverage for yet.

@ash211
Copy link

ash211 commented Mar 17, 2017

@mccheah scalastyle failure

@ash211 ash211 force-pushed the branch-2.1-kubernetes branch from d8245a1 to f9f5af4 Compare March 17, 2017 23:06
Copy link

@ash211 ash211 left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

All looks good!

@ash211 ash211 merged commit 078697f into branch-2.1-kubernetes Mar 20, 2017
@ash211 ash211 deleted the support-pem-ssl branch March 20, 2017 20:38
foxish pushed a commit that referenced this pull request Jul 24, 2017
* Support configuring SSL using PEM files.

* Address some missed comments

* Fix import ordering

* Slight rewording of comments

* Fix scalastyle
ifilonenko pushed a commit to ifilonenko/spark that referenced this pull request Feb 25, 2019
…e-spark-on-k8s#173)

* Support configuring SSL using PEM files.

* Address some missed comments

* Fix import ordering

* Slight rewording of comments

* Fix scalastyle

(cherry picked from commit 078697f)
ifilonenko pushed a commit to ifilonenko/spark that referenced this pull request Feb 25, 2019
puneetloya pushed a commit to puneetloya/spark that referenced this pull request Mar 11, 2019
…e-spark-on-k8s#173)

* Support configuring SSL using PEM files.

* Address some missed comments

* Fix import ordering

* Slight rewording of comments

* Fix scalastyle
Sign up for free to subscribe to this conversation on GitHub. Already have an account? Sign in.
Labels
None yet
Projects
None yet
Development

Successfully merging this pull request may close these issues.

4 participants