Repository: flume Updated Branches: refs/heads/flume-1.6 5a6adf8f2 -> d5c2cdac0
FLUME-2557. Make DatasetSink work with Kerberos (Joey Echeverria via Mike Percy) Project: http://git-wip-us.apache.org/repos/asf/flume/repo Commit: http://git-wip-us.apache.org/repos/asf/flume/commit/d5c2cdac Tree: http://git-wip-us.apache.org/repos/asf/flume/tree/d5c2cdac Diff: http://git-wip-us.apache.org/repos/asf/flume/diff/d5c2cdac Branch: refs/heads/flume-1.6 Commit: d5c2cdac0a600355a8dd72b8c805e7dc3e5667be Parents: 5a6adf8 Author: Mike Percy <[email protected]> Authored: Thu Nov 20 20:39:23 2014 -0800 Committer: Mike Percy <[email protected]> Committed: Thu Nov 20 20:54:12 2014 -0800 ---------------------------------------------------------------------- flume-ng-sinks/flume-dataset-sink/pom.xml | 20 ++- .../apache/flume/sink/kite/KerberosUtil.java | 15 ++- .../flume/sink/kite/TestKerberosUtil.java | 121 +++++++++++++++++++ .../src/test/resources/enable-kerberos.xml | 30 +++++ 4 files changed, 181 insertions(+), 5 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/flume/blob/d5c2cdac/flume-ng-sinks/flume-dataset-sink/pom.xml ---------------------------------------------------------------------- diff --git a/flume-ng-sinks/flume-dataset-sink/pom.xml b/flume-ng-sinks/flume-dataset-sink/pom.xml index 0b447a5..e929d60 100644 --- a/flume-ng-sinks/flume-dataset-sink/pom.xml +++ b/flume-ng-sinks/flume-dataset-sink/pom.xml @@ -16,7 +16,7 @@ See the License for the specific language governing permissions and limitations under the License. --> <project xmlns="http://maven.apache.org/POM/4.0.0" xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance" - xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/xsd/maven-4.0.0.xsd"> + xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/xsd/maven-4.0.0.xsd"> <modelVersion>4.0.0</modelVersion> @@ -36,6 +36,13 @@ limitations under the License. <groupId>org.apache.rat</groupId> <artifactId>apache-rat-plugin</artifactId> </plugin> + <plugin> + <groupId>org.apache.felix</groupId> + <artifactId>maven-bundle-plugin</artifactId> + <version>2.3.7</version> + <inherited>true</inherited> + <extensions>true</extensions> + </plugin> </plugins> </build> @@ -111,8 +118,8 @@ limitations under the License. <dependency> <!-- build will fail if this is not hadoop-common 2.* - because kite uses hflush. - --> + because kite uses hflush. + --> <groupId>org.apache.hadoop</groupId> <artifactId>hadoop-common</artifactId> <version>${hadoop2.version}</version> @@ -143,6 +150,13 @@ limitations under the License. </dependency> <dependency> + <groupId>org.apache.hadoop</groupId> + <artifactId>hadoop-minikdc</artifactId> + <version>${hadoop2.version}</version> + <scope>test</scope> + </dependency> + + <dependency> <groupId>org.slf4j</groupId> <artifactId>slf4j-log4j12</artifactId> <scope>test</scope> http://git-wip-us.apache.org/repos/asf/flume/blob/d5c2cdac/flume-ng-sinks/flume-dataset-sink/src/main/java/org/apache/flume/sink/kite/KerberosUtil.java ---------------------------------------------------------------------- diff --git a/flume-ng-sinks/flume-dataset-sink/src/main/java/org/apache/flume/sink/kite/KerberosUtil.java b/flume-ng-sinks/flume-dataset-sink/src/main/java/org/apache/flume/sink/kite/KerberosUtil.java index 92ad141..c0dbffb 100644 --- a/flume-ng-sinks/flume-dataset-sink/src/main/java/org/apache/flume/sink/kite/KerberosUtil.java +++ b/flume-ng-sinks/flume-dataset-sink/src/main/java/org/apache/flume/sink/kite/KerberosUtil.java @@ -82,6 +82,16 @@ public class KerberosUtil { */ public static synchronized UserGroupInformation login(String principal, String keytab) { + // If the principal or keytab isn't set, get the current (Linux) user + if (principal == null || keytab == null) { + try { + return UserGroupInformation.getCurrentUser(); + } catch (IOException ex) { + LOG.error("Can't get current user: {}", ex.getMessage()); + throw new RuntimeException(ex); + } + } + // resolve the requested principal, if it is present String finalPrincipal = null; if (principal != null && !principal.isEmpty()) { @@ -105,8 +115,9 @@ public class KerberosUtil { LOG.debug("Unable to get login user before Kerberos auth attempt", e); } - // if the current user is valid (matches the given principal) then use it - if (currentUser != null) { + // if the current user is valid (matches the given principal and has a TGT) + // then use it + if (currentUser != null && currentUser.hasKerberosCredentials()) { if (finalPrincipal == null || finalPrincipal.equals(currentUser.getUserName())) { LOG.debug("Using existing login for {}: {}", http://git-wip-us.apache.org/repos/asf/flume/blob/d5c2cdac/flume-ng-sinks/flume-dataset-sink/src/test/java/org/apache/flume/sink/kite/TestKerberosUtil.java ---------------------------------------------------------------------- diff --git a/flume-ng-sinks/flume-dataset-sink/src/test/java/org/apache/flume/sink/kite/TestKerberosUtil.java b/flume-ng-sinks/flume-dataset-sink/src/test/java/org/apache/flume/sink/kite/TestKerberosUtil.java new file mode 100644 index 0000000..f53ef73 --- /dev/null +++ b/flume-ng-sinks/flume-dataset-sink/src/test/java/org/apache/flume/sink/kite/TestKerberosUtil.java @@ -0,0 +1,121 @@ +/** + * 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.flume.sink.kite; + +import java.io.File; +import java.io.IOException; +import java.net.URL; +import java.util.Properties; +import org.apache.hadoop.conf.Configuration; +import org.apache.hadoop.minikdc.MiniKdc; +import org.apache.hadoop.security.UserGroupInformation; +import org.junit.AfterClass; +import org.junit.BeforeClass; +import org.junit.Test; +import static org.junit.Assert.*; + +public class TestKerberosUtil { + + private static MiniKdc kdc; + private static File workDir; + private static File flumeKeytab; + private static String flumePrincipal = "flume/localhost"; + private static File aliceKeytab; + private static String alicePrincipal = "alice"; + private static Properties conf; + + @BeforeClass + public static void startMiniKdc() throws Exception { + URL resource = Thread.currentThread() + .getContextClassLoader().getResource("enable-kerberos.xml"); + Configuration.addDefaultResource("enable-kerberos.xml"); + + workDir = new File(System.getProperty("test.dir", "target"), + TestKerberosUtil.class.getSimpleName()); + flumeKeytab = new File(workDir, "flume.keytab"); + aliceKeytab = new File(workDir, "alice.keytab"); + conf = MiniKdc.createConf(); + + kdc = new MiniKdc(conf, workDir); + kdc.start(); + + kdc.createPrincipal(flumeKeytab, flumePrincipal); + flumePrincipal = flumePrincipal + "@" + kdc.getRealm(); + + kdc.createPrincipal(aliceKeytab, alicePrincipal); + alicePrincipal = alicePrincipal + "@" + kdc.getRealm(); + } + + @AfterClass + public static void stopMiniKdc() { + if (kdc != null) { + kdc.stop(); + } + } + + @Test + public void testNullLogin() throws IOException { + String principal = null; + String keytab = null; + UserGroupInformation expResult = UserGroupInformation.getCurrentUser(); + UserGroupInformation result = KerberosUtil.login(principal, keytab); + assertEquals(expResult, result); + } + + @Test + public void testFlumeLogin() throws IOException { + String principal = flumePrincipal; + String keytab = flumeKeytab.getAbsolutePath(); + String expResult = principal; + + String result = KerberosUtil.login(principal, keytab).getUserName(); + assertEquals("Initial login failed", expResult, result); + + result = KerberosUtil.login(principal, keytab).getUserName(); + assertEquals("Re-login failed", expResult, result); + + principal = alicePrincipal; + keytab = aliceKeytab.getAbsolutePath(); + try { + result = KerberosUtil.login(principal, keytab).getUserName(); + fail("Login should have failed with a new principal: " + result); + } catch (KerberosUtil.SecurityException ex) { + assertTrue("Login with a new principal failed, but for an unexpected " + + "reason: " + ex.getMessage(), + ex.getMessage().contains("Cannot use multiple Kerberos principals: ")); + } + } + + @Test + public void testProxyAs() throws IOException { + String username = "alice"; + + UserGroupInformation login = UserGroupInformation.getCurrentUser(); + String expResult = username; + String result = KerberosUtil.proxyAs(username, login).getUserName(); + assertEquals("Proxy as didn't generate the expected username", expResult, result); + + login = KerberosUtil.login(flumePrincipal, flumeKeytab.getAbsolutePath()); + assertEquals("Login succeeded, but the principal doesn't match", + flumePrincipal, login.getUserName()); + + result = KerberosUtil.proxyAs(username, login).getUserName(); + assertEquals("Proxy as didn't generate the expected username", expResult, result); + } + +} http://git-wip-us.apache.org/repos/asf/flume/blob/d5c2cdac/flume-ng-sinks/flume-dataset-sink/src/test/resources/enable-kerberos.xml ---------------------------------------------------------------------- diff --git a/flume-ng-sinks/flume-dataset-sink/src/test/resources/enable-kerberos.xml b/flume-ng-sinks/flume-dataset-sink/src/test/resources/enable-kerberos.xml new file mode 100644 index 0000000..85b0447 --- /dev/null +++ b/flume-ng-sinks/flume-dataset-sink/src/test/resources/enable-kerberos.xml @@ -0,0 +1,30 @@ +<?xml version="1.0" encoding="UTF-8"?> +<!-- +Copyright 2014 Apache Software Foundation. + +Licensed 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. +--> + +<configuration> + + <property> + <name>hadoop.security.authentication</name> + <value>kerberos</value> + </property> + + <property> + <name>hadoop.security.authorization</name> + <value>true</value> + </property> + +</configuration>
