Bladeren bron

YARN-2446. Augmented Timeline service APIs to start taking in domains as a parameter while posting entities and events. Contributed by Zhijie Shen.

(cherry picked from commit 9e40de6af7959ac7bb5f4e4d2833ca14ea457614)
Vinod Kumar Vavilapalli 10 jaren geleden
bovenliggende
commit
ae7a3235be
17 gewijzigde bestanden met toevoegingen van 679 en 154 verwijderingen
  1. 3 0
      hadoop-yarn-project/CHANGES.txt
  2. 21 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timeline/TimelineEntity.java
  3. 11 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timeline/TimelinePutResponse.java
  4. 3 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/test/java/org/apache/hadoop/yarn/api/records/timeline/TestTimelineRecords.java
  5. 1 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/test/java/org/apache/hadoop/yarn/client/api/impl/TestTimelineClient.java
  6. 1 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/ApplicationHistoryServer.java
  7. 55 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/timeline/LeveldbTimelineStore.java
  8. 25 2
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/timeline/MemoryTimelineStore.java
  9. 58 8
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/timeline/TimelineDataManager.java
  10. 87 14
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/timeline/security/TimelineACLsManager.java
  11. 3 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/applicationhistoryservice/TestApplicationHistoryManagerOnTimelineStore.java
  12. 1 2
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/applicationhistoryservice/TestApplicationHistoryServer.java
  13. 26 19
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/TestLeveldbTimelineStore.java
  14. 167 79
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/TimelineStoreTestUtils.java
  15. 76 10
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/security/TestTimelineACLsManager.java
  16. 140 19
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/webapp/TestTimelineWebServices.java
  17. 1 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/webapp/TestTimelineWebServicesWithSSL.java

+ 3 - 0
hadoop-yarn-project/CHANGES.txt

@@ -76,6 +76,9 @@ Release 2.6.0 - UNRELEASED
     YARN-2613. Support retry in NMClient for rolling-upgrades. (Jian He via 
     junping_du)
 
+    YARN-2446. Augmented Timeline service APIs to start taking in domains as a
+    parameter while posting entities and events. (Zhijie Shen via vinodkv)
+
   IMPROVEMENTS
 
     YARN-2242. Improve exception information on AM launch crashes. (Li Lu 

+ 21 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timeline/TimelineEntity.java

@@ -64,6 +64,7 @@ public class TimelineEntity implements Comparable<TimelineEntity> {
       new HashMap<String, Set<Object>>();
   private Map<String, Object> otherInfo =
       new HashMap<String, Object>();
+  private String domainId;
 
   public TimelineEntity() {
 
@@ -325,6 +326,26 @@ public class TimelineEntity implements Comparable<TimelineEntity> {
     this.otherInfo = otherInfo;
   }
 
+  /**
+   * Get the ID of the domain that the entity is to be put
+   * 
+   * @return the domain ID
+   */
+  @XmlElement(name = "domain")
+  public String getDomainId() {
+    return domainId;
+  }
+
+  /**
+   * Set the ID of the domain that the entity is to be put
+   * 
+   * @param domainId
+   *          the name space ID
+   */
+  public void setDomainId(String domainId) {
+    this.domainId = domainId;
+  }
+
   @Override
   public int hashCode() {
     // generated by eclipse

+ 11 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timeline/TimelinePutResponse.java

@@ -118,6 +118,17 @@ public class TimelinePutResponse {
      */
     public static final int ACCESS_DENIED = 4;
 
+    /**
+     * Error code returned if the entity doesn't have an valid domain ID
+     */
+    public static final int NO_DOMAIN = 5;
+
+    /**
+     * Error code returned if the user is denied to relate the entity to another
+     * one in different domain
+     */
+    public static final int FORBIDDEN_RELATION = 6;
+
     private String entityId;
     private String entityType;
     private int errorCode;

+ 3 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/test/java/org/apache/hadoop/yarn/api/records/timeline/TestTimelineRecords.java

@@ -61,6 +61,7 @@ public class TestTimelineRecords {
       entity.addPrimaryFilter("pkey2", "pval2");
       entity.addOtherInfo("okey1", "oval1");
       entity.addOtherInfo("okey2", "oval2");
+      entity.setDomainId("domain id " + j);
       entities.addEntity(entity);
     }
     LOG.info("Entities in JSON:");
@@ -74,6 +75,7 @@ public class TestTimelineRecords {
     Assert.assertEquals(2, entity1.getEvents().size());
     Assert.assertEquals(2, entity1.getPrimaryFilters().size());
     Assert.assertEquals(2, entity1.getOtherInfo().size());
+    Assert.assertEquals("domain id 0", entity1.getDomainId());
     TimelineEntity entity2 = entities.getEntities().get(1);
     Assert.assertEquals("entity id 1", entity2.getEntityId());
     Assert.assertEquals("entity type 1", entity2.getEntityType());
@@ -81,6 +83,7 @@ public class TestTimelineRecords {
     Assert.assertEquals(2, entity2.getEvents().size());
     Assert.assertEquals(2, entity2.getPrimaryFilters().size());
     Assert.assertEquals(2, entity2.getOtherInfo().size());
+    Assert.assertEquals("domain id 1", entity2.getDomainId());
   }
 
   @Test

+ 1 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/test/java/org/apache/hadoop/yarn/client/api/impl/TestTimelineClient.java

@@ -240,6 +240,7 @@ public class TestTimelineClient {
     entity.addPrimaryFilter("pkey2", "pval2");
     entity.addOtherInfo("okey1", "oval1");
     entity.addOtherInfo("okey2", "oval2");
+    entity.setDomainId("domain id 1");
     return entity;
   }
 

+ 1 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/applicationhistoryservice/ApplicationHistoryServer.java

@@ -84,6 +84,7 @@ public class ApplicationHistoryServer extends CompositeService {
     secretManagerService = createTimelineDelegationTokenSecretManagerService(conf);
     addService(secretManagerService);
     timelineDataManager = createTimelineDataManager(conf);
+    addService(timelineDataManager);
 
     // init generic history service afterwards
     aclsManager = createApplicationACLsManager(conf);

+ 55 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/timeline/LeveldbTimelineStore.java

@@ -113,6 +113,9 @@ import com.google.common.annotations.VisibleForTesting;
  *     RELATED_ENTITIES_COLUMN + relatedentity type + relatedentity id
  *
  *   ENTITY_ENTRY_PREFIX + entity type + revstarttime + entity id +
+ *     DOMAIN_ID_COLUMN
+ *
+ *   ENTITY_ENTRY_PREFIX + entity type + revstarttime + entity id +
  *     INVISIBLE_REVERSE_RELATED_ENTITIES_COLUMN + relatedentity type +
  *     relatedentity id</pre>
  *
@@ -146,6 +149,7 @@ public class LeveldbTimelineStore extends AbstractService
   private static final byte[] RELATED_ENTITIES_COLUMN = "r".getBytes();
   private static final byte[] INVISIBLE_REVERSE_RELATED_ENTITIES_COLUMN =
       "z".getBytes();
+  private static final byte[] DOMAIN_ID_COLUMN = "d".getBytes();
 
   private static final byte[] DOMAIN_ENTRY_PREFIX = "d".getBytes();
   private static final byte[] OWNER_LOOKUP_PREFIX = "o".getBytes();
@@ -521,6 +525,10 @@ public class LeveldbTimelineStore extends AbstractService
             entity.addEvent(event);
           }
         }
+      } else if (key[prefixlen] == DOMAIN_ID_COLUMN[0]) {
+        byte[] v = iterator.peekNext().getValue();
+        String domainId = new String(v);
+        entity.setDomainId(domainId);
       } else {
         if (key[prefixlen] !=
             INVISIBLE_REVERSE_RELATED_ENTITIES_COLUMN[0]) {
@@ -793,6 +801,7 @@ public class LeveldbTimelineStore extends AbstractService
     List<EntityIdentifier> relatedEntitiesWithoutStartTimes =
         new ArrayList<EntityIdentifier>();
     byte[] revStartTime = null;
+    Map<String, Set<Object>> primaryFilters = null;
     try {
       writeBatch = db.createWriteBatch();
       List<TimelineEvent> events = entity.getEvents();
@@ -812,7 +821,7 @@ public class LeveldbTimelineStore extends AbstractService
       revStartTime = writeReverseOrderedLong(startAndInsertTime
           .startTime);
 
-      Map<String, Set<Object>> primaryFilters = entity.getPrimaryFilters();
+      primaryFilters = entity.getPrimaryFilters();
 
       // write entity marker
       byte[] markerKey = createEntityMarkerKey(entity.getEntityId(),
@@ -857,6 +866,21 @@ public class LeveldbTimelineStore extends AbstractService
               relatedEntitiesWithoutStartTimes.add(
                   new EntityIdentifier(relatedEntityId, relatedEntityType));
               continue;
+            } else {
+              byte[] domainIdBytes = db.get(createDomainIdKey(
+                  relatedEntityId, relatedEntityType, relatedEntityStartTime));
+              // This is the existing entity
+              String domainId = new String(domainIdBytes);
+              if (!domainId.equals(entity.getDomainId())) {
+                // in this case the entity will be put, but the relation will be
+                // ignored
+                TimelinePutError error = new TimelinePutError();
+                error.setEntityId(entity.getEntityId());
+                error.setEntityType(entity.getEntityType());
+                error.setErrorCode(TimelinePutError.FORBIDDEN_RELATION);
+                response.addError(error);
+                continue;
+              }
             }
             // write "forward" entry (related entity -> entity)
             key = createRelatedEntityKey(relatedEntityId,
@@ -893,6 +917,23 @@ public class LeveldbTimelineStore extends AbstractService
           writePrimaryFilterEntries(writeBatch, primaryFilters, key, value);
         }
       }
+
+      // write domain id entry
+      byte[] key = createDomainIdKey(entity.getEntityId(),
+          entity.getEntityType(), revStartTime);
+      if (entity.getDomainId() == null ||
+          entity.getDomainId().length() == 0) {
+        TimelinePutError error = new TimelinePutError();
+        error.setEntityId(entity.getEntityId());
+        error.setEntityType(entity.getEntityType());
+        error.setErrorCode(TimelinePutError.NO_DOMAIN);
+        response.addError(error);
+        return;
+      } else {
+        writeBatch.put(key, entity.getDomainId().getBytes());
+        writePrimaryFilterEntries(writeBatch, primaryFilters, key,
+            entity.getDomainId().getBytes());
+      }
       db.write(writeBatch);
     } catch (IOException e) {
       LOG.error("Error putting entity " + entity.getEntityId() +
@@ -920,6 +961,10 @@ public class LeveldbTimelineStore extends AbstractService
         }
         byte[] relatedEntityStartTime = writeReverseOrderedLong(
             relatedEntityStartAndInsertTime.startTime);
+          // This is the new entity, the domain should be the same
+        byte[] key = createDomainIdKey(relatedEntity.getId(),
+            relatedEntity.getType(), relatedEntityStartTime);
+        db.put(key, entity.getDomainId().getBytes());
         db.put(createRelatedEntityKey(relatedEntity.getId(),
             relatedEntity.getType(), relatedEntityStartTime,
             entity.getEntityId(), entity.getEntityType()), EMPTY_BYTES);
@@ -1265,6 +1310,15 @@ public class LeveldbTimelineStore extends AbstractService
         .add(relatedEntityType).add(relatedEntityId).getBytes();
   }
 
+  /**
+   * Creates a domain id key, serializing ENTITY_ENTRY_PREFIX +
+   * entity type + revstarttime + entity id + DOMAIN_ID_COLUMN.
+   */
+  private static byte[] createDomainIdKey(String entityId,
+      String entityType, byte[] revStartTime) throws IOException {
+    return KeyBuilder.newInstance().add(ENTITY_ENTRY_PREFIX).add(entityType)
+        .add(revStartTime).add(entityId).add(DOMAIN_ID_COLUMN).getBytes();
+  }
   /**
    * Clears the cache to test reloading start times from leveldb (only for
    * testing).

+ 25 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/timeline/MemoryTimelineStore.java

@@ -284,6 +284,16 @@ public class MemoryTimelineStore
         existingEntity.setEntityId(entity.getEntityId());
         existingEntity.setEntityType(entity.getEntityType());
         existingEntity.setStartTime(entity.getStartTime());
+        if (entity.getDomainId() == null ||
+            entity.getDomainId().length() == 0) {
+          TimelinePutError error = new TimelinePutError();
+          error.setEntityId(entityId.getId());
+          error.setEntityType(entityId.getType());
+          error.setErrorCode(TimelinePutError.NO_DOMAIN);
+          response.addError(error);
+          continue;
+        }
+        existingEntity.setDomainId(entity.getDomainId());
         entities.put(entityId, existingEntity);
         entityInsertTimes.put(entityId, System.currentTimeMillis());
       }
@@ -351,8 +361,19 @@ public class MemoryTimelineStore
               new EntityIdentifier(idStr, partRelatedEntities.getKey());
           TimelineEntity relatedEntity = entities.get(relatedEntityId);
           if (relatedEntity != null) {
-            relatedEntity.addRelatedEntity(
-                existingEntity.getEntityType(), existingEntity.getEntityId());
+            if (relatedEntity.getDomainId().equals(
+                existingEntity.getDomainId())) {
+              relatedEntity.addRelatedEntity(
+                  existingEntity.getEntityType(), existingEntity.getEntityId());
+            } else {
+              // in this case the entity will be put, but the relation will be
+              // ignored
+              TimelinePutError error = new TimelinePutError();
+              error.setEntityType(existingEntity.getEntityType());
+              error.setEntityId(existingEntity.getEntityId());
+              error.setErrorCode(TimelinePutError.FORBIDDEN_RELATION);
+              response.addError(error);
+            }
           } else {
             relatedEntity = new TimelineEntity();
             relatedEntity.setEntityId(relatedEntityId.getId());
@@ -360,6 +381,7 @@ public class MemoryTimelineStore
             relatedEntity.setStartTime(existingEntity.getStartTime());
             relatedEntity.addRelatedEntity(existingEntity.getEntityType(),
                 existingEntity.getEntityId());
+            relatedEntity.setDomainId(existingEntity.getDomainId());
             entities.put(relatedEntityId, relatedEntity);
             entityInsertTimes.put(relatedEntityId, System.currentTimeMillis());
           }
@@ -414,6 +436,7 @@ public class MemoryTimelineStore
     entityToReturn.setEntityId(entity.getEntityId());
     entityToReturn.setEntityType(entity.getEntityType());
     entityToReturn.setStartTime(entity.getStartTime());
+    entityToReturn.setDomainId(entity.getDomainId());
     // Deep copy
     if (fields.contains(Field.EVENTS)) {
       entityToReturn.addEvents(entity.getEvents());

+ 58 - 8
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/timeline/TimelineDataManager.java

@@ -30,7 +30,10 @@ import java.util.SortedSet;
 
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.security.UserGroupInformation;
+import org.apache.hadoop.service.AbstractService;
+import org.apache.hadoop.yarn.api.records.ApplicationAccessType;
 import org.apache.hadoop.yarn.api.records.timeline.TimelineEntities;
 import org.apache.hadoop.yarn.api.records.timeline.TimelineEntity;
 import org.apache.hadoop.yarn.api.records.timeline.TimelineEvents;
@@ -42,23 +45,49 @@ import org.apache.hadoop.yarn.server.timeline.TimelineReader.Field;
 import org.apache.hadoop.yarn.server.timeline.security.TimelineACLsManager;
 import org.apache.hadoop.yarn.util.timeline.TimelineUtils;
 
+import com.google.common.annotations.VisibleForTesting;
+
 /**
  * The class wrap over the timeline store and the ACLs manager. It does some non
  * trivial manipulation of the timeline data before putting or after getting it
  * from the timeline store, and checks the user's access to it.
  * 
  */
-public class TimelineDataManager {
+public class TimelineDataManager extends AbstractService {
 
   private static final Log LOG = LogFactory.getLog(TimelineDataManager.class);
+  @VisibleForTesting
+  public static final String DEFAULT_DOMAIN_ID = "DEFAULT";
 
   private TimelineStore store;
   private TimelineACLsManager timelineACLsManager;
 
   public TimelineDataManager(TimelineStore store,
       TimelineACLsManager timelineACLsManager) {
+    super(TimelineDataManager.class.getName());
     this.store = store;
     this.timelineACLsManager = timelineACLsManager;
+    timelineACLsManager.setTimelineStore(store);
+  }
+
+  @Override
+  protected void serviceInit(Configuration conf) throws Exception {
+    TimelineDomain domain = store.getDomain("DEFAULT");
+    // it is okay to reuse an existing domain even if it was created by another
+    // user of the timeline server before, because it allows everybody to access.
+    if (domain == null) {
+      // create a default domain, which allows everybody to access and modify
+      // the entities in it.
+      domain = new TimelineDomain();
+      domain.setId(DEFAULT_DOMAIN_ID);
+      domain.setDescription("System Default Domain");
+      domain.setOwner(
+          UserGroupInformation.getCurrentUser().getShortUserName());
+      domain.setReaders("*");
+      domain.setWriters("*");
+      store.put(domain);
+    }
+    super.serviceInit(conf);
   }
 
   /**
@@ -98,7 +127,8 @@ public class TimelineDataManager {
         TimelineEntity entity = entitiesItr.next();
         try {
           // check ACLs
-          if (!timelineACLsManager.checkAccess(callerUGI, entity)) {
+          if (!timelineACLsManager.checkAccess(
+              callerUGI, ApplicationAccessType.VIEW_APP, entity)) {
             entitiesItr.remove();
           } else {
             // clean up system data
@@ -141,7 +171,8 @@ public class TimelineDataManager {
         store.getEntity(entityId, entityType, fields);
     if (entity != null) {
       // check ACLs
-      if (!timelineACLsManager.checkAccess(callerUGI, entity)) {
+      if (!timelineACLsManager.checkAccess(
+          callerUGI, ApplicationAccessType.VIEW_APP, entity)) {
         entity = null;
       } else {
         // clean up the system data
@@ -189,7 +220,8 @@ public class TimelineDataManager {
               eventsOfOneEntity.getEntityType(),
               EnumSet.of(Field.PRIMARY_FILTERS));
           // check ACLs
-          if (!timelineACLsManager.checkAccess(callerUGI, entity)) {
+          if (!timelineACLsManager.checkAccess(
+              callerUGI, ApplicationAccessType.VIEW_APP, entity)) {
             eventsItr.remove();
           }
         } catch (Exception e) {
@@ -225,16 +257,29 @@ public class TimelineDataManager {
       EntityIdentifier entityID =
           new EntityIdentifier(entity.getEntityId(), entity.getEntityType());
 
+      // if the domain id is not specified, the entity will be put into
+      // the default domain
+      if (entity.getDomainId() == null ||
+          entity.getDomainId().length() == 0) {
+        entity.setDomainId(DEFAULT_DOMAIN_ID);
+      }
+
       // check if there is existing entity
       TimelineEntity existingEntity = null;
       try {
         existingEntity =
             store.getEntity(entityID.getId(), entityID.getType(),
                 EnumSet.of(Field.PRIMARY_FILTERS));
-        if (existingEntity != null
-            && !timelineACLsManager.checkAccess(callerUGI, existingEntity)) {
-          throw new YarnException("The timeline entity " + entityID
-              + " was not put by " + callerUGI + " before");
+        if (existingEntity != null &&
+            !existingEntity.getDomainId().equals(entity.getDomainId())) {
+          throw new YarnException("The domain of the timeline entity "
+            + entityID + " is not allowed to be changed.");
+        }
+        if (!timelineACLsManager.checkAccess(
+            callerUGI, ApplicationAccessType.MODIFY_APP, entity)) {
+          throw new YarnException(callerUGI
+              + " is not allowed to put the timeline entity " + entityID
+              + " into the domain " + entity.getDomainId() + ".");
         }
       } catch (Exception e) {
         // Skip the entity which already exists and was put by others
@@ -307,6 +352,11 @@ public class TimelineDataManager {
       domain.setOwner(existingDomain.getOwner());
     }
     store.put(domain);
+    // If the domain exists already, it is likely to be in the cache.
+    // We need to invalidate it.
+    if (existingDomain != null) {
+      timelineACLsManager.replaceIfExist(domain);
+    }
   }
 
   /**

+ 87 - 14
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/main/java/org/apache/hadoop/yarn/server/timeline/security/TimelineACLsManager.java

@@ -19,19 +19,26 @@
 package org.apache.hadoop.yarn.server.timeline.security;
 
 import java.io.IOException;
-import java.util.Set;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.Map;
 
+import org.apache.commons.collections.map.LRUMap;
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.classification.InterfaceAudience.Private;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.security.UserGroupInformation;
-import org.apache.hadoop.yarn.api.records.timeline.TimelineEntity;
+import org.apache.hadoop.security.authorize.AccessControlList;
+import org.apache.hadoop.yarn.api.records.ApplicationAccessType;
 import org.apache.hadoop.yarn.api.records.timeline.TimelineDomain;
+import org.apache.hadoop.yarn.api.records.timeline.TimelineEntity;
+import org.apache.hadoop.yarn.conf.YarnConfiguration;
 import org.apache.hadoop.yarn.exceptions.YarnException;
 import org.apache.hadoop.yarn.security.AdminACLsManager;
 import org.apache.hadoop.yarn.server.timeline.EntityIdentifier;
-import org.apache.hadoop.yarn.server.timeline.TimelineStore.SystemFilter;
+import org.apache.hadoop.yarn.server.timeline.TimelineStore;
+import org.apache.hadoop.yarn.util.StringHelper;
 
 import com.google.common.annotations.VisibleForTesting;
 
@@ -42,14 +49,58 @@ import com.google.common.annotations.VisibleForTesting;
 public class TimelineACLsManager {
 
   private static final Log LOG = LogFactory.getLog(TimelineACLsManager.class);
+  private static final int DOMAIN_ACCESS_ENTRY_CACHE_SIZE = 100;
 
   private AdminACLsManager adminAclsManager;
+  private Map<String, AccessControlListExt> aclExts;
+  private TimelineStore store;
 
+  @SuppressWarnings("unchecked")
   public TimelineACLsManager(Configuration conf) {
     this.adminAclsManager = new AdminACLsManager(conf);
+    aclExts = Collections.synchronizedMap(
+        new LRUMap(DOMAIN_ACCESS_ENTRY_CACHE_SIZE));
+  }
+
+  public void setTimelineStore(TimelineStore store) {
+    this.store = store;
+  }
+
+  private AccessControlListExt loadDomainFromTimelineStore(
+      String domainId) throws IOException {
+    if (store == null) {
+      return null;
+    }
+    TimelineDomain domain = store.getDomain(domainId);
+    if (domain == null) {
+      return null;
+    } else {
+      return putDomainIntoCache(domain);
+    }
+  }
+
+  public void replaceIfExist(TimelineDomain domain) {
+    if (aclExts.containsKey(domain.getId())) {
+      putDomainIntoCache(domain);
+    }
+  }
+
+  private AccessControlListExt putDomainIntoCache(
+      TimelineDomain domain) {
+    Map<ApplicationAccessType, AccessControlList> acls
+    = new HashMap<ApplicationAccessType, AccessControlList>(2);
+    acls.put(ApplicationAccessType.VIEW_APP,
+        new AccessControlList(StringHelper.cjoin(domain.getReaders())));
+    acls.put(ApplicationAccessType.MODIFY_APP,
+        new AccessControlList(StringHelper.cjoin(domain.getWriters())));
+    AccessControlListExt aclExt =
+        new AccessControlListExt(domain.getOwner(), acls);
+    aclExts.put(domain.getId(), aclExt);
+    return aclExt;
   }
 
   public boolean checkAccess(UserGroupInformation callerUGI,
+      ApplicationAccessType applicationAccessType,
       TimelineEntity entity) throws YarnException, IOException {
     if (LOG.isDebugEnabled()) {
       LOG.debug("Verifying the access of "
@@ -62,21 +113,33 @@ public class TimelineACLsManager {
       return true;
     }
 
-    Set<Object> values =
-        entity.getPrimaryFilters().get(
-            SystemFilter.ENTITY_OWNER.toString());
-    if (values == null || values.size() != 1) {
-      throw new YarnException("Owner information of the timeline entity "
+    // find domain owner and acls
+    AccessControlListExt aclExt = aclExts.get(entity.getDomainId());
+    if (aclExt == null) {
+      aclExt = loadDomainFromTimelineStore(entity.getDomainId());
+    }
+    if (aclExt == null) {
+      throw new YarnException("Domain information of the timeline entity "
           + new EntityIdentifier(entity.getEntityId(), entity.getEntityType())
-          + " is corrupted.");
+          + " doesn't exist.");
+    }
+    String owner = aclExt.owner;
+    AccessControlList domainACL = aclExt.acls.get(applicationAccessType);
+    if (domainACL == null) {
+      if (LOG.isDebugEnabled()) {
+        LOG.debug("ACL not found for access-type " + applicationAccessType
+            + " for domain " + entity.getDomainId() + " owned by "
+            + owner + ". Using default ["
+            + YarnConfiguration.DEFAULT_YARN_APP_ACL + "]");
+      }
+      domainACL =
+          new AccessControlList(YarnConfiguration.DEFAULT_YARN_APP_ACL);
     }
-    String owner = values.iterator().next().toString();
-    // TODO: Currently we just check the user is the admin or the timeline
-    // entity owner. In the future, we need to check whether the user is in the
-    // allowed user/group list
+
     if (callerUGI != null
         && (adminAclsManager.isAdmin(callerUGI) ||
-            callerUGI.getShortUserName().equals(owner))) {
+            callerUGI.getShortUserName().equals(owner) ||
+            domainACL.isUserAllowed(callerUGI))) {
       return true;
     }
     return false;
@@ -116,4 +179,14 @@ public class TimelineACLsManager {
     return oldAdminACLsManager;
   }
 
+  private static class AccessControlListExt {
+    private String owner;
+    private Map<ApplicationAccessType, AccessControlList> acls;
+
+    public AccessControlListExt(
+        String owner, Map<ApplicationAccessType, AccessControlList> acls) {
+      this.owner = owner;
+      this.acls = acls;
+    }
+  }
 }

+ 3 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/applicationhistoryservice/TestApplicationHistoryManagerOnTimelineStore.java

@@ -412,6 +412,7 @@ public class TestApplicationHistoryManagerOnTimelineStore {
     TimelineEntity entity = new TimelineEntity();
     entity.setEntityType(ApplicationMetricsConstants.ENTITY_TYPE);
     entity.setEntityId(appId.toString());
+    entity.setDomainId(TimelineDataManager.DEFAULT_DOMAIN_ID);
     entity.addPrimaryFilter(
         TimelineStore.SystemFilter.ENTITY_OWNER.toString(), "yarn");
     Map<String, Object> entityInfo = new HashMap<String, Object>();
@@ -456,6 +457,7 @@ public class TestApplicationHistoryManagerOnTimelineStore {
     TimelineEntity entity = new TimelineEntity();
     entity.setEntityType(AppAttemptMetricsConstants.ENTITY_TYPE);
     entity.setEntityId(appAttemptId.toString());
+    entity.setDomainId(TimelineDataManager.DEFAULT_DOMAIN_ID);
     entity.addPrimaryFilter(AppAttemptMetricsConstants.PARENT_PRIMARY_FILTER,
         appAttemptId.getApplicationId().toString());
     entity.addPrimaryFilter(
@@ -497,6 +499,7 @@ public class TestApplicationHistoryManagerOnTimelineStore {
     TimelineEntity entity = new TimelineEntity();
     entity.setEntityType(ContainerMetricsConstants.ENTITY_TYPE);
     entity.setEntityId(containerId.toString());
+    entity.setDomainId(TimelineDataManager.DEFAULT_DOMAIN_ID);
     entity.addPrimaryFilter(ContainerMetricsConstants.PARENT_PRIMARIY_FILTER,
         containerId.getApplicationAttemptId().toString());
     entity.addPrimaryFilter(

+ 1 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/applicationhistoryservice/TestApplicationHistoryServer.java

@@ -31,7 +31,6 @@ import org.apache.hadoop.yarn.conf.YarnConfiguration;
 import org.apache.hadoop.yarn.server.applicationhistoryservice.webapp.AHSWebApp;
 import org.apache.hadoop.yarn.server.timeline.security.TimelineAuthenticationFilterInitializer;
 import org.junit.After;
-import org.junit.Assert;
 import org.junit.Test;
 
 import java.util.HashMap;
@@ -48,7 +47,7 @@ public class TestApplicationHistoryServer {
     Configuration config = new YarnConfiguration();
     historyServer.init(config);
     assertEquals(STATE.INITED, historyServer.getServiceState());
-    assertEquals(4, historyServer.getServices().size());
+    assertEquals(5, historyServer.getServices().size());
     ApplicationHistoryClientService historyService =
         historyServer.getClientService();
     assertNotNull(historyServer.getClientService());

+ 26 - 19
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/TestLeveldbTimelineStore.java

@@ -160,21 +160,22 @@ public class TestLeveldbTimelineStore extends TimelineStoreTestUtils {
   @Test
   public void testGetEntityTypes() throws IOException {
     List<String> entityTypes = ((LeveldbTimelineStore)store).getEntityTypes();
-    assertEquals(4, entityTypes.size());
+    assertEquals(5, entityTypes.size());
     assertEquals(entityType1, entityTypes.get(0));
     assertEquals(entityType2, entityTypes.get(1));
     assertEquals(entityType4, entityTypes.get(2));
     assertEquals(entityType5, entityTypes.get(3));
+    assertEquals(entityType7, entityTypes.get(4));
   }
 
   @Test
   public void testDeleteEntities() throws IOException, InterruptedException {
-    assertEquals(2, getEntities("type_1").size());
+    assertEquals(3, getEntities("type_1").size());
     assertEquals(1, getEntities("type_2").size());
 
     assertEquals(false, deleteNextEntity(entityType1,
-        writeReverseOrderedLong(122l)));
-    assertEquals(2, getEntities("type_1").size());
+        writeReverseOrderedLong(60l)));
+    assertEquals(3, getEntities("type_1").size());
     assertEquals(1, getEntities("type_2").size());
 
     assertEquals(true, deleteNextEntity(entityType1,
@@ -183,16 +184,19 @@ public class TestLeveldbTimelineStore extends TimelineStoreTestUtils {
     assertEquals(1, entities.size());
     verifyEntityInfo(entityId2, entityType2, events2, Collections.singletonMap(
         entityType1, Collections.singleton(entityId1b)), EMPTY_PRIMARY_FILTERS,
-        EMPTY_MAP, entities.get(0));
+        EMPTY_MAP, entities.get(0), domainId1);
     entities = getEntitiesWithPrimaryFilter("type_1", userFilter);
-    assertEquals(1, entities.size());
+    assertEquals(2, entities.size());
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
+    // can retrieve entities across domains
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(1), domainId2);
 
     ((LeveldbTimelineStore)store).discardOldEntities(-123l);
-    assertEquals(1, getEntities("type_1").size());
+    assertEquals(2, getEntities("type_1").size());
     assertEquals(0, getEntities("type_2").size());
-    assertEquals(3, ((LeveldbTimelineStore)store).getEntityTypes().size());
+    assertEquals(4, ((LeveldbTimelineStore)store).getEntityTypes().size());
 
     ((LeveldbTimelineStore)store).discardOldEntities(123l);
     assertEquals(0, getEntities("type_1").size());
@@ -210,7 +214,7 @@ public class TestLeveldbTimelineStore extends TimelineStoreTestUtils {
     TimelineEntities atsEntities = new TimelineEntities();
     atsEntities.setEntities(Collections.singletonList(createEntity(entityId1b,
         entityType1, 789l, Collections.singletonList(ev2), null, primaryFilter,
-        null)));
+        null, domainId1)));
     TimelinePutResponse response = store.put(atsEntities);
     assertEquals(0, response.getErrors().size());
 
@@ -219,18 +223,21 @@ public class TestLeveldbTimelineStore extends TimelineStoreTestUtils {
         pfPair);
     assertEquals(1, entities.size());
     verifyEntityInfo(entityId1b, entityType1, Collections.singletonList(ev2),
-        EMPTY_REL_ENTITIES, primaryFilter, EMPTY_MAP, entities.get(0));
+        EMPTY_REL_ENTITIES, primaryFilter, EMPTY_MAP, entities.get(0),
+        domainId1);
 
     entities = getEntitiesWithPrimaryFilter("type_1", userFilter);
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
 
     ((LeveldbTimelineStore)store).discardOldEntities(-123l);
     assertEquals(1, getEntitiesWithPrimaryFilter("type_1", pfPair).size());
-    assertEquals(2, getEntitiesWithPrimaryFilter("type_1", userFilter).size());
+    assertEquals(3, getEntitiesWithPrimaryFilter("type_1", userFilter).size());
 
     ((LeveldbTimelineStore)store).discardOldEntities(123l);
     assertEquals(0, getEntities("type_1").size());
@@ -245,9 +252,9 @@ public class TestLeveldbTimelineStore extends TimelineStoreTestUtils {
   public void testFromTsWithDeletion()
       throws IOException, InterruptedException {
     long l = System.currentTimeMillis();
-    assertEquals(2, getEntitiesFromTs("type_1", l).size());
+    assertEquals(3, getEntitiesFromTs("type_1", l).size());
     assertEquals(1, getEntitiesFromTs("type_2", l).size());
-    assertEquals(2, getEntitiesFromTsWithPrimaryFilter("type_1", userFilter,
+    assertEquals(3, getEntitiesFromTsWithPrimaryFilter("type_1", userFilter,
         l).size());
     ((LeveldbTimelineStore)store).discardOldEntities(123l);
     assertEquals(0, getEntitiesFromTs("type_1", l).size());
@@ -263,9 +270,9 @@ public class TestLeveldbTimelineStore extends TimelineStoreTestUtils {
     assertEquals(0, getEntitiesFromTs("type_2", l).size());
     assertEquals(0, getEntitiesFromTsWithPrimaryFilter("type_1", userFilter,
         l).size());
-    assertEquals(2, getEntities("type_1").size());
+    assertEquals(3, getEntities("type_1").size());
     assertEquals(1, getEntities("type_2").size());
-    assertEquals(2, getEntitiesWithPrimaryFilter("type_1", userFilter).size());
+    assertEquals(3, getEntitiesWithPrimaryFilter("type_1", userFilter).size());
   }
   
   @Test

+ 167 - 79
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/TimelineStoreTestUtils.java

@@ -66,6 +66,10 @@ public class TimelineStoreTestUtils {
   protected String entityType4;
   protected String entityId5;
   protected String entityType5;
+  protected String entityId6;
+  protected String entityId7;
+  protected String entityType7;
+  
   protected Map<String, Set<Object>> primaryFilters;
   protected Map<String, Object> secondaryFilters;
   protected Map<String, Object> allFilters;
@@ -86,6 +90,8 @@ public class TimelineStoreTestUtils {
   protected List<TimelineEvent> events1;
   protected List<TimelineEvent> events2;
   protected long beforeTs;
+  protected String domainId1;
+  protected String domainId2;
 
   /**
    * Load test entity data into the given store
@@ -123,6 +129,9 @@ public class TimelineStoreTestUtils {
     String entityType4 = "type_4";
     String entityId5 = "id_5";
     String entityType5 = "type_5";
+    String entityId6 = "id_6";
+    String entityId7 = "id_7";
+    String entityType7 = "type_7";
 
     Map<String, Set<String>> relatedEntities =
         new HashMap<String, Set<String>>();
@@ -134,19 +143,19 @@ public class TimelineStoreTestUtils {
     events.add(ev3);
     events.add(ev4);
     entities.setEntities(Collections.singletonList(createEntity(entityId2,
-        entityType2, null, events, null, null, null)));
+        entityType2, null, events, null, null, null, "domain_id_1")));
     TimelinePutResponse response = store.put(entities);
     assertEquals(0, response.getErrors().size());
 
     TimelineEvent ev1 = createEvent(123l, "start_event", null);
     entities.setEntities(Collections.singletonList(createEntity(entityId1,
         entityType1, 123l, Collections.singletonList(ev1),
-        relatedEntities, primaryFilters, otherInfo1)));
+        relatedEntities, primaryFilters, otherInfo1, "domain_id_1")));
     response = store.put(entities);
     assertEquals(0, response.getErrors().size());
     entities.setEntities(Collections.singletonList(createEntity(entityId1b,
         entityType1, null, Collections.singletonList(ev1), relatedEntities,
-        primaryFilters, otherInfo1)));
+        primaryFilters, otherInfo1, "domain_id_1")));
     response = store.put(entities);
     assertEquals(0, response.getErrors().size());
 
@@ -157,17 +166,18 @@ public class TimelineStoreTestUtils {
     otherInfo2.put("info2", "val2");
     entities.setEntities(Collections.singletonList(createEntity(entityId1,
         entityType1, null, Collections.singletonList(ev2), null,
-        primaryFilters, otherInfo2)));
+        primaryFilters, otherInfo2, "domain_id_1")));
     response = store.put(entities);
     assertEquals(0, response.getErrors().size());
     entities.setEntities(Collections.singletonList(createEntity(entityId1b,
         entityType1, 789l, Collections.singletonList(ev2), null,
-        primaryFilters, otherInfo2)));
+        primaryFilters, otherInfo2, "domain_id_1")));
     response = store.put(entities);
     assertEquals(0, response.getErrors().size());
 
     entities.setEntities(Collections.singletonList(createEntity(
-        "badentityid", "badentity", null, null, null, null, otherInfo1)));
+        "badentityid", "badentity", null, null, null, null, otherInfo1,
+        "domain_id_1")));
     response = store.put(entities);
     assertEquals(1, response.getErrors().size());
     TimelinePutError error = response.getErrors().get(0);
@@ -178,9 +188,28 @@ public class TimelineStoreTestUtils {
     relatedEntities.clear();
     relatedEntities.put(entityType5, Collections.singleton(entityId5));
     entities.setEntities(Collections.singletonList(createEntity(entityId4,
-        entityType4, 42l, null, relatedEntities, null, null)));
+        entityType4, 42l, null, relatedEntities, null, null,
+        "domain_id_1")));
     response = store.put(entities);
-    assertEquals(0, response.getErrors().size());
+
+    relatedEntities.clear();
+    otherInfo1.put("info2", "val2");
+    entities.setEntities(Collections.singletonList(createEntity(entityId6,
+        entityType1, 61l, null, relatedEntities, primaryFilters, otherInfo1,
+        "domain_id_2")));
+    response = store.put(entities);
+
+    relatedEntities.clear();
+    relatedEntities.put(entityType1, Collections.singleton(entityId1));
+    entities.setEntities(Collections.singletonList(createEntity(entityId7,
+        entityType7, 62l, null, relatedEntities, null, null,
+        "domain_id_2")));
+    response = store.put(entities);
+    assertEquals(1, response.getErrors().size());
+    assertEquals(entityType7, response.getErrors().get(0).getEntityType());
+    assertEquals(entityId7, response.getErrors().get(0).getEntityId());
+    assertEquals(TimelinePutError.FORBIDDEN_RELATION,
+        response.getErrors().get(0).getErrorCode());
   }
 
   /**
@@ -235,6 +264,9 @@ public class TimelineStoreTestUtils {
     entityType4 = "type_4";
     entityId5 = "id_5";
     entityType5 = "type_5";
+    entityId6 = "id_6";
+    entityId7 = "id_7";
+    entityType7 = "type_7";
 
     ev1 = createEvent(123l, "start_event", null);
 
@@ -261,6 +293,9 @@ public class TimelineStoreTestUtils {
     events2 = new ArrayList<TimelineEvent>();
     events2.add(ev3);
     events2.add(ev4);
+
+    domainId1 = "domain_id_1";
+    domainId2 = "domain_id_2";
   }
 
   private TimelineDomain domain1;
@@ -282,7 +317,7 @@ public class TimelineStoreTestUtils {
     domain2.setDescription("description_2");
     domain2.setOwner("owner_2");
     domain2.setReaders("reader_user_2 reader_group_2");
-    domain2.setWriters("writer_user_2writer_group_2");
+    domain2.setWriters("writer_user_2 writer_group_2");
     store.put(domain2);
 
     // Wait a second before updating the domain information
@@ -311,50 +346,62 @@ public class TimelineStoreTestUtils {
   public void testGetSingleEntity() throws IOException {
     // test getting entity info
     verifyEntityInfo(null, null, null, null, null, null,
-        store.getEntity("id_1", "type_2", EnumSet.allOf(Field.class)));
+        store.getEntity("id_1", "type_2", EnumSet.allOf(Field.class)),
+        domainId1);
 
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
         primaryFilters, otherInfo, 123l, store.getEntity(entityId1,
-        entityType1, EnumSet.allOf(Field.class)));
+        entityType1, EnumSet.allOf(Field.class)), domainId1);
 
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
         primaryFilters, otherInfo, 123l, store.getEntity(entityId1b,
-        entityType1, EnumSet.allOf(Field.class)));
+        entityType1, EnumSet.allOf(Field.class)), domainId1);
 
     verifyEntityInfo(entityId2, entityType2, events2, relEntityMap,
         EMPTY_PRIMARY_FILTERS, EMPTY_MAP, -123l, store.getEntity(entityId2,
-        entityType2, EnumSet.allOf(Field.class)));
+        entityType2, EnumSet.allOf(Field.class)), domainId1);
 
     verifyEntityInfo(entityId4, entityType4, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
         EMPTY_PRIMARY_FILTERS, EMPTY_MAP, 42l, store.getEntity(entityId4,
-        entityType4, EnumSet.allOf(Field.class)));
+        entityType4, EnumSet.allOf(Field.class)), domainId1);
 
     verifyEntityInfo(entityId5, entityType5, EMPTY_EVENTS, relEntityMap2,
         EMPTY_PRIMARY_FILTERS, EMPTY_MAP, 42l, store.getEntity(entityId5,
-        entityType5, EnumSet.allOf(Field.class)));
+        entityType5, EnumSet.allOf(Field.class)), domainId1);
 
     // test getting single fields
     verifyEntityInfo(entityId1, entityType1, events1, null, null, null,
-        store.getEntity(entityId1, entityType1, EnumSet.of(Field.EVENTS)));
+        store.getEntity(entityId1, entityType1, EnumSet.of(Field.EVENTS)),
+        domainId1);
 
     verifyEntityInfo(entityId1, entityType1, Collections.singletonList(ev2),
         null, null, null, store.getEntity(entityId1, entityType1,
-        EnumSet.of(Field.LAST_EVENT_ONLY)));
+        EnumSet.of(Field.LAST_EVENT_ONLY)), domainId1);
 
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
         primaryFilters, otherInfo, store.getEntity(entityId1b, entityType1,
-        null));
+        null), domainId1);
 
     verifyEntityInfo(entityId1, entityType1, null, null, primaryFilters, null,
         store.getEntity(entityId1, entityType1,
-            EnumSet.of(Field.PRIMARY_FILTERS)));
+            EnumSet.of(Field.PRIMARY_FILTERS)), domainId1);
 
     verifyEntityInfo(entityId1, entityType1, null, null, null, otherInfo,
-        store.getEntity(entityId1, entityType1, EnumSet.of(Field.OTHER_INFO)));
+        store.getEntity(entityId1, entityType1, EnumSet.of(Field.OTHER_INFO)),
+        domainId1);
 
     verifyEntityInfo(entityId2, entityType2, null, relEntityMap, null, null,
         store.getEntity(entityId2, entityType2,
-            EnumSet.of(Field.RELATED_ENTITIES)));
+            EnumSet.of(Field.RELATED_ENTITIES)), domainId1);
+
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, store.getEntity(entityId6, entityType1,
+            EnumSet.allOf(Field.class)), domainId2);
+
+    // entity is created, but it doesn't relate to <entityType1, entityId1>
+    verifyEntityInfo(entityId7, entityType7, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        EMPTY_PRIMARY_FILTERS, EMPTY_MAP, store.getEntity(entityId7, entityType7,
+            EnumSet.allOf(Field.class)), domainId2);
   }
 
   protected List<TimelineEntity> getEntities(String entityType)
@@ -438,28 +485,30 @@ public class TimelineStoreTestUtils {
         getEntitiesWithPrimaryFilter("type_6", userFilter).size());
 
     List<TimelineEntity> entities = getEntities("type_1");
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
 
     entities = getEntities("type_2");
     assertEquals(1, entities.size());
     verifyEntityInfo(entityId2, entityType2, events2, relEntityMap,
-        EMPTY_PRIMARY_FILTERS, EMPTY_MAP, entities.get(0));
+        EMPTY_PRIMARY_FILTERS, EMPTY_MAP, entities.get(0), domainId1);
 
     entities = getEntities("type_1", 1l, null, null, null,
         EnumSet.allOf(Field.class));
     assertEquals(1, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
 
     entities = getEntities("type_1", 1l, 0l, null, null,
         EnumSet.allOf(Field.class));
     assertEquals(1, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
 
     entities = getEntities("type_1", null, 234l, null, null,
         EnumSet.allOf(Field.class));
@@ -475,35 +524,48 @@ public class TimelineStoreTestUtils {
 
     entities = getEntities("type_1", null, null, 345l, null,
         EnumSet.allOf(Field.class));
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
 
     entities = getEntities("type_1", null, null, 123l, null,
         EnumSet.allOf(Field.class));
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
   }
 
   public void testGetEntitiesWithFromId() throws IOException {
     List<TimelineEntity> entities = getEntitiesFromId("type_1", entityId1);
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
 
     entities = getEntitiesFromId("type_1", entityId1b);
-    assertEquals(1, entities.size());
+    assertEquals(2, entities.size());
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(1), domainId2);
+
+    entities = getEntitiesFromId("type_1", entityId6);
+    assertEquals(1, entities.size());
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(0), domainId2);
 
-    entities = getEntitiesFromIdWithWindow("type_1", 0l, entityId1);
+    entities = getEntitiesFromIdWithWindow("type_1", 0l, entityId6);
     assertEquals(0, entities.size());
 
     entities = getEntitiesFromId("type_2", "a");
@@ -512,7 +574,7 @@ public class TimelineStoreTestUtils {
     entities = getEntitiesFromId("type_2", entityId2);
     assertEquals(1, entities.size());
     verifyEntityInfo(entityId2, entityType2, events2, relEntityMap,
-        EMPTY_PRIMARY_FILTERS, EMPTY_MAP, entities.get(0));
+        EMPTY_PRIMARY_FILTERS, EMPTY_MAP, entities.get(0), domainId1);
 
     entities = getEntitiesFromIdWithWindow("type_2", -456l, null);
     assertEquals(0, entities.size());
@@ -529,20 +591,30 @@ public class TimelineStoreTestUtils {
     // same tests with primary filters
     entities = getEntitiesFromIdWithPrimaryFilter("type_1", userFilter,
         entityId1);
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
 
     entities = getEntitiesFromIdWithPrimaryFilter("type_1", userFilter,
         entityId1b);
-    assertEquals(1, entities.size());
+    assertEquals(2, entities.size());
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(1), domainId2);
+
+    entities = getEntitiesFromIdWithPrimaryFilter("type_1", userFilter,
+        entityId6);
+    assertEquals(1, entities.size());
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(0), domainId2);
 
     entities = getEntitiesFromIdWithPrimaryFilterAndWindow("type_1", 0l,
-        entityId1, userFilter);
+        entityId6, userFilter);
     assertEquals(0, entities.size());
 
     entities = getEntitiesFromIdWithPrimaryFilter("type_2", userFilter, "a");
@@ -555,13 +627,13 @@ public class TimelineStoreTestUtils {
     assertEquals(0, getEntitiesFromTsWithPrimaryFilter("type_1", userFilter,
         beforeTs).size());
     long afterTs = System.currentTimeMillis();
-    assertEquals(2, getEntitiesFromTs("type_1", afterTs).size());
+    assertEquals(3, getEntitiesFromTs("type_1", afterTs).size());
     assertEquals(1, getEntitiesFromTs("type_2", afterTs).size());
-    assertEquals(2, getEntitiesFromTsWithPrimaryFilter("type_1", userFilter,
+    assertEquals(3, getEntitiesFromTsWithPrimaryFilter("type_1", userFilter,
         afterTs).size());
-    assertEquals(2, getEntities("type_1").size());
+    assertEquals(3, getEntities("type_1").size());
     assertEquals(1, getEntities("type_2").size());
-    assertEquals(2, getEntitiesWithPrimaryFilter("type_1", userFilter).size());
+    assertEquals(3, getEntitiesWithPrimaryFilter("type_1", userFilter).size());
     // check insert time is not overwritten
     long beforeTs = this.beforeTs;
     loadTestEntityData();
@@ -569,9 +641,9 @@ public class TimelineStoreTestUtils {
     assertEquals(0, getEntitiesFromTs("type_2", beforeTs).size());
     assertEquals(0, getEntitiesFromTsWithPrimaryFilter("type_1", userFilter,
         beforeTs).size());
-    assertEquals(2, getEntitiesFromTs("type_1", afterTs).size());
+    assertEquals(3, getEntitiesFromTs("type_1", afterTs).size());
     assertEquals(1, getEntitiesFromTs("type_2", afterTs).size());
-    assertEquals(2, getEntitiesFromTsWithPrimaryFilter("type_1", userFilter,
+    assertEquals(3, getEntitiesFromTsWithPrimaryFilter("type_1", userFilter,
         afterTs).size());
   }
 
@@ -589,32 +661,40 @@ public class TimelineStoreTestUtils {
 
     List<TimelineEntity> entities = getEntitiesWithPrimaryFilter("type_1",
         userFilter);
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
 
     entities = getEntitiesWithPrimaryFilter("type_1", numericFilter1);
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
 
     entities = getEntitiesWithPrimaryFilter("type_1", numericFilter2);
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
 
     entities = getEntitiesWithPrimaryFilter("type_1", numericFilter3);
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
 
     entities = getEntitiesWithPrimaryFilter("type_2", userFilter);
     assertEquals(0, entities.size());
@@ -622,12 +702,12 @@ public class TimelineStoreTestUtils {
     entities = getEntities("type_1", 1l, null, null, userFilter, null);
     assertEquals(1, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
 
     entities = getEntities("type_1", 1l, 0l, null, userFilter, null);
     assertEquals(1, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
 
     entities = getEntities("type_1", null, 234l, null, userFilter, null);
     assertEquals(0, entities.size());
@@ -636,29 +716,35 @@ public class TimelineStoreTestUtils {
     assertEquals(0, entities.size());
 
     entities = getEntities("type_1", null, null, 345l, userFilter, null);
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
   }
 
   public void testGetEntitiesWithSecondaryFilters() throws IOException {
     // test using secondary filter
     List<TimelineEntity> entities = getEntitiesWithFilters("type_1", null,
         goodTestingFilters);
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
 
     entities = getEntitiesWithFilters("type_1", userFilter, goodTestingFilters);
-    assertEquals(2, entities.size());
+    assertEquals(3, entities.size());
     verifyEntityInfo(entityId1, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(0));
+        primaryFilters, otherInfo, entities.get(0), domainId1);
     verifyEntityInfo(entityId1b, entityType1, events1, EMPTY_REL_ENTITIES,
-        primaryFilters, otherInfo, entities.get(1));
+        primaryFilters, otherInfo, entities.get(1), domainId1);
+    verifyEntityInfo(entityId6, entityType1, EMPTY_EVENTS, EMPTY_REL_ENTITIES,
+        primaryFilters, otherInfo, entities.get(2), domainId2);
 
     entities = getEntitiesWithFilters("type_1", null,
         Collections.singleton(new NameValuePair("user", "none")));
@@ -737,10 +823,10 @@ public class TimelineStoreTestUtils {
   protected static void verifyEntityInfo(String entityId, String entityType,
       List<TimelineEvent> events, Map<String, Set<String>> relatedEntities,
       Map<String, Set<Object>> primaryFilters, Map<String, Object> otherInfo,
-      Long startTime, TimelineEntity retrievedEntityInfo) {
+      Long startTime, TimelineEntity retrievedEntityInfo, String domainId) {
 
     verifyEntityInfo(entityId, entityType, events, relatedEntities,
-        primaryFilters, otherInfo, retrievedEntityInfo);
+        primaryFilters, otherInfo, retrievedEntityInfo, domainId);
     assertEquals(startTime, retrievedEntityInfo.getStartTime());
   }
 
@@ -750,13 +836,14 @@ public class TimelineStoreTestUtils {
   protected static void verifyEntityInfo(String entityId, String entityType,
       List<TimelineEvent> events, Map<String, Set<String>> relatedEntities,
       Map<String, Set<Object>> primaryFilters, Map<String, Object> otherInfo,
-      TimelineEntity retrievedEntityInfo) {
+      TimelineEntity retrievedEntityInfo, String domainId) {
     if (entityId == null) {
       assertNull(retrievedEntityInfo);
       return;
     }
     assertEquals(entityId, retrievedEntityInfo.getEntityId());
     assertEquals(entityType, retrievedEntityInfo.getEntityType());
+    assertEquals(domainId, retrievedEntityInfo.getDomainId());
     if (events == null) {
       assertNull(retrievedEntityInfo.getEvents());
     } else {
@@ -801,7 +888,7 @@ public class TimelineStoreTestUtils {
       Long startTime, List<TimelineEvent> events,
       Map<String, Set<String>> relatedEntities,
       Map<String, Set<Object>> primaryFilters,
-      Map<String, Object> otherInfo) {
+      Map<String, Object> otherInfo, String domainId) {
     TimelineEntity entity = new TimelineEntity();
     entity.setEntityId(entityId);
     entity.setEntityType(entityType);
@@ -818,6 +905,7 @@ public class TimelineStoreTestUtils {
     }
     entity.setPrimaryFilters(primaryFilters);
     entity.setOtherInfo(otherInfo);
+    entity.setDomainId(domainId);
     return entity;
   }
 

+ 76 - 10
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/security/TestTimelineACLsManager.java

@@ -18,32 +18,54 @@
 
 package org.apache.hadoop.yarn.server.timeline.security;
 
+import java.io.IOException;
+
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.security.UserGroupInformation;
-import org.apache.hadoop.yarn.api.records.timeline.TimelineEntity;
+import org.apache.hadoop.yarn.api.records.ApplicationAccessType;
 import org.apache.hadoop.yarn.api.records.timeline.TimelineDomain;
+import org.apache.hadoop.yarn.api.records.timeline.TimelineEntity;
 import org.apache.hadoop.yarn.conf.YarnConfiguration;
 import org.apache.hadoop.yarn.exceptions.YarnException;
+import org.apache.hadoop.yarn.server.timeline.MemoryTimelineStore;
 import org.apache.hadoop.yarn.server.timeline.TimelineStore;
 import org.junit.Assert;
 import org.junit.Test;
 
 public class TestTimelineACLsManager {
 
+  private static TimelineDomain domain;
+
+  static {
+    domain = new TimelineDomain();
+    domain.setId("domain_id_1");
+    domain.setOwner("owner");
+    domain.setReaders("reader");
+    domain.setWriters("writer");
+  }
+
   @Test
   public void testYarnACLsNotEnabledForEntity() throws Exception {
     Configuration conf = new YarnConfiguration();
     conf.setBoolean(YarnConfiguration.YARN_ACL_ENABLE, false);
     TimelineACLsManager timelineACLsManager =
         new TimelineACLsManager(conf);
+    timelineACLsManager.setTimelineStore(new TestTimelineStore());
     TimelineEntity entity = new TimelineEntity();
     entity.addPrimaryFilter(
         TimelineStore.SystemFilter.ENTITY_OWNER
             .toString(), "owner");
+    entity.setDomainId("domain_id_1");
     Assert.assertTrue(
         "Always true when ACLs are not enabled",
         timelineACLsManager.checkAccess(
-            UserGroupInformation.createRemoteUser("user"), entity));
+            UserGroupInformation.createRemoteUser("user"),
+            ApplicationAccessType.VIEW_APP, entity));
+    Assert.assertTrue(
+        "Always true when ACLs are not enabled",
+        timelineACLsManager.checkAccess(
+            UserGroupInformation.createRemoteUser("user"),
+            ApplicationAccessType.MODIFY_APP, entity));
   }
 
   @Test
@@ -53,22 +75,53 @@ public class TestTimelineACLsManager {
     conf.set(YarnConfiguration.YARN_ADMIN_ACL, "admin");
     TimelineACLsManager timelineACLsManager =
         new TimelineACLsManager(conf);
+    timelineACLsManager.setTimelineStore(new TestTimelineStore());
     TimelineEntity entity = new TimelineEntity();
     entity.addPrimaryFilter(
         TimelineStore.SystemFilter.ENTITY_OWNER
             .toString(), "owner");
+    entity.setDomainId("domain_id_1");
     Assert.assertTrue(
-        "Owner should be allowed to access",
+        "Owner should be allowed to view",
         timelineACLsManager.checkAccess(
-            UserGroupInformation.createRemoteUser("owner"), entity));
+            UserGroupInformation.createRemoteUser("owner"),
+            ApplicationAccessType.VIEW_APP, entity));
+    Assert.assertTrue(
+        "Reader should be allowed to view",
+        timelineACLsManager.checkAccess(
+            UserGroupInformation.createRemoteUser("reader"),
+            ApplicationAccessType.VIEW_APP, entity));
     Assert.assertFalse(
-        "Other shouldn't be allowed to access",
+        "Other shouldn't be allowed to view",
         timelineACLsManager.checkAccess(
-            UserGroupInformation.createRemoteUser("other"), entity));
+            UserGroupInformation.createRemoteUser("other"),
+            ApplicationAccessType.VIEW_APP, entity));
     Assert.assertTrue(
-        "Admin should be allowed to access",
+        "Admin should be allowed to view",
+        timelineACLsManager.checkAccess(
+            UserGroupInformation.createRemoteUser("admin"),
+            ApplicationAccessType.VIEW_APP, entity));
+
+    Assert.assertTrue(
+        "Owner should be allowed to modify",
         timelineACLsManager.checkAccess(
-            UserGroupInformation.createRemoteUser("admin"), entity));
+            UserGroupInformation.createRemoteUser("owner"),
+            ApplicationAccessType.MODIFY_APP, entity));
+    Assert.assertTrue(
+        "Writer should be allowed to modify",
+        timelineACLsManager.checkAccess(
+            UserGroupInformation.createRemoteUser("writer"),
+            ApplicationAccessType.MODIFY_APP, entity));
+    Assert.assertFalse(
+        "Other shouldn't be allowed to modify",
+        timelineACLsManager.checkAccess(
+            UserGroupInformation.createRemoteUser("other"),
+            ApplicationAccessType.MODIFY_APP, entity));
+    Assert.assertTrue(
+        "Admin should be allowed to modify",
+        timelineACLsManager.checkAccess(
+            UserGroupInformation.createRemoteUser("admin"),
+            ApplicationAccessType.MODIFY_APP, entity));
   }
 
   @Test
@@ -78,14 +131,16 @@ public class TestTimelineACLsManager {
     conf.set(YarnConfiguration.YARN_ADMIN_ACL, "owner");
     TimelineACLsManager timelineACLsManager =
         new TimelineACLsManager(conf);
+    timelineACLsManager.setTimelineStore(new TestTimelineStore());
     TimelineEntity entity = new TimelineEntity();
     try {
       timelineACLsManager.checkAccess(
-          UserGroupInformation.createRemoteUser("owner"), entity);
+          UserGroupInformation.createRemoteUser("owner"),
+          ApplicationAccessType.VIEW_APP, entity);
       Assert.fail("Exception is expected");
     } catch (YarnException e) {
       Assert.assertTrue("It's not the exact expected exception", e.getMessage()
-          .contains("is corrupted."));
+          .contains("doesn't exist."));
     }
   }
 
@@ -144,4 +199,15 @@ public class TestTimelineACLsManager {
     }
   }
 
+  private static class TestTimelineStore extends MemoryTimelineStore {
+    @Override
+    public TimelineDomain getDomain(
+        String domainId) throws IOException {
+      if (domainId == null) {
+        return null;
+      } else {
+        return domain;
+      }
+    }
+  }
 }

+ 140 - 19
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/webapp/TestTimelineWebServices.java

@@ -25,6 +25,7 @@ import static org.mockito.Mockito.mock;
 import static org.mockito.Mockito.spy;
 import static org.mockito.Mockito.when;
 
+import java.util.Collections;
 import java.util.Enumeration;
 import java.util.HashMap;
 import java.util.HashSet;
@@ -95,11 +96,14 @@ public class TestTimelineWebServices extends JerseyTest {
       Configuration conf = new YarnConfiguration();
       conf.setBoolean(YarnConfiguration.YARN_ACL_ENABLE, false);
       timelineACLsManager = new TimelineACLsManager(conf);
+      timelineACLsManager.setTimelineStore(store);
       conf.setBoolean(YarnConfiguration.YARN_ACL_ENABLE, true);
       conf.set(YarnConfiguration.YARN_ADMIN_ACL, "admin");
       adminACLsManager = new AdminACLsManager(conf);
       TimelineDataManager timelineDataManager =
           new TimelineDataManager(store, timelineACLsManager);
+      timelineDataManager.init(conf);
+      timelineDataManager.start();
       bind(TimelineDataManager.class).toInstance(timelineDataManager);
       serve("/*").with(GuiceContainer.class);
       TimelineAuthenticationFilter taFilter =
@@ -182,7 +186,7 @@ public class TestTimelineWebServices extends JerseyTest {
 
   private static void verifyEntities(TimelineEntities entities) {
     Assert.assertNotNull(entities);
-    Assert.assertEquals(2, entities.getEntities().size());
+    Assert.assertEquals(3, entities.getEntities().size());
     TimelineEntity entity1 = entities.getEntities().get(0);
     Assert.assertNotNull(entity1);
     Assert.assertEquals("id_1", entity1.getEntityId());
@@ -199,6 +203,14 @@ public class TestTimelineWebServices extends JerseyTest {
     Assert.assertEquals(2, entity2.getEvents().size());
     Assert.assertEquals(4, entity2.getPrimaryFilters().size());
     Assert.assertEquals(4, entity2.getOtherInfo().size());
+    TimelineEntity entity3 = entities.getEntities().get(2);
+    Assert.assertNotNull(entity2);
+    Assert.assertEquals("id_6", entity3.getEntityId());
+    Assert.assertEquals("type_1", entity3.getEntityType());
+    Assert.assertEquals(61l, entity3.getStartTime().longValue());
+    Assert.assertEquals(0, entity3.getEvents().size());
+    Assert.assertEquals(4, entity3.getPrimaryFilters().size());
+    Assert.assertEquals(4, entity3.getOtherInfo().size());
   }
 
   @Test
@@ -220,7 +232,7 @@ public class TestTimelineWebServices extends JerseyTest {
         .accept(MediaType.APPLICATION_JSON)
         .get(ClientResponse.class);
     assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
-    assertEquals(1, response.getEntity(TimelineEntities.class).getEntities()
+    assertEquals(2, response.getEntity(TimelineEntities.class).getEntities()
         .size());
 
     response = r.path("ws").path("v1").path("timeline")
@@ -228,7 +240,7 @@ public class TestTimelineWebServices extends JerseyTest {
         .accept(MediaType.APPLICATION_JSON)
         .get(ClientResponse.class);
     assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
-    assertEquals(2, response.getEntity(TimelineEntities.class).getEntities()
+    assertEquals(3, response.getEntity(TimelineEntities.class).getEntities()
         .size());
   }
 
@@ -249,7 +261,7 @@ public class TestTimelineWebServices extends JerseyTest {
         .accept(MediaType.APPLICATION_JSON)
         .get(ClientResponse.class);
     assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
-    assertEquals(2, response.getEntity(TimelineEntities.class).getEntities()
+    assertEquals(3, response.getEntity(TimelineEntities.class).getEntities()
         .size());
   }
 
@@ -439,6 +451,7 @@ public class TestTimelineWebServices extends JerseyTest {
     entity.setEntityId("test id 1");
     entity.setEntityType("test type 1");
     entity.setStartTime(System.currentTimeMillis());
+    entity.setDomainId("domain_id_1");
     entities.addEntity(entity);
     WebResource r = resource();
     // No owner, will be rejected
@@ -482,10 +495,11 @@ public class TestTimelineWebServices extends JerseyTest {
       entity.setEntityId("test id 2");
       entity.setEntityType("test type 2");
       entity.setStartTime(System.currentTimeMillis());
+      entity.setDomainId("domain_id_1");
       entities.addEntity(entity);
       WebResource r = resource();
       ClientResponse response = r.path("ws").path("v1").path("timeline")
-          .queryParam("user.name", "tester")
+          .queryParam("user.name", "writer_user_1")
           .accept(MediaType.APPLICATION_JSON)
           .type(MediaType.APPLICATION_JSON)
           .post(ClientResponse.class, entities);
@@ -497,7 +511,7 @@ public class TestTimelineWebServices extends JerseyTest {
 
       // override/append timeline data in the same entity with different user
       response = r.path("ws").path("v1").path("timeline")
-          .queryParam("user.name", "other")
+          .queryParam("user.name", "writer_user_3")
           .accept(MediaType.APPLICATION_JSON)
           .type(MediaType.APPLICATION_JSON)
           .post(ClientResponse.class, entities);
@@ -507,6 +521,82 @@ public class TestTimelineWebServices extends JerseyTest {
       Assert.assertEquals(1, putResponse.getErrors().size());
       Assert.assertEquals(TimelinePutResponse.TimelinePutError.ACCESS_DENIED,
           putResponse.getErrors().get(0).getErrorCode());
+
+      // Cross domain relationship will be rejected
+      entities = new TimelineEntities();
+      entity = new TimelineEntity();
+      entity.setEntityId("test id 3");
+      entity.setEntityType("test type 2");
+      entity.setStartTime(System.currentTimeMillis());
+      entity.setDomainId("domain_id_2");
+      entity.setRelatedEntities(Collections.singletonMap(
+          "test type 2", Collections.singleton("test id 2")));
+      entities.addEntity(entity);
+      r = resource();
+      response = r.path("ws").path("v1").path("timeline")
+          .queryParam("user.name", "writer_user_3")
+          .accept(MediaType.APPLICATION_JSON)
+          .type(MediaType.APPLICATION_JSON)
+          .post(ClientResponse.class, entities);
+      assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
+      putResponse = response.getEntity(TimelinePutResponse.class);
+      Assert.assertNotNull(putResponse);
+      Assert.assertEquals(1, putResponse.getErrors().size());
+      Assert.assertEquals(TimelinePutError.FORBIDDEN_RELATION,
+          putResponse.getErrors().get(0).getErrorCode());
+
+      // Make sure the entity has been added anyway even though the
+      // relationship is been excluded
+      response = r.path("ws").path("v1").path("timeline")
+          .path("test type 2").path("test id 3")
+          .queryParam("user.name", "reader_user_3")
+          .accept(MediaType.APPLICATION_JSON)
+          .get(ClientResponse.class);
+      assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
+      entity = response.getEntity(TimelineEntity.class);
+      Assert.assertNotNull(entity);
+      Assert.assertEquals("test id 3", entity.getEntityId());
+      Assert.assertEquals("test type 2", entity.getEntityType());
+    } finally {
+      timelineACLsManager.setAdminACLsManager(oldAdminACLsManager);
+    }
+  }
+
+  @Test
+  public void testPostEntitiesToDefaultDomain() throws Exception {
+    AdminACLsManager oldAdminACLsManager =
+        timelineACLsManager.setAdminACLsManager(adminACLsManager);
+    try {
+      TimelineEntities entities = new TimelineEntities();
+      TimelineEntity entity = new TimelineEntity();
+      entity.setEntityId("test id 7");
+      entity.setEntityType("test type 7");
+      entity.setStartTime(System.currentTimeMillis());
+      entities.addEntity(entity);
+      WebResource r = resource();
+      ClientResponse response = r.path("ws").path("v1").path("timeline")
+          .queryParam("user.name", "anybody_1")
+          .accept(MediaType.APPLICATION_JSON)
+          .type(MediaType.APPLICATION_JSON)
+          .post(ClientResponse.class, entities);
+      assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
+      TimelinePutResponse putResposne =
+          response.getEntity(TimelinePutResponse.class);
+      Assert.assertNotNull(putResposne);
+      Assert.assertEquals(0, putResposne.getErrors().size());
+      // verify the entity exists in the store
+      response = r.path("ws").path("v1").path("timeline")
+          .path("test type 7").path("test id 7")
+          .queryParam("user.name", "any_body_2")
+          .accept(MediaType.APPLICATION_JSON)
+          .get(ClientResponse.class);
+      assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
+      entity = response.getEntity(TimelineEntity.class);
+      Assert.assertNotNull(entity);
+      Assert.assertEquals("test id 7", entity.getEntityId());
+      Assert.assertEquals("test type 7", entity.getEntityType());
+      Assert.assertEquals(TimelineDataManager.DEFAULT_DOMAIN_ID,
+          entity.getDomainId());
     } finally {
       timelineACLsManager.setAdminACLsManager(oldAdminACLsManager);
     }
@@ -522,18 +612,23 @@ public class TestTimelineWebServices extends JerseyTest {
       entity.setEntityId("test id 3");
       entity.setEntityType("test type 3");
       entity.setStartTime(System.currentTimeMillis());
+      entity.setDomainId("domain_id_1");
       entities.addEntity(entity);
       WebResource r = resource();
       ClientResponse response = r.path("ws").path("v1").path("timeline")
-          .queryParam("user.name", "tester")
+          .queryParam("user.name", "writer_user_1")
           .accept(MediaType.APPLICATION_JSON)
           .type(MediaType.APPLICATION_JSON)
           .post(ClientResponse.class, entities);
+      assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
+      TimelinePutResponse putResponse =
+          response.getEntity(TimelinePutResponse.class);
+      Assert.assertEquals(0, putResponse.getErrors().size());
       // verify the system data will not be exposed
       // 1. No field specification
       response = r.path("ws").path("v1").path("timeline")
           .path("test type 3").path("test id 3")
-          .queryParam("user.name", "tester")
+          .queryParam("user.name", "reader_user_1")
           .accept(MediaType.APPLICATION_JSON)
           .get(ClientResponse.class);
       assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
@@ -544,7 +639,7 @@ public class TestTimelineWebServices extends JerseyTest {
       response = r.path("ws").path("v1").path("timeline")
           .path("test type 3").path("test id 3")
           .queryParam("fields", "relatedentities")
-          .queryParam("user.name", "tester")
+          .queryParam("user.name", "reader_user_1")
           .accept(MediaType.APPLICATION_JSON)
           .get(ClientResponse.class);
       assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
@@ -555,7 +650,7 @@ public class TestTimelineWebServices extends JerseyTest {
       response = r.path("ws").path("v1").path("timeline")
           .path("test type 3").path("test id 3")
           .queryParam("fields", "primaryfilters")
-          .queryParam("user.name", "tester")
+          .queryParam("user.name", "reader_user_1")
           .accept(MediaType.APPLICATION_JSON)
           .get(ClientResponse.class);
       assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
@@ -566,7 +661,7 @@ public class TestTimelineWebServices extends JerseyTest {
       // get entity with other user
       response = r.path("ws").path("v1").path("timeline")
           .path("test type 3").path("test id 3")
-          .queryParam("user.name", "other")
+          .queryParam("user.name", "reader_user_2")
           .accept(MediaType.APPLICATION_JSON)
           .get(ClientResponse.class);
       assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
@@ -582,42 +677,55 @@ public class TestTimelineWebServices extends JerseyTest {
     AdminACLsManager oldAdminACLsManager =
         timelineACLsManager.setAdminACLsManager(adminACLsManager);
     try {
+      // Put entity [4, 4] in domain 1
       TimelineEntities entities = new TimelineEntities();
       TimelineEntity entity = new TimelineEntity();
       entity.setEntityId("test id 4");
       entity.setEntityType("test type 4");
       entity.setStartTime(System.currentTimeMillis());
+      entity.setDomainId("domain_id_1");
       entities.addEntity(entity);
       WebResource r = resource();
       ClientResponse response = r.path("ws").path("v1").path("timeline")
-          .queryParam("user.name", "tester")
+          .queryParam("user.name", "writer_user_1")
           .accept(MediaType.APPLICATION_JSON)
           .type(MediaType.APPLICATION_JSON)
           .post(ClientResponse.class, entities);
+      assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
+      TimelinePutResponse putResponse =
+          response.getEntity(TimelinePutResponse.class);
+      Assert.assertEquals(0, putResponse.getErrors().size());
 
+      // Put entity [4, 5] in domain 2
       entities = new TimelineEntities();
       entity = new TimelineEntity();
       entity.setEntityId("test id 5");
       entity.setEntityType("test type 4");
       entity.setStartTime(System.currentTimeMillis());
+      entity.setDomainId("domain_id_2");
       entities.addEntity(entity);
       r = resource();
       response = r.path("ws").path("v1").path("timeline")
-          .queryParam("user.name", "other")
+          .queryParam("user.name", "writer_user_3")
           .accept(MediaType.APPLICATION_JSON)
           .type(MediaType.APPLICATION_JSON)
           .post(ClientResponse.class, entities);
+      assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
+      putResponse = response.getEntity(TimelinePutResponse.class);
+      Assert.assertEquals(0, putResponse.getErrors().size());
 
+      // Query entities of type 4
       response = r.path("ws").path("v1").path("timeline")
-          .queryParam("user.name", "other")
+          .queryParam("user.name", "reader_user_1")
           .path("test type 4")
           .accept(MediaType.APPLICATION_JSON)
           .get(ClientResponse.class);
       assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
       entities = response.getEntity(TimelineEntities.class);
+      // Reader 1 should just have the access to entity [4, 4]
       assertEquals(1, entities.getEntities().size());
       assertEquals("test type 4", entities.getEntities().get(0).getEntityType());
-      assertEquals("test id 5", entities.getEntities().get(0).getEntityId());
+      assertEquals("test id 4", entities.getEntities().get(0).getEntityId());
     } finally {
       timelineACLsManager.setAdminACLsManager(oldAdminACLsManager);
     }
@@ -628,11 +736,13 @@ public class TestTimelineWebServices extends JerseyTest {
     AdminACLsManager oldAdminACLsManager =
         timelineACLsManager.setAdminACLsManager(adminACLsManager);
     try {
+      // Put entity [5, 5] in domain 1
       TimelineEntities entities = new TimelineEntities();
       TimelineEntity entity = new TimelineEntity();
       entity.setEntityId("test id 5");
       entity.setEntityType("test type 5");
       entity.setStartTime(System.currentTimeMillis());
+      entity.setDomainId("domain_id_1");
       TimelineEvent event = new TimelineEvent();
       event.setEventType("event type 1");
       event.setTimestamp(System.currentTimeMillis());
@@ -640,16 +750,22 @@ public class TestTimelineWebServices extends JerseyTest {
       entities.addEntity(entity);
       WebResource r = resource();
       ClientResponse response = r.path("ws").path("v1").path("timeline")
-          .queryParam("user.name", "tester")
+          .queryParam("user.name", "writer_user_1")
           .accept(MediaType.APPLICATION_JSON)
           .type(MediaType.APPLICATION_JSON)
           .post(ClientResponse.class, entities);
+      assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
+      TimelinePutResponse putResponse =
+          response.getEntity(TimelinePutResponse.class);
+      Assert.assertEquals(0, putResponse.getErrors().size());
 
+      // Put entity [5, 6] in domain 2
       entities = new TimelineEntities();
       entity = new TimelineEntity();
       entity.setEntityId("test id 6");
       entity.setEntityType("test type 5");
       entity.setStartTime(System.currentTimeMillis());
+      entity.setDomainId("domain_id_2");
       event = new TimelineEvent();
       event.setEventType("event type 2");
       event.setTimestamp(System.currentTimeMillis());
@@ -657,21 +773,26 @@ public class TestTimelineWebServices extends JerseyTest {
       entities.addEntity(entity);
       r = resource();
       response = r.path("ws").path("v1").path("timeline")
-          .queryParam("user.name", "other")
+          .queryParam("user.name", "writer_user_3")
           .accept(MediaType.APPLICATION_JSON)
           .type(MediaType.APPLICATION_JSON)
           .post(ClientResponse.class, entities);
+      assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
+      putResponse = response.getEntity(TimelinePutResponse.class);
+      Assert.assertEquals(0, putResponse.getErrors().size());
 
+      // Query events belonging to the entities of type 4
       response = r.path("ws").path("v1").path("timeline")
           .path("test type 5").path("events")
-          .queryParam("user.name", "other")
+          .queryParam("user.name", "reader_user_1")
           .queryParam("entityId", "test id 5,test id 6")
           .accept(MediaType.APPLICATION_JSON)
           .get(ClientResponse.class);
       assertEquals(MediaType.APPLICATION_JSON_TYPE, response.getType());
       TimelineEvents events = response.getEntity(TimelineEvents.class);
+      // Reader 1 should just have the access to the events of entity [5, 5]
       assertEquals(1, events.getAllEvents().size());
-      assertEquals("test id 6", events.getAllEvents().get(0).getEntityId());
+      assertEquals("test id 5", events.getAllEvents().get(0).getEntityId());
     } finally {
       timelineACLsManager.setAdminACLsManager(oldAdminACLsManager);
     }

+ 1 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-applicationhistoryservice/src/test/java/org/apache/hadoop/yarn/server/timeline/webapp/TestTimelineWebServicesWithSSL.java

@@ -95,6 +95,7 @@ public class TestTimelineWebServicesWithSSL {
       TimelineEntity expectedEntity = new TimelineEntity();
       expectedEntity.setEntityType("test entity type");
       expectedEntity.setEntityId("test entity id");
+      expectedEntity.setDomainId("test domain id");
       TimelineEvent event = new TimelineEvent();
       event.setEventType("test event type");
       event.setTimestamp(0L);