You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@apex.apache.org by pr...@apache.org on 2016/12/15 22:23:37 UTC

[1/2] apex-core git commit: APEXCORE-572 - Remove dependency on hadoop-common test.jar. Implemented similar functionality in JarHelper.

Repository: apex-core
Updated Branches:
  refs/heads/master b74d68967 -> 3f06ce71c


APEXCORE-572 - Remove dependency on hadoop-common test.jar. Implemented similar functionality in JarHelper.


Project: http://git-wip-us.apache.org/repos/asf/apex-core/repo
Commit: http://git-wip-us.apache.org/repos/asf/apex-core/commit/5ee715ea
Tree: http://git-wip-us.apache.org/repos/asf/apex-core/tree/5ee715ea
Diff: http://git-wip-us.apache.org/repos/asf/apex-core/diff/5ee715ea

Branch: refs/heads/master
Commit: 5ee715ea5ec49130f6d9fae005404a332b420471
Parents: a9e4e05
Author: Vlad Rozov <v....@datatorrent.com>
Authored: Fri Nov 4 15:01:08 2016 -0700
Committer: Vlad Rozov <v....@datatorrent.com>
Committed: Tue Dec 13 08:59:57 2016 -0800

----------------------------------------------------------------------
 .../org/apache/apex/common/util/JarHelper.java  | 186 +++++++++++++++++++
 .../apache/apex/common/util/package-info.java   |  22 +++
 .../apache/apex/common/util/JarHelperTest.java  | 110 +++++++++++
 engine/pom.xml                                  |  24 ---
 .../java/com/datatorrent/stram/StramClient.java |  23 +--
 .../datatorrent/stram/StramMiniClusterTest.java |   9 +-
 .../datatorrent/stram/StramRecoveryTest.java    |   4 +-
 7 files changed, 330 insertions(+), 48 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/apex-core/blob/5ee715ea/common/src/main/java/org/apache/apex/common/util/JarHelper.java
----------------------------------------------------------------------
diff --git a/common/src/main/java/org/apache/apex/common/util/JarHelper.java b/common/src/main/java/org/apache/apex/common/util/JarHelper.java
new file mode 100644
index 0000000..91ba117
--- /dev/null
+++ b/common/src/main/java/org/apache/apex/common/util/JarHelper.java
@@ -0,0 +1,186 @@
+/**
+ * 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.apex.common.util;
+
+import java.io.File;
+import java.io.FileInputStream;
+import java.io.FileOutputStream;
+import java.io.IOException;
+import java.net.JarURLConnection;
+import java.net.URL;
+import java.nio.file.FileVisitResult;
+import java.nio.file.Files;
+import java.nio.file.Path;
+import java.nio.file.SimpleFileVisitor;
+import java.nio.file.attribute.BasicFileAttributes;
+import java.security.CodeSource;
+import java.util.HashMap;
+import java.util.Map;
+import java.util.jar.JarEntry;
+import java.util.jar.JarFile;
+import java.util.jar.JarOutputStream;
+import java.util.jar.Manifest;
+
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import org.apache.commons.io.IOUtils;
+import org.apache.hadoop.classification.InterfaceAudience;
+import org.apache.hadoop.classification.InterfaceStability;
+
+@InterfaceAudience.Public
+@InterfaceStability.Evolving
+public class JarHelper
+{
+  private static final Logger logger = LoggerFactory.getLogger(JarHelper.class);
+
+  private final Map<URL, String> sourceToJar = new HashMap<>();
+
+  public static String createJar(String prefix, File dir, boolean deleteOnExit) throws IOException
+  {
+    if (!dir.exists() || !dir.isDirectory()) {
+      throw new IllegalArgumentException(String.format("dir %s must be an existing directory.", dir));
+    }
+    File temp = File.createTempFile(prefix, ".jar");
+    if (deleteOnExit) {
+      temp.deleteOnExit();
+    }
+    new JarCreator(temp).createJar(dir);
+    return temp.getAbsolutePath();
+  }
+
+  public String getJar(Class<?> jarClass)
+  {
+    String jar = null;
+    final CodeSource codeSource = jarClass.getProtectionDomain().getCodeSource();
+    if (codeSource != null) {
+      URL location = codeSource.getLocation();
+      jar = sourceToJar.get(location);
+      if (jar == null) {
+        // don't create jar file from folders multiple times
+        if ("jar".equals(location.getProtocol())) {
+          try {
+            location = ((JarURLConnection)location.openConnection()).getJarFileURL();
+          } catch (IOException e) {
+            throw new AssertionError("Cannot resolve jar file for " + jarClass, e);
+          }
+        }
+        if ("file".equals(location.getProtocol())) {
+          jar = location.getFile();
+          final File dir = new File(jar);
+          if (dir.isDirectory()) {
+            try {
+              jar = createJar("apex-", dir, false);
+            } catch (IOException e) {
+              throw new AssertionError("Cannot resolve jar file for " + jarClass + ". URL " + location, e);
+            }
+          }
+        } else {
+          throw new AssertionError("Cannot resolve jar file for " + jarClass + ". URL " + location);
+        }
+        sourceToJar.put(location, jar);
+        logger.debug("added sourceLocation {} as {}", location, jar);
+      }
+      if (jar == null) {
+        throw new AssertionError("Cannot resolve jar file for " + jarClass);
+      }
+    }
+    return jar;
+  }
+
+  private static class JarCreator
+  {
+
+    private final JarOutputStream jos;
+
+    private JarCreator(File file) throws IOException
+    {
+      jos = new JarOutputStream(new FileOutputStream(file));
+    }
+
+    private void createJar(File dir) throws IOException
+    {
+      try {
+        File manifestFile = new File(dir, JarFile.MANIFEST_NAME);
+        if (!manifestFile.exists()) {
+          jos.putNextEntry(new JarEntry(JarFile.MANIFEST_NAME));
+          new Manifest().write(jos);
+          jos.closeEntry();
+        } else {
+          addEntry(manifestFile, JarFile.MANIFEST_NAME);
+        }
+        final Path root = dir.toPath();
+        Files.walkFileTree(root,
+            new SimpleFileVisitor<Path>()
+            {
+              String relativePath;
+
+              @Override
+              public FileVisitResult preVisitDirectory(Path dir, BasicFileAttributes attrs) throws IOException
+              {
+                relativePath = root.relativize(dir).toString();
+                if (!relativePath.isEmpty()) {
+                  if (!relativePath.endsWith("/")) {
+                    relativePath += "/";
+                  }
+                  addEntry(dir.toFile(), relativePath);
+                }
+                return super.preVisitDirectory(dir, attrs);
+              }
+
+              @Override
+              public FileVisitResult visitFile(Path file, BasicFileAttributes attrs) throws IOException
+              {
+                String name = relativePath + file.getFileName().toString();
+                if (!JarFile.MANIFEST_NAME.equals(name)) {
+                  addEntry(file.toFile(), relativePath + file.getFileName().toString());
+                }
+                return super.visitFile(file, attrs);
+              }
+
+              @Override
+              public FileVisitResult postVisitDirectory(Path dir, IOException exc) throws IOException
+              {
+                relativePath = root.relativize(dir.getParent()).toString();
+                if (!relativePath.isEmpty() && !relativePath.endsWith("/")) {
+                  relativePath += "/";
+                }
+                return super.postVisitDirectory(dir, exc);
+              }
+            }
+        );
+      } finally {
+        jos.close();
+      }
+    }
+
+    private void addEntry(File file, String name) throws IOException
+    {
+      final JarEntry ze = new JarEntry(name);
+      ze.setTime(file.lastModified());
+      jos.putNextEntry(ze);
+      if (file.isFile()) {
+        try (final FileInputStream input = new FileInputStream(file)) {
+          IOUtils.copy(input, jos);
+        }
+      }
+      jos.closeEntry();
+    }
+  }
+}

http://git-wip-us.apache.org/repos/asf/apex-core/blob/5ee715ea/common/src/main/java/org/apache/apex/common/util/package-info.java
----------------------------------------------------------------------
diff --git a/common/src/main/java/org/apache/apex/common/util/package-info.java b/common/src/main/java/org/apache/apex/common/util/package-info.java
new file mode 100644
index 0000000..24839d0
--- /dev/null
+++ b/common/src/main/java/org/apache/apex/common/util/package-info.java
@@ -0,0 +1,22 @@
+/**
+ * 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.
+ */
+/**
+ * Shared utilities
+ */
+package org.apache.apex.common.util;

http://git-wip-us.apache.org/repos/asf/apex-core/blob/5ee715ea/common/src/test/java/org/apache/apex/common/util/JarHelperTest.java
----------------------------------------------------------------------
diff --git a/common/src/test/java/org/apache/apex/common/util/JarHelperTest.java b/common/src/test/java/org/apache/apex/common/util/JarHelperTest.java
new file mode 100644
index 0000000..ef142db
--- /dev/null
+++ b/common/src/test/java/org/apache/apex/common/util/JarHelperTest.java
@@ -0,0 +1,110 @@
+/**
+ * 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.apex.common.util;
+
+import java.io.FileOutputStream;
+import java.nio.file.Files;
+import java.util.jar.Attributes;
+import java.util.jar.JarFile;
+import java.util.jar.Manifest;
+import java.util.zip.ZipEntry;
+
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import static org.junit.Assert.assertArrayEquals;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertNotNull;
+import static org.junit.Assert.assertNull;
+import static org.junit.Assert.assertSame;
+import static org.junit.Assert.assertTrue;
+
+public class JarHelperTest
+{
+  private static final Logger logger = LoggerFactory.getLogger(JarHelperTest.class);
+
+  private static final String file = "file";
+  private static final byte[] data = "data".getBytes();
+  private static final String dir = "dir/";
+  private static final String META = "META-INF/";
+  private static final String version = "1.0";
+
+  @Rule
+  public TemporaryFolder folder = new TemporaryFolder();
+
+  @Test
+  public void createJar() throws Exception
+  {
+    JarFile jar = new JarFile(JarHelper.createJar("apex", folder.getRoot(), true));
+    logger.debug("Created jar {} with MANIFEST only.", jar.getName());
+    assertNotNull("MANIFEST exists", jar.getManifest());
+    assertNull(jar.getEntry(file));
+    assertNull(jar.getEntry(dir));
+    jar.close();
+
+    Files.write(folder.newFile(file).toPath(), data);
+    folder.newFolder(dir);
+    jar = new JarFile(JarHelper.createJar("apex", folder.getRoot(), true));
+    logger.debug("Created jar {} with a file and a directory.", jar.getName());
+    assertNotNull("MANIFEST exists", jar.getManifest());
+    ZipEntry entry = jar.getEntry(file);
+    assertNotNull(entry);
+    assertFalse(entry.isDirectory());
+    byte[] data = new byte[JarHelperTest.data.length];
+    jar.getInputStream(entry).read(data);
+    assertArrayEquals(data, JarHelperTest.data);
+    assertTrue(jar.getEntry(dir).isDirectory());
+    jar.close();
+
+    folder.newFolder(META);
+    Manifest manifest = new Manifest();
+    manifest.getMainAttributes().put(Attributes.Name.MANIFEST_VERSION, version);
+    manifest.write(new FileOutputStream(folder.newFile(JarFile.MANIFEST_NAME)));
+    jar = new JarFile(JarHelper.createJar(JarHelperTest.class.getSimpleName(), folder.getRoot(), true));
+    logger.debug("Created jar {} with a file, a directory and a MANIFEST.", jar.getName());
+    assertEquals("MANIFEST version", jar.getManifest().getMainAttributes().getValue(Attributes.Name.MANIFEST_VERSION), version);
+    entry = jar.getEntry(file);
+    assertNotNull(entry);
+    assertFalse(entry.isDirectory());
+    jar.getInputStream(entry).read(data);
+    assertArrayEquals(data, JarHelperTest.data);
+    assertTrue(jar.getEntry(dir).isDirectory());
+    jar.close();
+  }
+
+  @Test
+  public void getJar() throws Exception
+  {
+    final JarHelper jarHelper = new JarHelper();
+
+    assertNull("System jar is null", jarHelper.getJar(Class.class));
+
+    String jar = jarHelper.getJar(JarHelper.class);
+    assertNotNull("JarHelper jar is not null", jar);
+    assertSame(jar, jarHelper.getJar(JarHelper.class));
+
+    jar = jarHelper.getJar(JarHelperTest.class);
+    assertNotNull("JarHelperTest jar is not null", jar);
+    assertSame(jar, jarHelper.getJar(JarHelperTest.class));
+  }
+}

http://git-wip-us.apache.org/repos/asf/apex-core/blob/5ee715ea/engine/pom.xml
----------------------------------------------------------------------
diff --git a/engine/pom.xml b/engine/pom.xml
index 0630931..fa7e5c7 100644
--- a/engine/pom.xml
+++ b/engine/pom.xml
@@ -254,30 +254,6 @@
       <version>1.1.9</version>
     </dependency>
     <dependency>
-      <groupId>org.apache.hadoop</groupId>
-      <artifactId>hadoop-common</artifactId>
-      <version>${hadoop.version}</version>
-      <type>test-jar</type>
-      <exclusions>
-        <exclusion>
-          <groupId>org.codehaus.jackson</groupId>
-          <artifactId>jackson-core-asl</artifactId>
-        </exclusion>
-        <exclusion>
-          <groupId>org.codehaus.jackson</groupId>
-          <artifactId>jackson-mapper-asl</artifactId>
-        </exclusion>
-        <exclusion>
-          <groupId>commons-beanutils</groupId>
-          <artifactId>commons-beanutils</artifactId>
-        </exclusion>
-        <exclusion>
-          <groupId>commons-beanutils</groupId>
-          <artifactId>commons-beanutils-core</artifactId>
-        </exclusion>
-      </exclusions>
-    </dependency>
-    <dependency>
       <groupId>org.mockito</groupId>
       <artifactId>mockito-core</artifactId>
       <version>1.10.19</version>

http://git-wip-us.apache.org/repos/asf/apex-core/blob/5ee715ea/engine/src/main/java/com/datatorrent/stram/StramClient.java
----------------------------------------------------------------------
diff --git a/engine/src/main/java/com/datatorrent/stram/StramClient.java b/engine/src/main/java/com/datatorrent/stram/StramClient.java
index 45e3fbd..d6bcfb4 100644
--- a/engine/src/main/java/com/datatorrent/stram/StramClient.java
+++ b/engine/src/main/java/com/datatorrent/stram/StramClient.java
@@ -36,6 +36,8 @@ import java.util.Set;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
+import org.apache.apex.common.util.JarHelper;
+
 import org.apache.commons.io.IOUtils;
 import org.apache.commons.lang.ArrayUtils;
 import org.apache.commons.lang.StringUtils;
@@ -51,7 +53,6 @@ import org.apache.hadoop.io.DataOutputBuffer;
 import org.apache.hadoop.security.Credentials;
 import org.apache.hadoop.security.UserGroupInformation;
 import org.apache.hadoop.security.token.Token;
-import org.apache.hadoop.util.JarFinder;
 import org.apache.hadoop.yarn.api.ApplicationConstants;
 import org.apache.hadoop.yarn.api.records.ApplicationId;
 import org.apache.hadoop.yarn.api.records.ApplicationReport;
@@ -204,25 +205,13 @@ public class StramClient
     }
 
     LinkedHashSet<String> localJarFiles = new LinkedHashSet<>(); // avoid duplicates
-    HashMap<String, String> sourceToJar = new HashMap<>();
+    JarHelper jarHelper = new JarHelper();
 
     for (Class<?> jarClass : jarClasses) {
-      if (jarClass.getProtectionDomain().getCodeSource() == null) {
-        // system class
-        continue;
-      }
-      String sourceLocation = jarClass.getProtectionDomain().getCodeSource().getLocation().toString();
-      String jar = sourceToJar.get(sourceLocation);
-      if (jar == null) {
-        // don't create jar file from folders multiple times
-        jar = JarFinder.getJar(jarClass);
-        sourceToJar.put(sourceLocation, jar);
-        LOG.debug("added sourceLocation {} as {}", sourceLocation, jar);
-      }
-      if (jar == null) {
-        throw new AssertionError("Cannot resolve jar file for " + jarClass);
+      String jar = jarHelper.getJar(jarClass);
+      if (jar != null) {
+        localJarFiles.add(jar);
       }
-      localJarFiles.add(jar);
     }
 
     String libJarsPath = dag.getValue(Context.DAGContext.LIBRARY_JARS);

http://git-wip-us.apache.org/repos/asf/apex-core/blob/5ee715ea/engine/src/test/java/com/datatorrent/stram/StramMiniClusterTest.java
----------------------------------------------------------------------
diff --git a/engine/src/test/java/com/datatorrent/stram/StramMiniClusterTest.java b/engine/src/test/java/com/datatorrent/stram/StramMiniClusterTest.java
index e2bb362..3ec9882 100644
--- a/engine/src/test/java/com/datatorrent/stram/StramMiniClusterTest.java
+++ b/engine/src/test/java/com/datatorrent/stram/StramMiniClusterTest.java
@@ -43,9 +43,9 @@ import org.junit.Test;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
+import org.apache.apex.common.util.JarHelper;
 import org.apache.commons.lang.StringUtils;
 import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.util.JarFinder;
 import org.apache.hadoop.yarn.api.ApplicationMasterProtocol;
 import org.apache.hadoop.yarn.api.protocolrecords.AllocateRequest;
 import org.apache.hadoop.yarn.api.protocolrecords.AllocateResponse;
@@ -192,10 +192,9 @@ public class StramMiniClusterTest
       LOG.info("Number containers: {}", nr.getNumContainers());
     }
 
-    String appMasterJar = JarFinder.getJar(StreamingAppMaster.class);
-    LOG.info("appmaster jar: " + appMasterJar);
-    String testJar = JarFinder.getJar(StramMiniClusterTest.class);
-    LOG.info("testJar: " + testJar);
+    JarHelper jarHelper = new JarHelper();
+    LOG.info("engine jar: {}", jarHelper.getJar(StreamingAppMaster.class));
+    LOG.info("engine test jar: {}", jarHelper.getJar(StramMiniClusterTest.class));
 
     // create test application
     Properties dagProps = new Properties();

http://git-wip-us.apache.org/repos/asf/apex-core/blob/5ee715ea/engine/src/test/java/com/datatorrent/stram/StramRecoveryTest.java
----------------------------------------------------------------------
diff --git a/engine/src/test/java/com/datatorrent/stram/StramRecoveryTest.java b/engine/src/test/java/com/datatorrent/stram/StramRecoveryTest.java
index 0f56fa6..74e18ee 100644
--- a/engine/src/test/java/com/datatorrent/stram/StramRecoveryTest.java
+++ b/engine/src/test/java/com/datatorrent/stram/StramRecoveryTest.java
@@ -20,6 +20,7 @@ package com.datatorrent.stram;
 
 import java.io.ByteArrayInputStream;
 import java.io.ByteArrayOutputStream;
+import java.io.Closeable;
 import java.io.DataInputStream;
 import java.io.DataOutputStream;
 import java.io.File;
@@ -49,7 +50,6 @@ import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.ipc.RPC;
 import org.apache.hadoop.ipc.RPC.Server;
 import org.apache.hadoop.net.NetUtils;
-import org.apache.hadoop.test.MockitoUtil;
 
 import com.google.common.collect.Lists;
 
@@ -472,7 +472,7 @@ public class StramRecoveryTest
     Configuration conf = new Configuration(false);
     final AtomicBoolean timedout = new AtomicBoolean();
 
-    StreamingContainerUmbilicalProtocol impl = MockitoUtil.mockProtocol(StreamingContainerUmbilicalProtocol.class);
+    StreamingContainerUmbilicalProtocol impl = Mockito.mock(StreamingContainerUmbilicalProtocol.class, Mockito.withSettings().extraInterfaces(Closeable.class));
 
     Mockito.doAnswer(new org.mockito.stubbing.Answer<Void>()
     {


[2/2] apex-core git commit: Merge branch 'APEXCORE-572' of github.com:vrozov/apex-core

Posted by pr...@apache.org.
Merge branch 'APEXCORE-572' of github.com:vrozov/apex-core


Project: http://git-wip-us.apache.org/repos/asf/apex-core/repo
Commit: http://git-wip-us.apache.org/repos/asf/apex-core/commit/3f06ce71
Tree: http://git-wip-us.apache.org/repos/asf/apex-core/tree/3f06ce71
Diff: http://git-wip-us.apache.org/repos/asf/apex-core/diff/3f06ce71

Branch: refs/heads/master
Commit: 3f06ce71cf88c790070d4067766ec5662ac0231f
Parents: b74d689 5ee715e
Author: Pramod Immaneni <pr...@datatorrent.com>
Authored: Thu Dec 15 14:14:04 2016 -0800
Committer: Pramod Immaneni <pr...@datatorrent.com>
Committed: Thu Dec 15 14:14:04 2016 -0800

----------------------------------------------------------------------
 .../org/apache/apex/common/util/JarHelper.java  | 186 +++++++++++++++++++
 .../apache/apex/common/util/package-info.java   |  22 +++
 .../apache/apex/common/util/JarHelperTest.java  | 110 +++++++++++
 engine/pom.xml                                  |  24 ---
 .../java/com/datatorrent/stram/StramClient.java |  23 +--
 .../datatorrent/stram/StramMiniClusterTest.java |   9 +-
 .../datatorrent/stram/StramRecoveryTest.java    |   4 +-
 7 files changed, 330 insertions(+), 48 deletions(-)
----------------------------------------------------------------------