Переглянути джерело

HADOOP-8943. Support multiple group mapping providers. Contributed by Kai Zheng

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/trunk@1605857 13f79535-47bb-0310-9956-ffa450edef68
Brandon Li 11 роки тому
батько
коміт
f194aaa0d1

+ 2 - 0
hadoop-common-project/hadoop-common/CHANGES.txt

@@ -378,6 +378,8 @@ Release 2.5.0 - UNRELEASED
 
     HADOOP-9704. Write metrics sink plugin for Hadoop/Graphite (Chu Tong, Alex Newman and Babak Behzad via raviprak)
 
+    HADOOP-8943. Support multiple group mapping providers. (Kai Zheng via brandonli)
+
   IMPROVEMENTS
 
     HADOOP-10451. Remove unused field and imports from SaslRpcServer.

+ 166 - 0
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/security/CompositeGroupsMapping.java

@@ -0,0 +1,166 @@
+/**
+ * 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.hadoop.security;
+
+import java.io.IOException;
+import java.util.ArrayList;
+import java.util.Iterator;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.util.TreeSet;
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+import org.apache.hadoop.conf.Configurable;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.util.ReflectionUtils;
+
+/**
+ * An implementation of {@link GroupMappingServiceProvider} which
+ * composites other group mapping providers for determining group membership.
+ * This allows to combine existing provider implementations and composite 
+ * a virtually new provider without customized development to deal with complex situation. 
+ */
+@InterfaceAudience.LimitedPrivate({"HDFS", "MapReduce"})
+@InterfaceStability.Evolving
+public class CompositeGroupsMapping
+    implements GroupMappingServiceProvider, Configurable {
+  
+  public static final String MAPPING_PROVIDERS_CONFIG_KEY = GROUP_MAPPING_CONFIG_PREFIX + ".providers";
+  public static final String MAPPING_PROVIDERS_COMBINED_CONFIG_KEY = MAPPING_PROVIDERS_CONFIG_KEY + ".combined";
+  public static final String MAPPING_PROVIDER_CONFIG_PREFIX = GROUP_MAPPING_CONFIG_PREFIX + ".provider";
+  
+  private static final Log LOG = LogFactory.getLog(CompositeGroupsMapping.class);
+
+  private List<GroupMappingServiceProvider> providersList = 
+		  new ArrayList<GroupMappingServiceProvider>();
+  
+  private Configuration conf;
+  private boolean combined;
+
+
+
+  /**
+   * Returns list of groups for a user.
+   * 
+   * @param user get groups for this user
+   * @return list of groups for a given user
+   */
+  @Override
+  public synchronized List<String> getGroups(String user) throws IOException {
+    Set<String> groupSet = new TreeSet<String>();
+
+    List<String> groups = null;
+    for (GroupMappingServiceProvider provider : providersList) {
+      try {
+        groups = provider.getGroups(user);
+      } catch (Exception e) {
+        //LOG.warn("Exception trying to get groups for user " + user, e);      
+      }        
+      if (groups != null && ! groups.isEmpty()) {
+        groupSet.addAll(groups);
+        if (!combined) break;
+      }
+    }
+
+    List<String> results = new ArrayList<String>(groupSet.size());
+    results.addAll(groupSet);
+    return results;
+  }
+  
+  /**
+   * Caches groups, no need to do that for this provider
+   */
+  @Override
+  public void cacheGroupsRefresh() throws IOException {
+    // does nothing in this provider of user to groups mapping
+  }
+
+  /** 
+   * Adds groups to cache, no need to do that for this provider
+   *
+   * @param groups unused
+   */
+  @Override
+  public void cacheGroupsAdd(List<String> groups) throws IOException {
+    // does nothing in this provider of user to groups mapping
+  }
+
+  @Override
+  public synchronized Configuration getConf() {
+    return conf;
+  }
+
+  @Override
+  public synchronized void setConf(Configuration conf) {
+    this.conf = conf;
+    
+    this.combined = conf.getBoolean(MAPPING_PROVIDERS_COMBINED_CONFIG_KEY, true);
+    
+    loadMappingProviders();
+  }
+  
+  private void loadMappingProviders() {
+    String[] providerNames = conf.getStrings(MAPPING_PROVIDERS_CONFIG_KEY, new String[]{});
+
+    String providerKey;
+    for (String name : providerNames) {
+      providerKey = MAPPING_PROVIDER_CONFIG_PREFIX + "." + name;
+      Class<?> providerClass = conf.getClass(providerKey, null);
+      if (providerClass == null) {
+        LOG.error("The mapping provider, " + name + " does not have a valid class");
+      } else {
+        addMappingProvider(name, providerClass);
+      }
+    }
+  }
+    
+  private void addMappingProvider(String providerName, Class<?> providerClass) {
+    Configuration newConf = prepareConf(providerName);
+    GroupMappingServiceProvider provider = 
+        (GroupMappingServiceProvider) ReflectionUtils.newInstance(providerClass, newConf);
+    providersList.add(provider);
+
+  }
+
+  /*
+   * For any provider specific configuration properties, such as "hadoop.security.group.mapping.ldap.url" 
+   * and the like, allow them to be configured as "hadoop.security.group.mapping.provider.PROVIDER-X.ldap.url",
+   * so that a provider such as LdapGroupsMapping can be used to composite a complex one with other providers.
+   */
+  private Configuration prepareConf(String providerName) {
+    Configuration newConf = new Configuration();
+    Iterator<Map.Entry<String, String>> entries = conf.iterator();
+    String providerKey = MAPPING_PROVIDER_CONFIG_PREFIX + "." + providerName;
+    while (entries.hasNext()) {
+      Map.Entry<String, String> entry = entries.next();
+      String key = entry.getKey();
+      // get a property like "hadoop.security.group.mapping.provider.PROVIDER-X.ldap.url"
+      if (key.startsWith(providerKey) && !key.equals(providerKey)) {
+        // restore to be the one like "hadoop.security.group.mapping.ldap.url" 
+        // so that can be used by original provider.
+        key = key.replace(".provider." + providerName, "");
+        newConf.set(key, entry.getValue());
+      }
+    }
+    return newConf;
+  }  
+}

+ 2 - 0
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/security/GroupMappingServiceProvider.java

@@ -22,6 +22,7 @@ import java.util.List;
 
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
+import org.apache.hadoop.fs.CommonConfigurationKeysPublic;
 
 /**
  * An interface for the implementation of a user-to-groups mapping service
@@ -30,6 +31,7 @@ import org.apache.hadoop.classification.InterfaceStability;
 @InterfaceAudience.Public
 @InterfaceStability.Evolving
 public interface GroupMappingServiceProvider {
+  public static final String GROUP_MAPPING_CONFIG_PREFIX = CommonConfigurationKeysPublic.HADOOP_SECURITY_GROUP_MAPPING;
   
   /**
    * Get all various group memberships of a given user.

+ 92 - 0
hadoop-common-project/hadoop-common/src/main/resources/core-default.xml

@@ -94,6 +94,98 @@
   </description>
 </property>
 
+<!-- 
+=== Multiple group mapping providers configuration sample === 
+  This sample illustrates a typical use case for CompositeGroupsMapping where
+Hadoop authentication uses MIT Kerberos which trusts an AD realm. In this case, service
+principals such as hdfs, mapred, hbase, hive, oozie and etc can be placed in In MIT Kerberos,
+but end users are just from the trusted AD. For the service principals, ShellBasedUnixGroupsMapping
+provider can be used to query their groups for efficiency, and for end users, LdapGroupsMapping 
+provider can be used. This avoids to add group entries in AD for service principals when only using 
+LdapGroupsMapping provider.
+  In case multiple ADs are involved and trusted by the MIT Kerberos in this use case, LdapGroupsMapping
+provider can be used more times with different AD specific configurations. This sample also shows how
+to do that. Here are the necessary configurations.
+
+<property>
+  <name>hadoop.security.group.mapping</name>
+  <value>org.apache.hadoop.security.CompositeGroupsMapping</value>
+  <description>
+    Class for user to group mapping (get groups for a given user) for ACL, which 
+    makes use of other multiple providers to provide the service.
+  </description>
+</property>
+
+<property>
+  <name>hadoop.security.group.mapping.providers</name>
+  <value>shell4services,ad4usersX,ad4usersY</value>
+  <description>
+    Comma separated of names of other providers to provide user to group mapping. 
+  </description>
+</property>
+
+<property>
+  <name>hadoop.security.group.mapping.providers.combined</name>
+  <value>true</value>
+  <description>
+    true or false to indicate whether groups from the providers are combined or not. The default value is true
+    If true, then all the providers will be tried to get groups and all the groups are combined to return as
+    the final results. Otherwise, providers are tried one by one in the configured list order, and if any
+    groups are retrieved from any provider, then the groups will be returned without trying the left ones.
+  </description>
+</property>
+
+<property>
+  <name>hadoop.security.group.mapping.provider.shell4services</name>
+  <value>org.apache.hadoop.security.ShellBasedUnixGroupsMapping</value>
+  <description>
+    Class for group mapping provider named by 'shell4services'. The name can then be referenced 
+    by hadoop.security.group.mapping.providers property.
+  </description>
+</property>
+
+<property>
+  <name>hadoop.security.group.mapping.provider.ad4usersX</name>
+  <value>org.apache.hadoop.security.LdapGroupsMapping</value>
+  <description>
+    Class for group mapping provider named by 'ad4usersX'. The name can then be referenced 
+    by hadoop.security.group.mapping.providers property.
+  </description>
+</property>
+
+<property>
+  <name>hadoop.security.group.mapping.provider.ad4usersY</name>
+  <value>org.apache.hadoop.security.LdapGroupsMapping</value>
+  <description>
+    Class for group mapping provider named by 'ad4usersY'. The name can then be referenced 
+    by hadoop.security.group.mapping.providers property.
+  </description>
+</property>
+
+<property>
+<name>hadoop.security.group.mapping.provider.ad4usersX.ldap.url</name>
+<value>ldap://ad-host-for-users-X:389</value>
+  <description>
+    ldap url for the provider named by 'ad4usersX'. Note this property comes from 
+    'hadoop.security.group.mapping.ldap.url'.
+  </description>
+</property>
+
+<property>
+<name>hadoop.security.group.mapping.provider.ad4usersY.ldap.url</name>
+<value>ldap://ad-host-for-users-Y:389</value>
+  <description>
+    ldap url for the provider named by 'ad4usersY'. Note this property comes from 
+    'hadoop.security.group.mapping.ldap.url'.
+  </description>
+</property>
+
+You also need to configure other properties like
+  hadoop.security.group.mapping.ldap.bind.password.file and etc.
+for ldap providers in the same way as above does.
+
+-->
+ 
 <property>
   <name>hadoop.security.groups.cache.secs</name>
   <value>300</value>

+ 185 - 0
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/security/TestCompositeGroupMapping.java

@@ -0,0 +1,185 @@
+/**
+ * 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.hadoop.security;
+
+import static org.junit.Assert.assertTrue;
+
+import java.io.IOException;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.List;
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.conf.Configurable;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.fs.CommonConfigurationKeys;
+import org.junit.Test;
+
+
+public class TestCompositeGroupMapping {
+  public static final Log LOG = LogFactory.getLog(TestCompositeGroupMapping.class);
+  private static Configuration conf = new Configuration();
+  
+  private static class TestUser {
+    String name;
+    String group;
+    String group2;
+    
+    public TestUser(String name, String group) {
+      this.name = name;
+      this.group = group;
+    }
+
+    public TestUser(String name, String group, String group2) {
+      this(name, group);
+      this.group2 = group2;
+    }
+  };
+  
+  private static TestUser john = new TestUser("John", "user-group");
+  private static TestUser hdfs = new TestUser("hdfs", "supergroup");
+  private static TestUser jack = new TestUser("Jack", "user-group", "dev-group-1");
+  
+  private static final String PROVIDER_SPECIFIC_CONF = ".test.prop";
+  private static final String PROVIDER_SPECIFIC_CONF_KEY = 
+      GroupMappingServiceProvider.GROUP_MAPPING_CONFIG_PREFIX + PROVIDER_SPECIFIC_CONF;
+  private static final String PROVIDER_SPECIFIC_CONF_VALUE_FOR_USER = "value-for-user";
+  private static final String PROVIDER_SPECIFIC_CONF_VALUE_FOR_CLUSTER = "value-for-cluster";
+  
+  private static abstract class GroupMappingProviderBase 
+    implements GroupMappingServiceProvider, Configurable {
+    
+    private Configuration conf;
+    
+    @Override
+    public void setConf(Configuration conf) {
+      this.conf = conf;
+    }
+
+    @Override
+    public Configuration getConf() {
+      return this.conf;
+    }
+
+    @Override
+    public void cacheGroupsRefresh() throws IOException {
+      
+    }
+
+    @Override
+    public void cacheGroupsAdd(List<String> groups) throws IOException {
+      
+    }
+    
+    protected List<String> toList(String group) {
+      if (group != null) {
+        return Arrays.asList(new String[] {group});
+      }
+      return new ArrayList<String>();
+    }
+    
+    protected void checkTestConf(String expectedValue) {
+      String configValue = getConf().get(PROVIDER_SPECIFIC_CONF_KEY);
+      if (configValue == null || !configValue.equals(expectedValue)) {
+        throw new RuntimeException("Failed to find mandatory configuration of " + PROVIDER_SPECIFIC_CONF_KEY);
+      }
+    }
+  };
+  
+  private static class UserProvider extends GroupMappingProviderBase {
+    @Override
+    public List<String> getGroups(String user) throws IOException {
+      checkTestConf(PROVIDER_SPECIFIC_CONF_VALUE_FOR_USER);
+      
+      String group = null;
+      if (user.equals(john.name)) {
+        group = john.group;
+      } else if (user.equals(jack.name)) {
+        group = jack.group;
+      }
+      
+      return toList(group);
+    }
+  }
+  
+  private static class ClusterProvider extends GroupMappingProviderBase {    
+    @Override
+    public List<String> getGroups(String user) throws IOException {
+      checkTestConf(PROVIDER_SPECIFIC_CONF_VALUE_FOR_CLUSTER);
+      
+      String group = null;
+      if (user.equals(hdfs.name)) {
+        group = hdfs.group;
+      } else if (user.equals(jack.name)) { // jack has another group from clusterProvider
+        group = jack.group2;
+      }
+      
+      return toList(group);
+    }
+  }
+  
+  static {
+    conf.setClass(CommonConfigurationKeys.HADOOP_SECURITY_GROUP_MAPPING,
+      CompositeGroupsMapping.class, GroupMappingServiceProvider.class);
+    conf.set(CompositeGroupsMapping.MAPPING_PROVIDERS_CONFIG_KEY, "userProvider,clusterProvider");
+
+    conf.setClass(CompositeGroupsMapping.MAPPING_PROVIDER_CONFIG_PREFIX + ".userProvider", 
+        UserProvider.class, GroupMappingServiceProvider.class);
+
+    conf.setClass(CompositeGroupsMapping.MAPPING_PROVIDER_CONFIG_PREFIX + ".clusterProvider", 
+        ClusterProvider.class, GroupMappingServiceProvider.class);
+
+    conf.set(CompositeGroupsMapping.MAPPING_PROVIDER_CONFIG_PREFIX + 
+        ".clusterProvider" + PROVIDER_SPECIFIC_CONF, PROVIDER_SPECIFIC_CONF_VALUE_FOR_CLUSTER);
+
+    conf.set(CompositeGroupsMapping.MAPPING_PROVIDER_CONFIG_PREFIX + 
+        ".userProvider" + PROVIDER_SPECIFIC_CONF, PROVIDER_SPECIFIC_CONF_VALUE_FOR_USER);
+  }
+
+  @Test
+  public void TestMultipleGroupsMapping() throws Exception {
+    Groups groups = new Groups(conf);
+
+    assertTrue(groups.getGroups(john.name).get(0).equals(john.group));
+    assertTrue(groups.getGroups(hdfs.name).get(0).equals(hdfs.group));
+  }
+
+  @Test
+  public void TestMultipleGroupsMappingWithCombined() throws Exception {
+    conf.set(CompositeGroupsMapping.MAPPING_PROVIDERS_COMBINED_CONFIG_KEY, "true");
+    Groups groups = new Groups(conf);
+
+    assertTrue(groups.getGroups(jack.name).size() == 2);
+    // the configured providers list in order is "userProvider,clusterProvider"
+    // group -> userProvider, group2 -> clusterProvider
+    assertTrue(groups.getGroups(jack.name).contains(jack.group));
+    assertTrue(groups.getGroups(jack.name).contains(jack.group2));
+  }
+
+  @Test
+  public void TestMultipleGroupsMappingWithoutCombined() throws Exception {
+    conf.set(CompositeGroupsMapping.MAPPING_PROVIDERS_COMBINED_CONFIG_KEY, "false");
+    Groups groups = new Groups(conf);
+
+    // the configured providers list in order is "userProvider,clusterProvider"
+    // group -> userProvider, group2 -> clusterProvider
+    assertTrue(groups.getGroups(jack.name).size() == 1);
+    assertTrue(groups.getGroups(jack.name).get(0).equals(jack.group));
+  }
+}