18
18
package org .apache .spark .deploy .yarn
19
19
20
20
import java .io .File
21
+ import java .lang .reflect .UndeclaredThrowableException
21
22
import java .nio .charset .StandardCharsets .UTF_8
23
+ import java .security .PrivilegedExceptionAction
22
24
import java .util .regex .Matcher
23
25
import java .util .regex .Pattern
24
26
@@ -156,7 +158,7 @@ class YarnSparkHadoopUtil extends SparkHadoopUtil {
156
158
*/
157
159
def obtainTokenForHiveMetastore (conf : Configuration ): Option [Token [DelegationTokenIdentifier ]] = {
158
160
try {
159
- obtainTokenForHiveMetastoreInner(conf, UserGroupInformation .getCurrentUser().getUserName )
161
+ obtainTokenForHiveMetastoreInner(conf)
160
162
} catch {
161
163
case e : ClassNotFoundException =>
162
164
logInfo(s " Hive class not found $e" )
@@ -171,8 +173,8 @@ class YarnSparkHadoopUtil extends SparkHadoopUtil {
171
173
* @param username the username of the principal requesting the delegating token.
172
174
* @return a delegation token
173
175
*/
174
- private [yarn] def obtainTokenForHiveMetastoreInner (conf : Configuration ,
175
- username : String ) : Option [Token [DelegationTokenIdentifier ]] = {
176
+ private [yarn] def obtainTokenForHiveMetastoreInner (conf : Configuration ) :
177
+ Option [Token [DelegationTokenIdentifier ]] = {
176
178
val mirror = universe.runtimeMirror(Utils .getContextOrSparkClassLoader)
177
179
178
180
// the hive configuration class is a subclass of Hadoop Configuration, so can be cast down
@@ -187,11 +189,12 @@ class YarnSparkHadoopUtil extends SparkHadoopUtil {
187
189
188
190
// Check for local metastore
189
191
if (metastoreUri.nonEmpty) {
190
- require(username.nonEmpty, " Username undefined" )
191
192
val principalKey = " hive.metastore.kerberos.principal"
192
193
val principal = hiveConf.getTrimmed(principalKey, " " )
193
194
require(principal.nonEmpty, " Hive principal $principalKey undefined" )
194
- logDebug(s " Getting Hive delegation token for $username against $principal at $metastoreUri" )
195
+ val currentUser = UserGroupInformation .getCurrentUser()
196
+ logDebug(s " Getting Hive delegation token for ${currentUser.getUserName()} against " +
197
+ s " $principal at $metastoreUri" )
195
198
val hiveClass = mirror.classLoader.loadClass(" org.apache.hadoop.hive.ql.metadata.Hive" )
196
199
val closeCurrent = hiveClass.getMethod(" closeCurrent" )
197
200
try {
@@ -200,12 +203,14 @@ class YarnSparkHadoopUtil extends SparkHadoopUtil {
200
203
classOf [String ], classOf [String ])
201
204
val getHive = hiveClass.getMethod(" get" , hiveConfClass)
202
205
203
- // invoke
204
- val hive = getHive.invoke(null , hiveConf)
205
- val tokenStr = getDelegationToken.invoke(hive, username, principal).asInstanceOf [String ]
206
- val hive2Token = new Token [DelegationTokenIdentifier ]()
207
- hive2Token.decodeFromUrlString(tokenStr)
208
- Some (hive2Token)
206
+ doAsRealUser {
207
+ val hive = getHive.invoke(null , hiveConf)
208
+ val tokenStr = getDelegationToken.invoke(hive, currentUser.getUserName(), principal)
209
+ .asInstanceOf [String ]
210
+ val hive2Token = new Token [DelegationTokenIdentifier ]()
211
+ hive2Token.decodeFromUrlString(tokenStr)
212
+ Some (hive2Token)
213
+ }
209
214
} finally {
210
215
Utils .tryLogNonFatalError {
211
216
closeCurrent.invoke(null )
@@ -216,6 +221,26 @@ class YarnSparkHadoopUtil extends SparkHadoopUtil {
216
221
None
217
222
}
218
223
}
224
+
225
+ /**
226
+ * Run some code as the real logged in user (which may differ from the current user, for
227
+ * example, when using proxying).
228
+ */
229
+ private def doAsRealUser [T ](fn : => T ): T = {
230
+ val currentUser = UserGroupInformation .getCurrentUser()
231
+ val realUser = Option (currentUser.getRealUser()).getOrElse(currentUser)
232
+
233
+ // For some reason the Scala-generated anonymous class ends up causing an
234
+ // UndeclaredThrowableException, even if you annotate the method with @throws.
235
+ try {
236
+ realUser.doAs(new PrivilegedExceptionAction [T ]() {
237
+ override def run (): T = fn
238
+ })
239
+ } catch {
240
+ case e : UndeclaredThrowableException => throw Option (e.getCause()).getOrElse(e)
241
+ }
242
+ }
243
+
219
244
}
220
245
221
246
object YarnSparkHadoopUtil {
0 commit comments