Repository: flink
Updated Branches:
refs/heads/master 786a6cbb3 -> 8afadd459
[FLINK-7778] [build] Shade ZooKeeper dependency (followups)
- Rename the 'flink-shaded-curator-recipes' module to 'flink-shaded-curator',
because it actually contains more curator code than just the recipes.
- Move the exception handling logic of 'ZooKeeperAccess' directly into the
ZooKeeperStateHandleStore
Project: http://git-wip-us.apache.org/repos/asf/flink/repo
Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/8afadd45
Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/8afadd45
Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/8afadd45
Branch: refs/heads/master
Commit: 8afadd459294bad8c8eecd1b0c75f773bfe27704
Parents: d368a07
Author: Stephan Ewen <[email protected]>
Authored: Thu Nov 2 14:47:39 2017 +0100
Committer: Stephan Ewen <[email protected]>
Committed: Thu Nov 2 16:51:25 2017 +0100
----------------------------------------------------------------------
.../store/ZooKeeperMesosWorkerStore.java | 18 ++--
.../zookeeper/ZooKeeperStateHandleStore.java | 10 ++-
.../runtime/zookeeper/ZookeeperAccess.java | 66 ---------------
flink-shaded-curator-recipes/pom.xml | 88 --------------------
flink-shaded-curator/pom.xml | 88 ++++++++++++++++++++
pom.xml | 2 +-
6 files changed, 102 insertions(+), 170 deletions(-)
----------------------------------------------------------------------
http://git-wip-us.apache.org/repos/asf/flink/blob/8afadd45/flink-mesos/src/main/java/org/apache/flink/mesos/runtime/clusterframework/store/ZooKeeperMesosWorkerStore.java
----------------------------------------------------------------------
diff --git
a/flink-mesos/src/main/java/org/apache/flink/mesos/runtime/clusterframework/store/ZooKeeperMesosWorkerStore.java
b/flink-mesos/src/main/java/org/apache/flink/mesos/runtime/clusterframework/store/ZooKeeperMesosWorkerStore.java
index 738f99e..9f2fa44 100644
---
a/flink-mesos/src/main/java/org/apache/flink/mesos/runtime/clusterframework/store/ZooKeeperMesosWorkerStore.java
+++
b/flink-mesos/src/main/java/org/apache/flink/mesos/runtime/clusterframework/store/ZooKeeperMesosWorkerStore.java
@@ -25,10 +25,10 @@ import
org.apache.flink.runtime.zookeeper.ZooKeeperSharedCount;
import org.apache.flink.runtime.zookeeper.ZooKeeperSharedValue;
import org.apache.flink.runtime.zookeeper.ZooKeeperStateHandleStore;
import org.apache.flink.runtime.zookeeper.ZooKeeperVersionedValue;
-import org.apache.flink.runtime.zookeeper.ZookeeperAccess;
import org.apache.flink.util.FlinkException;
import org.apache.mesos.Protos;
+
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@@ -214,19 +214,11 @@ public class ZooKeeperMesosWorkerStore implements
MesosWorkerStore {
int currentVersion = workersInZooKeeper.exists(path);
if (currentVersion == -1) {
- try {
- workersInZooKeeper.addAndLock(path,
worker);
- LOG.debug("Added {} in ZooKeeper.",
worker);
- } catch (Exception ex) {
- throw
ZookeeperAccess.wrapIfZooKeeperNodeExistsException(ex);
- }
+ workersInZooKeeper.addAndLock(path, worker);
+ LOG.debug("Added {} in ZooKeeper.", worker);
} else {
- try {
- workersInZooKeeper.replace(path,
currentVersion, worker);
- LOG.debug("Updated {} in ZooKeeper.",
worker);
- } catch (Exception ex) {
- throw
ZookeeperAccess.wrapIfZooKeeperNoNodeException(ex);
- }
+ workersInZooKeeper.replace(path,
currentVersion, worker);
+ LOG.debug("Updated {} in ZooKeeper.", worker);
}
}
}
http://git-wip-us.apache.org/repos/asf/flink/blob/8afadd45/flink-runtime/src/main/java/org/apache/flink/runtime/zookeeper/ZooKeeperStateHandleStore.java
----------------------------------------------------------------------
diff --git
a/flink-runtime/src/main/java/org/apache/flink/runtime/zookeeper/ZooKeeperStateHandleStore.java
b/flink-runtime/src/main/java/org/apache/flink/runtime/zookeeper/ZooKeeperStateHandleStore.java
index dc3f7d7..f0d67fd 100644
---
a/flink-runtime/src/main/java/org/apache/flink/runtime/zookeeper/ZooKeeperStateHandleStore.java
+++
b/flink-runtime/src/main/java/org/apache/flink/runtime/zookeeper/ZooKeeperStateHandleStore.java
@@ -41,6 +41,7 @@ import java.io.Serializable;
import java.util.ArrayList;
import java.util.Collection;
import java.util.Collections;
+import java.util.ConcurrentModificationException;
import java.util.List;
import java.util.UUID;
import java.util.concurrent.Executor;
@@ -163,6 +164,9 @@ public class ZooKeeperStateHandleStore<T extends
Serializable> {
success = true;
return storeHandle;
}
+ catch (KeeperException.NodeExistsException e) {
+ throw new ConcurrentModificationException("ZooKeeper
unexpectedly modified", e);
+ }
finally {
if (!success) {
// Cleanup the state handle if it was not
written to ZooKeeper.
@@ -202,8 +206,10 @@ public class ZooKeeperStateHandleStore<T extends
Serializable> {
.withVersion(expectedVersion)
.forPath(path, serializedStateHandle);
success = true;
+ } catch (KeeperException.NoNodeException e) {
+ throw new ConcurrentModificationException("ZooKeeper
unexpectedly modified", e);
} finally {
- if(success) {
+ if (success) {
oldStateHandle.discardState();
} else {
newStateHandle.discardState();
@@ -673,7 +679,7 @@ public class ZooKeeperStateHandleStore<T extends
Serializable> {
}
}
- };
+ }
/**
* Callback interface for remove calls
http://git-wip-us.apache.org/repos/asf/flink/blob/8afadd45/flink-runtime/src/main/java/org/apache/flink/runtime/zookeeper/ZookeeperAccess.java
----------------------------------------------------------------------
diff --git
a/flink-runtime/src/main/java/org/apache/flink/runtime/zookeeper/ZookeeperAccess.java
b/flink-runtime/src/main/java/org/apache/flink/runtime/zookeeper/ZookeeperAccess.java
deleted file mode 100644
index 2c6160a..0000000
---
a/flink-runtime/src/main/java/org/apache/flink/runtime/zookeeper/ZookeeperAccess.java
+++ /dev/null
@@ -1,66 +0,0 @@
-/*
- * 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.flink.runtime.zookeeper;
-
-import org.apache.zookeeper.KeeperException;
-
-import java.util.ConcurrentModificationException;
-
-/**
- * Utility class providing access to relocated zookeeper classes.
- *
- * <p>This class is necessary as flink-runtime relocates its ZooKeeper
dependency.
- * Other modules may still depend on this dependency but will encounter a
ClassNotFoundException
- * on access as they don't apply the relocation pattern of flink-runtime.
- */
-public final class ZookeeperAccess {
-
- private ZookeeperAccess(){
- }
-
- /**
- * Wraps and returns the given exception in a {@link
ConcurrentModificationException} if it is a
- * {@link org.apache.zookeeper.KeeperException.NodeExistsException}.
Otherwise the
- * given exception is returned.
- *
- * @param ex exception to wrap
- * @return wrapping ConcurrentModificationException if it is a
NodeExistsException, otherwise the given exception
- */
- public static Exception wrapIfZooKeeperNodeExistsException(Exception
ex) {
- if (ex instanceof KeeperException.NodeExistsException) {
- return new ConcurrentModificationException("ZooKeeper
unexpectedly modified", ex);
- }
- return ex;
- }
-
- /**
- * Wraps and returns the given exception in a {@link
ConcurrentModificationException} if it is a
- * {@link org.apache.zookeeper.KeeperException.NoNodeException}.
Otherwise the
- * given exception is returned.
- *
- * @param ex exception to wrap
- * @return wrapping ConcurrentModificationException if it is a
NoNodeException, otherwise the given exception
- */
- public static Exception wrapIfZooKeeperNoNodeException(Exception ex) {
- if (ex instanceof KeeperException.NoNodeException) {
- return new ConcurrentModificationException("ZooKeeper
unexpectedly modified", ex);
- }
- return ex;
- }
-}
http://git-wip-us.apache.org/repos/asf/flink/blob/8afadd45/flink-shaded-curator-recipes/pom.xml
----------------------------------------------------------------------
diff --git a/flink-shaded-curator-recipes/pom.xml
b/flink-shaded-curator-recipes/pom.xml
deleted file mode 100644
index 06d3821..0000000
--- a/flink-shaded-curator-recipes/pom.xml
+++ /dev/null
@@ -1,88 +0,0 @@
-<?xml version="1.0" encoding="UTF-8"?>
-<!--
-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.
--->
-<project xmlns="http://maven.apache.org/POM/4.0.0"
xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
- xsi:schemaLocation="http://maven.apache.org/POM/4.0.0
http://maven.apache.org/maven-v4_0_0.xsd">
-
- <modelVersion>4.0.0</modelVersion>
-
- <parent>
- <groupId>org.apache.flink</groupId>
- <artifactId>flink-parent</artifactId>
- <version>1.4-SNAPSHOT</version>
- <relativePath>..</relativePath>
- </parent>
-
- <artifactId>flink-shaded-curator-recipes</artifactId>
- <name>flink-shaded-curator-recipes</name>
-
- <packaging>jar</packaging>
-
-
- <dependencies>
- <dependency>
- <groupId>org.apache.curator</groupId>
- <artifactId>curator-recipes</artifactId>
- <version>${curator.version}</version>
- </dependency>
- </dependencies>
-
- <build>
- <plugins>
- <plugin>
- <groupId>org.apache.maven.plugins</groupId>
- <artifactId>maven-shade-plugin</artifactId>
- <executions>
- <execution>
- <id>shade-flink</id>
- <phase>package</phase>
- <goals>
- <goal>shade</goal>
- </goals>
- <configuration>
- <artifactSet>
- <includes>
-
<include>com.google.guava:guava</include>
-
<include>org.apache.curator:*</include>
- </includes>
- </artifactSet>
- <filters>
- <filter>
-
<artifact>com.google.guava:guava</artifact>
- <!--
Shade guava classes that are not included by curator -->
-
<includes>
-
<include>com/google/common/base/Function.class</include>
-
<include>com/google/common/base/Predicate.class</include>
-
<include>com/google/common/reflect/TypeToken.class</include>
-
</includes>
- </filter>
- </filters>
- <relocations>
- <relocation>
-
<pattern>com.google.common</pattern>
-
<shadedPattern>org.apache.flink.curator.shaded.com.google.common</shadedPattern>
- </relocation>
- </relocations>
- </configuration>
- </execution>
- </executions>
- </plugin>
- </plugins>
- </build>
-</project>
http://git-wip-us.apache.org/repos/asf/flink/blob/8afadd45/flink-shaded-curator/pom.xml
----------------------------------------------------------------------
diff --git a/flink-shaded-curator/pom.xml b/flink-shaded-curator/pom.xml
new file mode 100644
index 0000000..3c2e80f
--- /dev/null
+++ b/flink-shaded-curator/pom.xml
@@ -0,0 +1,88 @@
+<?xml version="1.0" encoding="UTF-8"?>
+<!--
+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.
+-->
+<project xmlns="http://maven.apache.org/POM/4.0.0"
xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
+ xsi:schemaLocation="http://maven.apache.org/POM/4.0.0
http://maven.apache.org/maven-v4_0_0.xsd">
+
+ <modelVersion>4.0.0</modelVersion>
+
+ <parent>
+ <groupId>org.apache.flink</groupId>
+ <artifactId>flink-parent</artifactId>
+ <version>1.4-SNAPSHOT</version>
+ <relativePath>..</relativePath>
+ </parent>
+
+ <artifactId>flink-shaded-curator</artifactId>
+ <name>flink-shaded-curator</name>
+
+ <packaging>jar</packaging>
+
+
+ <dependencies>
+ <dependency>
+ <groupId>org.apache.curator</groupId>
+ <artifactId>curator-recipes</artifactId>
+ <version>${curator.version}</version>
+ </dependency>
+ </dependencies>
+
+ <build>
+ <plugins>
+ <plugin>
+ <groupId>org.apache.maven.plugins</groupId>
+ <artifactId>maven-shade-plugin</artifactId>
+ <executions>
+ <execution>
+ <id>shade-flink</id>
+ <phase>package</phase>
+ <goals>
+ <goal>shade</goal>
+ </goals>
+ <configuration>
+ <artifactSet>
+ <includes>
+
<include>com.google.guava:guava</include>
+
<include>org.apache.curator:*</include>
+ </includes>
+ </artifactSet>
+ <filters>
+ <filter>
+
<artifact>com.google.guava:guava</artifact>
+ <!--
Shade guava classes that are not included by curator -->
+
<includes>
+
<include>com/google/common/base/Function.class</include>
+
<include>com/google/common/base/Predicate.class</include>
+
<include>com/google/common/reflect/TypeToken.class</include>
+
</includes>
+ </filter>
+ </filters>
+ <relocations>
+ <relocation>
+
<pattern>com.google.common</pattern>
+
<shadedPattern>org.apache.flink.curator.shaded.com.google.common</shadedPattern>
+ </relocation>
+ </relocations>
+ </configuration>
+ </execution>
+ </executions>
+ </plugin>
+ </plugins>
+ </build>
+</project>
http://git-wip-us.apache.org/repos/asf/flink/blob/8afadd45/pom.xml
----------------------------------------------------------------------
diff --git a/pom.xml b/pom.xml
index faddab4..9cf603a 100644
--- a/pom.xml
+++ b/pom.xml
@@ -54,7 +54,7 @@ under the License.
<module>tools/force-shading</module>
<module>flink-annotations</module>
<module>flink-shaded-hadoop</module>
- <module>flink-shaded-curator-recipes</module>
+ <module>flink-shaded-curator</module>
<module>flink-core</module>
<module>flink-java</module>
<module>flink-java8</module>