Quellcode durchsuchen

HDFS-1096. allow dfsadmin/mradmin refresh of superuser proxy group mappings

git-svn-id: https://svn.apache.org/repos/asf/hadoop/hdfs/trunk@951086 13f79535-47bb-0310-9956-ffa450edef68
Boris Shkolnik vor 15 Jahren
Ursprung
Commit
fe5c3e64f9

+ 3 - 0
CHANGES.txt

@@ -9,6 +9,9 @@ Trunk (unreleased changes)
 
   IMPROVEMENTS
 
+    HDFS-1096. allow dfsadmin/mradmin refresh of superuser proxy group
+     mappings (boryas)
+
     HDFS-1146. Javadoc for getDelegationTokenSecretManager in FSNamesystem (jnp via boryas)
 
     HDFS-1132. Refactor TestFileStatus (Eli Collins via cos)

+ 3 - 3
src/java/org/apache/hadoop/hdfs/HDFSPolicyProvider.java

@@ -22,7 +22,7 @@ import org.apache.hadoop.hdfs.protocol.ClientProtocol;
 import org.apache.hadoop.hdfs.server.protocol.DatanodeProtocol;
 import org.apache.hadoop.hdfs.server.protocol.InterDatanodeProtocol;
 import org.apache.hadoop.hdfs.server.protocol.NamenodeProtocol;
-import org.apache.hadoop.security.RefreshUserToGroupMappingsProtocol;
+import org.apache.hadoop.security.RefreshUserMappingsProtocol;
 import org.apache.hadoop.security.authorize.PolicyProvider;
 import org.apache.hadoop.security.authorize.RefreshAuthorizationPolicyProtocol;
 import org.apache.hadoop.security.authorize.Service;
@@ -42,8 +42,8 @@ public class HDFSPolicyProvider extends PolicyProvider {
     new Service("security.namenode.protocol.acl", NamenodeProtocol.class),
     new Service("security.refresh.policy.protocol.acl", 
                 RefreshAuthorizationPolicyProtocol.class),
-    new Service("security.refresh.usertogroups.mappings.protocol.acl", 
-                RefreshUserToGroupMappingsProtocol.class),
+    new Service("security.refresh.user.mappings.protocol.acl", 
+                RefreshUserMappingsProtocol.class),
   };
   
   @Override

+ 1 - 1
src/java/org/apache/hadoop/hdfs/protocol/ClientProtocol.java

@@ -599,7 +599,7 @@ public interface ClientProtocol extends VersionedProtocol {
    *                <li>2 get safe mode state.</li></ul>
    * @return <ul><li>0 if the safe mode is OFF or</li> 
    *         <li>1 if the safe mode is ON.</li></ul>
-   ¥
+   *                   
    * @throws IOException
    */
   public boolean setSafeMode(FSConstants.SafeModeAction action) 

+ 16 - 9
src/java/org/apache/hadoop/hdfs/server/namenode/NameNode.java

@@ -31,17 +31,19 @@ import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.ContentSummary;
 import org.apache.hadoop.fs.CreateFlag;
-import org.apache.hadoop.fs.FileContext;
 import org.apache.hadoop.fs.FileStatus;
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.FsServerDefaults;
+import org.apache.hadoop.fs.Options;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.fs.Trash;
-import org.apache.hadoop.fs.Options;
 import org.apache.hadoop.fs.UnresolvedLinkException;
 import org.apache.hadoop.fs.permission.FsPermission;
 import org.apache.hadoop.fs.permission.PermissionStatus;
+import org.apache.hadoop.hdfs.DFSConfigKeys;
+import org.apache.hadoop.hdfs.DFSUtil;
 import org.apache.hadoop.hdfs.HDFSPolicyProvider;
+import org.apache.hadoop.hdfs.HdfsConfiguration;
 import org.apache.hadoop.hdfs.protocol.Block;
 import org.apache.hadoop.hdfs.protocol.BlockListAsLongs;
 import org.apache.hadoop.hdfs.protocol.ClientProtocol;
@@ -55,6 +57,7 @@ import org.apache.hadoop.hdfs.protocol.LocatedBlocks;
 import org.apache.hadoop.hdfs.protocol.UnregisteredNodeException;
 import org.apache.hadoop.hdfs.protocol.UnresolvedPathException;
 import org.apache.hadoop.hdfs.security.token.block.ExportedBlockKeys;
+import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier;
 import org.apache.hadoop.hdfs.server.common.IncorrectVersionException;
 import org.apache.hadoop.hdfs.server.common.UpgradeStatusReport;
 import org.apache.hadoop.hdfs.server.common.HdfsConstants.NamenodeRole;
@@ -71,9 +74,6 @@ import org.apache.hadoop.hdfs.server.protocol.NamenodeRegistration;
 import org.apache.hadoop.hdfs.server.protocol.NamespaceInfo;
 import org.apache.hadoop.hdfs.server.protocol.NodeRegistration;
 import org.apache.hadoop.hdfs.server.protocol.UpgradeCommand;
-import org.apache.hadoop.hdfs.DFSConfigKeys;
-import org.apache.hadoop.hdfs.DFSUtil;
-import org.apache.hadoop.hdfs.HdfsConfiguration;
 import org.apache.hadoop.http.HttpServer;
 import org.apache.hadoop.io.EnumSetWritable;
 import org.apache.hadoop.io.Text;
@@ -84,14 +84,14 @@ import org.apache.hadoop.net.NetworkTopology;
 import org.apache.hadoop.net.Node;
 import org.apache.hadoop.security.AccessControlException;
 import org.apache.hadoop.security.Groups;
-import org.apache.hadoop.security.RefreshUserToGroupMappingsProtocol;
+import org.apache.hadoop.security.RefreshUserMappingsProtocol;
 import org.apache.hadoop.security.UserGroupInformation;
 import org.apache.hadoop.security.authorize.AuthorizationException;
+import org.apache.hadoop.security.authorize.ProxyUsers;
 import org.apache.hadoop.security.authorize.RefreshAuthorizationPolicyProtocol;
 import org.apache.hadoop.security.authorize.ServiceAuthorizationManager;
 import org.apache.hadoop.security.token.Token;
 import org.apache.hadoop.security.token.SecretManager.InvalidToken;
-import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier;
 import org.apache.hadoop.util.ServicePlugin;
 import org.apache.hadoop.util.StringUtils;
 
@@ -145,8 +145,8 @@ public class NameNode implements NamenodeProtocols, FSConstants {
       return NamenodeProtocol.versionID;
     } else if (protocol.equals(RefreshAuthorizationPolicyProtocol.class.getName())){
       return RefreshAuthorizationPolicyProtocol.versionID;
-    } else if (protocol.equals(RefreshUserToGroupMappingsProtocol.class.getName())){
-      return RefreshUserToGroupMappingsProtocol.versionID;
+    } else if (protocol.equals(RefreshUserMappingsProtocol.class.getName())){
+      return RefreshUserMappingsProtocol.versionID;
     } else {
       throw new IOException("Unknown protocol to name node: " + protocol);
     }
@@ -1272,6 +1272,13 @@ public class NameNode implements NamenodeProtocols, FSConstants {
     Groups.getUserToGroupsMappingService(conf).refresh();
   }
 
+  @Override
+  public void refreshSuperUserGroupsConfiguration(Configuration conf) {
+    LOG.info("Refreshing SuperUser proxy group mapping list ");
+
+    ProxyUsers.refreshSuperUserGroupsConfiguration(conf);
+  }
+
   private static void printUsage() {
     System.err.println(
       "Usage: java NameNode [" +

+ 2 - 2
src/java/org/apache/hadoop/hdfs/server/protocol/NamenodeProtocols.java

@@ -20,7 +20,7 @@ package org.apache.hadoop.hdfs.server.protocol;
 
 import org.apache.hadoop.hdfs.protocol.ClientProtocol;
 import org.apache.hadoop.security.authorize.RefreshAuthorizationPolicyProtocol;
-import org.apache.hadoop.security.RefreshUserToGroupMappingsProtocol;
+import org.apache.hadoop.security.RefreshUserMappingsProtocol;
 
 /** The full set of RPC methods implemented by the Namenode.  */
 public interface NamenodeProtocols
@@ -28,5 +28,5 @@ public interface NamenodeProtocols
           DatanodeProtocol,
           NamenodeProtocol,
           RefreshAuthorizationPolicyProtocol,
-          RefreshUserToGroupMappingsProtocol {
+          RefreshUserMappingsProtocol {
 }

+ 54 - 9
src/java/org/apache/hadoop/hdfs/tools/DFSAdmin.java

@@ -44,7 +44,7 @@ import org.apache.hadoop.hdfs.server.namenode.NameNode;
 import org.apache.hadoop.ipc.RPC;
 import org.apache.hadoop.ipc.RemoteException;
 import org.apache.hadoop.net.NetUtils;
-import org.apache.hadoop.security.RefreshUserToGroupMappingsProtocol;
+import org.apache.hadoop.security.RefreshUserMappingsProtocol;
 import org.apache.hadoop.security.UserGroupInformation;
 import org.apache.hadoop.security.authorize.RefreshAuthorizationPolicyProtocol;
 import org.apache.hadoop.util.StringUtils;
@@ -475,6 +475,7 @@ public class DFSAdmin extends FsShell {
       "\t[" + ClearSpaceQuotaCommand.USAGE +"]\n" +
       "\t[-refreshServiceAcl]\n" +
       "\t[-refreshUserToGroupsMappings]\n" +
+      "\t[refreshSuperUserGroupsConfiguration]\n" +
       "\t[-printTopology]\n" +
       "\t[-help [cmd]]\n";
 
@@ -532,6 +533,9 @@ public class DFSAdmin extends FsShell {
     String refreshUserToGroupsMappings = 
       "-refreshUserToGroupsMappings: Refresh user-to-groups mappings\n";
     
+    String refreshSuperUserGroupsConfiguration = 
+      "-refreshSuperUserGroupsConfiguration: Refresh superuser proxy groups mappings\n";
+
     String printTopology = "-printTopology: Print a tree of the racks and their\n" +
                            "\t\tnodes as reported by the Namenode\n";
     
@@ -566,6 +570,8 @@ public class DFSAdmin extends FsShell {
       System.out.println(refreshServiceAcl);
     } else if ("refreshUserToGroupsMappings".equals(cmd)) {
       System.out.println(refreshUserToGroupsMappings);
+    } else if ("refreshSuperUserGroupsConfiguration".equals(cmd)) {
+      System.out.println(refreshSuperUserGroupsConfiguration);
     } else if ("printTopology".equals(cmd)) {
       System.out.println(printTopology);
     } else if ("help".equals(cmd)) {
@@ -585,6 +591,8 @@ public class DFSAdmin extends FsShell {
       System.out.println(SetSpaceQuotaCommand.DESCRIPTION);
       System.out.println(ClearSpaceQuotaCommand.DESCRIPTION);
       System.out.println(refreshServiceAcl);
+      System.out.println(refreshUserToGroupsMappings);
+      System.out.println(refreshSuperUserGroupsConfiguration);
       System.out.println(printTopology);
       System.out.println(help);
       System.out.println();
@@ -767,20 +775,51 @@ public class DFSAdmin extends FsShell {
         conf.get(DFSConfigKeys.DFS_NAMENODE_USER_NAME_KEY, ""));
  
     // Create the client
-    RefreshUserToGroupMappingsProtocol refreshProtocol = 
-      (RefreshUserToGroupMappingsProtocol) 
-      RPC.getProxy(RefreshUserToGroupMappingsProtocol.class, 
-                   RefreshUserToGroupMappingsProtocol.versionID, 
-                   NameNode.getAddress(conf), getUGI(), conf,
-                   NetUtils.getSocketFactory(conf, 
-                                             RefreshUserToGroupMappingsProtocol.class));
-    
+    RefreshUserMappingsProtocol refreshProtocol = 
+      (RefreshUserMappingsProtocol) 
+      RPC.getProxy(RefreshUserMappingsProtocol.class, 
+          RefreshUserMappingsProtocol.versionID, 
+          NameNode.getAddress(conf), getUGI(), conf,
+          NetUtils.getSocketFactory(conf, 
+              RefreshUserMappingsProtocol.class));
+
     // Refresh the user-to-groups mappings
     refreshProtocol.refreshUserToGroupsMappings(conf);
     
     return 0;
   }
   
+
+  /**
+   * refreshSuperUserGroupsConfiguration {@link NameNode}.
+   * @return exitcode 0 on success, non-zero on failure
+   * @throws IOException
+   */
+  public int refreshSuperUserGroupsConfiguration() throws IOException {
+    // Get the current configuration
+    Configuration conf = getConf();
+
+    // for security authorization
+    // server principal for this call 
+    // should be NAMENODE's one.
+    conf.set(CommonConfigurationKeys.HADOOP_SECURITY_SERVICE_USER_NAME_KEY, 
+        conf.get(DFSConfigKeys.DFS_NAMENODE_USER_NAME_KEY, ""));
+
+    // Create the client
+    RefreshUserMappingsProtocol refreshProtocol = 
+      (RefreshUserMappingsProtocol) 
+      RPC.getProxy(RefreshUserMappingsProtocol.class, 
+          RefreshUserMappingsProtocol.versionID, 
+          NameNode.getAddress(conf), getUGI(), conf,
+          NetUtils.getSocketFactory(conf, 
+              RefreshUserMappingsProtocol.class));
+
+    // Refresh the user-to-groups mappings
+    refreshProtocol.refreshSuperUserGroupsConfiguration(conf);
+
+    return 0;
+  }
+
   /**
    * Displays format of commands.
    * @param cmd The command that is being executed.
@@ -828,6 +867,9 @@ public class DFSAdmin extends FsShell {
     } else if ("-refreshUserToGroupsMappings".equals(cmd)) {
       System.err.println("Usage: java DFSAdmin"
                          + " [-refreshUserToGroupsMappings]");
+    } else if ("-refreshSuperUserGroupsConfiguration".equals(cmd)) {
+      System.err.println("Usage: java DFSAdmin"
+                         + " [-refreshSuperUserGroupsConfiguration]");
     } else if ("-printTopology".equals(cmd)) {
       System.err.println("Usage: java DFSAdmin"
                          + " [-printTopology]");
@@ -843,6 +885,7 @@ public class DFSAdmin extends FsShell {
       System.err.println("           [-metasave filename]");
       System.err.println("           [-refreshServiceAcl]");
       System.err.println("           [-refreshUserToGroupsMappings]");
+      System.err.println("           [-refreshSuperUserGroupsConfiguration]");
       System.err.println("           [-printTopology]");
       System.err.println("           ["+SetQuotaCommand.USAGE+"]");
       System.err.println("           ["+ClearQuotaCommand.USAGE+"]");
@@ -973,6 +1016,8 @@ public class DFSAdmin extends FsShell {
         exitCode = refreshServiceAcl();
       } else if ("-refreshUserToGroupsMappings".equals(cmd)) {
         exitCode = refreshUserToGroupsMappings();
+      } else if ("-refreshSuperUserGroupsConfiguration".equals(cmd)) {
+        exitCode = refreshSuperUserGroupsConfiguration();
       } else if ("-printTopology".equals(cmd)) {
         exitCode = printTopology();
       } else if ("-help".equals(cmd)) {

+ 92 - 13
src/test/hdfs/org/apache/hadoop/security/TestGroupMappingServiceRefresh.java → src/test/hdfs/org/apache/hadoop/security/TestRefreshUserMappings.java

@@ -21,6 +21,9 @@ package org.apache.hadoop.security;
 
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.fail;
+import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.when;
 
 import java.io.IOException;
 import java.util.ArrayList;
@@ -28,17 +31,18 @@ import java.util.Arrays;
 import java.util.List;
 
 import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.fs.CommonConfigurationKeys;
 import org.apache.hadoop.fs.FileSystem;
-import org.apache.hadoop.hdfs.HdfsConfiguration;
 import org.apache.hadoop.hdfs.MiniDFSCluster;
+import org.apache.hadoop.hdfs.server.namenode.NameNode;
 import org.apache.hadoop.hdfs.tools.DFSAdmin;
+import org.apache.hadoop.security.authorize.AuthorizationException;
+import org.apache.hadoop.security.authorize.ProxyUsers;
 import org.junit.After;
 import org.junit.Before;
 import org.junit.Test;
-import org.apache.hadoop.security.GroupMappingServiceProvider;
 
-public class TestGroupMappingServiceRefresh {
+
+public class TestRefreshUserMappings {
   private MiniDFSCluster cluster;
   Configuration config;
   private static long groupRefreshTimeoutSec = 1;
@@ -48,7 +52,7 @@ public class TestGroupMappingServiceRefresh {
     
     @Override
     public List<String> getGroups(String user) throws IOException {
-      System.err.println("Getting groups in MockUnixGroupsMapping");
+      System.out.println("Getting groups in MockUnixGroupsMapping");
       String g1 = user + (10 * i + 1);
       String g2 = user + (10 * i + 2);
       List<String> l = new ArrayList<String>(2);
@@ -61,14 +65,13 @@ public class TestGroupMappingServiceRefresh {
   
   @Before
   public void setUp() throws Exception {
-    config = new HdfsConfiguration();
-    config.setClass(CommonConfigurationKeys.HADOOP_SECURITY_GROUP_MAPPING,
-        TestGroupMappingServiceRefresh.MockUnixGroupsMapping.class,
+    config = new Configuration();
+    config.setClass("hadoop.security.group.mapping",
+        TestRefreshUserMappings.MockUnixGroupsMapping.class,
         GroupMappingServiceProvider.class);
-    config.setLong(CommonConfigurationKeys.HADOOP_SECURITY_GROUPS_CACHE_SECS, 
-        groupRefreshTimeoutSec);
-    
+    config.setLong("hadoop.security.groups.cache.secs", groupRefreshTimeoutSec);
     Groups.getUserToGroupsMappingService(config);
+    
     FileSystem.setDefaultUri(config, "hdfs://localhost:" + "0");
     cluster = new MiniDFSCluster(0, config, 1, true, true, true,  null, null, null, null);
     cluster.waitActive();
@@ -80,13 +83,13 @@ public class TestGroupMappingServiceRefresh {
       cluster.shutdown();
     }
   }
-  
+    
   @Test
   public void testGroupMappingRefresh() throws Exception {
     DFSAdmin admin = new DFSAdmin(config);
     String [] args =  new String[]{"-refreshUserToGroupsMappings"};
     Groups groups = Groups.getUserToGroupsMappingService(config);
-    String user = UserGroupInformation.getCurrentUser().getShortUserName();
+    String user = UserGroupInformation.getCurrentUser().getUserName();
     System.out.println("first attempt:");
     List<String> g1 = groups.getGroups(user);
     String [] str_groups = new String [g1.size()];
@@ -120,4 +123,80 @@ public class TestGroupMappingServiceRefresh {
       assertFalse("Should be different group ", g3.get(i).equals(g4.get(i)));
     }
   }
+  
+  @Test
+  public void testRefreshSuperUserGroupsConfiguration() throws Exception {
+    final String SUPER_USER = "super_user";
+    final String [] GROUP_NAMES1 = new String [] {"gr1" , "gr2"};
+    final String [] GROUP_NAMES2 = new String [] {"gr3" , "gr4"};
+    
+    //keys in conf
+    String userKeyGroups = ProxyUsers.getProxySuperuserGroupConfKey(SUPER_USER);
+    String userKeyHosts = ProxyUsers.getProxySuperuserIpConfKey (SUPER_USER);
+    
+    config.set(userKeyGroups, "gr3,gr4,gr5"); // superuser can proxy for this group
+    config.set(userKeyHosts,"127.0.0.1");
+    
+    UserGroupInformation ugi1 = mock(UserGroupInformation.class);
+    UserGroupInformation ugi2 = mock(UserGroupInformation.class);
+    UserGroupInformation suUgi = mock(UserGroupInformation.class);
+    when(ugi1.getRealUser()).thenReturn(suUgi);
+    when(ugi2.getRealUser()).thenReturn(suUgi);
+
+    when(suUgi.getShortUserName()).thenReturn(SUPER_USER); // super user
+    when(suUgi.getUserName()).thenReturn(SUPER_USER+"L"); // super user
+     
+    when(ugi1.getShortUserName()).thenReturn("user1");
+    when(ugi2.getShortUserName()).thenReturn("user2");
+    
+    when(ugi1.getUserName()).thenReturn("userL1");
+    when(ugi2.getUserName()).thenReturn("userL2");
+   
+    // set groups for users
+    when(ugi1.getGroupNames()).thenReturn(GROUP_NAMES1);
+    when(ugi2.getGroupNames()).thenReturn(GROUP_NAMES2);
+   
+    
+    // check before
+    try {
+      ProxyUsers.authorize(ugi1, "127.0.0.1", config);
+      fail("first auth for " + ugi1.getShortUserName() + " should've failed ");
+    } catch (AuthorizationException e) {
+      // expected
+      System.err.println("auth for " + ugi1.getUserName() + " failed");
+    }
+    try {
+      ProxyUsers.authorize(ugi2, "127.0.0.1", config);
+      System.err.println("auth for " + ugi2.getUserName() + " succeeded");
+      // expected
+    } catch (AuthorizationException e) {
+      fail("first auth for " + ugi2.getShortUserName() + " should've succeeded: " + e.getLocalizedMessage());
+    }
+    
+    DFSAdmin admin = new DFSAdmin(config);
+    String [] args = new String[]{"-refreshSuperUserGroupsConfiguration"};
+    NameNode nn = cluster.getNameNode();
+    Configuration conf = new Configuration(config);
+    conf.set(userKeyGroups, "gr2"); // superuser can proxy for this group
+    admin.setConf(conf);
+    admin.run(args);
+    
+    //check after...
+    
+    try {
+      ProxyUsers.authorize(ugi2, "127.0.0.1", config);
+      fail("second auth for " + ugi2.getShortUserName() + " should've failed ");
+    } catch (AuthorizationException e) {
+      // expected
+      System.err.println("auth for " + ugi2.getUserName() + " failed");
+    }
+    try {
+      ProxyUsers.authorize(ugi1, "127.0.0.1", config);
+      System.err.println("auth for " + ugi1.getUserName() + " succeeded");
+      // expected
+    } catch (AuthorizationException e) {
+      fail("second auth for " + ugi1.getShortUserName() + " should've succeeded: " + e.getLocalizedMessage());
+    }    
+  }
+
 }