Browse Source

svn merge -r 1359774:1359777 FIXES: HADOOP-8525. Provide Improved Traceability for Configuration (bobby)

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/branch-0.23@1359789 13f79535-47bb-0310-9956-ffa450edef68
Robert Joseph Evans 13 years ago
parent
commit
0b4d88b493

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

@@ -23,6 +23,8 @@ Release 0.23.3 - UNRELEASED
     HADOOP-8350. Improve NetUtils.getInputStream to return a stream which has
     a tunable timeout. (todd)
 
+    HADOOP-8525. Provide Improved Traceability for Configuration (bobby)
+
   OPTIMIZATIONS
 
   BUG FIXES

+ 179 - 50
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/conf/Configuration.java

@@ -33,12 +33,14 @@ import java.io.Writer;
 import java.net.InetSocketAddress;
 import java.net.URL;
 import java.util.ArrayList;
+import java.util.Arrays;
 import java.util.Collection;
 import java.util.Collections;
 import java.util.Enumeration;
 import java.util.HashMap;
 import java.util.HashSet;
 import java.util.Iterator;
+import java.util.LinkedList;
 import java.util.List;
 import java.util.ListIterator;
 import java.util.Map;
@@ -74,7 +76,6 @@ import org.apache.hadoop.util.ReflectionUtils;
 import org.apache.hadoop.util.StringUtils;
 import org.codehaus.jackson.JsonFactory;
 import org.codehaus.jackson.JsonGenerator;
-import org.w3c.dom.Comment;
 import org.w3c.dom.DOMException;
 import org.w3c.dom.Document;
 import org.w3c.dom.Element;
@@ -157,17 +158,45 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
 
   private boolean quietmode = true;
   
+  private static class Resource {
+    private final Object resource;
+    private final String name;
+    
+    public Resource(Object resource) {
+      this(resource, resource.toString());
+    }
+    
+    public Resource(Object resource, String name) {
+      this.resource = resource;
+      this.name = name;
+    }
+    
+    public String getName(){
+      return name;
+    }
+    
+    public Object getResource() {
+      return resource;
+    }
+    
+    @Override
+    public String toString() {
+      return name;
+    }
+  }
+  
   /**
    * List of configuration resources.
    */
-  private ArrayList<Object> resources = new ArrayList<Object>();
-
+  private ArrayList<Resource> resources = new ArrayList<Resource>();
+  
   /**
    * The value reported as the setting resource when a key is set
-   * by code rather than a file resource.
+   * by code rather than a file resource by dumpConfiguration.
    */
   static final String UNKNOWN_RESOURCE = "Unknown";
 
+
   /**
    * List of configuration parameters marked <b>final</b>. 
    */
@@ -195,7 +224,7 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
    * Stores the mapping of key to the resource which modifies or loads 
    * the key most recently
    */
-  private HashMap<String, String> updatingResource;
+  private HashMap<String, String[]> updatingResource;
  
   /**
    * Class to keep the information about the keys which replace the deprecated
@@ -406,7 +435,7 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
    */
   public Configuration(boolean loadDefaults) {
     this.loadDefaults = loadDefaults;
-    updatingResource = new HashMap<String, String>();
+    updatingResource = new HashMap<String, String[]>();
     synchronized(Configuration.class) {
       REGISTRY.put(this, null);
     }
@@ -419,7 +448,7 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
    */
   @SuppressWarnings("unchecked")
   public Configuration(Configuration other) {
-   this.resources = (ArrayList)other.resources.clone();
+   this.resources = (ArrayList<Resource>) other.resources.clone();
    synchronized(other) {
      if (other.properties != null) {
        this.properties = (Properties)other.properties.clone();
@@ -429,7 +458,7 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
        this.overlay = (Properties)other.overlay.clone();
      }
 
-     this.updatingResource = new HashMap<String, String>(other.updatingResource);
+     this.updatingResource = new HashMap<String, String[]>(other.updatingResource);
    }
    
     this.finalParameters = new HashSet<String>(other.finalParameters);
@@ -467,7 +496,7 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
    *             with that name.
    */
   public void addResource(String name) {
-    addResourceObject(name);
+    addResourceObject(new Resource(name));
   }
 
   /**
@@ -481,7 +510,7 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
    *            the classpath.
    */
   public void addResource(URL url) {
-    addResourceObject(url);
+    addResourceObject(new Resource(url));
   }
 
   /**
@@ -495,7 +524,7 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
    *             the classpath.
    */
   public void addResource(Path file) {
-    addResourceObject(file);
+    addResourceObject(new Resource(file));
   }
 
   /**
@@ -507,7 +536,21 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
    * @param in InputStream to deserialize the object from. 
    */
   public void addResource(InputStream in) {
-    addResourceObject(in);
+    addResourceObject(new Resource(in));
+  }
+
+  /**
+   * Add a configuration resource. 
+   * 
+   * The properties of this resource will override properties of previously 
+   * added resources, unless they were marked <a href="#Final">final</a>. 
+   * 
+   * @param in InputStream to deserialize the object from.
+   * @param name the name of the resource because InputStream.toString is not
+   * very descriptive some times.  
+   */
+  public void addResource(InputStream in, String name) {
+    addResourceObject(new Resource(in, name));
   }
   
   
@@ -524,7 +567,7 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
     finalParameters.clear();                      // clear site-limits
   }
   
-  private synchronized void addResourceObject(Object resource) {
+  private synchronized void addResourceObject(Resource resource) {
     resources.add(resource);                      // add to resources
     reloadConfiguration();
   }
@@ -628,20 +671,42 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
    * @param value property value.
    */
   public void set(String name, String value) {
+    set(name, value, null);
+  }
+  
+  /** 
+   * Set the <code>value</code> of the <code>name</code> property. If 
+   * <code>name</code> is deprecated or there is a deprecated name associated to it,
+   * it sets the value to both names.
+   * 
+   * @param name property name.
+   * @param value property value.
+   * @param source the place that this configuration value came from 
+   * (For debugging).
+   */
+  public void set(String name, String value, String source) {
     if (deprecatedKeyMap.isEmpty()) {
       getProps();
     }
     if (!isDeprecated(name)) {
       getOverlay().setProperty(name, value);
       getProps().setProperty(name, value);
-      updatingResource.put(name, UNKNOWN_RESOURCE);
+      if(source == null) {
+        updatingResource.put(name, new String[] {"programatically"});
+      } else {
+        updatingResource.put(name, new String[] {source});
+      }
     }
     else {
       DeprecatedKeyInfo keyInfo = deprecatedKeyMap.get(name);
+      String altSource = "because " + name + " is deprecated";
       LOG.warn(keyInfo.getWarningMessage(name));
       for (String newKey : keyInfo.newKeys) {
-        getOverlay().setProperty(newKey, value);
-        getProps().setProperty(newKey, value);
+        if(!newKey.equals(name)) {
+          getOverlay().setProperty(newKey, value);
+          getProps().setProperty(newKey, value);
+          updatingResource.put(newKey, new String[] {altSource});
+        }
       }
     }
   }
@@ -933,6 +998,43 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
     }
   }
 
+  /**
+   * Gets information about why a property was set.  Typically this is the 
+   * path to the resource objects (file, URL, etc.) the property came from, but
+   * it can also indicate that it was set programatically, or because of the
+   * command line.
+   *
+   * @param name - The property name to get the source of.
+   * @return null - If the property or its source wasn't found. Otherwise, 
+   * returns a list of the sources of the resource.  The older sources are
+   * the first ones in the list.  So for example if a configuration is set from
+   * the command line, and then written out to a file that is read back in the
+   * first entry would indicate that it was set from the command line, while
+   * the second one would indicate the file that the new configuration was read
+   * in from.
+   */
+  @InterfaceStability.Unstable
+  public synchronized String[] getPropertySources(String name) {
+    if (properties == null) {
+      // If properties is null, it means a resource was newly added
+      // but the props were cleared so as to load it upon future
+      // requests. So lets force a load by asking a properties list.
+      getProps();
+    }
+    // Return a null right away if our properties still
+    // haven't loaded or the resource mapping isn't defined
+    if (properties == null || updatingResource == null) {
+      return null;
+    } else {
+      String[] source = updatingResource.get(name);
+      if(source == null) {
+        return null;
+      } else {
+        return Arrays.copyOf(source, source.length);
+      }
+    }
+  }
+
   /**
    * A class that represents a set of positive integer ranges. It parses 
    * strings of the form: "2-3,5,7-" where ranges are separated by comma and 
@@ -1531,11 +1633,14 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
   protected synchronized Properties getProps() {
     if (properties == null) {
       properties = new Properties();
+      HashMap<String, String[]> backup = 
+        new HashMap<String, String[]>(updatingResource);
       loadResources(properties, resources, quietmode);
       if (overlay!= null) {
         properties.putAll(overlay);
         for (Map.Entry<Object,Object> item: overlay.entrySet()) {
-          updatingResource.put((String) item.getKey(), UNKNOWN_RESOURCE);
+          String key = (String)item.getKey();
+          updatingResource.put(key, backup.get(key));
         }
       }
     }
@@ -1581,25 +1686,25 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
   }
 
   private void loadResources(Properties properties,
-                             ArrayList resources,
+                             ArrayList<Resource> resources,
                              boolean quiet) {
     if(loadDefaults) {
       for (String resource : defaultResources) {
-        loadResource(properties, resource, quiet);
+        loadResource(properties, new Resource(resource), quiet);
       }
     
       //support the hadoop-site.xml as a deprecated case
       if(getResource("hadoop-site.xml")!=null) {
-        loadResource(properties, "hadoop-site.xml", quiet);
+        loadResource(properties, new Resource("hadoop-site.xml"), quiet);
       }
     }
     
-    for (Object resource : resources) {
+    for (Resource resource : resources) {
       loadResource(properties, resource, quiet);
     }
   }
   
-  private void loadResource(Properties properties, Object name, boolean quiet) {
+  private void loadResource(Properties properties, Resource wrapper, boolean quiet) {
     try {
       DocumentBuilderFactory docBuilderFactory 
         = DocumentBuilderFactory.newInstance();
@@ -1620,26 +1725,29 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
       Document doc = null;
       Element root = null;
 
-      if (name instanceof URL) {                  // an URL resource
-        URL url = (URL)name;
+      Object resource = wrapper.getResource();
+      String name = wrapper.getName();
+      
+      if (resource instanceof URL) {                  // an URL resource
+        URL url = (URL)resource;
         if (url != null) {
           if (!quiet) {
             LOG.info("parsing " + url);
           }
           doc = builder.parse(url.toString());
         }
-      } else if (name instanceof String) {        // a CLASSPATH resource
-        URL url = getResource((String)name);
+      } else if (resource instanceof String) {        // a CLASSPATH resource
+        URL url = getResource((String)resource);
         if (url != null) {
           if (!quiet) {
             LOG.info("parsing " + url);
           }
           doc = builder.parse(url.toString());
         }
-      } else if (name instanceof Path) {          // a file resource
+      } else if (resource instanceof Path) {          // a file resource
         // Can't use FileSystem API or we get an infinite loop
         // since FileSystem uses Configuration API.  Use java.io.File instead.
-        File file = new File(((Path)name).toUri().getPath())
+        File file = new File(((Path)resource).toUri().getPath())
           .getAbsoluteFile();
         if (file.exists()) {
           if (!quiet) {
@@ -1652,20 +1760,20 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
             in.close();
           }
         }
-      } else if (name instanceof InputStream) {
+      } else if (resource instanceof InputStream) {
         try {
-          doc = builder.parse((InputStream)name);
+          doc = builder.parse((InputStream)resource);
         } finally {
-          ((InputStream)name).close();
+          ((InputStream)resource).close();
         }
-      } else if (name instanceof Element) {
-        root = (Element)name;
+      } else if (resource instanceof Element) {
+        root = (Element)resource;
       }
 
       if (doc == null && root == null) {
         if (quiet)
           return;
-        throw new RuntimeException(name + " not found");
+        throw new RuntimeException(resource + " not found");
       }
 
       if (root == null) {
@@ -1680,7 +1788,7 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
           continue;
         Element prop = (Element)propNode;
         if ("configuration".equals(prop.getTagName())) {
-          loadResource(properties, prop, quiet);
+          loadResource(properties, new Resource(prop, name), quiet);
           continue;
         }
         if (!"property".equals(prop.getTagName()))
@@ -1689,6 +1797,7 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
         String attr = null;
         String value = null;
         boolean finalParameter = false;
+        LinkedList<String> source = new LinkedList<String>();
         for (int j = 0; j < fields.getLength(); j++) {
           Node fieldNode = fields.item(j);
           if (!(fieldNode instanceof Element))
@@ -1700,7 +1809,10 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
             value = ((Text)field.getFirstChild()).getData();
           if ("final".equals(field.getTagName()) && field.hasChildNodes())
             finalParameter = "true".equals(((Text)field.getFirstChild()).getData());
+          if ("source".equals(field.getTagName()) && field.hasChildNodes())
+            source.add(((Text)field.getFirstChild()).getData());
         }
+        source.add(name);
         
         // Ignore this parameter if it has already been marked as 'final'
         if (attr != null) {
@@ -1709,11 +1821,13 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
             keyInfo.accessed = false;
             for (String key:keyInfo.newKeys) {
               // update new keys with deprecated key's value 
-              loadProperty(properties, name, key, value, finalParameter);
+              loadProperty(properties, name, key, value, finalParameter, 
+                  source.toArray(new String[source.size()]));
             }
           }
           else {
-            loadProperty(properties, name, attr, value, finalParameter);
+            loadProperty(properties, name, attr, value, finalParameter, 
+                source.toArray(new String[source.size()]));
           }
         }
       }
@@ -1733,12 +1847,12 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
     }
   }
 
-  private void loadProperty(Properties properties, Object name, String attr,
-      String value, boolean finalParameter) {
+  private void loadProperty(Properties properties, String name, String attr,
+      String value, boolean finalParameter, String[] source) {
     if (value != null) {
       if (!finalParameters.contains(attr)) {
         properties.setProperty(attr, value);
-        updatingResource.put(attr, name.toString());
+        updatingResource.put(attr, source);
       } else {
         LOG.warn(name+":an attempt to override final parameter: "+attr
             +";  Ignoring.");
@@ -1810,11 +1924,6 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
       Element propNode = doc.createElement("property");
       conf.appendChild(propNode);
 
-      if (updatingResource != null) {
-        Comment commentNode = doc.createComment(
-          "Loaded from " + updatingResource.get(name));
-        propNode.appendChild(commentNode);
-      }
       Element nameNode = doc.createElement("name");
       nameNode.appendChild(doc.createTextNode(name));
       propNode.appendChild(nameNode);
@@ -1823,6 +1932,17 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
       valueNode.appendChild(doc.createTextNode(value));
       propNode.appendChild(valueNode);
 
+      if (updatingResource != null) {
+        String[] sources = updatingResource.get(name);
+        if(sources != null) {
+          for(String s : sources) {
+            Element sourceNode = doc.createElement("source");
+            sourceNode.appendChild(doc.createTextNode(s));
+            propNode.appendChild(sourceNode);
+          }
+        }
+      }
+      
       conf.appendChild(doc.createTextNode("\n"));
     }
     return doc;
@@ -1855,8 +1975,12 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
                                        config.get((String) item.getKey()));
         dumpGenerator.writeBooleanField("isFinal",
                                         config.finalParameters.contains(item.getKey()));
-        dumpGenerator.writeStringField("resource",
-                                       config.updatingResource.get(item.getKey()));
+        String[] resources = config.updatingResource.get(item.getKey());
+        String resource = UNKNOWN_RESOURCE;
+        if(resources != null && resources.length > 0) {
+          resource = resources[0];
+        }
+        dumpGenerator.writeStringField("resource", resource);
         dumpGenerator.writeEndObject();
       }
     }
@@ -1896,7 +2020,7 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
     toString(resources, sb);
     return sb.toString();
   }
-
+  
   private <T> void toString(List<T> resources, StringBuilder sb) {
     ListIterator<T> i = resources.listIterator();
     while (i.hasNext()) {
@@ -1933,8 +2057,11 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
     clear();
     int size = WritableUtils.readVInt(in);
     for(int i=0; i < size; ++i) {
-      set(org.apache.hadoop.io.Text.readString(in), 
-          org.apache.hadoop.io.Text.readString(in));
+      String key = org.apache.hadoop.io.Text.readString(in);
+      String value = org.apache.hadoop.io.Text.readString(in);
+      set(key, value); 
+      String sources[] = WritableUtils.readCompressedStringArray(in);
+      updatingResource.put(key, sources);
     }
   }
 
@@ -1945,6 +2072,8 @@ public class Configuration implements Iterable<Map.Entry<String,String>>,
     for(Map.Entry<Object, Object> item: props.entrySet()) {
       org.apache.hadoop.io.Text.writeString(out, (String) item.getKey());
       org.apache.hadoop.io.Text.writeString(out, (String) item.getValue());
+      WritableUtils.writeCompressedStringArray(out, 
+          updatingResource.get(item.getKey()));
     }
   }
   

+ 10 - 6
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/util/GenericOptionsParser.java

@@ -268,7 +268,8 @@ public class GenericOptionsParser {
     }
 
     if (line.hasOption("jt")) {
-      conf.set("mapred.job.tracker", line.getOptionValue("jt"));
+      conf.set("mapred.job.tracker", line.getOptionValue("jt"), 
+          "from -jt command line option");
     }
     if (line.hasOption("conf")) {
       String[] values = line.getOptionValues("conf");
@@ -278,7 +279,8 @@ public class GenericOptionsParser {
     }
     if (line.hasOption("libjars")) {
       conf.set("tmpjars", 
-               validateFiles(line.getOptionValue("libjars"), conf));
+               validateFiles(line.getOptionValue("libjars"), conf),
+               "from -libjars command line option");
       //setting libjars in client classpath
       URL[] libjars = getLibJars(conf);
       if(libjars!=null && libjars.length>0) {
@@ -290,18 +292,20 @@ public class GenericOptionsParser {
     }
     if (line.hasOption("files")) {
       conf.set("tmpfiles", 
-               validateFiles(line.getOptionValue("files"), conf));
+               validateFiles(line.getOptionValue("files"), conf),
+               "from -files command line option");
     }
     if (line.hasOption("archives")) {
       conf.set("tmparchives", 
-                validateFiles(line.getOptionValue("archives"), conf));
+                validateFiles(line.getOptionValue("archives"), conf),
+                "from -archives command line option");
     }
     if (line.hasOption('D')) {
       String[] property = line.getOptionValues('D');
       for(String prop : property) {
         String[] keyval = prop.split("=", 2);
         if (keyval.length == 2) {
-          conf.set(keyval[0], keyval[1]);
+          conf.set(keyval[0], keyval[1], "from command line");
         }
       }
     }
@@ -320,7 +324,7 @@ public class GenericOptionsParser {
         LOG.debug("setting conf tokensFile: " + fileName);
       }
       conf.set("mapreduce.job.credentials.json", localFs.makeQualified(p)
-          .toString());
+          .toString(), "from -tokenCacheFile command line option");
 
     }
   }

+ 1 - 1
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/conf/TestConfServlet.java

@@ -64,7 +64,7 @@ public class TestConfServlet extends TestCase {
       String resource = (String)propertyInfo.get("resource");
       System.err.println("k: " + key + " v: " + val + " r: " + resource);
       if (TEST_KEY.equals(key) && TEST_VAL.equals(val)
-          && Configuration.UNKNOWN_RESOURCE.equals(resource)) {
+          && "programatically".equals(resource)) {
         foundSetting = true;
       }
     }

+ 51 - 3
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/conf/TestConfiguration.java

@@ -168,7 +168,8 @@ public class TestConfiguration extends TestCase {
     appendProperty(name, val, false);
   }
  
-  void appendProperty(String name, String val, boolean isFinal)
+  void appendProperty(String name, String val, boolean isFinal, 
+      String ... sources)
     throws IOException {
     out.write("<property>");
     out.write("<name>");
@@ -180,6 +181,11 @@ public class TestConfiguration extends TestCase {
     if (isFinal) {
       out.write("<final>true</final>");
     }
+    for(String s : sources) {
+      out.write("<source>");
+      out.write(s);
+      out.write("</source>");
+    }
     out.write("</property>\n");
   }
   
@@ -640,7 +646,49 @@ public class TestConfiguration extends TestCase {
                  conf.getPattern("test.pattern3", defaultPattern).pattern());
   }
 
-  public void testSocketAddress() throws IOException {
+  public void testPropertySource() throws IOException {
+    out = new BufferedWriter(new FileWriter(CONFIG));
+    startConfig();
+    appendProperty("test.foo", "bar");
+    endConfig();
+    Path fileResource = new Path(CONFIG);
+    conf.addResource(fileResource);
+    conf.set("fs.defaultFS", "value");
+    String [] sources = conf.getPropertySources("test.foo");
+    assertEquals(1, sources.length);
+    assertEquals(
+        "Resource string returned for a file-loaded property" +
+        " must be a proper absolute path",
+        fileResource,
+        new Path(sources[0]));
+    assertArrayEquals("Resource string returned for a set() property must be " +
+    		"\"programatically\"",
+        new String[]{"programatically"},
+        conf.getPropertySources("fs.defaultFS"));
+    assertEquals("Resource string returned for an unset property must be null",
+        null, conf.getPropertySources("fs.defaultFoo"));
+  }
+  
+  public void testMultiplePropertySource() throws IOException {
+    out = new BufferedWriter(new FileWriter(CONFIG));
+    startConfig();
+    appendProperty("test.foo", "bar", false, "a", "b", "c");
+    endConfig();
+    Path fileResource = new Path(CONFIG);
+    conf.addResource(fileResource);
+    String [] sources = conf.getPropertySources("test.foo");
+    assertEquals(4, sources.length);
+    assertEquals("a", sources[0]);
+    assertEquals("b", sources[1]);
+    assertEquals("c", sources[2]);
+    assertEquals(
+        "Resource string returned for a file-loaded property" +
+        " must be a proper absolute path",
+        fileResource,
+        new Path(sources[3]));
+  }
+
+  public void testSocketAddress() {
     Configuration conf = new Configuration();
     final String defaultAddr = "host:1";
     final int defaultPort = 2;
@@ -886,7 +934,7 @@ public class TestConfiguration extends TestCase {
       confDump.put(prop.getKey(), prop);
     }
     assertEquals("value5",confDump.get("test.key6").getValue());
-    assertEquals("Unknown", confDump.get("test.key4").getResource());
+    assertEquals("programatically", confDump.get("test.key4").getResource());
     outWriter.close();
   }