瀏覽代碼

YARN-3041. Added the overall data model of timeline service next gen. Contributed by Zhijie Shen.

(cherry picked from commit fdff5d2625c129c20c44f90a0517b28cee1522bd)
Zhijie Shen 10 年之前
父節點
當前提交
c77f86bf63
共有 14 個文件被更改,包括 1131 次插入0 次删除
  1. 35 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/ApplicationAttemptEntity.java
  2. 47 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/ApplicationEntity.java
  3. 36 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/ClusterEntity.java
  4. 35 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/ContainerEntity.java
  5. 81 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/FlowEntity.java
  6. 117 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/HierarchicalTimelineEntity.java
  7. 260 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/TimelineEntity.java
  8. 71 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/TimelineEntityType.java
  9. 77 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/TimelineEvent.java
  10. 116 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/TimelineMetric.java
  11. 35 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/TimelineQueue.java
  12. 35 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/TimelineUser.java
  13. 21 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/package-info.java
  14. 165 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/test/java/org/apache/hadoop/yarn/api/records/timelineservice/TestTimelineServiceRecords.java

+ 35 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/ApplicationAttemptEntity.java

@@ -0,0 +1,35 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlRootElement;
+
+@XmlRootElement(name = "appattempt")
+@XmlAccessorType(XmlAccessType.NONE)
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public class ApplicationAttemptEntity extends HierarchicalTimelineEntity {
+  public ApplicationAttemptEntity() {
+    super(TimelineEntityType.YARN_APPLICATION_ATTEMPT.toString());
+  }
+}

+ 47 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/ApplicationEntity.java

@@ -0,0 +1,47 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlElement;
+import javax.xml.bind.annotation.XmlRootElement;
+
+@XmlRootElement(name = "application")
+@XmlAccessorType(XmlAccessType.NONE)
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public class ApplicationEntity extends HierarchicalTimelineEntity {
+  private String queue;
+
+  public ApplicationEntity() {
+    super(TimelineEntityType.YARN_APPLICATION.toString());
+  }
+
+  @XmlElement(name = "queue")
+  public String getQueue() {
+    return queue;
+  }
+
+  public void setQueue(String queue) {
+    this.queue = queue;
+  }
+}

+ 36 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/ClusterEntity.java

@@ -0,0 +1,36 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlRootElement;
+
+@XmlRootElement(name = "cluster")
+@XmlAccessorType(XmlAccessType.NONE)
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public class ClusterEntity extends HierarchicalTimelineEntity {
+  public ClusterEntity() {
+    super(TimelineEntityType.YARN_CLUSTER.toString());
+  }
+
+}

+ 35 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/ContainerEntity.java

@@ -0,0 +1,35 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlRootElement;
+
+@XmlRootElement(name = "container")
+@XmlAccessorType(XmlAccessType.NONE)
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public class ContainerEntity extends HierarchicalTimelineEntity {
+  public ContainerEntity() {
+    super(TimelineEntityType.YARN_CONTAINER.toString());
+  }
+}

+ 81 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/FlowEntity.java

@@ -0,0 +1,81 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlElement;
+import javax.xml.bind.annotation.XmlRootElement;
+
+@XmlRootElement(name = "flow")
+@XmlAccessorType(XmlAccessType.NONE)
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public class FlowEntity extends HierarchicalTimelineEntity {
+  private String user;
+  private String version;
+  private String run;
+
+  public FlowEntity() {
+    super(TimelineEntityType.YARN_FLOW.toString());
+  }
+
+  @Override
+  public String getId() {
+    //Flow id schema: user@flow_name(or id)/version/run
+    StringBuilder sb = new StringBuilder();
+    sb.append(user);
+    sb.append('@');
+    sb.append(super.getId());
+    sb.append('/');
+    sb.append(version);
+    sb.append('/');
+    sb.append(run);
+    return sb.toString();
+  }
+
+  @XmlElement(name = "user")
+  public String getUser() {
+    return user;
+  }
+
+  public void setUser(String user) {
+    this.user = user;
+  }
+
+  @XmlElement(name = "version")
+  public String getVersion() {
+    return version;
+  }
+
+  public void setVersion(String version) {
+    this.version = version;
+  }
+
+  @XmlElement(name = "run")
+  public String getRun() {
+    return run;
+  }
+
+  public void setRun(String run) {
+    this.run = run;
+  }
+}

+ 117 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/HierarchicalTimelineEntity.java

@@ -0,0 +1,117 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlElement;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.Map;
+import java.util.Set;
+
+@XmlAccessorType(XmlAccessType.NONE)
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public abstract class HierarchicalTimelineEntity extends TimelineEntity {
+  private Identifier parent;
+  private Map<String, Set<String>> children = new HashMap<>();
+
+  HierarchicalTimelineEntity(String type) {
+    super(type);
+  }
+
+  @XmlElement(name = "parent")
+  public Identifier getParent() {
+    return parent;
+  }
+
+  public void setParent(Identifier parent) {
+    validateParent(parent.getType());
+    this.parent = parent;
+  }
+
+  public void setParent(String type, String id) {
+    validateParent(type);
+    parent = new Identifier();
+    parent.setType(type);
+    parent.setId(id);
+  }
+
+  @XmlElement(name = "children")
+  public Map<String, Set<String>> getChildren() {
+    return children;
+  }
+
+  public void setChildren(Map<String, Set<String>> children) {
+    validateChildren(children);
+    this.children = children;
+  }
+
+  public void addChildren(Map<String, Set<String>> children) {
+    validateChildren(children);
+    for (Map.Entry<String, Set<String>> entry : children.entrySet()) {
+      Set<String> ids = this.children.get(entry.getKey());
+      if (ids == null) {
+        ids = new HashSet<>();
+        this.children.put(entry.getKey(), ids);
+      }
+      ids.addAll(entry.getValue());
+    }
+  }
+
+  public void addChild(String type, String id) {
+    TimelineEntityType thisType = TimelineEntityType.valueOf(getType());
+    TimelineEntityType childType = TimelineEntityType.valueOf(type);
+    if (thisType.isChild(childType)) {
+      Set<String> ids = children.get(type);
+      if (ids == null) {
+        ids = new HashSet<>();
+        children.put(type, ids);
+      }
+      ids.add(id);
+    } else {
+      throw new IllegalArgumentException(
+          type + " is not the acceptable child of " + this.getType());
+    }
+  }
+
+  private void validateParent(String type) {
+    TimelineEntityType parentType = TimelineEntityType.valueOf(type);
+    TimelineEntityType thisType = TimelineEntityType.valueOf(getType());
+    if (!thisType.isParent(parentType)) {
+      throw new IllegalArgumentException(
+          type + " is not the acceptable parent of " + this.getType());
+    }
+  }
+
+  private void validateChildren(Map<String, Set<String>> children) {
+    TimelineEntityType thisType = TimelineEntityType.valueOf(getType());
+    for (Map.Entry<String, Set<String>> entry : children.entrySet()) {
+      TimelineEntityType childType = TimelineEntityType.valueOf(entry.getKey());
+      if (!thisType.isChild(childType)) {
+        throw new IllegalArgumentException(
+            entry.getKey() + " is not the acceptable child of " +
+                this.getType());
+      }
+    }
+  }
+}

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

@@ -0,0 +1,260 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlElement;
+import javax.xml.bind.annotation.XmlRootElement;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.Map;
+import java.util.Set;
+
+@XmlRootElement(name = "entity")
+@XmlAccessorType(XmlAccessType.NONE)
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public class TimelineEntity {
+
+  @XmlRootElement(name = "identifier")
+  @XmlAccessorType(XmlAccessType.NONE)
+  public static class Identifier {
+    private String type;
+    private String id;
+
+    public Identifier() {
+
+    }
+
+    @XmlElement(name = "type")
+    public String getType() {
+      return type;
+    }
+
+    public void setType(String type) {
+      this.type = type;
+    }
+
+    @XmlElement(name = "id")
+    public String getId() {
+      return id;
+    }
+
+    public void setId(String id) {
+      this.id = id;
+    }
+  }
+
+  private Identifier identifier;
+  private Map<String, Object> info = new HashMap<>();
+  private Map<String, Object> configs = new HashMap<>();
+  private Set<TimelineMetric> metrics = new HashSet<>();
+  private Set<TimelineEvent> events = new HashSet<>();
+  private Map<String, Set<String>> isRelatedToEntities = new HashMap<>();
+  private Map<String, Set<String>> relatesToEntities = new HashMap<>();
+  private long createdTime;
+  private long modifiedTime;
+
+  public TimelineEntity() {
+    identifier = new Identifier();
+  }
+
+  protected TimelineEntity(String type) {
+    this();
+    identifier.type = type;
+  }
+
+  @XmlElement(name = "type")
+  public String getType() {
+    return identifier.type;
+  }
+
+  public void setType(String type) {
+    identifier.type = type;
+  }
+
+  @XmlElement(name = "id")
+  public String getId() {
+    return identifier.id;
+  }
+
+  public void setId(String id) {
+    identifier.id = id;
+  }
+
+  public Identifier getIdentifier() {
+    return identifier;
+  }
+
+  public void setIdentifier(Identifier identifier) {
+    this.identifier = identifier;
+  }
+
+  @XmlElement(name = "info")
+  public Map<String, Object> getInfo() {
+    return info;
+  }
+
+  public void setInfo(Map<String, Object> info) {
+    this.info = info;
+  }
+
+  public void addInfo(Map<String, Object> info) {
+    this.info.putAll(info);
+  }
+
+  public void addInfo(String key, Object value) {
+    info.put(key, value);
+  }
+
+  @XmlElement(name = "configs")
+  public Map<String, Object> getConfigs() {
+    return configs;
+  }
+
+  public void setConfigs(Map<String, Object> configs) {
+    this.configs = configs;
+  }
+
+  public void addConfigs(Map<String, Object> configs) {
+    this.configs.putAll(configs);
+  }
+
+  public void addConfig(String key, Object value) {
+    configs.put(key, value);
+  }
+
+  @XmlElement(name = "metrics")
+  public Set<TimelineMetric> getMetrics() {
+    return metrics;
+  }
+
+  public void setMetrics(Set<TimelineMetric> metrics) {
+    this.metrics = metrics;
+  }
+
+  public void addMetrics(Set<TimelineMetric> metrics) {
+    this.metrics.addAll(metrics);
+  }
+
+  public void addMetric(TimelineMetric metric) {
+    metrics.add(metric);
+  }
+
+  @XmlElement(name = "events")
+  public Set<TimelineEvent> getEvents() {
+    return events;
+  }
+
+  public void setEvents(Set<TimelineEvent> events) {
+    this.events = events;
+  }
+
+  public void addEvents(Set<TimelineEvent> events) {
+    this.events.addAll(events);
+  }
+
+  public void addEvent(TimelineEvent event) {
+    events.add(event);
+  }
+
+  @XmlElement(name = "isrelatedto")
+  public Map<String, Set<String>> getIsRelatedToEntities() {
+    return isRelatedToEntities;
+  }
+
+  public void setIsRelatedToEntities(
+      Map<String, Set<String>> isRelatedToEntities) {
+    this.isRelatedToEntities = isRelatedToEntities;
+  }
+
+  public void addIsRelatedToEntities(
+      Map<String, Set<String>> isRelatedToEntities) {
+    for (Map.Entry<String, Set<String>> entry : isRelatedToEntities
+        .entrySet()) {
+      Set<String> ids = this.isRelatedToEntities.get(entry.getKey());
+      if (ids == null) {
+        ids = new HashSet<>();
+        this.isRelatedToEntities.put(entry.getKey(), ids);
+      }
+      ids.addAll(entry.getValue());
+    }
+  }
+
+  public void addIsRelatedToEntity(String type, String id) {
+    Set<String> ids = isRelatedToEntities.get(type);
+    if (ids == null) {
+      ids = new HashSet<>();
+      isRelatedToEntities.put(type, ids);
+    }
+    ids.add(id);
+  }
+
+  @XmlElement(name = "relatesto")
+  public Map<String, Set<String>> getRelatesToEntities() {
+    return relatesToEntities;
+  }
+
+  public void addRelatesToEntities(Map<String, Set<String>> relatesToEntities) {
+    for (Map.Entry<String, Set<String>> entry : relatesToEntities.entrySet()) {
+      Set<String> ids = this.relatesToEntities.get(entry.getKey());
+      if (ids == null) {
+        ids = new HashSet<>();
+        this.relatesToEntities.put(entry.getKey(), ids);
+      }
+      ids.addAll(entry.getValue());
+    }
+  }
+
+  public void addRelatesToEntity(String type, String id) {
+    Set<String> ids = relatesToEntities.get(type);
+    if (ids == null) {
+      ids = new HashSet<>();
+      relatesToEntities.put(type, ids);
+    }
+    ids.add(id);
+  }
+
+  public void setRelatesToEntities(Map<String, Set<String>> relatesToEntities) {
+    this.relatesToEntities = relatesToEntities;
+  }
+
+  @XmlElement(name = "createdtime")
+  public long getCreatedTime() {
+    return createdTime;
+  }
+
+  public void setCreatedTime(long createdTime) {
+    this.createdTime = createdTime;
+  }
+
+  @XmlElement(name = "modifiedtime")
+  public long getModifiedTime() {
+    return modifiedTime;
+  }
+
+  public void setModifiedTime(long modifiedTime) {
+    this.modifiedTime = modifiedTime;
+  }
+
+
+}

+ 71 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/TimelineEntityType.java

@@ -0,0 +1,71 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public enum TimelineEntityType {
+  YARN_CLUSTER,
+  YARN_FLOW,
+  YARN_APPLICATION,
+  YARN_APPLICATION_ATTEMPT,
+  YARN_CONTAINER,
+  YARN_USER,
+  YARN_QUEUE;
+
+  public boolean isParent(TimelineEntityType type) {
+    switch (this) {
+      case YARN_CLUSTER:
+        return false;
+      case YARN_FLOW:
+        return YARN_FLOW == type || YARN_CLUSTER == type;
+      case YARN_APPLICATION:
+        return YARN_FLOW == type || YARN_CLUSTER == type;
+      case YARN_APPLICATION_ATTEMPT:
+        return YARN_APPLICATION == type;
+      case YARN_CONTAINER:
+        return YARN_APPLICATION_ATTEMPT == type;
+      case YARN_QUEUE:
+        return YARN_QUEUE == type;
+      default:
+        return false;
+    }
+  }
+
+  public boolean isChild(TimelineEntityType type) {
+    switch (this) {
+      case YARN_CLUSTER:
+        return YARN_FLOW == type || YARN_APPLICATION == type;
+      case YARN_FLOW:
+        return YARN_FLOW == type || YARN_APPLICATION == type;
+      case YARN_APPLICATION:
+        return YARN_APPLICATION_ATTEMPT == type;
+      case YARN_APPLICATION_ATTEMPT:
+        return YARN_CONTAINER == type;
+      case YARN_CONTAINER:
+        return false;
+      case YARN_QUEUE:
+        return YARN_QUEUE == type;
+      default:
+        return false;
+    }
+  }
+}

+ 77 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/TimelineEvent.java

@@ -0,0 +1,77 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlElement;
+import javax.xml.bind.annotation.XmlRootElement;
+import java.util.HashMap;
+import java.util.Map;
+
+@XmlRootElement(name = "event")
+@XmlAccessorType(XmlAccessType.NONE)
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public class TimelineEvent {
+  private String id;
+  private Map<String, Object> info = new HashMap<>();
+  private long timestamp;
+
+  public TimelineEvent() {
+
+  }
+
+  @XmlElement(name = "id")
+  public String getId() {
+    return id;
+  }
+
+  public void setId(String id) {
+    this.id = id;
+  }
+
+  @XmlElement(name = "info")
+  public Map<String, Object> getInfo() {
+    return info;
+  }
+
+  public void setInfo(Map<String, Object> info) {
+    this.info = info;
+  }
+
+  public void addInfo(Map<String, Object> info) {
+    this.info.putAll(info);
+  }
+
+  public void addInfo(String key, Object value) {
+    info.put(key, value);
+  }
+
+  @XmlElement(name = "timestamp")
+  public long getTimestamp() {
+    return timestamp;
+  }
+
+  public void setTimestamp(long timestamp) {
+    this.timestamp = timestamp;
+  }
+}

+ 116 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/TimelineMetric.java

@@ -0,0 +1,116 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlElement;
+import javax.xml.bind.annotation.XmlRootElement;
+import java.util.HashMap;
+import java.util.LinkedHashMap;
+import java.util.Map;
+
+@XmlRootElement(name = "metric")
+@XmlAccessorType(XmlAccessType.NONE)
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public class TimelineMetric {
+  private String id;
+  private Map<String, Object> info = new HashMap<>();
+  private Object singleData;
+  private Map<Long, Object> timeSeries = new LinkedHashMap<>();
+  private long startTime;
+  private long endTime;
+
+  public TimelineMetric() {
+
+  }
+
+  @XmlElement(name = "id")
+  public String getId() {
+    return id;
+  }
+
+  public void setId(String id) {
+    this.id = id;
+  }
+
+  @XmlElement(name = "info")
+  public Map<String, Object> getInfo() {
+    return info;
+  }
+
+  public void setInfo(Map<String, Object> info) {
+    this.info = info;
+  }
+
+  public void addInfo(Map<String, Object> info) {
+    this.info.putAll(info);
+  }
+
+  public void addInfo(String key, Object value) {
+    info.put(key, value);
+  }
+
+  @XmlElement(name = "data")
+  public Object getSingleData() {
+    return singleData;
+  }
+
+  public void setSingleData(Object singleData) {
+    this.singleData = singleData;
+  }
+
+  @XmlElement(name = "timeseries")
+  public Map<Long, Object> getTimeSeries() {
+    return timeSeries;
+  }
+
+  public void setTimeSeries(Map<Long, Object> timeSeries) {
+    this.timeSeries = timeSeries;
+  }
+
+  public void addTimeSeries(Map<Long, Object> timeSeries) {
+    this.timeSeries.putAll(timeSeries);
+  }
+
+  public void addTimeSeriesData(long timestamp, Object value) {
+    timeSeries.put(timestamp, value);
+  }
+
+  @XmlElement(name = "starttime")
+  public long getStartTime() {
+    return startTime;
+  }
+
+  public void setStartTime(long startTime) {
+    this.startTime = startTime;
+  }
+
+  @XmlElement(name = "endtime")
+  public long getEndTime() {
+    return endTime;
+  }
+
+  public void setEndTime(long endTime) {
+    this.endTime = endTime;
+  }
+}

+ 35 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/TimelineQueue.java

@@ -0,0 +1,35 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlRootElement;
+
+@XmlRootElement(name = "queue")
+@XmlAccessorType(XmlAccessType.NONE)
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public class TimelineQueue extends HierarchicalTimelineEntity {
+  public TimelineQueue() {
+    super(TimelineEntityType.YARN_QUEUE.toString());
+  }
+}

+ 35 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/TimelineUser.java

@@ -0,0 +1,35 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+import javax.xml.bind.annotation.XmlAccessType;
+import javax.xml.bind.annotation.XmlAccessorType;
+import javax.xml.bind.annotation.XmlRootElement;
+
+@XmlRootElement(name = "user")
+@XmlAccessorType(XmlAccessType.NONE)
+@InterfaceAudience.Public
+@InterfaceStability.Unstable
+public class TimelineUser extends TimelineEntity {
+  public TimelineUser() {
+    super(TimelineEntityType.YARN_USER.toString());
+  }
+}

+ 21 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-api/src/main/java/org/apache/hadoop/yarn/api/records/timelineservice/package-info.java

@@ -0,0 +1,21 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+@InterfaceAudience.Public package org.apache.hadoop.yarn.api.records.timelineservice;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+

+ 165 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/test/java/org/apache/hadoop/yarn/api/records/timelineservice/TestTimelineServiceRecords.java

@@ -0,0 +1,165 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.yarn.api.records.timelineservice;
+
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.yarn.api.records.ApplicationAttemptId;
+import org.apache.hadoop.yarn.api.records.ApplicationId;
+import org.apache.hadoop.yarn.api.records.ContainerId;
+import org.apache.hadoop.yarn.util.timeline.TimelineUtils;
+import org.junit.Test;
+
+
+public class TestTimelineServiceRecords {
+  private static final Log LOG =
+      LogFactory.getLog(TestTimelineServiceRecords.class);
+
+  @Test
+  public void testTimelineEntities() throws Exception {
+    TimelineEntity entity = new TimelineEntity();
+    entity.setType("test type 1");
+    entity.setId("test id 1");
+    entity.addInfo("test info key 1", "test info value 1");
+    entity.addInfo("test info key 2", "test info value 2");
+    entity.addConfig("test config key 1", "test config value 1");
+    entity.addConfig("test config key 2", "test config value 2");
+    TimelineMetric metric1 = new TimelineMetric();
+    metric1.setId("test metric id 1");
+    metric1.addInfo("test info key 1", "test info value 1");
+    metric1.addInfo("test info key 2", "test info value 2");
+    metric1.addTimeSeriesData(1L, "test time series 1");
+    metric1.addTimeSeriesData(2L, "test time series 2");
+    metric1.setStartTime(0L);
+    metric1.setEndTime(1L);
+    entity.addMetric(metric1);
+    TimelineMetric metric2 = new TimelineMetric();
+    metric2.setId("test metric id 1");
+    metric2.addInfo("test info key 1", "test info value 1");
+    metric2.addInfo("test info key 2", "test info value 2");
+    metric2.setSingleData("test info value 3");
+    metric1.setStartTime(0L);
+    metric1.setEndTime(1L);
+    entity.addMetric(metric2);
+    TimelineEvent event1 = new TimelineEvent();
+    event1.setId("test event id 1");
+    event1.addInfo("test info key 1", "test info value 1");
+    event1.addInfo("test info key 2", "test info value 2");
+    event1.setTimestamp(0L);
+    entity.addEvent(event1);
+    TimelineEvent event2 = new TimelineEvent();
+    event2.setId("test event id 2");
+    event2.addInfo("test info key 1", "test info value 1");
+    event2.addInfo("test info key 2", "test info value 2");
+    event2.setTimestamp(1L);
+    entity.addEvent(event2);
+    entity.setCreatedTime(0L);
+    entity.setModifiedTime(1L);
+    entity.addRelatesToEntity("test type 2", "test id 2");
+    entity.addRelatesToEntity("test type 3", "test id 3");
+    entity.addIsRelatedToEntity("test type 4", "test id 4");
+    entity.addIsRelatedToEntity("test type 5", "test id 5");
+    LOG.info(TimelineUtils.dumpTimelineRecordtoJSON(entity, true));
+  }
+
+  @Test
+  public void testFirstClassCitizenEntities() throws Exception {
+    TimelineUser user = new TimelineUser();
+    user.setId("test user id");
+
+    TimelineQueue queue = new TimelineQueue();
+    queue.setId("test queue id");
+
+
+    ClusterEntity cluster = new ClusterEntity();
+    cluster.setId("test cluster id");
+
+    FlowEntity flow1 = new FlowEntity();
+    flow1.setId("test flow id");
+    flow1.setUser(user.getId());
+    flow1.setVersion("test flow version");
+    flow1.setRun("test run 1");
+
+    FlowEntity flow2 = new FlowEntity();
+    flow2.setId("test flow run id2");
+    flow2.setUser(user.getId());
+    flow1.setVersion("test flow version2");
+    flow2.setRun("test run 2");
+
+    ApplicationEntity app = new ApplicationEntity();
+    app.setId(ApplicationId.newInstance(0, 1).toString());
+    app.setQueue(queue.getId());
+
+    ApplicationAttemptEntity appAttempt = new ApplicationAttemptEntity();
+    appAttempt.setId(ApplicationAttemptId.newInstance(
+        ApplicationId.newInstance(0, 1), 1).toString());
+
+    ContainerEntity container = new ContainerEntity();
+    container.setId(ContainerId.newContainerId(
+        ApplicationAttemptId.newInstance(
+            ApplicationId.newInstance(0, 1), 1), 1).toString());
+
+    cluster.addChild(TimelineEntityType.YARN_FLOW.toString(), flow1.getId());
+    flow1
+        .setParent(TimelineEntityType.YARN_CLUSTER.toString(), cluster.getId());
+    flow1.addChild(TimelineEntityType.YARN_FLOW.toString(), flow2.getId());
+    flow2.setParent(TimelineEntityType.YARN_FLOW.toString(), flow1.getId());
+    flow2.addChild(TimelineEntityType.YARN_APPLICATION.toString(), app.getId());
+    app.setParent(TimelineEntityType.YARN_FLOW.toString(), flow2.getId());
+    app.addChild(TimelineEntityType.YARN_APPLICATION_ATTEMPT.toString(),
+        appAttempt.getId());
+    appAttempt
+        .setParent(TimelineEntityType.YARN_APPLICATION.toString(), app.getId());
+    appAttempt.addChild(TimelineEntityType.YARN_CONTAINER.toString(),
+        container.getId());
+    container.setParent(TimelineEntityType.YARN_APPLICATION_ATTEMPT.toString(),
+        appAttempt.getId());
+
+    LOG.info(TimelineUtils.dumpTimelineRecordtoJSON(cluster, true));
+    LOG.info(TimelineUtils.dumpTimelineRecordtoJSON(flow1, true));
+    LOG.info(TimelineUtils.dumpTimelineRecordtoJSON(flow2, true));
+    LOG.info(TimelineUtils.dumpTimelineRecordtoJSON(app, true));
+    LOG.info(TimelineUtils.dumpTimelineRecordtoJSON(appAttempt, true));
+    LOG.info(TimelineUtils.dumpTimelineRecordtoJSON(container, true));
+  }
+
+  @Test
+  public void testUser() throws Exception {
+    TimelineUser user = new TimelineUser();
+    user.setId("test user id");
+    user.addInfo("test info key 1", "test info value 1");
+    user.addInfo("test info key 2", "test info value 2");
+    LOG.info(TimelineUtils.dumpTimelineRecordtoJSON(user, true));
+  }
+
+  @Test
+  public void testQueue() throws Exception {
+    TimelineQueue queue = new TimelineQueue();
+    queue.setId("test queue id");
+    queue.addInfo("test info key 1", "test info value 1");
+    queue.addInfo("test info key 2", "test info value 2");
+    queue.setParent(TimelineEntityType.YARN_QUEUE.toString(),
+        "test parent queue id");
+    queue.addChild(TimelineEntityType.YARN_QUEUE.toString(),
+        "test child queue id 1");
+    queue.addChild(TimelineEntityType.YARN_QUEUE.toString(),
+        "test child queue id 2");
+    LOG.info(TimelineUtils.dumpTimelineRecordtoJSON(queue, true));
+  }
+}