-
Notifications
You must be signed in to change notification settings - Fork 117
Basic Secure HDFS Support [514] #540
Changes from 6 commits
7612bf5
50f47d0
87df4a7
67856a5
7cdae31
04aa26f
765455d
488b37e
37feb22
4e44027
86c7b8f
64b0af7
ba2bafc
0c99503
a9d074b
a3b12a7
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,87 @@ | ||
/* | ||
* 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.k8s | ||
|
||
import java.io.File | ||
|
||
import scala.collection.JavaConverters._ | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Imports are not consistent with the rest of the project. Order should be as follows everywhere:
Please look over all files and fix all imports. There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I was curious about the import order. According to http://spark.apache.org/contributing.html, the recommended import order is slightly different. scala.* and other 3rd parties libraries are separated by an empty space. Do we know which one is correct?
An example from the same page:
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Actually @kimoonkim and I think our code is incorrect in most places. There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. @mccheah Cool. Should we then follow the import order suggested in http://spark.apache.org/contributing.html going forward? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Yes we should. We can fix the ordering as we merge upstream. |
||
|
||
import io.fabric8.kubernetes.api.model.{ContainerBuilder, KeyToPathBuilder, PodBuilder} | ||
|
||
import org.apache.spark.deploy.k8s.constants._ | ||
import org.apache.spark.internal.Logging | ||
|
||
/** | ||
* This is separated out from the HadoopConf steps API because this component can be reused to | ||
* set up the Hadoop Configuration for executors as well. | ||
*/ | ||
private[spark] trait HadoopConfBootstrap { | ||
/** | ||
* Bootstraps a main container with the ConfigMaps containing Hadoop config files | ||
* mounted as volumes and an ENV variable pointing to the mounted file. | ||
*/ | ||
def bootstrapMainContainerAndVolumes( | ||
originalPodWithMainContainer: PodWithMainContainer) | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Can this all fit on one line? |
||
: PodWithMainContainer | ||
} | ||
|
||
private[spark] class HadoopConfBootstrapImpl( | ||
hadoopConfConfigMapName: String, | ||
hadoopConfigFiles: Seq[File], | ||
hadoopUGI: HadoopUGIUtil) extends HadoopConfBootstrap with Logging{ | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Add a space after |
||
|
||
override def bootstrapMainContainerAndVolumes( | ||
originalPodWithMainContainer: PodWithMainContainer) | ||
: PodWithMainContainer = { | ||
logInfo("HADOOP_CONF_DIR defined. Mounting Hadoop specific .xml files") | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Should we filter |
||
val keyPaths = hadoopConfigFiles.map{ file => | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. nit: empty space after |
||
val fileStringPath = file.toPath.getFileName.toString | ||
new KeyToPathBuilder() | ||
.withKey(fileStringPath) | ||
.withPath(fileStringPath) | ||
.build() } | ||
val hadoopSupportedPod = new PodBuilder(originalPodWithMainContainer.pod) | ||
.editSpec() | ||
.addNewVolume() | ||
.withName(HADOOP_FILE_VOLUME) | ||
.withNewConfigMap() | ||
.withName(hadoopConfConfigMapName) | ||
.withItems(keyPaths.asJava) | ||
.endConfigMap() | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Wrong indention. |
||
.endVolume() | ||
.endSpec() | ||
.build() | ||
val hadoopSupportedContainer = new ContainerBuilder( | ||
originalPodWithMainContainer.mainContainer) | ||
.addNewVolumeMount() | ||
.withName(HADOOP_FILE_VOLUME) | ||
.withMountPath(HADOOP_CONF_DIR_PATH) | ||
.endVolumeMount() | ||
.addNewEnv() | ||
.withName(ENV_HADOOP_CONF_DIR) | ||
.withValue(HADOOP_CONF_DIR_PATH) | ||
.endEnv() | ||
.addNewEnv() | ||
.withName(ENV_SPARK_USER) | ||
.withValue(hadoopUGI.getShortName) | ||
.endEnv() | ||
.build() | ||
originalPodWithMainContainer.copy( | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. nit: put an empty line before the returned value. |
||
pod = hadoopSupportedPod, | ||
mainContainer = hadoopSupportedContainer) | ||
} | ||
} |
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,80 @@ | ||
/* | ||
* 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.k8s | ||
|
||
import java.io.{ByteArrayInputStream, ByteArrayOutputStream, DataInputStream, DataOutputStream} | ||
|
||
import scala.util.Try | ||
|
||
import org.apache.hadoop.conf.Configuration | ||
import org.apache.hadoop.fs.FileSystem | ||
import org.apache.hadoop.security.{Credentials, UserGroupInformation} | ||
import org.apache.hadoop.security.token.{Token, TokenIdentifier} | ||
import org.apache.hadoop.security.token.delegation.AbstractDelegationTokenIdentifier | ||
|
||
|
||
// Function of this class is merely for mocking reasons | ||
private[spark] class HadoopUGIUtil{ | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Put a |
||
def getCurrentUser: UserGroupInformation = UserGroupInformation.getCurrentUser | ||
|
||
def getShortName: String = getCurrentUser.getShortUserName | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Maybe we don't need this method. There is only one caller. And the caller can easily just do There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. This is used purely for mocking There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. We should wrap a minimal set of methods for mocking. And we already wrap |
||
|
||
def isSecurityEnabled: Boolean = UserGroupInformation.isSecurityEnabled | ||
|
||
def loginUserFromKeytabAndReturnUGI(principal: String, keytab: String): UserGroupInformation = | ||
UserGroupInformation.loginUserFromKeytabAndReturnUGI(principal, keytab) | ||
|
||
def dfsAddDelegationToken(hadoopConf: Configuration, renewer: String, creds: Credentials) | ||
: Iterable[Token[_ <: TokenIdentifier]] = | ||
FileSystem.get(hadoopConf).addDelegationTokens(renewer, creds) | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Just return a There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. How exactly can this be done? This has been tripping me up, as I am trying to mock this FileSystem object but with no luck (while ensuring that it passes Integration tests) There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I think you can add a method to this class like:
|
||
|
||
def getCurrentTime: Long = System.currentTimeMillis() | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Logically, getting current time does not belong to HadoopUGI. Move to some other class? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Once again, used purely for mocking of the HadoopUGI... There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I think mocking this system interaction makes sense. I'm just pointing out that we can have multiple utility classes for mocking, and time-related methods don't seem to belong to UGI. There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Use |
||
|
||
// Functions that should be in Core with Rebase to 2.3 | ||
@deprecated("Moved to core in 2.3", "2.3") | ||
def getTokenRenewalInterval( | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Line 30 says "Function of this class is merely for mocking reasons". But it seems this function has real business logic, more than just mocking purpose. Move it to some other class? |
||
renewedTokens: Iterable[Token[_ <: TokenIdentifier]], | ||
hadoopConf: Configuration): Option[Long] = { | ||
val renewIntervals = renewedTokens.filter { | ||
_.decodeIdentifier().isInstanceOf[AbstractDelegationTokenIdentifier] | ||
}.flatMap { token => | ||
Try { | ||
val newExpiration = token.renew(hadoopConf) | ||
val identifier = token.decodeIdentifier().asInstanceOf[AbstractDelegationTokenIdentifier] | ||
val interval = newExpiration - identifier.getIssueDate | ||
interval | ||
}.toOption | ||
} | ||
renewIntervals.reduceLeftOption(_ min _) | ||
} | ||
|
||
@deprecated("Moved to core in 2.3", "2.3") | ||
def serialize(creds: Credentials): Array[Byte] = { | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Ditto. It has business logic that should be tested than just being mocked. Move to some other class? |
||
val byteStream = new ByteArrayOutputStream | ||
val dataStream = new DataOutputStream(byteStream) | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Is There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. handled this below There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Use |
||
creds.writeTokenStorageToStream(dataStream) | ||
dataStream.close() | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. We need to make sure this is called even if There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. +1 |
||
byteStream.toByteArray | ||
} | ||
|
||
@deprecated("Moved to core in 2.3", "2.3") | ||
def deserialize(tokenBytes: Array[Byte]): Credentials = { | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Same here. Move to some other class? |
||
val creds = new Credentials() | ||
creds.readTokenStorageStream(new DataInputStream(new ByteArrayInputStream(tokenBytes))) | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. The |
||
creds | ||
} | ||
} |
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,76 @@ | ||
/* | ||
* 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.k8s | ||
|
||
import io.fabric8.kubernetes.api.model.{ContainerBuilder, PodBuilder} | ||
|
||
import org.apache.spark.deploy.k8s.constants._ | ||
import org.apache.spark.internal.Logging | ||
|
||
|
||
/** | ||
* This is separated out from the HadoopConf steps API because this component can be reused to | ||
* mounted the DT secret for executors as well. | ||
*/ | ||
private[spark] trait KerberosTokenConfBootstrap { | ||
// Bootstraps a main container with the Secret mounted as volumes and an ENV variable | ||
// pointing to the mounted file containing the DT for Secure HDFS interaction | ||
def bootstrapMainContainerAndVolumes( | ||
originalPodWithMainContainer: PodWithMainContainer) | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Indentation is off here - think we want this line and the next line indented in one more. |
||
: PodWithMainContainer | ||
} | ||
|
||
private[spark] class KerberosTokenConfBootstrapImpl( | ||
secretName: String, | ||
secretItemKey: String, | ||
userName: String) extends KerberosTokenConfBootstrap with Logging{ | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Space after |
||
|
||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Nit. Have one empty line instead of two? |
||
override def bootstrapMainContainerAndVolumes( | ||
originalPodWithMainContainer: PodWithMainContainer) | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Indentation is off here, indent in one more along with the line below. |
||
: PodWithMainContainer = { | ||
logInfo("Mounting HDFS DT from Secret for Secure HDFS") | ||
val secretMountedPod = new PodBuilder(originalPodWithMainContainer.pod) | ||
.editOrNewSpec() | ||
.addNewVolume() | ||
.withName(SPARK_APP_HADOOP_SECRET_VOLUME_NAME) | ||
.withNewSecret() | ||
.withSecretName(secretName) | ||
.endSecret() | ||
.endVolume() | ||
.endSpec() | ||
.build() | ||
// TODO: ENV_HADOOP_TOKEN_FILE_LOCATION should point to the latest token data item key. | ||
val secretMountedContainer = new ContainerBuilder( | ||
originalPodWithMainContainer.mainContainer) | ||
.addNewVolumeMount() | ||
.withName(SPARK_APP_HADOOP_SECRET_VOLUME_NAME) | ||
.withMountPath(SPARK_APP_HADOOP_CREDENTIALS_BASE_DIR) | ||
.endVolumeMount() | ||
.addNewEnv() | ||
.withName(ENV_HADOOP_TOKEN_FILE_LOCATION) | ||
.withValue(s"$SPARK_APP_HADOOP_CREDENTIALS_BASE_DIR/$secretItemKey") | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I just realized we have an edge case that this will fail. Imagine a job ran for many weeks and the refresh server added new weekly tokens. And also imagine the dynamic allocation is enabled and new executors are launching. Those new executors should use the latest token, not the initial token. i.e. ENV_HADOOP_TOKEN_FILE_LOCATION should point to the latest token data item key. I don't know how we can solve this yet. And we should probably address it later in a follow-up PR that we'll write for picking up the new token. Maybe add a TODO here so we don't forget this? |
||
.endEnv() | ||
.addNewEnv() | ||
.withName(ENV_SPARK_USER) | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. SPARK_USER could be different from Job User so yes we are overwriting it. There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Can we resolve it in one place and set it consistently everywhere? Right now the ordering and overwriting is ambiguous. There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Its not ambiguous as there are scenarios where the UGI could be either the Job User or taken from the TGT |
||
.withValue(userName) | ||
.endEnv() | ||
.build() | ||
originalPodWithMainContainer.copy( | ||
pod = secretMountedPod, | ||
mainContainer = secretMountedContainer) | ||
} | ||
} |
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -496,6 +496,49 @@ package object config extends Logging { | |
|
||
private[spark] val KUBERNETES_NODE_SELECTOR_PREFIX = "spark.kubernetes.node.selector." | ||
|
||
private[spark] val KUBERNETES_KERBEROS_SUPPORT = | ||
ConfigBuilder("spark.kubernetes.kerberos.enabled") | ||
.doc("Specify whether your job is a job that will require a Delegation Token to access HDFS") | ||
.booleanConf | ||
.createWithDefault(false) | ||
|
||
private[spark] val KUBERNETES_KERBEROS_KEYTAB = | ||
ConfigBuilder("spark.kubernetes.kerberos.keytab") | ||
.doc("Specify the location of keytab" + | ||
" for Kerberos in order to access Secure HDFS") | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. nit: empty space at the end of the first part. Ditto below. |
||
.stringConf | ||
.createOptional | ||
|
||
private[spark] val KUBERNETES_KERBEROS_PRINCIPAL = | ||
ConfigBuilder("spark.kubernetes.kerberos.principal") | ||
.doc("Specify the principal" + | ||
" for Kerberos in order to access Secure HDFS") | ||
.stringConf | ||
.createOptional | ||
|
||
private[spark] val KUBERNETES_KERBEROS_RENEWER_PRINCIPAL = | ||
ConfigBuilder("spark.kubernetes.kerberos.renewer.principal") | ||
.doc("Specify the principal" + | ||
" you wish to renew and retrieve your Kerberos values with") | ||
.stringConf | ||
.createOptional | ||
|
||
private[spark] val KUBERNETES_KERBEROS_DT_SECRET_NAME = | ||
ConfigBuilder("spark.kubernetes.kerberos.tokensecret.name") | ||
.doc("Specify the name of the secret where " + | ||
" your existing delegation token is stored. This removes the need" + | ||
" for the job user to provide any keytab for launching a job") | ||
.stringConf | ||
.createOptional | ||
|
||
private[spark] val KUBERNETES_KERBEROS_DT_SECRET_ITEM_KEY = | ||
ConfigBuilder("spark.kubernetes.kerberos.tokensecret.itemkey") | ||
.doc("Specify the item key of the data where " + | ||
" your existing delegation token is stored. This removes the need" + | ||
" for the job user to provide any keytab for launching a job") | ||
.stringConf | ||
.createOptional | ||
|
||
private[spark] def resolveK8sMaster(rawMasterString: String): String = { | ||
if (!rawMasterString.startsWith("k8s://")) { | ||
throw new IllegalArgumentException("Master URL should start with k8s:// in Kubernetes mode.") | ||
|
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Curious. Is the token refresh server supposed to renew this pre-populated token as well? Or is it supposed to be renewed by the job user? We may want to comment on that.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
The token refresh server is supposed to renew this pre-populated token. The assumption is that if you supply a pre-populated token it will be automatically updated by either an administrator or the token refresh server. In the later PR if you think, you should probably note this.