[2/2] storm git commit: Merge branch 'STORM-3276' of github.com:revans2/incubator-storm into STORM-3276
Merge branch 'STORM-3276' of github.com:revans2/incubator-storm into STORM-3276 STORM-3276: Updated Flux to deal with storm local correctly This closes #2908 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/074fb35a Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/074fb35a Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/074fb35a Branch: refs/heads/master Commit: 074fb35a0a5ab746d5508d2a60b48a7948a2cdda Parents: cf00a53 095bdbc Author: Robert Evans Authored: Fri Dec 21 12:20:57 2018 -0600 Committer: Robert Evans Committed: Fri Dec 21 12:20:57 2018 -0600 -- bin/storm.py| 13 +++- flux/README.md | 60 +- .../main/java/org/apache/storm/flux/Flux.java | 47 +++--- .../java/org/apache/storm/LocalCluster.java | 66 4 files changed, 106 insertions(+), 80 deletions(-) --
[1/2] storm git commit: STORM-3276: Updated Flux to deal with storm local correctly
Repository: storm Updated Branches: refs/heads/master cf00a537b -> 074fb35a0 STORM-3276: Updated Flux to deal with storm local correctly Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/095bdbc2 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/095bdbc2 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/095bdbc2 Branch: refs/heads/master Commit: 095bdbc267fe4354c28baa84186807dcd3be2a31 Parents: cf00a53 Author: Robert (Bobby) Evans Authored: Thu Nov 15 14:12:48 2018 -0600 Committer: Robert (Bobby) Evans Committed: Tue Dec 18 11:21:16 2018 -0600 -- bin/storm.py| 13 +++- flux/README.md | 60 +- .../main/java/org/apache/storm/flux/Flux.java | 47 +++--- .../java/org/apache/storm/LocalCluster.java | 66 4 files changed, 106 insertions(+), 80 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/095bdbc2/bin/storm.py -- diff --git a/bin/storm.py b/bin/storm.py index 013e141..058ada8 100755 --- a/bin/storm.py +++ b/bin/storm.py @@ -338,12 +338,16 @@ def local(jarfile, klass, *args): local also adds in the option --local-ttl which sets the number of seconds the local cluster will run for before it shuts down. +--local-zookeeper if using an external zookeeper sets the connection string to use for it. + --java-debug lets you turn on java debugging and set the parameters passed to -agentlib:jdwp on the JDK --java-debug transport=dt_socket,address=localhost:8000 will open up a debugging server on port 8000. """ -[ttl, debug_args, args] = parse_local_opts(args) +[ttl, lzk, debug_args, args] = parse_local_opts(args) extrajvmopts = ["-Dstorm.local.sleeptime=" + ttl] +if lzk != None: +extrajvmopts = extrajvmopts + ["-Dstorm.local.zookeeper=" + lzk] if debug_args != None: extrajvmopts = extrajvmopts + ["-agentlib:jdwp=" + debug_args] run_client_jar(jarfile, "org.apache.storm.LocalCluster", [klass] + list(args), client=False, daemon=False, extrajvmopts=extrajvmopts) @@ -980,19 +984,22 @@ def parse_local_opts(args): curr = list(args[:]) curr.reverse() ttl = "20" +lzk = None debug_args = None args_list = [] while len(curr) > 0: token = curr.pop() -if token == "--local-ttl": +if token == "--local-zookeeper": +lzk = curr.pop() +elif token == "--local-ttl": ttl = curr.pop() elif token == "--java-debug": debug_args = curr.pop() else: args_list.append(token) -return ttl, debug_args, args_list +return ttl, lzk, debug_args, args_list def parse_jar_opts(args): http://git-wip-us.apache.org/repos/asf/storm/blob/095bdbc2/flux/README.md -- diff --git a/flux/README.md b/flux/README.md index 58ed25d..47bc7d6 100644 --- a/flux/README.md +++ b/flux/README.md @@ -17,36 +17,7 @@ order to change configuration. Flux is a framework and set of utilities that make defining and deploying Apache Storm topologies less painful and deveoper-intensive. -Have you ever found yourself repeating this pattern?: - -```java - -public static void main(String[] args) throws Exception { -// logic to determine if we're running locally or not... -// create necessary config options... -boolean runLocal = shouldRunLocal(); -if(runLocal){ -LocalCluster cluster = new LocalCluster(); -cluster.submitTopology(name, conf, topology); -} else { -StormSubmitter.submitTopology(name, conf, topology); -} -} -``` - -Wouldn't something like this be easier: - -```bash -storm jar mytopology.jar org.apache.storm.flux.Flux --local config.yaml -``` - -or: - -```bash -storm jar mytopology.jar org.apache.storm.flux.Flux --remote config.yaml -``` - -Another pain point often mentioned is the fact that the wiring for a Topology graph is often tied up in Java code, +A Major pain point often mentioned is the fact that the wiring for a Topology graph is often tied up in Java code, and that any changes require recompilation and repackaging of the topology jar file. Flux aims to alleviate that pain by allowing you to package all your Storm components in a single jar, and use an external text file to define the layout and configuration of your topologies. @@ -202,13 +173,13 @@ The example below illustrates Flux usage with the Maven shade plugin: ``` ### Deploying and Running a Flux Topolo
[3/3] storm git commit: Merge branch 'agresch_blacklist' of github.com:agresch/storm into STORM-3295
Merge branch 'agresch_blacklist' of github.com:agresch/storm into STORM-3295 STORM-3295: allow blacklist scheduling to function properly with multiple supervisors on a host This closes #2918 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/cf00a537 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/cf00a537 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/cf00a537 Branch: refs/heads/master Commit: cf00a537b4ce34fdd3a2823642e7d4a3158684c6 Parents: 41dd0a7 f594c20 Author: Robert Evans Authored: Tue Dec 18 10:06:40 2018 -0600 Committer: Robert Evans Committed: Tue Dec 18 10:06:40 2018 -0600 -- .../strategies/DefaultBlacklistStrategy.java| 44 ++-- .../strategies/RasBlacklistStrategy.java| 34 --- 2 files changed, 51 insertions(+), 27 deletions(-) --
[1/3] storm git commit: STORM-3295 allow blacklist scheduling to function properly with multiple supervisors on a host
Repository: storm Updated Branches: refs/heads/master 41dd0a7b3 -> cf00a537b STORM-3295 allow blacklist scheduling to function properly with multiple supervisors on a host Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/b4d3df21 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/b4d3df21 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/b4d3df21 Branch: refs/heads/master Commit: b4d3df2167b2962bdf7039ba064ad7a646f32644 Parents: 94cd157 Author: Aaron Gresch Authored: Tue Dec 4 16:23:35 2018 -0600 Committer: Aaron Gresch Committed: Tue Dec 4 16:23:35 2018 -0600 -- .../strategies/RasBlacklistStrategy.java| 51 ++-- 1 file changed, 37 insertions(+), 14 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/b4d3df21/storm-server/src/main/java/org/apache/storm/scheduler/blacklist/strategies/RasBlacklistStrategy.java -- diff --git a/storm-server/src/main/java/org/apache/storm/scheduler/blacklist/strategies/RasBlacklistStrategy.java b/storm-server/src/main/java/org/apache/storm/scheduler/blacklist/strategies/RasBlacklistStrategy.java index 6c823ab..ac7abb1 100644 --- a/storm-server/src/main/java/org/apache/storm/scheduler/blacklist/strategies/RasBlacklistStrategy.java +++ b/storm-server/src/main/java/org/apache/storm/scheduler/blacklist/strategies/RasBlacklistStrategy.java @@ -22,6 +22,8 @@ import java.util.HashSet; import java.util.List; import java.util.Map; import java.util.Set; +import java.util.TreeMap; + import org.apache.storm.generated.InvalidTopologyException; import org.apache.storm.scheduler.Cluster; import org.apache.storm.scheduler.SupervisorDetails; @@ -79,25 +81,46 @@ public class RasBlacklistStrategy extends DefaultBlacklistStrategy { if (shortage.areAnyOverZero() || shortageSlots > 0) { LOG.info("Need {} and {} slots more. Releasing some blacklisted nodes to cover it.", shortage, shortageSlots); -//release earliest blacklist -for (String supervisor : blacklistedNodeIds) { -SupervisorDetails sd = availableSupervisors.get(supervisor); -if (sd != null) { -NormalizedResourcesWithMemory sdAvailable = cluster.getAvailableResources(sd); -int sdAvailableSlots = cluster.getAvailablePorts(sd).size(); -readyToRemove.add(supervisor); -shortage.remove(sdAvailable, cluster.getResourceMetrics()); -shortageSlots -= sdAvailableSlots; -LOG.debug("Releasing {} with {} and {} slots leaving {} and {} slots to go", supervisor, -sdAvailable, sdAvailableSlots, shortage, shortageSlots); -if (!shortage.areAnyOverZero() && shortageSlots <= 0) { -// we have enough resources now... -break; + +//release earliest blacklist - but release all supervisors on a given blacklisted host. +Map> hostToSupervisorIds = createHostToSupervisorMap(blacklistedNodeIds, cluster); +for (Set supervisorIds : hostToSupervisorIds.values()) { +for (String supervisorId : supervisorIds) { +SupervisorDetails sd = availableSupervisors.get(supervisorId); +if (sd != null) { +NormalizedResourcesWithMemory sdAvailable = cluster.getAvailableResources(sd); +int sdAvailableSlots = cluster.getAvailablePorts(sd).size(); +readyToRemove.add(supervisorId); +shortage.remove(sdAvailable, cluster.getResourceMetrics()); +shortageSlots -= sdAvailableSlots; +LOG.info("Releasing {} with {} and {} slots leaving {} and {} slots to go", supervisorId, +sdAvailable, sdAvailableSlots, shortage, shortageSlots); } } +// make sure we've handled all supervisors on the host before we break +if (!shortage.areAnyOverZero() && shortageSlots <= 0) { +// we have enough resources now... +break; +} } } } return readyToRemove; } + +private Map> createHostToSupervisorMap(final List blacklistedNodeIds, Cluster cluster) { +Map> hostToSupervisorMap = new TreeMap<>(); +for (String supervisorId : blacklistedNodeIds) { +String hostname = clust
[2/3] storm git commit: STORM-3295 Fix blacklisting of multiple supervisors per host using DefaultBlacklistStrategy
STORM-3295 Fix blacklisting of multiple supervisors per host using DefaultBlacklistStrategy Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/f594c20d Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/f594c20d Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/f594c20d Branch: refs/heads/master Commit: f594c20dc5f3547578835e32c2a48cc37f2d6cfd Parents: b4d3df2 Author: Aaron Gresch Authored: Thu Dec 6 11:02:10 2018 -0600 Committer: Aaron Gresch Committed: Thu Dec 6 11:02:10 2018 -0600 -- .../strategies/DefaultBlacklistStrategy.java| 44 ++-- .../strategies/RasBlacklistStrategy.java| 17 2 files changed, 31 insertions(+), 30 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/f594c20d/storm-server/src/main/java/org/apache/storm/scheduler/blacklist/strategies/DefaultBlacklistStrategy.java -- diff --git a/storm-server/src/main/java/org/apache/storm/scheduler/blacklist/strategies/DefaultBlacklistStrategy.java b/storm-server/src/main/java/org/apache/storm/scheduler/blacklist/strategies/DefaultBlacklistStrategy.java index a0c5b6c..7748d6f 100644 --- a/storm-server/src/main/java/org/apache/storm/scheduler/blacklist/strategies/DefaultBlacklistStrategy.java +++ b/storm-server/src/main/java/org/apache/storm/scheduler/blacklist/strategies/DefaultBlacklistStrategy.java @@ -24,7 +24,6 @@ import org.apache.storm.scheduler.Cluster; import org.apache.storm.scheduler.SupervisorDetails; import org.apache.storm.scheduler.Topologies; import org.apache.storm.scheduler.TopologyDetails; -import org.apache.storm.scheduler.WorkerSlot; import org.apache.storm.scheduler.blacklist.reporters.IReporter; import org.apache.storm.scheduler.blacklist.reporters.LogReporter; import org.apache.storm.utils.ObjectReader; @@ -142,20 +141,23 @@ public class DefaultBlacklistStrategy implements IBlacklistStrategy { if (shortageSlots > 0) { LOG.info("Need {} slots more. Releasing some blacklisted nodes to cover it.", shortageSlots); -//release earliest blacklist -for (String supervisor : blacklistedNodeIds) { -SupervisorDetails sd = availableSupervisors.get(supervisor); -if (sd != null) { -int sdAvailableSlots = cluster.getAvailablePorts(sd).size(); -readyToRemove.add(supervisor); -shortageSlots -= sdAvailableSlots; -LOG.debug("Releasing {} with {} slots leaving {} slots to go", supervisor, -sdAvailableSlots, shortageSlots); -if (shortageSlots <= 0) { -// we have enough resources now... -break; +//release earliest blacklist - but release all supervisors on a given blacklisted host. +Map> hostToSupervisorIds = createHostToSupervisorMap(blacklistedNodeIds, cluster); +for (Set supervisorIds : hostToSupervisorIds.values()) { +for (String supervisorId : supervisorIds) { +SupervisorDetails sd = availableSupervisors.get(supervisorId); +if (sd != null) { +int sdAvailableSlots = cluster.getAvailablePorts(sd).size(); +readyToRemove.add(supervisorId); +shortageSlots -= sdAvailableSlots; +LOG.debug("Releasing {} with {} slots leaving {} slots to go", supervisorId, +sdAvailableSlots, shortageSlots); } } +if (shortageSlots <= 0) { +// we have enough resources now... +break; +} } } } @@ -176,4 +178,20 @@ public class DefaultBlacklistStrategy implements IBlacklistStrategy { throw new RuntimeException(e); } } + +protected Map> createHostToSupervisorMap(final List blacklistedNodeIds, Cluster cluster) { +Map> hostToSupervisorMap = new TreeMap<>(); +for (String supervisorId : blacklistedNodeIds) { +String hostname = cluster.getHost(supervisorId); +if (hostname != null) { +Set supervisorIds = hostToSupervisorMap.get(hostname); +if (supervisorIds == null) { +supervisorIds = new HashSet<>(); +hostToSupervisorMap.put(hostname, supervisorIds); +} +supervisorIds.add(supervisorId); +} +} +
[1/2] storm git commit: STORM-3303 adjust some logging priorities, log topology info
Repository: storm Updated Branches: refs/heads/master 339b1e6f3 -> 41dd0a7b3 STORM-3303 adjust some logging priorities, log topology info Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/abca1540 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/abca1540 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/abca1540 Branch: refs/heads/master Commit: abca1540be9e6875f93dc0b9a276710717bc0ff6 Parents: f5b3f93 Author: Aaron Gresch Authored: Tue Dec 11 14:18:23 2018 -0600 Committer: Aaron Gresch Committed: Tue Dec 11 14:18:23 2018 -0600 -- .../jvm/org/apache/storm/pacemaker/PacemakerClient.java | 2 +- .../utils/StormBoundedExponentialBackoffRetry.java | 4 ++-- .../java/org/apache/storm/daemon/nimbus/Nimbus.java | 12 ++-- .../org/apache/storm/daemon/supervisor/Supervisor.java | 2 +- .../security/auth/workertoken/WorkerTokenManager.java | 2 +- 5 files changed, 11 insertions(+), 11 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/abca1540/storm-client/src/jvm/org/apache/storm/pacemaker/PacemakerClient.java -- diff --git a/storm-client/src/jvm/org/apache/storm/pacemaker/PacemakerClient.java b/storm-client/src/jvm/org/apache/storm/pacemaker/PacemakerClient.java index 5390a2c..e01c7c0 100644 --- a/storm-client/src/jvm/org/apache/storm/pacemaker/PacemakerClient.java +++ b/storm-client/src/jvm/org/apache/storm/pacemaker/PacemakerClient.java @@ -187,7 +187,7 @@ public class PacemakerClient implements ISaslClient { throw new PacemakerConnectionException("couldn't get response after " + maxRetries + " attempts."); } retry--; -LOG.error("Not getting response or getting null response. Making {} more attempts for {}.", retry, host); +LOG.warn("Not getting response or getting null response. Making {} more attempts for {}.", retry, host); } } } http://git-wip-us.apache.org/repos/asf/storm/blob/abca1540/storm-client/src/jvm/org/apache/storm/utils/StormBoundedExponentialBackoffRetry.java -- diff --git a/storm-client/src/jvm/org/apache/storm/utils/StormBoundedExponentialBackoffRetry.java b/storm-client/src/jvm/org/apache/storm/utils/StormBoundedExponentialBackoffRetry.java index 768c83c..969a9c6 100644 --- a/storm-client/src/jvm/org/apache/storm/utils/StormBoundedExponentialBackoffRetry.java +++ b/storm-client/src/jvm/org/apache/storm/utils/StormBoundedExponentialBackoffRetry.java @@ -57,13 +57,13 @@ public class StormBoundedExponentialBackoffRetry extends BoundedExponentialBacko int exp = 1 << retryCount; int jitter = random.nextInt(exp); long sleepTimeMs = super.getBaseSleepTimeMs() + exp + jitter; -LOG.warn("WILL SLEEP FOR {}ms (NOT MAX)", sleepTimeMs); +LOG.debug("WILL SLEEP FOR {}ms (NOT MAX)", sleepTimeMs); return sleepTimeMs; } else { int stepJitter = random.nextInt(stepSize); long sleepTimeMs = Math.min(super.getMaxSleepTimeMs(), (linearBaseSleepMs + (stepSize * (retryCount - expRetriesThreshold)) + stepJitter)); -LOG.warn("WILL SLEEP FOR {}ms (MAX)", sleepTimeMs); +LOG.debug("WILL SLEEP FOR {}ms (MAX)", sleepTimeMs); return sleepTimeMs; } } http://git-wip-us.apache.org/repos/asf/storm/blob/abca1540/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java -- diff --git a/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java b/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java index 9568c04..90d43aa 100644 --- a/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java +++ b/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java @@ -1682,9 +1682,9 @@ public class Nimbus implements Iface, Shutdownable, DaemonCommon { && codeCount < minReplicationCount && confCount < minReplicationCount) { if (maxWaitTime > 0 && totalWaitTime > maxWaitTime) { -LOG.info("desired replication count of {} not achieved but we have hit the max wait time {}" +LOG.info("desired replication count of {} not achieved for {} but we have hit the max wait time {}" + " so moving on with replication count for conf key = {} for code key = {} for jar key = ", - minReplicationCount, maxWaitTime, confCount, codeCount, jarCount); +
[2/2] storm git commit: Merge branch 'agresch_logging' of github.com:agresch/storm into STORM-3303
Merge branch 'agresch_logging' of github.com:agresch/storm into STORM-3303 STORM-3303: adjust some logging priorities, log topology info This closes #2926 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/41dd0a7b Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/41dd0a7b Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/41dd0a7b Branch: refs/heads/master Commit: 41dd0a7b333c3f1fbe9538394541ff8bc03451b0 Parents: 339b1e6 abca154 Author: Robert Evans Authored: Tue Dec 18 09:33:47 2018 -0600 Committer: Robert Evans Committed: Tue Dec 18 09:33:47 2018 -0600 -- .../jvm/org/apache/storm/pacemaker/PacemakerClient.java | 2 +- .../utils/StormBoundedExponentialBackoffRetry.java | 4 ++-- .../java/org/apache/storm/daemon/nimbus/Nimbus.java | 12 ++-- .../org/apache/storm/daemon/supervisor/Supervisor.java | 2 +- .../security/auth/workertoken/WorkerTokenManager.java | 2 +- 5 files changed, 11 insertions(+), 11 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/41dd0a7b/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java --
[1/2] storm git commit: STORM-3302: Ensures we close sockets to HDFS
Repository: storm Updated Branches: refs/heads/master a9c2f3adb -> 339b1e6f3 STORM-3302: Ensures we close sockets to HDFS Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/845787d6 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/845787d6 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/845787d6 Branch: refs/heads/master Commit: 845787d671dcdf4475493ea5888f2021b1422ba9 Parents: 730c1a3 Author: Derek Dagit Authored: Wed Dec 12 10:43:28 2018 -0600 Committer: Derek Dagit Committed: Wed Dec 12 10:43:28 2018 -0600 -- .../org/apache/storm/blobstore/BlobStore.java | 14 ++--- .../storm/dependency/DependencyUploader.java| 20 +--- .../apache/storm/blobstore/BlobStoreUtils.java | 33 .../storm/nimbus/LeaderListenerCallback.java| 3 +- 4 files changed, 46 insertions(+), 24 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/845787d6/storm-client/src/jvm/org/apache/storm/blobstore/BlobStore.java -- diff --git a/storm-client/src/jvm/org/apache/storm/blobstore/BlobStore.java b/storm-client/src/jvm/org/apache/storm/blobstore/BlobStore.java index cb2928c..6cf9df9 100644 --- a/storm-client/src/jvm/org/apache/storm/blobstore/BlobStore.java +++ b/storm-client/src/jvm/org/apache/storm/blobstore/BlobStore.java @@ -289,12 +289,16 @@ public abstract class BlobStore implements Shutdownable, AutoCloseable { out.write(buffer, 0, len); } out.close(); -} catch (AuthorizationException | IOException | RuntimeException e) { -if (out != null) { -out.cancel(); -} +out = null; } finally { -in.close(); +try { +if (out != null) { +out.cancel(); +} +in.close(); +} catch (IOException throwaway) { +// Ignored +} } } http://git-wip-us.apache.org/repos/asf/storm/blob/845787d6/storm-client/src/jvm/org/apache/storm/dependency/DependencyUploader.java -- diff --git a/storm-client/src/jvm/org/apache/storm/dependency/DependencyUploader.java b/storm-client/src/jvm/org/apache/storm/dependency/DependencyUploader.java index 41c1d86..d8f8c5a 100644 --- a/storm-client/src/jvm/org/apache/storm/dependency/DependencyUploader.java +++ b/storm-client/src/jvm/org/apache/storm/dependency/DependencyUploader.java @@ -154,11 +154,23 @@ public class DependencyUploader { acls.add(new AccessControl(AccessControlType.OTHER, BlobStoreAclHandler.READ)); -AtomicOutputStream blob = getBlobStore().createBlob(key, new SettableBlobMeta(acls)); -Files.copy(dependency.toPath(), blob); -blob.close(); +AtomicOutputStream blob = null; +try { +blob = getBlobStore().createBlob(key, new SettableBlobMeta(acls)); +Files.copy(dependency.toPath(), blob); +blob.close(); +blob = null; -uploadNew = true; +uploadNew = true; +} finally { +try { +if (blob != null) { +blob.cancel(); +} +} catch (IOException throwaway) { +// Ignore. +} +} } return uploadNew; http://git-wip-us.apache.org/repos/asf/storm/blob/845787d6/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java -- diff --git a/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java b/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java index b9f93db..00d833f 100644 --- a/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java +++ b/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java @@ -122,7 +122,6 @@ public class BlobStoreUtils { throws TTransportException { ReadableBlobMeta rbm; ClientBlobStore remoteBlobStore; -InputStreamWithMeta in; boolean isSuccess = false; LOG.debug("Download blob NimbusInfos {}", nimbusInfos); for (NimbusInfo nimbusInfo : nimbusInfos) { @@ -134,8 +133,9 @@ public class BlobStoreUtils { rbm = client.getClient().getBlobMeta(key); remoteBlobStore = new NimbusBlobStore(); remoteBlobStore.setClient(conf, client); -in = remoteBlobStore.getBlob(key); -b
[2/2] storm git commit: Merge branch 'storm-3302-fix-socket-leaks' of github.com:d2r/storm into STORM-3302
Merge branch 'storm-3302-fix-socket-leaks' of github.com:d2r/storm into STORM-3302 STORM-3302: Ensures we close streams to HDFS this closes #2925 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/339b1e6f Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/339b1e6f Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/339b1e6f Branch: refs/heads/master Commit: 339b1e6f34d8209ff8c1afcdea815781bfbe22eb Parents: a9c2f3a 845787d Author: Robert Evans Authored: Tue Dec 18 09:04:29 2018 -0600 Committer: Robert Evans Committed: Tue Dec 18 09:04:29 2018 -0600 -- .../org/apache/storm/blobstore/BlobStore.java | 14 ++--- .../storm/dependency/DependencyUploader.java| 20 +--- .../apache/storm/blobstore/BlobStoreUtils.java | 33 .../storm/nimbus/LeaderListenerCallback.java| 3 +- 4 files changed, 46 insertions(+), 24 deletions(-) --
[2/2] storm git commit: Merge branch 'STORM-2891' of https://github.com/srdo/storm into STORM-2891
Merge branch 'STORM-2891' of https://github.com/srdo/storm into STORM-2891 STORM-2891: Upgrade checkstyle plugin to 3.0.0 This closes #2905 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/f17b3dad Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/f17b3dad Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/f17b3dad Branch: refs/heads/master Commit: f17b3dad8572f96217a8bf058022b26fe147a0b5 Parents: 4ebb640 2e73e40 Author: Robert Evans Authored: Tue Nov 13 11:14:44 2018 -0600 Committer: Robert Evans Committed: Tue Nov 13 11:14:44 2018 -0600 -- pom.xml | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) --
[1/2] storm git commit: STORM-2891: Upgrade checkstyle plugin to 3.0.0
Repository: storm Updated Branches: refs/heads/master 4ebb6406e -> f17b3dad8 STORM-2891: Upgrade checkstyle plugin to 3.0.0 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/2e73e408 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/2e73e408 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/2e73e408 Branch: refs/heads/master Commit: 2e73e4086cfdc37c919120b0b3bc9fb74494dd17 Parents: 98ed0a8 Author: Stig Rohde Døssing Authored: Sun Nov 11 17:26:27 2018 +0100 Committer: Stig Rohde Døssing Committed: Sun Nov 11 17:26:27 2018 +0100 -- pom.xml | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/2e73e408/pom.xml -- diff --git a/pom.xml b/pom.xml index d7191ae..9836fab 100644 --- a/pom.xml +++ b/pom.xml @@ -1131,7 +1131,7 @@ org.apache.maven.plugins maven-checkstyle-plugin -2.17 +3.0.0 org.apache.storm
[1/2] storm git commit: Add inheritance to cgroups
Repository: storm Updated Branches: refs/heads/master 98ed0a815 -> 4ebb6406e Add inheritance to cgroups Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/9178c102 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/9178c102 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/9178c102 Branch: refs/heads/master Commit: 9178c10211874f8928ebc5ce4fdbf04ae698bcb5 Parents: 19fbfb9 Author: Kishor Patil Authored: Thu Nov 8 17:40:37 2018 + Committer: Kishor Patil Committed: Thu Nov 8 13:05:53 2018 -0500 -- conf/defaults.yaml| 1 + .../main/java/org/apache/storm/DaemonConfig.java | 8 .../storm/container/cgroup/CgroupManager.java | 18 ++ 3 files changed, 27 insertions(+) -- http://git-wip-us.apache.org/repos/asf/storm/blob/9178c102/conf/defaults.yaml -- diff --git a/conf/defaults.yaml b/conf/defaults.yaml index d9ad56c..ec81d22 100644 --- a/conf/defaults.yaml +++ b/conf/defaults.yaml @@ -371,6 +371,7 @@ storm.resource.isolation.plugin: "org.apache.storm.container.cgroup.CgroupManage # If storm.resource.isolation.plugin.enable is set to false the unit tests for cgroups will not run storm.resource.isolation.plugin.enable: false storm.cgroup.memory.enforcement.enable: false +storm.cgroup.inherit.cpuset.configs: false # Configs for CGroup support storm.cgroup.hierarchy.dir: "/cgroup/storm_resources" http://git-wip-us.apache.org/repos/asf/storm/blob/9178c102/storm-server/src/main/java/org/apache/storm/DaemonConfig.java -- diff --git a/storm-server/src/main/java/org/apache/storm/DaemonConfig.java b/storm-server/src/main/java/org/apache/storm/DaemonConfig.java index 6c12fbb..4b51ef7 100644 --- a/storm-server/src/main/java/org/apache/storm/DaemonConfig.java +++ b/storm-server/src/main/java/org/apache/storm/DaemonConfig.java @@ -1068,6 +1068,14 @@ public class DaemonConfig implements Validated { public static String STORM_CGROUP_MEMORY_LIMIT_TOLERANCE_MARGIN_MB = "storm.cgroup.memory.limit.tolerance.margin.mb"; /** + * To determine whether or not to cgroups should inherit cpuset.cpus and cpuset.mems config values form parent cgroup + * Note that cpuset.cpus and cpuset.mems configs in a cgroup must be initialized (i.e. contain a valid value) prior to + * being able to launch processes in that cgroup. The common use case for this config is when the linux distribution + * that is used does not support the cgroup.clone_children config. + */ +@isBoolean +public static String STORM_CGROUP_INHERIT_CPUSET_CONFIGS = "storm.cgroup.inherit.cpuset.configs"; +/** * Java does not always play nicely with cgroups. It is coming but not fully implemented and not for the way storm uses cgroups. In the * short term you can disable the hard memory enforcement by cgroups and let the supervisor handle shooting workers going over their * limit in a kinder way. http://git-wip-us.apache.org/repos/asf/storm/blob/9178c102/storm-server/src/main/java/org/apache/storm/container/cgroup/CgroupManager.java -- diff --git a/storm-server/src/main/java/org/apache/storm/container/cgroup/CgroupManager.java b/storm-server/src/main/java/org/apache/storm/container/cgroup/CgroupManager.java index a68e18e..fbdb364 100644 --- a/storm-server/src/main/java/org/apache/storm/container/cgroup/CgroupManager.java +++ b/storm-server/src/main/java/org/apache/storm/container/cgroup/CgroupManager.java @@ -32,6 +32,7 @@ import org.apache.storm.Config; import org.apache.storm.DaemonConfig; import org.apache.storm.container.ResourceIsolationInterface; import org.apache.storm.container.cgroup.core.CpuCore; +import org.apache.storm.container.cgroup.core.CpusetCore; import org.apache.storm.container.cgroup.core.MemoryCore; import org.apache.storm.utils.ObjectReader; import org.slf4j.Logger; @@ -201,6 +202,23 @@ public class CgroupManager implements ResourceIsolationInterface { } } } + +if ((boolean) this.conf.get(DaemonConfig.STORM_CGROUP_INHERIT_CPUSET_CONFIGS)) { +if (workerGroup.getParent().getCores().containsKey(SubSystemType.cpuset)) { +CpusetCore parentCpusetCore = (CpusetCore) workerGroup.getParent().getCores().get(SubSystemType.cpuset); +CpusetCore cpusetCore = (CpusetCore) workerGroup.getCores().get(SubSystemType.cpuset); +try { +cpusetCore.setCpus(parentCpusetCore.getCpus()); +} catch (IOException e) { +throw new RuntimeExcept
[2/2] storm git commit: Merge branch 'add-cgroup-inheritance' of https://github.com/kishorvpatil/incubator-storm into STORM-3284
Merge branch 'add-cgroup-inheritance' of https://github.com/kishorvpatil/incubator-storm into STORM-3284 STORM-3284: Add inheritance to cgroups This closes #2903 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/4ebb6406 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/4ebb6406 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/4ebb6406 Branch: refs/heads/master Commit: 4ebb6406ea850cdd5b710fd9185fe211bbdf746d Parents: 98ed0a8 9178c10 Author: Robert Evans Authored: Tue Nov 13 10:45:14 2018 -0600 Committer: Robert Evans Committed: Tue Nov 13 10:45:14 2018 -0600 -- conf/defaults.yaml| 1 + .../main/java/org/apache/storm/DaemonConfig.java | 8 .../storm/container/cgroup/CgroupManager.java | 18 ++ 3 files changed, 27 insertions(+) --
[2/2] storm git commit: Merge branch 'storm3275' of https://github.com/kishorvpatil/incubator-storm into STORM-3275
Merge branch 'storm3275' of https://github.com/kishorvpatil/incubator-storm into STORM-3275 STORM-3275: Fix UIHelpers timeout while starting profiler This closes #2895 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/02662ea6 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/02662ea6 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/02662ea6 Branch: refs/heads/master Commit: 02662ea6641a6763b8e285ad292dbb789be6fc61 Parents: fb63089 6b42587 Author: Robert Evans Authored: Fri Nov 2 14:11:24 2018 -0500 Committer: Robert Evans Committed: Fri Nov 2 14:11:24 2018 -0500 -- .../org/apache/storm/daemon/ui/UIHelpers.java | 20 ++-- 1 file changed, 10 insertions(+), 10 deletions(-) --
[1/2] storm git commit: Fix UIHelpers timeout while starting profiler
Repository: storm Updated Branches: refs/heads/master fb6308992 -> 02662ea66 Fix UIHelpers timeout while starting profiler Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/6b42587d Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/6b42587d Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/6b42587d Branch: refs/heads/master Commit: 6b42587dfb1f99c8dfb6066a2bbc83e29552cff8 Parents: 730c1a3 Author: Kishor Patil Authored: Wed Oct 24 17:50:59 2018 -0400 Committer: Kishor Patil Committed: Wed Oct 24 17:50:59 2018 -0400 -- .../org/apache/storm/daemon/ui/UIHelpers.java | 20 ++-- 1 file changed, 10 insertions(+), 10 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/6b42587d/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java -- diff --git a/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java b/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java index ff121ae..5935e72 100644 --- a/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java +++ b/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java @@ -2164,7 +2164,7 @@ public class UIHelpers { } /** - * getTopologyProfilingAction. + * setTopologyProfilingAction. * @param client client * @param id id * @param hostPort hostPort @@ -2173,7 +2173,7 @@ public class UIHelpers { * @param profileAction profileAction * @throws TException TException */ -public static void getTopologyProfilingAction( +public static void setTopologyProfilingAction( Nimbus.Iface client, String id, String hostPort, Long timestamp, Map config, ProfileAction profileAction) throws TException { @@ -2200,9 +2200,9 @@ public class UIHelpers { public static Map getTopologyProfilingStart(Nimbus.Iface client, String id, String hostPort, String timeout, Map config) throws TException { -getTopologyProfilingAction( -client, id , hostPort, System.currentTimeMillis() + Long.valueOf(timeout), -config, ProfileAction.JPROFILE_START); +setTopologyProfilingAction( +client, id , hostPort, System.currentTimeMillis() + (Long.valueOf(timeout) * 60_000), +config, ProfileAction.JPROFILE_STOP); Map result = new HashMap(); String host = hostPort.split(":")[0]; String port = hostPort.split(":")[1]; @@ -2225,7 +2225,7 @@ public class UIHelpers { public static Map getTopologyProfilingStop(Nimbus.Iface client, String id, String hostPort, Map config) throws TException { -getTopologyProfilingAction(client, id , hostPort, 0L, config, ProfileAction.JPROFILE_STOP); +setTopologyProfilingAction(client, id , hostPort, 0L, config, ProfileAction.JPROFILE_STOP); Map result = new HashMap(); result.put("status", "ok"); result.put("id", hostPort); @@ -2254,7 +2254,7 @@ public class UIHelpers { */ public static Map getTopologyProfilingDump(Nimbus.Iface client, String id, String hostPort, Map config) throws TException { -getTopologyProfilingAction( +setTopologyProfilingAction( client, id , hostPort, System.currentTimeMillis(), config, ProfileAction.JPROFILE_DUMP ); @@ -2267,7 +2267,7 @@ public class UIHelpers { public static Map getTopologyProfilingDumpJstack(Nimbus.Iface client, String id, String hostPort, Map config) throws TException { -getTopologyProfilingAction( +setTopologyProfilingAction( client, id , hostPort, System.currentTimeMillis(), config, ProfileAction.JSTACK_DUMP ); Map result = new HashMap(); @@ -2288,7 +2288,7 @@ public class UIHelpers { public static Map getTopologyProfilingRestartWorker(Nimbus.Iface client, String id, String hostPort, Map config) throws TException { -getTopologyProfilingAction( +setTopologyProfilingAction( client, id , hostPort, System.currentTimeMillis(), config, ProfileAction.JVM_RESTART ); Map result = new HashMap(); @@ -2308,7 +23
[2/2] storm git commit: Merge branch 'master' of https://github.com/jnioche/storm into PULL-2897
Merge branch 'master' of https://github.com/jnioche/storm into PULL-2897 bugfix ConfigurableTopology completely overwrites preexisting config This closes #2897 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/fb630899 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/fb630899 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/fb630899 Branch: refs/heads/master Commit: fb630899207f23ad0d3fbe91d532aca7ff604447 Parents: 00cd3d3 a88e449 Author: Robert Evans Authored: Fri Nov 2 12:32:13 2018 -0500 Committer: Robert Evans Committed: Fri Nov 2 12:32:13 2018 -0500 -- .../src/jvm/org/apache/storm/topology/ConfigurableTopology.java| 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) --
[1/2] storm git commit: bugfix ConfigurableTopology completely overwrites preexisting config
Repository: storm Updated Branches: refs/heads/master 00cd3d323 -> fb6308992 bugfix ConfigurableTopology completely overwrites preexisting config Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/a88e4491 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/a88e4491 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/a88e4491 Branch: refs/heads/master Commit: a88e4491c39e7dded5a16974ed2fcacc33892ae7 Parents: 730c1a3 Author: Julien Nioche Authored: Fri Oct 26 16:41:03 2018 +0100 Committer: Julien Nioche Committed: Fri Oct 26 16:41:03 2018 +0100 -- .../src/jvm/org/apache/storm/topology/ConfigurableTopology.java| 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/a88e4491/storm-client/src/jvm/org/apache/storm/topology/ConfigurableTopology.java -- diff --git a/storm-client/src/jvm/org/apache/storm/topology/ConfigurableTopology.java b/storm-client/src/jvm/org/apache/storm/topology/ConfigurableTopology.java index 088a382..3915198 100644 --- a/storm-client/src/jvm/org/apache/storm/topology/ConfigurableTopology.java +++ b/storm-client/src/jvm/org/apache/storm/topology/ConfigurableTopology.java @@ -86,7 +86,7 @@ public abstract class ConfigurableTopology { if (ret.size() == 1) { Object confNode = ret.get("config"); if (confNode != null && confNode instanceof Map) { -ret = (Map) ret; +ret = (Map) confNode; } } }
[3/3] storm git commit: Merge branch 'STORM-3261' of https://github.com/contradictioned/storm
Merge branch 'STORM-3261' of https://github.com/contradictioned/storm STORM-3261: Fix broken links in documentation This closes #2896 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/dcf637ff Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/dcf637ff Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/dcf637ff Branch: refs/heads/master Commit: dcf637ff666e3c08ef81911034576019d35ca215 Parents: 9e44d48 64ce3e0 Author: Robert Evans Authored: Wed Oct 31 12:58:13 2018 -0500 Committer: Robert Evans Committed: Wed Oct 31 12:58:13 2018 -0500 -- docs/Concepts.md| 2 +- docs/Trident-tutorial.md| 2 +- docs/Tutorial.md| 2 +- docs/Windowing.md | 2 +- docs/flux.md| 2 +- docs/storm-kinesis.md | 6 ++ docs/storm-opentsdb.md | 8 ++-- docs/storm-pmml.md | 7 ++- docs/storm-rocketmq.md | 6 +- docs/storm-sql-reference.md | 4 ++-- 10 files changed, 30 insertions(+), 11 deletions(-) --
[1/3] storm git commit: Added missing YAML headers, so jekyll generates the html files.
Repository: storm Updated Branches: refs/heads/master 9e44d4899 -> dcf637ff6 Added missing YAML headers, so jekyll generates the html files. Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/e996abce Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/e996abce Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/e996abce Branch: refs/heads/master Commit: e996abce19ce535599dab1bb90977d0056a9ad97 Parents: 730c1a3 Author: Manuel Dossinger Authored: Fri Oct 26 11:18:03 2018 +0200 Committer: Manuel Dossinger Committed: Fri Oct 26 11:18:03 2018 +0200 -- docs/storm-kinesis.md | 6 ++ docs/storm-opentsdb.md | 8 ++-- docs/storm-pmml.md | 7 ++- docs/storm-rocketmq.md | 6 +- 4 files changed, 23 insertions(+), 4 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/e996abce/docs/storm-kinesis.md -- diff --git a/docs/storm-kinesis.md b/docs/storm-kinesis.md index 0d7a73e..0921875 100644 --- a/docs/storm-kinesis.md +++ b/docs/storm-kinesis.md @@ -1,3 +1,9 @@ +--- +title: Storm Kinesis +layout: documentation +documentation: true +--- + #Storm Kinesis Spout Provides core storm spout for consuming data from a stream in Amazon Kinesis Streams. It stores the sequence numbers that can be committed in zookeeper and starts consuming records after that sequence number on restart by default. Below is the code sample to create a sample topology that uses the spout. Each http://git-wip-us.apache.org/repos/asf/storm/blob/e996abce/docs/storm-opentsdb.md -- diff --git a/docs/storm-opentsdb.md b/docs/storm-opentsdb.md index 11995ce..695621b 100644 --- a/docs/storm-opentsdb.md +++ b/docs/storm-opentsdb.md @@ -1,5 +1,9 @@ -# Storm OpenTSDB Bolt and TridentState - +--- +title: Storm OpenTSDB Bolt and TridentState +layout: documentation +documentation: true +--- + OpenTSDB offers a scalable and highly available storage for time series data. It consists of a Time Series Daemon (TSD) servers along with command line utilities. Each TSD connects to the configured HBase cluster to push/query the data. http://git-wip-us.apache.org/repos/asf/storm/blob/e996abce/docs/storm-pmml.md -- diff --git a/docs/storm-pmml.md b/docs/storm-pmml.md index 5051489..0326fb6 100644 --- a/docs/storm-pmml.md +++ b/docs/storm-pmml.md @@ -1,4 +1,9 @@ -# Storm PMML Bolt +--- +title: Storm PMML Bolt +layout: documentation +documentation: true +--- + Storm integration to load PMML models and compute predictive scores for running tuples. The PMML model represents the machine learning (predictive) model used to do prediction on raw input data. The model is typically loaded into a runtime environment, which will score the raw data that comes in the tuples. http://git-wip-us.apache.org/repos/asf/storm/blob/e996abce/docs/storm-rocketmq.md -- diff --git a/docs/storm-rocketmq.md b/docs/storm-rocketmq.md index 2661ddf..bac0cb7 100644 --- a/docs/storm-rocketmq.md +++ b/docs/storm-rocketmq.md @@ -1,4 +1,8 @@ -# Storm RocketMQ +--- +title: Storm RocketMQ +layout: documentation +documentation: true +--- Storm/Trident integration for [RocketMQ](https://rocketmq.incubator.apache.org/). This package includes the core spout, bolt and trident states that allows a storm topology to either write storm tuples into a topic or read from topics in a storm topology.
[2/3] storm git commit: Fixed broken links
Fixed broken links Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/64ce3e08 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/64ce3e08 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/64ce3e08 Branch: refs/heads/master Commit: 64ce3e0853531e550f31280fe4285f16aaebda1f Parents: e996abc Author: Manuel Dossinger Authored: Fri Oct 26 11:18:27 2018 +0200 Committer: Manuel Dossinger Committed: Fri Oct 26 11:18:27 2018 +0200 -- docs/Concepts.md| 2 +- docs/Trident-tutorial.md| 2 +- docs/Tutorial.md| 2 +- docs/Windowing.md | 2 +- docs/flux.md| 2 +- docs/storm-sql-reference.md | 4 ++-- 6 files changed, 7 insertions(+), 7 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/64ce3e08/docs/Concepts.md -- diff --git a/docs/Concepts.md b/docs/Concepts.md index 8516b9e..f810d36 100644 --- a/docs/Concepts.md +++ b/docs/Concepts.md @@ -116,5 +116,5 @@ Topologies execute across one or more worker processes. Each worker process is a ### Performance Tuning -Refer to [performance tuning guide](Performance.md) +Refer to [performance tuning guide](Performance.html). http://git-wip-us.apache.org/repos/asf/storm/blob/64ce3e08/docs/Trident-tutorial.md -- diff --git a/docs/Trident-tutorial.md b/docs/Trident-tutorial.md index 7df2594..ce26bbe 100644 --- a/docs/Trident-tutorial.md +++ b/docs/Trident-tutorial.md @@ -237,7 +237,7 @@ Trident solves this problem by doing two things: With these two primitives, you can achieve exactly-once semantics with your state updates. Rather than store just the count in the database, what you can do instead is store the transaction id with the count in the database as an atomic value. Then, when updating the count, you can just compare the transaction id in the database with the transaction id for the current batch. If they're the same, you skip the update â because of the strong ordering, you know for sure that the value in the database incorporates the current batch. If they're different, you increment the count. -Of course, you don't have to do this logic manually in your topologies. This logic is wrapped by the State abstraction and done automatically. Nor is your State object required to implement the transaction id trick: if you don't want to pay the cost of storing the transaction id in the database, you don't have to. In that case the State will have at-least-once-processing semantics in the case of failures (which may be fine for your application). You can read more about how to implement a State and the various fault-tolerance tradeoffs possible [in this doc](/documentation/Trident-state.html). +Of course, you don't have to do this logic manually in your topologies. This logic is wrapped by the State abstraction and done automatically. Nor is your State object required to implement the transaction id trick: if you don't want to pay the cost of storing the transaction id in the database, you don't have to. In that case the State will have at-least-once-processing semantics in the case of failures (which may be fine for your application). You can read more about how to implement a State and the various fault-tolerance tradeoffs possible [in this doc](Trident-state.html). A State is allowed to use whatever strategy it wants to store state. So it could store state in an external database or it could keep the state in-memory but backed by HDFS (like how HBase works). State's are not required to hold onto state forever. For example, you could have an in-memory State implementation that only keeps the last X hours of data available and drops anything older. Take a look at the implementation of the [Memcached integration](https://github.com/nathanmarz/trident-memcached/blob/master/src/jvm/trident/memcached/MemcachedState.java) for an example State implementation. http://git-wip-us.apache.org/repos/asf/storm/blob/64ce3e08/docs/Tutorial.md -- diff --git a/docs/Tutorial.md b/docs/Tutorial.md index aea38d3..ccc415d 100644 --- a/docs/Tutorial.md +++ b/docs/Tutorial.md @@ -218,7 +218,7 @@ A stream grouping tells a topology how to send tuples between two components. Re When a task for Bolt A emits a tuple to Bolt B, which task should it send the tuple to? -A "stream grouping" answers this question by telling Storm how to send tuples between sets of tasks. Before we dig into the different kinds of stream groupings, let's take a look at another topology from [storm-starter](http://github.com/apache/storm/blob/{{page.version}}/examples/storm-starter). Thi
[2/2] storm git commit: Merge branch 'STORM-3273' of https://github.com/revans2/incubator-storm into STORM-3273
Merge branch 'STORM-3273' of https://github.com/revans2/incubator-storm into STORM-3273 STORM-3273: Remove storm.local.hostname from topology conf. This closes #2894 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/730c1a30 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/730c1a30 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/730c1a30 Branch: refs/heads/master Commit: 730c1a302d857121542ce27b6a40b05b90f7f3ed Parents: 441f210 72be708 Author: Robert Evans Authored: Wed Oct 24 09:51:51 2018 -0500 Committer: Robert Evans Committed: Wed Oct 24 09:51:51 2018 -0500 -- .../src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java | 6 ++ 1 file changed, 6 insertions(+) -- http://git-wip-us.apache.org/repos/asf/storm/blob/730c1a30/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java --
[1/2] storm git commit: STORM-3273: Remove storm.local.hostname from topology conf.
Repository: storm Updated Branches: refs/heads/master 441f21039 -> 730c1a302 STORM-3273: Remove storm.local.hostname from topology conf. Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/72be7085 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/72be7085 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/72be7085 Branch: refs/heads/master Commit: 72be7085f8d9357295cd0991d7b4d0124d614cd7 Parents: e2563a1 Author: Robert (Bobby) Evans Authored: Tue Oct 23 09:11:28 2018 -0500 Committer: Robert (Bobby) Evans Committed: Tue Oct 23 09:11:28 2018 -0500 -- .../src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java | 6 ++ 1 file changed, 6 insertions(+) -- http://git-wip-us.apache.org/repos/asf/storm/blob/72be7085/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java -- diff --git a/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java b/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java index 9392ac3..5da37f8 100644 --- a/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java +++ b/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java @@ -3054,6 +3054,8 @@ public class Nimbus implements Iface, Shutdownable, DaemonCommon { Map otherConf = Utils.getConfigFromClasspath(cp, conf); Map totalConfToSave = Utils.merge(otherConf, topoConf); Map totalConf = Utils.merge(conf, totalConfToSave); + + //When reading the conf in nimbus we want to fall back to our own settings // if the other config does not have it set. topology = normalizeTopology(totalConf, topology); @@ -3072,6 +3074,9 @@ public class Nimbus implements Iface, Shutdownable, DaemonCommon { LOG.debug("{} set to: {}", Config.TOPOLOGY_EVENTLOGGER_EXECUTORS, numEventLoggerExecs); } +//Remove any configs that are specific to a host that might mess with the running topology. +totalConfToSave.remove(Config.STORM_LOCAL_HOSTNAME); //Don't override the host name, or everything looks like it is on nimbus + IStormClusterState state = stormClusterState; if (creds == null && workerTokenManager != null) { @@ -3238,6 +3243,7 @@ public class Nimbus implements Iface, Shutdownable, DaemonCommon { if ((boolean) conf.getOrDefault(DaemonConfig.STORM_TOPOLOGY_CLASSPATH_BEGINNING_ENABLED, false)) { topoConfigOverrides.remove(Config.TOPOLOGY_CLASSPATH_BEGINNING); } +topoConfigOverrides.remove(Config.STORM_LOCAL_HOSTNAME); options.set_topology_conf_overrides(JSONValue.toJSONString(topoConfigOverrides)); } Subject subject = getSubject();
[3/3] storm git commit: Merge branch 'agresch_storm-3272' of https://github.com/agresch/storm into STORM-3272
Merge branch 'agresch_storm-3272' of https://github.com/agresch/storm into STORM-3272 STORM-3272: allow worker-launcher to delete dead symlinks This closes #2893 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/441f2103 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/441f2103 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/441f2103 Branch: refs/heads/master Commit: 441f2103905979605d5586c3bc8fa04fee4fd31e Parents: 4d7099f 3fe0021 Author: Robert Evans Authored: Wed Oct 24 09:26:40 2018 -0500 Committer: Robert Evans Committed: Wed Oct 24 09:26:40 2018 -0500 -- .../native/worker-launcher/impl/worker-launcher.c | 17 - 1 file changed, 12 insertions(+), 5 deletions(-) --
[2/3] storm git commit: STORM-3272 log stat failure
STORM-3272 log stat failure Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/3fe00213 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/3fe00213 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/3fe00213 Branch: refs/heads/master Commit: 3fe0021379c1b491bc45bd5cc8efbc91cedd8536 Parents: 1313447 Author: Aaron Gresch Authored: Tue Oct 23 09:59:35 2018 -0500 Committer: Aaron Gresch Committed: Tue Oct 23 09:59:35 2018 -0500 -- storm-core/src/native/worker-launcher/impl/worker-launcher.c | 6 +- 1 file changed, 5 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/3fe00213/storm-core/src/native/worker-launcher/impl/worker-launcher.c -- diff --git a/storm-core/src/native/worker-launcher/impl/worker-launcher.c b/storm-core/src/native/worker-launcher/impl/worker-launcher.c index d085315..5155ed8 100644 --- a/storm-core/src/native/worker-launcher/impl/worker-launcher.c +++ b/storm-core/src/native/worker-launcher/impl/worker-launcher.c @@ -597,8 +597,12 @@ int recursive_delete(const char *path, int supervisor_owns_dir) { if(access(path, F_OK) != 0) { if(errno == ENOENT) { + if(lstat(path, &file_stat) != 0) { + fprintf(LOGFILE, "Failed to stat %s: %s", path, strerror(errno)); + return 0; + } // we need to handle symlinks that target missing files. - if((lstat(path, &file_stat) != 0) || ((file_stat.st_mode & S_IFMT) != S_IFLNK)) { + if((file_stat.st_mode & S_IFMT) != S_IFLNK) { return 0; } }
[1/3] storm git commit: STORM-3272 allow worker-launcher to delete dead symlinks
Repository: storm Updated Branches: refs/heads/master 4d7099f3c -> 441f21039 STORM-3272 allow worker-launcher to delete dead symlinks Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/13134477 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/13134477 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/13134477 Branch: refs/heads/master Commit: 131344777ae4cbc4ca47952b3a95f35b11713f23 Parents: e2563a1 Author: Aaron Gresch Authored: Tue Oct 23 08:57:52 2018 -0500 Committer: Aaron Gresch Committed: Tue Oct 23 09:00:38 2018 -0500 -- .../src/native/worker-launcher/impl/worker-launcher.c | 13 - 1 file changed, 8 insertions(+), 5 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/13134477/storm-core/src/native/worker-launcher/impl/worker-launcher.c -- diff --git a/storm-core/src/native/worker-launcher/impl/worker-launcher.c b/storm-core/src/native/worker-launcher/impl/worker-launcher.c index 02c0bec..d085315 100644 --- a/storm-core/src/native/worker-launcher/impl/worker-launcher.c +++ b/storm-core/src/native/worker-launcher/impl/worker-launcher.c @@ -593,14 +593,17 @@ int recursive_delete(const char *path, int supervisor_owns_dir) { return UNABLE_TO_BUILD_PATH; } + struct stat file_stat; + if(access(path, F_OK) != 0) { if(errno == ENOENT) { - return 0; -} -// Can probably return here, but we'll try to lstat anyway. - } + // we need to handle symlinks that target missing files. + if((lstat(path, &file_stat) != 0) || ((file_stat.st_mode & S_IFMT) != S_IFLNK)) { + return 0; + } + } + } - struct stat file_stat; if(lstat(path, &file_stat) != 0) { fprintf(LOGFILE, "Failed to delete %s: %s", path, strerror(errno)); return UNABLE_TO_STAT_FILE;
[1/3] storm git commit: Added in better docs for local mode testing.
Repository: storm Updated Branches: refs/heads/master 6bca7a949 -> 4d7099f3c Added in better docs for local mode testing. Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/46fe41e4 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/46fe41e4 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/46fe41e4 Branch: refs/heads/master Commit: 46fe41e418ce06f507488c2debb3b04033273006 Parents: ce984cd Author: Robert (Bobby) Evans Authored: Mon Oct 22 10:57:49 2018 -0500 Committer: Robert (Bobby) Evans Committed: Mon Oct 22 10:57:49 2018 -0500 -- docs/Local-mode.md | 4 +++- 1 file changed, 3 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/46fe41e4/docs/Local-mode.md -- diff --git a/docs/Local-mode.md b/docs/Local-mode.md index 199d690..31afcc3 100644 --- a/docs/Local-mode.md +++ b/docs/Local-mode.md @@ -7,7 +7,9 @@ Local mode simulates a Storm cluster in process and is useful for developing and To run a topology in local mode you have two options. The most common option is to run your topology with `storm local` instead of `storm jar` -This will bring up a local simulated cluster and force all interactions with nimbus to go through the simulated cluster instead of going to a separate process. +This will bring up a local simulated cluster and force all interactions with nimbus to go through the simulated cluster instead of going to a separate process. By default this will run the process for 20 seconds before tearing down the entire cluster. You can override this by including a `--local-ttl` command line option which sets the number of seconds it should run for. + +### Programatic If you want to do some automated testing but without actually launching a storm cluster you can use the same classes internally that `storm local` does.
[2/3] storm git commit: Addressed review comments
Addressed review comments Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/617b03a6 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/617b03a6 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/617b03a6 Branch: refs/heads/master Commit: 617b03a6af580c00aaa8d31f4cdde1a8bd7bc5c9 Parents: 46fe41e Author: Robert (Bobby) Evans Authored: Tue Oct 23 11:11:55 2018 -0500 Committer: Robert (Bobby) Evans Committed: Tue Oct 23 11:11:55 2018 -0500 -- docs/Local-mode.md | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/617b03a6/docs/Local-mode.md -- diff --git a/docs/Local-mode.md b/docs/Local-mode.md index 31afcc3..3b83d54 100644 --- a/docs/Local-mode.md +++ b/docs/Local-mode.md @@ -9,7 +9,7 @@ To run a topology in local mode you have two options. The most common option is This will bring up a local simulated cluster and force all interactions with nimbus to go through the simulated cluster instead of going to a separate process. By default this will run the process for 20 seconds before tearing down the entire cluster. You can override this by including a `--local-ttl` command line option which sets the number of seconds it should run for. -### Programatic +### Programmatic If you want to do some automated testing but without actually launching a storm cluster you can use the same classes internally that `storm local` does. @@ -79,7 +79,7 @@ You can see a full list of configurations [here](javadocs/org/apache/storm/Confi 1. **Config.TOPOLOGY_MAX_TASK_PARALLELISM**: This config puts a ceiling on the number of threads spawned for a single component. Oftentimes production topologies have a lot of parallelism (hundreds of threads) which places unreasonable load when trying to test the topology in local mode. This config lets you easy control that parallelism. 2. **Config.TOPOLOGY_DEBUG**: When this is set to true, Storm will log a message every time a tuple is emitted from any spout or bolt. This is extremely useful for debugging.A -These, like all other configs, can be set on the command line when launching your toplogy with the `-c` flag. The flag is of the form `-c =` so to enable debugging when launching your topology in local mode you could run +These, like all other configs, can be set on the command line when launching your topology with the `-c` flag. The flag is of the form `-c =` so to enable debugging when launching your topology in local mode you could run ``` storm local topology.jar -c topology.debug=true
[3/3] storm git commit: Merge branch 'LOCAL_MODE_DOCS' of https://github.com/revans2/incubator-storm
Merge branch 'LOCAL_MODE_DOCS' of https://github.com/revans2/incubator-storm Added in better docs for local mode testing. This closes #2892 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/4d7099f3 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/4d7099f3 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/4d7099f3 Branch: refs/heads/master Commit: 4d7099f3c21ce3dae981414882f327d1b5ab29a1 Parents: 6bca7a9 617b03a Author: Robert Evans Authored: Tue Oct 23 12:31:46 2018 -0500 Committer: Robert Evans Committed: Tue Oct 23 12:31:46 2018 -0500 -- docs/Local-mode.md | 6 -- 1 file changed, 4 insertions(+), 2 deletions(-) --
[2/2] storm git commit: Merge branch 'STORM-3269' of https://github.com/revans2/incubator-storm into STORM-3269
Merge branch 'STORM-3269' of https://github.com/revans2/incubator-storm into STORM-3269 STORM-3269: Update version of httpclient, and fix version dep issue This closes #2891 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/6bca7a94 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/6bca7a94 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/6bca7a94 Branch: refs/heads/master Commit: 6bca7a94945e9f24fc9259db6efca939180f272f Parents: 71aa68f 4bba9c4 Author: Robert Evans Authored: Tue Oct 23 12:06:30 2018 -0500 Committer: Robert Evans Committed: Tue Oct 23 12:06:30 2018 -0500 -- pom.xml | 2 +- .../src/jvm/org/apache/storm/utils/VersionInfo.java | 16 +--- .../rocksdb/ReadOnlyStringMetadataCache.java| 3 --- .../metricstore/rocksdb/RocksDbMetricsWriter.java | 4 +--- .../rocksdb/WritableStringMetadataCache.java| 3 +-- 5 files changed, 16 insertions(+), 12 deletions(-) --
[1/2] storm git commit: STORM-3269: Update version of httpclient, and fix version dep issue
Repository: storm Updated Branches: refs/heads/master 71aa68f09 -> 6bca7a949 STORM-3269: Update version of httpclient, and fix version dep issue Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/4bba9c4e Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/4bba9c4e Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/4bba9c4e Branch: refs/heads/master Commit: 4bba9c4e8ba962dd6f35c7fd25cfa466050f5607 Parents: ce984cd Author: Robert (Bobby) Evans Authored: Mon Oct 22 10:44:18 2018 -0500 Committer: Robert (Bobby) Evans Committed: Mon Oct 22 10:44:18 2018 -0500 -- pom.xml | 2 +- .../src/jvm/org/apache/storm/utils/VersionInfo.java | 16 +--- .../rocksdb/ReadOnlyStringMetadataCache.java| 3 --- .../metricstore/rocksdb/RocksDbMetricsWriter.java | 4 +--- .../rocksdb/WritableStringMetadataCache.java| 3 +-- 5 files changed, 16 insertions(+), 12 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/4bba9c4e/pom.xml -- diff --git a/pom.xml b/pom.xml index f5bc4fe..d7191ae 100644 --- a/pom.xml +++ b/pom.xml @@ -263,7 +263,7 @@ 0.2.3 1.5.0 1.11 -4.3.3 +4.5.6 2.0.1 0.9.0 16.0.1 http://git-wip-us.apache.org/repos/asf/storm/blob/4bba9c4e/storm-client/src/jvm/org/apache/storm/utils/VersionInfo.java -- diff --git a/storm-client/src/jvm/org/apache/storm/utils/VersionInfo.java b/storm-client/src/jvm/org/apache/storm/utils/VersionInfo.java index 9805d8b..bd8ff51 100644 --- a/storm-client/src/jvm/org/apache/storm/utils/VersionInfo.java +++ b/storm-client/src/jvm/org/apache/storm/utils/VersionInfo.java @@ -38,7 +38,8 @@ import org.slf4j.LoggerFactory; public final class VersionInfo { private static final Logger LOG = LoggerFactory.getLogger(VersionInfo.class); private static final String STORM_CORE_PROPERTIES_NAME = "storm-core-version-info.properties"; -public static final IVersionInfo OUR_FULL_VERSION = new VersionInfoImpl("storm-core"); +private static final String STORM_CLIENT_PROPERTIES_NAME = "storm-client-version-info.properties"; +public static final IVersionInfo OUR_FULL_VERSION = new VersionInfoImpl("storm-client"); public static final SimpleVersion OUR_VERSION = new SimpleVersion(OUR_FULL_VERSION.getVersion()); private static class VersionInfoImpl implements IVersionInfo { @@ -122,11 +123,20 @@ public final class VersionInfo { * @return the IVersionInfo or null. */ public static IVersionInfo getFromClasspath(List classpath) { +IVersionInfo ret = getFromClasspath(classpath, STORM_CLIENT_PROPERTIES_NAME); +if (ret == null) { +//storm-core is needed here for backwards compatibility. +ret = getFromClasspath(classpath, STORM_CORE_PROPERTIES_NAME); +} +return ret; +} + +private static IVersionInfo getFromClasspath(List classpath, final String propFileName) { IVersionInfo ret = null; for (String part: classpath) { Path p = Paths.get(part); if (Files.isDirectory(p)) { -Path child = p.resolve(STORM_CORE_PROPERTIES_NAME); +Path child = p.resolve(propFileName); if (Files.exists(child) && !Files.isDirectory(child)) { try (FileReader reader = new FileReader(child.toFile())) { Properties info = new Properties(); @@ -144,7 +154,7 @@ public final class VersionInfo { Enumeration zipEnums = jf.entries(); while (zipEnums.hasMoreElements()) { ZipEntry entry = zipEnums.nextElement(); -if (!entry.isDirectory() && entry.getName().equals(STORM_CORE_PROPERTIES_NAME)) { +if (!entry.isDirectory() && entry.getName().equals(propFileName)) { try (InputStreamReader reader = new InputStreamReader(jf.getInputStream(entry))) { Properties info = new Properties(); info.load(reader); http://git-wip-us.apache.org/repos/asf/storm/blob/4bba9c4e/storm-server/src/main/java/org/apache/storm/metricstore/rocksdb/ReadOnlyStringMetadataCache.java -- diff --git a/storm-server/src/main/java/org/apache/storm/metricstore/rocksdb/ReadOnlyStringMetadataCache.java b/storm-server/src/main/java/org/apache/storm/metricstore/rocksdb/ReadOnlyStringMetadataCa
[1/2] storm git commit: STORM-3260: Add in support to print some state
Repository: storm Updated Branches: refs/heads/master e2563a195 -> 71aa68f09 STORM-3260: Add in support to print some state Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/6828ecae Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/6828ecae Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/6828ecae Branch: refs/heads/master Commit: 6828ecae38c79522c3b7c9ab590d54ececf6d4c3 Parents: 66b8f50 Author: Robert (Bobby) Evans Authored: Wed Oct 17 17:02:40 2018 -0500 Committer: Robert (Bobby) Evans Committed: Wed Oct 17 17:02:40 2018 -0500 -- .../org/apache/storm/blobstore/BlobStore.java | 7 +- .../org/apache/storm/command/AdminCommands.java | 190 ++- .../org/apache/storm/daemon/nimbus/Nimbus.java | 3 - 3 files changed, 187 insertions(+), 13 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/6828ecae/storm-client/src/jvm/org/apache/storm/blobstore/BlobStore.java -- diff --git a/storm-client/src/jvm/org/apache/storm/blobstore/BlobStore.java b/storm-client/src/jvm/org/apache/storm/blobstore/BlobStore.java index 00b632d..cb2928c 100644 --- a/storm-client/src/jvm/org/apache/storm/blobstore/BlobStore.java +++ b/storm-client/src/jvm/org/apache/storm/blobstore/BlobStore.java @@ -48,7 +48,7 @@ import org.slf4j.LoggerFactory; * Modifying the replication factor only works for HdfsBlobStore as for the LocalFsBlobStore the replication is dependent on the number of * Nimbodes available. */ -public abstract class BlobStore implements Shutdownable { +public abstract class BlobStore implements Shutdownable, AutoCloseable { protected static final String BASE_BLOBS_DIR_NAME = "blobs"; private static final Logger LOG = LoggerFactory.getLogger(BlobStore.class); private static final KeyFilter TO_TOPO_ID = (key) -> ConfigUtils.getIdFromBlobKey(key); @@ -192,6 +192,11 @@ public abstract class BlobStore implements Shutdownable { public abstract int updateBlobReplication(String key, int replication, Subject who) throws AuthorizationException, KeyNotFoundException, IOException; +@Override +public void close() { +shutdown(); +} + /** * Filters keys based on the KeyFilter passed as the argument. * http://git-wip-us.apache.org/repos/asf/storm/blob/6828ecae/storm-core/src/jvm/org/apache/storm/command/AdminCommands.java -- diff --git a/storm-core/src/jvm/org/apache/storm/command/AdminCommands.java b/storm-core/src/jvm/org/apache/storm/command/AdminCommands.java index 8fabe6c..564a01c 100644 --- a/storm-core/src/jvm/org/apache/storm/command/AdminCommands.java +++ b/storm-core/src/jvm/org/apache/storm/command/AdminCommands.java @@ -19,20 +19,30 @@ package org.apache.storm.command; import com.google.common.collect.Sets; +import java.io.File; import java.io.PrintStream; import java.util.Arrays; +import java.util.Collection; import java.util.HashMap; import java.util.HashSet; import java.util.Map; import java.util.Set; +import org.apache.commons.io.FileUtils; import org.apache.storm.blobstore.BlobStore; import org.apache.storm.cluster.ClusterStateContext; import org.apache.storm.cluster.ClusterUtils; import org.apache.storm.cluster.DaemonType; import org.apache.storm.cluster.IStormClusterState; +import org.apache.storm.daemon.nimbus.Nimbus; +import org.apache.storm.generated.Assignment; import org.apache.storm.generated.Credentials; +import org.apache.storm.generated.StormTopology; +import org.apache.storm.generated.SupervisorInfo; import org.apache.storm.nimbus.NimbusInfo; import org.apache.storm.shade.org.apache.zookeeper.ZkCli; +import org.apache.storm.thrift.TBase; +import org.apache.storm.thrift.TFieldIdEnum; +import org.apache.storm.thrift.meta_data.FieldMetaData; import org.apache.storm.utils.ConfigUtils; import org.apache.storm.utils.ServerUtils; import org.apache.storm.utils.Utils; @@ -61,16 +71,17 @@ public class AdminCommands { private static class RemoveCorruptTopologies implements AdminCommand { @Override public void run(String[] args, Map conf, String command) throws Exception { -BlobStore nimbusBlobStore = ServerUtils.getNimbusBlobStore(conf, NimbusInfo.fromConf(conf), null); -IStormClusterState stormClusterState = ClusterUtils.mkStormClusterState(conf, new ClusterStateContext(DaemonType.NIMBUS, conf)); +try (BlobStore nimbusBlobStore = ServerUtils.getNimbusBlobStore(conf, NimbusInfo.fromConf(conf), null)) { +IStormClusterState stormClusterState = ClusterUtils.mkStormClusterState(conf, new ClusterStateContext(DaemonType.NIMBUS, conf));
[2/2] storm git commit: Merge branch 'STORM-3260' of https://github.com/revans2/incubator-storm into STORM-3260
Merge branch 'STORM-3260' of https://github.com/revans2/incubator-storm into STORM-3260 STORM-3260: Add in support to print some state This closes #2882 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/71aa68f0 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/71aa68f0 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/71aa68f0 Branch: refs/heads/master Commit: 71aa68f09c00dcd70c01bc918ee018accde7eb43 Parents: e2563a1 6828eca Author: Robert Evans Authored: Tue Oct 23 11:09:35 2018 -0500 Committer: Robert Evans Committed: Tue Oct 23 11:09:35 2018 -0500 -- .../org/apache/storm/blobstore/BlobStore.java | 7 +- .../org/apache/storm/command/AdminCommands.java | 190 ++- .../org/apache/storm/daemon/nimbus/Nimbus.java | 3 - 3 files changed, 187 insertions(+), 13 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/71aa68f0/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java --
[1/2] storm git commit: [STORM-3263] Fix sorting by owner resources in the UI
Repository: storm Updated Branches: refs/heads/master 3d315b854 -> e2563a195 [STORM-3263] Fix sorting by owner resources in the UI Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/6b037370 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/6b037370 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/6b037370 Branch: refs/heads/master Commit: 6b0373703ab89648943f52ec87c43531e2defc84 Parents: ce984cd Author: Jacob Tolar Authored: Fri Oct 19 14:14:32 2018 -0500 Committer: Jacob Tolar Committed: Fri Oct 19 14:14:32 2018 -0500 -- .../apache/storm/daemon/ui/WEB-INF/js/script.js | 25 +--- 1 file changed, 22 insertions(+), 3 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/6b037370/storm-webapp/src/main/java/org/apache/storm/daemon/ui/WEB-INF/js/script.js -- diff --git a/storm-webapp/src/main/java/org/apache/storm/daemon/ui/WEB-INF/js/script.js b/storm-webapp/src/main/java/org/apache/storm/daemon/ui/WEB-INF/js/script.js index 49b3518..78016c0 100644 --- a/storm-webapp/src/main/java/org/apache/storm/daemon/ui/WEB-INF/js/script.js +++ b/storm-webapp/src/main/java/org/apache/storm/daemon/ui/WEB-INF/js/script.js @@ -21,6 +21,21 @@ $(function () { //Add in custom sorting for some data types $.extend( $.fn.dataTableExt.oSort, { + // The 'resource-num' type is for sorting resource guarantees + // that have 'N/A' in the type. It's just the same as 'num', but + // we preprocess to sort N/A below 0. Otherwise, if we just use + // type 'num', DataTables always puts N/A at the *end* of the list + // (for both asc and desc sort), which is confusing. + "resource-num-pre": function(raw) { +if (raw == 'N/A') { + return Number.NEGATIVE_INFINITY; +} else { + return $.fn.dataTableExt.oSort['num-pre'](raw); +} + }, + "resource-num-asc": $.fn.dataTableExt.oSort['num-asc'], + "resource-num-desc": $.fn.dataTableExt.oSort['num-desc'], + "time-str-pre": function (raw) { var s = $(raw).text(); if (s == "") { @@ -549,10 +564,12 @@ var makeOwnerSummaryTable = function(response, elId, parentId) { if (showCpu) { columns.push({ -data: 'memoryGuarantee' +data: 'memoryGuarantee', +type: 'resource-num' }); columns.push({ data: 'memoryGuaranteeRemaining', +type: 'resource-num', render: function(data, type, row) { return getResourceGuaranteeRemainingFormat(type, data); } @@ -561,10 +578,12 @@ var makeOwnerSummaryTable = function(response, elId, parentId) { data: 'totalCpuUsage' }); columns.push({ -data: 'cpuGuarantee' +data: 'cpuGuarantee', +type: 'resource-num' }); columns.push({ data: 'cpuGuaranteeRemaining', +type: 'resource-num', render: function(data, type, row) { return getResourceGuaranteeRemainingFormat(type, data); } @@ -585,4 +604,4 @@ var makeOwnerSummaryTable = function(response, elId, parentId) { function getPageRenderedTimestamp(eId) { document.getElementById(eId).innerHTML = "Page rendered at: " + Date(); -}; \ No newline at end of file +};
[2/2] storm git commit: Merge branch 'STORM-3263' of https://github.com/jacobtolar/storm into STORM-3263
Merge branch 'STORM-3263' of https://github.com/jacobtolar/storm into STORM-3263 STORM-3263: Fix sorting by owner resources in the UI This closes #2888 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/e2563a19 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/e2563a19 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/e2563a19 Branch: refs/heads/master Commit: e2563a195fffa18b3299fd5903ec8bdbae8467c8 Parents: 3d315b8 6b03737 Author: Robert Evans Authored: Mon Oct 22 12:50:53 2018 -0500 Committer: Robert Evans Committed: Mon Oct 22 12:50:53 2018 -0500 -- .../apache/storm/daemon/ui/WEB-INF/js/script.js | 25 +--- 1 file changed, 22 insertions(+), 3 deletions(-) --
[1/2] storm git commit: STORM-3262 prevent falsely reporting leadership
Repository: storm Updated Branches: refs/heads/master 0ef21a85f -> 80999f405 STORM-3262 prevent falsely reporting leadership Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/02404865 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/02404865 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/02404865 Branch: refs/heads/master Commit: 02404865f241d2e737eb5dabe9d745d2393f3d9c Parents: 66b8f50 Author: Aaron Gresch Authored: Thu Oct 18 11:23:28 2018 -0500 Committer: Aaron Gresch Committed: Thu Oct 18 11:23:28 2018 -0500 -- .../main/java/org/apache/storm/daemon/nimbus/Nimbus.java | 10 -- 1 file changed, 8 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/02404865/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java -- diff --git a/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java b/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java index 3fa2f5d..9392ac3 100644 --- a/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java +++ b/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java @@ -2720,8 +2720,13 @@ public class Nimbus implements Iface, Shutdownable, DaemonCommon { NimbusInfo leader = leaderElector.getLeader(); for (NimbusSummary nimbusSummary : nimbuses) { nimbusSummary.set_uptime_secs(Time.deltaSecs(nimbusSummary.get_uptime_secs())); - nimbusSummary.set_isLeader(leader.getHost().equals(nimbusSummary.get_host()) - && leader.getPort() == nimbusSummary.get_port()); +// sometimes Leader election indicates the current nimbus is leader, but the host was recently restarted, +// and is currently not a leader. +boolean isLeader = leader.getHost().equals(nimbusSummary.get_host()) && leader.getPort() == nimbusSummary.get_port(); +if (isLeader && this.nimbusHostPortInfo.getHost().equals(leader.getHost()) && !this.isLeader()) { +isLeader = false; +} +nimbusSummary.set_isLeader(isLeader); } List topologySummaries = new ArrayList<>(); @@ -3787,6 +3792,7 @@ public class Nimbus implements Iface, Shutdownable, DaemonCommon { public String beginFileUpload() throws AuthorizationException, TException { try { beginFileUploadCalls.mark(); +assertIsLeader(); checkAuthorization(null, null, "fileUpload"); String fileloc = getInbox() + "/stormjar-" + Utils.uuid() + ".jar"; uploaders.put(fileloc, new TimedWritableByteChannel(Channels.newChannel(new FileOutputStream(fileloc)), fileUploadDuration));
[2/2] storm git commit: Merge branch 'agresch_storm-3262' of https://github.com/agresch/storm into STORM-3262
Merge branch 'agresch_storm-3262' of https://github.com/agresch/storm into STORM-3262 STORM-3262: prevent falsely reporting leadership This closes #2887 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/80999f40 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/80999f40 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/80999f40 Branch: refs/heads/master Commit: 80999f40536b06694465080d3f6c3f6bab6efc4a Parents: 0ef21a8 0240486 Author: Robert Evans Authored: Mon Oct 22 11:08:11 2018 -0500 Committer: Robert Evans Committed: Mon Oct 22 11:08:11 2018 -0500 -- .../main/java/org/apache/storm/daemon/nimbus/Nimbus.java | 10 -- 1 file changed, 8 insertions(+), 2 deletions(-) --
[2/2] storm git commit: Merge branch 'master' of https://github.com/contradictioned/storm
Merge branch 'master' of https://github.com/contradictioned/storm Fix Markdown errors and minor typos in the documentation. This closes #2885 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/ce984cd3 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/ce984cd3 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/ce984cd3 Branch: refs/heads/master Commit: ce984cd31a16e7fe4b983659005f1f7648455404 Parents: 0a01432 379d9a4 Author: Robert Evans Authored: Fri Oct 19 10:31:55 2018 -0500 Committer: Robert Evans Committed: Fri Oct 19 10:31:55 2018 -0500 -- docs/Kestrel-and-Storm.md | 2 + docs/SECURITY.md| 56 ++-- ...nding-the-parallelism-of-a-Storm-topology.md | 2 +- docs/index.md | 2 +- 4 files changed, 33 insertions(+), 29 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/ce984cd3/docs/SECURITY.md --
[1/2] storm git commit: Fix Markdown errors and minor typos in the documentation.
Repository: storm Updated Branches: refs/heads/master 0a01432fb -> ce984cd31 Fix Markdown errors and minor typos in the documentation. Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/379d9a41 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/379d9a41 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/379d9a41 Branch: refs/heads/master Commit: 379d9a4171409c88634a5aae9f1fb409eee73a39 Parents: 66b8f50 Author: Manuel Dossinger Authored: Thu Oct 18 14:55:58 2018 +0200 Committer: Manuel Dossinger Committed: Thu Oct 18 14:55:58 2018 +0200 -- docs/Kestrel-and-Storm.md | 2 + docs/SECURITY.md| 56 ++-- ...nding-the-parallelism-of-a-Storm-topology.md | 2 +- docs/index.md | 2 +- 4 files changed, 33 insertions(+), 29 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/379d9a41/docs/Kestrel-and-Storm.md -- diff --git a/docs/Kestrel-and-Storm.md b/docs/Kestrel-and-Storm.md index ff48995..1efe397 100644 --- a/docs/Kestrel-and-Storm.md +++ b/docs/Kestrel-and-Storm.md @@ -58,6 +58,7 @@ At first, we need to have a program that can add items to a Kestrel queue. The f ## Remove items from Kestrel This method dequeues items from a queue without removing them. + ``` private static void dequeueItems(KestrelClient kestrelClient, String queueName) throws IOException, ParseError { @@ -80,6 +81,7 @@ This method dequeues items from a queue without removing them. ``` This method dequeues items from a queue and then removes them. + ``` private static void dequeueAndRemoveItems(KestrelClient kestrelClient, String queueName) throws IOException, ParseError http://git-wip-us.apache.org/repos/asf/storm/blob/379d9a41/docs/SECURITY.md -- diff --git a/docs/SECURITY.md b/docs/SECURITY.md index 38375a8..4be619e 100644 --- a/docs/SECURITY.md +++ b/docs/SECURITY.md @@ -506,43 +506,45 @@ nimbus.groups: ### DRPC - Storm provides the Access Control List for the DRPC Authorizer.Users can see [org.apache.storm.security.auth.authorizer.DRPCSimpleACLAuthorizer](javadocs/org/apache/storm/security/auth/authorizer/DRPCSimpleACLAuthorizer.html) for more details. +Storm provides the Access Control List for the DRPC Authorizer.Users can see [org.apache.storm.security.auth.authorizer.DRPCSimpleACLAuthorizer](javadocs/org/apache/storm/security/auth/authorizer/DRPCSimpleACLAuthorizer.html) for more details. - There are several DRPC ACL related configurations. +There are several DRPC ACL related configurations. - | YAML Setting | Description | - ||--| - | drpc.authorizer.acl | A class that will perform authorization for DRPC operations. Set this to org.apache.storm.security.auth.authorizer.DRPCSimpleACLAuthorizer when using security.| - | drpc.authorizer.acl.filename | This is the name of a file that the ACLs will be loaded from. It is separate from storm.yaml to allow the file to be updated without bringing down a DRPC server. Defaults to drpc-auth-acl.yaml | - | drpc.authorizer.acl.strict| It is useful to set this to false for staging where users may want to experiment, but true for production where you want users to be secure. Defaults to false. | +| YAML Setting | Description | +||--| +| drpc.authorizer.acl | A class that will perform authorization for DRPC operations. Set this to org.apache.storm.security.auth.authorizer.DRPCSimpleACLAuthorizer when using security.| +| drpc.authorizer.acl.filename | This is the name of a file that the ACLs will be loaded from. It is separate from storm.yaml to allow the file to be updated without bringing down a DRPC server. Defaults to drpc-auth-acl.yaml | +| drpc.authorizer.acl.strict| It is useful to set this to false for staging where users may want to experiment, but true for production where you want users to be secure. Defaults to false. | - The file pointed to by drpc.authorizer.acl.filename will have only one config in it drpc.authorizer.acl this should be of the form +The file pointed to by drpc.authorizer.acl.filename will have only one config in it drpc.authorizer.acl this should be of the form - drpc.authorizer.acl: +```yaml +drpc.authorizer.acl: "functionName1": "client.users": - "alice" - "bob" "invocation.user": "bob" - - In this the users bob and alice as client.users are allowed to run DRPC requests against functionName1, but only bob as the invocation.user is allowed to run the topology that actually processes those requests. +``` + +In
[1/2] storm git commit: Clarify spout non-serializable field exception message
Repository: storm Updated Branches: refs/heads/master 14a483e0e -> 0a01432fb Clarify spout non-serializable field exception message ISpout has an `open` method instead of a `prepare` method. Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/e33cf7c0 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/e33cf7c0 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/e33cf7c0 Branch: refs/heads/master Commit: e33cf7c089748d21c2857c19005a1bb60e39bd55 Parents: 66b8f50 Author: Edward Samson Authored: Thu Oct 18 19:27:38 2018 +0800 Committer: Edward Samson Committed: Thu Oct 18 19:33:16 2018 +0800 -- docs/FAQ.md| 2 +- .../src/jvm/org/apache/storm/topology/TopologyBuilder.java | 2 +- 2 files changed, 2 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/e33cf7c0/docs/FAQ.md -- diff --git a/docs/FAQ.md b/docs/FAQ.md index ce9130e..525514e 100644 --- a/docs/FAQ.md +++ b/docs/FAQ.md @@ -64,7 +64,7 @@ At time of writing, you can't emit to multiple output streams from Trident -- se ### Why am I getting a NotSerializableException/IllegalStateException when my topology is being started up? -Within the Storm lifecycle, the topology is instantiated and then serialized to byte format to be stored in ZooKeeper, prior to the topology being executed. Within this step, if a spout or bolt within the topology has an initialized unserializable property, serialization will fail. If there is a need for a field that is unserializable, initialize it within the bolt or spout's prepare method, which is run after the topology is delivered to the worker. +Within the Storm lifecycle, the topology is instantiated and then serialized to byte format to be stored in ZooKeeper, prior to the topology being executed. Within this step, if a spout or bolt within the topology has an initialized unserializable property, serialization will fail. If there is a need for a field that is unserializable, initialize it within the bolt's `prepare` or spout's `open` method, which is run after the topology is delivered to the worker. ## Spouts http://git-wip-us.apache.org/repos/asf/storm/blob/e33cf7c0/storm-client/src/jvm/org/apache/storm/topology/TopologyBuilder.java -- diff --git a/storm-client/src/jvm/org/apache/storm/topology/TopologyBuilder.java b/storm-client/src/jvm/org/apache/storm/topology/TopologyBuilder.java index 9f2545b..eafc728 100644 --- a/storm-client/src/jvm/org/apache/storm/topology/TopologyBuilder.java +++ b/storm-client/src/jvm/org/apache/storm/topology/TopologyBuilder.java @@ -134,7 +134,7 @@ public class TopologyBuilder { throw new IllegalStateException( "Spout '" + spoutId + "' contains a non-serializable field of type " + wrapperCause.getCause().getMessage() + ", " + "which was instantiated prior to topology creation. " + wrapperCause.getCause().getMessage() + " " + -"should be instantiated within the prepare method of '" + spoutId + " at the earliest.", wrapperCause); +"should be instantiated within the open method of '" + spoutId + " at the earliest.", wrapperCause); } throw wrapperCause; }
[2/2] storm git commit: Merge branch 'master' of https://github.com/esamson/storm
Merge branch 'master' of https://github.com/esamson/storm Clarify spout non-serializable field exception message This closes #2884 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/0a01432f Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/0a01432f Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/0a01432f Branch: refs/heads/master Commit: 0a01432fb5de1f8fbbea47cf679a707f629fb71b Parents: 14a483e e33cf7c Author: Robert Evans Authored: Fri Oct 19 10:26:31 2018 -0500 Committer: Robert Evans Committed: Fri Oct 19 10:26:31 2018 -0500 -- docs/FAQ.md| 2 +- .../src/jvm/org/apache/storm/topology/TopologyBuilder.java | 2 +- 2 files changed, 2 insertions(+), 2 deletions(-) --
[3/3] storm git commit: Merge branch 'UI_SECURITY_DOCS' of https://github.com/revans2/incubator-storm
Merge branch 'UI_SECURITY_DOCS' of https://github.com/revans2/incubator-storm Improve the security docs on how to configure something like SSO for UI this closes #2886 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/14a483e0 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/14a483e0 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/14a483e0 Branch: refs/heads/master Commit: 14a483e0e90cdd04cc852a2a64a703f29bcf60a2 Parents: a100c12 05f8adc Author: Robert Evans Authored: Fri Oct 19 10:23:13 2018 -0500 Committer: Robert Evans Committed: Fri Oct 19 10:23:13 2018 -0500 -- docs/SECURITY.md | 47 +-- 1 file changed, 37 insertions(+), 10 deletions(-) --
[2/3] storm git commit: Spell Check
Spell Check Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/05f8adcd Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/05f8adcd Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/05f8adcd Branch: refs/heads/master Commit: 05f8adcdaa457e1b5880b803cc304f199688b945 Parents: ebb41d7 Author: Robert (Bobby) Evans Authored: Thu Oct 18 11:03:58 2018 -0500 Committer: Robert (Bobby) Evans Committed: Thu Oct 18 11:03:58 2018 -0500 -- docs/SECURITY.md | 20 ++-- 1 file changed, 10 insertions(+), 10 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/05f8adcd/docs/SECURITY.md -- diff --git a/docs/SECURITY.md b/docs/SECURITY.md index 9363d71..6abe9c0 100644 --- a/docs/SECURITY.md +++ b/docs/SECURITY.md @@ -60,17 +60,17 @@ logviewer.filter.params: "param1":"value1" ``` The `ui.filter` is an instance of `javax.servlet.Filter` that is intended to -filter all incomming requests to the UI and authenticate the request mapping +filter all incoming requests to the UI and authenticate the request mapping it to a "user". Typically this is done by modifying or wrapping the `HttpServletRequest` to return the user principal through the `getUserPrincipal()` method or returning the user name through the -`getRemoteUser()` method. If your filter authenticates in a differnt way you +`getRemoteUser()` method. If your filter authenticates in a different way you can look at setting `ui.http.creds.plugin` to point to an instance of `IHttpCredentialsPlugin` that can take the `HttpServletRequest` and return a user name and populate the needed fields in the current `ReqContext`. These are advanced features and you may want to look at the `DefaultHttpCredentialsPlugin` as an example of how to do this. -These same settings apply to the logviewer too. If you want to have separate controle +These same settings apply to the logviewer too. If you want to have separate control over how authentication works in the logviewer you may optionally set `logviewer.filter` instead and it will override any `ui.filter` settings for the logviewer process. @@ -82,7 +82,7 @@ logviewer.port set to the port of the proxy in its storm.yaml, while the logview must have it set to the actual port that they are going to bind to. The servlet filters are preferred because it allows individual topologies to -specificy who is and who is not allowed to access the pages associated with +specify who is and who is not allowed to access the pages associated with them. Storm UI (or logviewer) can be configured to use AuthenticationFilter from hadoop-auth. @@ -118,7 +118,7 @@ on that endpoint similar to the ui/logviewer. The `drpc.http.filter` and `drpc.http.filter.params` configs can be used to setup a `Filter` for the DRPC server. Unlike the logviewer it does not fall back to the UI configs as the DRPC server is intended to be REST only and often will be hit by headless users. -The `drpc.http.creds.plugin` confg can be used in cases where the default plugin is not good enough because of how authentication happens. +The `drpc.http.creds.plugin` config can be used in cases where the default plugin is not good enough because of how authentication happens. ## UI / DRPC / LOGVIEWER SSL @@ -141,7 +141,7 @@ optional config 8. ui.https.truststore.type (example "jks") If users want to setup 2-way auth -9. ui.https.want.client.auth (If this set to true server requests for client certifcate authentication, but keeps the connection if no authentication provided) +9. ui.https.want.client.auth (If this set to true server requests for client certificate authentication, but keeps the connection if no authentication provided) 10. ui.https.need.client.auth (If this set to true server requires client to provide authentication) @@ -162,7 +162,7 @@ optional config 8. drpc.https.truststore.type (example "jks") If users want to setup 2-way auth -9. drpc.https.want.client.auth (If this set to true server requests for client certifcate authentication, but keeps the connection if no authentication provided) +9. drpc.https.want.client.auth (If this set to true server requests for client certificate authentication, but keeps the connection if no authentication provided) 10. drpc.https.need.client.auth (If this set to true server requires client to provide authentication) @@ -183,7 +183,7 @@ optional config 8. logviewer.https.truststore.type (example "jks") If users want to setup 2-way auth -9. logviewer.https.want.client.auth (If this set to true server requests for client certifcate authentication, but keeps the connection if no au
[1/3] storm git commit: Improve the security docs on how to configure something like SSO for UI
Repository: storm Updated Branches: refs/heads/master a100c1241 -> 14a483e0e Improve the security docs on how to configure something like SSO for UI Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/ebb41d7f Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/ebb41d7f Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/ebb41d7f Branch: refs/heads/master Commit: ebb41d7faf1ebefaece3cb746608a1855b794593 Parents: 66b8f50 Author: Robert (Bobby) Evans Authored: Thu Oct 18 10:44:33 2018 -0500 Committer: Robert (Bobby) Evans Committed: Thu Oct 18 10:44:33 2018 -0500 -- docs/SECURITY.md | 35 +++ 1 file changed, 31 insertions(+), 4 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/ebb41d7f/docs/SECURITY.md -- diff --git a/docs/SECURITY.md b/docs/SECURITY.md index 38375a8..9363d71 100644 --- a/docs/SECURITY.md +++ b/docs/SECURITY.md @@ -50,7 +50,7 @@ The UI and logviewer processes provide a way to not only see what a cluster is doing, but also manipulate running topologies. In general these processes should not be exposed except to users of the cluster. -Some form of Authentication is typically required, with using java servlet filters +Some form of Authentication is typically required, and can be done using a java servlet filter ```yaml ui.filter: "filter.class" @@ -58,9 +58,25 @@ ui.filter.params: "param1":"value1" logviewer.filter: "filter.class" logviewer.filter.params: "param1":"value1" ``` -or by restricting the UI/log viewers ports to only accept connections from local -hosts, and then front them with another web server, like Apache httpd, that can -authenticate/authorize incoming connections and + +The `ui.filter` is an instance of `javax.servlet.Filter` that is intended to +filter all incomming requests to the UI and authenticate the request mapping +it to a "user". Typically this is done by modifying or wrapping the +`HttpServletRequest` to return the user principal through the +`getUserPrincipal()` method or returning the user name through the +`getRemoteUser()` method. If your filter authenticates in a differnt way you +can look at setting `ui.http.creds.plugin` to point to an instance of `IHttpCredentialsPlugin` +that can take the `HttpServletRequest` and return a user name and populate the needed fields +in the current `ReqContext`. These are advanced features and you may want to look at the +`DefaultHttpCredentialsPlugin` as an example of how to do this. + +These same settings apply to the logviewer too. If you want to have separate controle +over how authentication works in the logviewer you may optionally set `logviewer.filter` +instead and it will override any `ui.filter` settings for the logviewer process. + +If the cluster is single tenant you might want to just restrict access to the UI/log +viewers ports to only accept connections from local hosts, and then front them with +another web server, like Apache httpd, that can authenticate/authorize incoming connections and proxy the connection to the storm process. To make this work the ui process must have logviewer.port set to the port of the proxy in its storm.yaml, while the logviewers must have it set to the actual port that they are going to bind to. @@ -94,6 +110,17 @@ curl -i --negotiate -u:anyUser -b ~/cookiejar.txt -c ~/cookiejar.txt http://s **Caution**: In AD MIT Keberos setup the key size is bigger than the default UI jetty server request header size. Make sure you set ui.header.buffer.bytes to 65536 in storm.yaml. More details are on [STORM-633](https://issues.apache.org/jira/browse/STORM-633) +## DRPC HTTP + +The DRPC server optionally supports a REST endpoint as well, and you can configure authentication +on that endpoint similar to the ui/logviewer. + +The `drpc.http.filter` and `drpc.http.filter.params` configs can be used to setup a `Filter` for the DRPC server. Unlike the logviewer +it does not fall back to the UI configs as the DRPC server is intended to be REST only and often will be hit by headless users. + +The `drpc.http.creds.plugin` confg can be used in cases where the default plugin is not good enough because of how authentication happens. + + ## UI / DRPC / LOGVIEWER SSL UI,DRPC and LOGVIEWER allows users to configure ssl .
[1/2] storm git commit: STORM-3258 reduce blobstore logging
Repository: storm Updated Branches: refs/heads/master 66b8f5075 -> a100c1241 STORM-3258 reduce blobstore logging Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/56d3752c Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/56d3752c Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/56d3752c Branch: refs/heads/master Commit: 56d3752c001f021ebb723c904d134fd01024cad8 Parents: a0b8a6a Author: Aaron Gresch Authored: Tue Oct 16 08:40:00 2018 -0500 Committer: Aaron Gresch Committed: Tue Oct 16 08:40:00 2018 -0500 -- .../storm/hdfs/blobstore/HdfsBlobStoreImpl.java | 23 ++-- 1 file changed, 12 insertions(+), 11 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/56d3752c/external/storm-hdfs-blobstore/src/main/java/org/apache/storm/hdfs/blobstore/HdfsBlobStoreImpl.java -- diff --git a/external/storm-hdfs-blobstore/src/main/java/org/apache/storm/hdfs/blobstore/HdfsBlobStoreImpl.java b/external/storm-hdfs-blobstore/src/main/java/org/apache/storm/hdfs/blobstore/HdfsBlobStoreImpl.java index 702a16f..8a97572 100644 --- a/external/storm-hdfs-blobstore/src/main/java/org/apache/storm/hdfs/blobstore/HdfsBlobStoreImpl.java +++ b/external/storm-hdfs-blobstore/src/main/java/org/apache/storm/hdfs/blobstore/HdfsBlobStoreImpl.java @@ -15,8 +15,17 @@ * See the License for the specific language governing permissions and * limitations under the License. */ + package org.apache.storm.hdfs.blobstore; +import java.io.FileNotFoundException; +import java.io.IOException; +import java.util.ArrayList; +import java.util.Iterator; +import java.util.Map; +import java.util.NoSuchElementException; +import java.util.Timer; +import java.util.TimerTask; import org.apache.hadoop.conf.Configuration; import org.apache.hadoop.fs.FileStatus; import org.apache.hadoop.fs.FileSystem; @@ -28,14 +37,6 @@ import org.apache.storm.utils.ObjectReader; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import java.io.FileNotFoundException; -import java.io.IOException; -import java.util.ArrayList; -import java.util.Iterator; -import java.util.Map; -import java.util.NoSuchElementException; -import java.util.Timer; -import java.util.TimerTask; /** * HDFS blob store impl. @@ -43,7 +44,7 @@ import java.util.TimerTask; public class HdfsBlobStoreImpl { private static final Logger LOG = LoggerFactory.getLogger(HdfsBlobStoreImpl.class); -private static final long FULL_CLEANUP_FREQ = 60 * 60 * 1000l; +private static final long FULL_CLEANUP_FREQ = 60 * 60 * 1000L; private static final int BUCKETS = 1024; private static final String BLOBSTORE_DATA = "data"; @@ -125,7 +126,7 @@ public class HdfsBlobStoreImpl { public HdfsBlobStoreImpl(Path path, Map conf, Configuration hconf) throws IOException { -LOG.info("Blob store based in {}", path); +LOG.debug("Blob store based in {}", path); _fullPath = path; _hadoopConf = hconf; _fs = path.getFileSystem(_hadoopConf); @@ -206,7 +207,7 @@ public class HdfsBlobStoreImpl { } /** - * Delete a key from the blob store + * Delete a key from the blob store. * * @param key the key to delete * @throws IOException on any error
[2/2] storm git commit: Merge branch 'agresch_storm-3258' of https://github.com/agresch/storm into STORM-3258
Merge branch 'agresch_storm-3258' of https://github.com/agresch/storm into STORM-3258 STORM-3258: reduce blobstore logging This closes #2879 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/a100c124 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/a100c124 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/a100c124 Branch: refs/heads/master Commit: a100c1241d689595bf1ec8ae21aaa97fd1aed951 Parents: 66b8f50 56d3752 Author: Robert Evans Authored: Fri Oct 19 09:49:31 2018 -0500 Committer: Robert Evans Committed: Fri Oct 19 09:49:31 2018 -0500 -- .../storm/hdfs/blobstore/HdfsBlobStoreImpl.java | 23 ++-- 1 file changed, 12 insertions(+), 11 deletions(-) --
[1/2] storm git commit: Revert "[STORM-3233] Updated zookeeper client to version 3.4.13 which fixes various issues including ZOOKEEPER-2184 that prevents ZooKeeper Java clients working in dynamic IP (
Repository: storm Updated Branches: refs/heads/1.1.x-branch 66a429abf -> f21d4e585 Revert "[STORM-3233] Updated zookeeper client to version 3.4.13 which fixes various issues including ZOOKEEPER-2184 that prevents ZooKeeper Java clients working in dynamic IP (container / cloud) environment." This reverts commit ac763b8bbd23f7fdc0aa9990bd0e61ce17706e77. Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/8315d31c Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/8315d31c Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/8315d31c Branch: refs/heads/1.1.x-branch Commit: 8315d31c689ce23eec6c171500411f818b828f53 Parents: 66a429a Author: Robert (Bobby) Evans Authored: Thu Oct 11 10:18:55 2018 -0500 Committer: Robert (Bobby) Evans Committed: Thu Oct 11 10:18:55 2018 -0500 -- pom.xml | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/8315d31c/pom.xml -- diff --git a/pom.xml b/pom.xml index 05dd00a..7e08c64 100644 --- a/pom.xml +++ b/pom.xml @@ -254,7 +254,7 @@ 0.2.3 1.9.5 2.1.3 -3.4.13 +3.4.6 0.1.0 1.2.0 0.14.0
[2/2] storm git commit: Merge branch 'STORM-3233-revert-1.1.x' of https://github.com/revans2/incubator-storm into 1.1.x-branch
Merge branch 'STORM-3233-revert-1.1.x' of https://github.com/revans2/incubator-storm into 1.1.x-branch Revert "[STORM-3233] Updated zookeeper client to version 3.4.13 whichâ¦" This closes #2870 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/f21d4e58 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/f21d4e58 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/f21d4e58 Branch: refs/heads/1.1.x-branch Commit: f21d4e5850459749f4b484fe7e4731c9d85691d7 Parents: 66a429a 8315d31 Author: Robert Evans Authored: Mon Oct 15 11:00:28 2018 -0500 Committer: Robert Evans Committed: Mon Oct 15 11:00:28 2018 -0500 -- pom.xml | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) --
[2/2] storm git commit: Merge branch 'STORM-3233-revert-1.x' of https://github.com/revans2/incubator-storm into 1.x-branch
Merge branch 'STORM-3233-revert-1.x' of https://github.com/revans2/incubator-storm into 1.x-branch Revert "[STORM-3233] Updated zookeeper client to version 3.4.13 whichâ¦" This closes #2869 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/2f3a04dc Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/2f3a04dc Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/2f3a04dc Branch: refs/heads/1.x-branch Commit: 2f3a04dcc5f1271f0b857171db97b346f8750e49 Parents: ce1d862 aca89ed Author: Robert Evans Authored: Mon Oct 15 10:59:29 2018 -0500 Committer: Robert Evans Committed: Mon Oct 15 10:59:29 2018 -0500 -- pom.xml | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) --
[1/2] storm git commit: Revert "[STORM-3233] Updated zookeeper client to version 3.4.13 which fixes various issues including ZOOKEEPER-2184 that prevents ZooKeeper Java clients working in dynamic IP (
Repository: storm Updated Branches: refs/heads/1.x-branch ce1d862ee -> 2f3a04dcc Revert "[STORM-3233] Updated zookeeper client to version 3.4.13 which fixes various issues including ZOOKEEPER-2184 that prevents ZooKeeper Java clients working in dynamic IP (container / cloud) environment." This reverts commit 6d77f0a977d2276daa2f985864b41abac4f59c3f. Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/aca89eda Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/aca89eda Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/aca89eda Branch: refs/heads/1.x-branch Commit: aca89eda345fb6e3a43c96885b35717398257441 Parents: ce1d862 Author: Robert (Bobby) Evans Authored: Thu Oct 11 10:17:16 2018 -0500 Committer: Robert (Bobby) Evans Committed: Thu Oct 11 10:17:16 2018 -0500 -- pom.xml | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/aca89eda/pom.xml -- diff --git a/pom.xml b/pom.xml index 8fa189e..cff8bc7 100644 --- a/pom.xml +++ b/pom.xml @@ -253,7 +253,7 @@ 0.2.3 1.9.5 2.1.3 -3.4.13 +3.4.6 0.1.0 1.2.0 0.14.0
[1/2] storm git commit: Revert "[STORM-3233] Updated zookeeper client to version 3.4.13 which fixes various issues including ZOOKEEPER-2184 that prevents ZooKeeper Java clients working in dynamic IP (
Repository: storm Updated Branches: refs/heads/master 438e02e4f -> a0b8a6a8d Revert "[STORM-3233] Updated zookeeper client to version 3.4.13 which fixes various issues including ZOOKEEPER-2184 that prevents ZooKeeper Java clients working in dynamic IP (container / cloud) environment." This reverts commit b187971e0da0238c6beba200b01fb738a6f848d8. Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/238f1c18 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/238f1c18 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/238f1c18 Branch: refs/heads/master Commit: 238f1c18f3ef5fa5240439a7ffd9f0a89a200473 Parents: 438e02e Author: Robert (Bobby) Evans Authored: Mon Oct 15 10:28:57 2018 -0500 Committer: Robert (Bobby) Evans Committed: Mon Oct 15 10:28:57 2018 -0500 -- pom.xml | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/238f1c18/pom.xml -- diff --git a/pom.xml b/pom.xml index 22fc32c..f5bc4fe 100644 --- a/pom.xml +++ b/pom.xml @@ -275,7 +275,7 @@ 1.7.21 3.2.6 2.19.0 -3.4.13 +3.4.6 0.9.94 2.3.3 2.6.1
[2/2] storm git commit: Merge branch 'STORM-3233-revert' of https://github.com/revans2/incubator-storm
Merge branch 'STORM-3233-revert' of https://github.com/revans2/incubator-storm Revert "[STORM-3233] Updated zookeeper client to version 3.4.13 whichâ¦" This closes #2868 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/a0b8a6a8 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/a0b8a6a8 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/a0b8a6a8 Branch: refs/heads/master Commit: a0b8a6a8d6746700fb356ed0687f88844a146b9a Parents: 438e02e 238f1c1 Author: Robert Evans Authored: Mon Oct 15 10:36:10 2018 -0500 Committer: Robert Evans Committed: Mon Oct 15 10:36:10 2018 -0500 -- pom.xml | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) --
[1/2] storm git commit: Don't wait for localization of blobs if assignments change.
Repository: storm Updated Branches: refs/heads/master f671f8930 -> 438e02e4f Don't wait for localization of blobs if assignments change. Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/23dab727 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/23dab727 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/23dab727 Branch: refs/heads/master Commit: 23dab727ed23ed739e066d414c546652cb8ac4d1 Parents: 20d5581 Author: Kishor Patil Authored: Thu Oct 11 18:04:23 2018 -0400 Committer: Kishor Patil Committed: Thu Oct 11 18:04:23 2018 -0400 -- .../src/main/java/org/apache/storm/daemon/supervisor/Slot.java | 6 +++--- 1 file changed, 3 insertions(+), 3 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/23dab727/storm-server/src/main/java/org/apache/storm/daemon/supervisor/Slot.java -- diff --git a/storm-server/src/main/java/org/apache/storm/daemon/supervisor/Slot.java b/storm-server/src/main/java/org/apache/storm/daemon/supervisor/Slot.java index e27f8f4..8b1b2f5 100644 --- a/storm-server/src/main/java/org/apache/storm/daemon/supervisor/Slot.java +++ b/storm-server/src/main/java/org/apache/storm/daemon/supervisor/Slot.java @@ -430,9 +430,6 @@ public class Slot extends Thread implements AutoCloseable, BlobChangingCallback dynamicState = informChangedBlobs(dynamicState, dynamicState.pendingLocalization); } -// Wait until time out -dynamicState.pendingDownload.get(1000, TimeUnit.MILLISECONDS); -//Downloading of all blobs finished. This is the precondition for all codes below. if (!equivalent(dynamicState.newAssignment, dynamicState.pendingLocalization)) { //Scheduling changed staticState.localizer.releaseSlotFor(dynamicState.pendingLocalization, staticState.port); @@ -442,6 +439,9 @@ public class Slot extends Thread implements AutoCloseable, BlobChangingCallback staticState); } +// Wait until time out +dynamicState.pendingDownload.get(1000, TimeUnit.MILLISECONDS); +//Downloading of all blobs finished. This is the precondition for all codes below. if (!dynamicState.pendingChangingBlobs.isEmpty()) { LOG.info("There are pending changes, waiting for them to finish before launching container..."); //We cannot launch the container yet the resources may still be updating
[2/2] storm git commit: Merge branch 'storm3254' of https://github.com/kishorvpatil/incubator-storm into STORM-3254
Merge branch 'storm3254' of https://github.com/kishorvpatil/incubator-storm into STORM-3254 STORM-3254: Don't wait for localization of blobs if assignments change. This closes #2876 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/438e02e4 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/438e02e4 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/438e02e4 Branch: refs/heads/master Commit: 438e02e4fd57ac8c7a7933f71500336cc07012d8 Parents: f671f89 23dab72 Author: Robert Evans Authored: Mon Oct 15 08:56:06 2018 -0500 Committer: Robert Evans Committed: Mon Oct 15 08:56:06 2018 -0500 -- .../src/main/java/org/apache/storm/daemon/supervisor/Slot.java | 6 +++--- 1 file changed, 3 insertions(+), 3 deletions(-) --
[1/3] storm git commit: Uses dev-zookeeper in travis-ci
Repository: storm Updated Branches: refs/heads/master c52cedf9b -> f671f8930 Uses dev-zookeeper in travis-ci Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/cafe61ff Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/cafe61ff Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/cafe61ff Branch: refs/heads/master Commit: cafe61ffaa98c9656b489b4bc38d1f1689e28535 Parents: 2244074 Author: Derek Dagit Authored: Fri Oct 12 13:02:52 2018 -0500 Committer: Derek Dagit Committed: Fri Oct 12 13:02:52 2018 -0500 -- integration-test/run-it.sh | 6 +- 1 file changed, 5 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/cafe61ff/integration-test/run-it.sh -- diff --git a/integration-test/run-it.sh b/integration-test/run-it.sh index 217a77a..d2988a4 100755 --- a/integration-test/run-it.sh +++ b/integration-test/run-it.sh @@ -79,7 +79,6 @@ echo "Using storm version:" ${STORM_VERSION} # setup storm cluster list_storm_processes || true sudo bash "${SCRIPT_DIR}/config/common.sh" -sudo bash "${SCRIPT_DIR}/config/install-zookeeper.sh" "$zookeeper_version" sudo bash "${SCRIPT_DIR}/config/install-storm.sh" "$storm_binary_zip" if [[ "$TRAVIS_JDK_VERSION" == "oraclejdk10" ]] || [[ "${JDK_VERSION}" == "10" ]] then @@ -91,6 +90,11 @@ function start_storm_process() { echo starting: storm $1 sudo su storm -c "export JAVA_HOME=\"${JAVA_HOME}\" && cd /usr/share/storm && storm $1" & } +if [[ "${USER}" == 'travis' ]]; then +start_storm_process dev-zookeeper +else +sudo bash "${SCRIPT_DIR}/config/install-zookeeper.sh" "$zookeeper_version" +fi start_storm_process nimbus start_storm_process ui start_storm_process supervisor
[3/3] storm git commit: Merge branch 'storm-3255-use-dev-zk-in-travis' of https://github.com/d2r/storm
Merge branch 'storm-3255-use-dev-zk-in-travis' of https://github.com/d2r/storm STORM-3255: Uses dev-zookeeper in travis-ci This closes #2877 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/f671f893 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/f671f893 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/f671f893 Branch: refs/heads/master Commit: f671f89301058419dfa059318770e5a646eabb7a Parents: c52cedf 3920502 Author: Robert Evans Authored: Mon Oct 15 08:52:47 2018 -0500 Committer: Robert Evans Committed: Mon Oct 15 08:52:47 2018 -0500 -- integration-test/config/install-zookeeper.sh | 21 - integration-test/run-it.sh | 4 +--- 2 files changed, 1 insertion(+), 24 deletions(-) --
[2/3] storm git commit: Uses dev-zookeeper even when not on travis
Uses dev-zookeeper even when not on travis Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/39205027 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/39205027 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/39205027 Branch: refs/heads/master Commit: 39205027d9fc066310617931a741b51905d27fcd Parents: cafe61f Author: Derek Dagit Authored: Fri Oct 12 14:54:32 2018 -0500 Committer: Derek Dagit Committed: Fri Oct 12 14:54:32 2018 -0500 -- integration-test/config/install-zookeeper.sh | 21 - integration-test/run-it.sh | 8 +--- 2 files changed, 1 insertion(+), 28 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/39205027/integration-test/config/install-zookeeper.sh -- diff --git a/integration-test/config/install-zookeeper.sh b/integration-test/config/install-zookeeper.sh deleted file mode 100644 index 98253d7..000 --- a/integration-test/config/install-zookeeper.sh +++ /dev/null @@ -1,21 +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. -# -# $1 is the Zookeeper version to install -apt-get --yes install zookeeper=$1 zookeeperd=$1 -service zookeeper stop -echo maxClientCnxns=200 >> /etc/zookeeper/conf/zoo.cfg -service zookeeper start http://git-wip-us.apache.org/repos/asf/storm/blob/39205027/integration-test/run-it.sh -- diff --git a/integration-test/run-it.sh b/integration-test/run-it.sh index d2988a4..94e22f8 100755 --- a/integration-test/run-it.sh +++ b/integration-test/run-it.sh @@ -61,14 +61,12 @@ if [[ "${USER}" == "vagrant" ]]; then # install oracle jdk8 or jdk10 java -version mvn --version export MAVEN_OPTS="-Xmx3000m" -zookeeper_version=3.4.8* else ( while true; do echo "heartbeat"; sleep 300; done ) & #heartbeat needed by travis ci if [[ "${USER}" == "travis" ]]; then ( cd "${STORM_SRC_DIR}/storm-dist/binary" && mvn clean package -Dgpg.skip=true ) fi (( $(find "${STORM_SRC_DIR}/storm-dist/binary" -iname 'apache-storm*.zip' | wc -l) == 1 )) || die "expected exactly one zip file, did you run: cd ${STORM_SRC_DIR}/storm-dist/binary && mvn clean package -Dgpg.skip=true" -zookeeper_version=3.4.5* fi storm_binary_zip=$(find "${STORM_SRC_DIR}/storm-dist" -iname '*.zip') @@ -90,11 +88,7 @@ function start_storm_process() { echo starting: storm $1 sudo su storm -c "export JAVA_HOME=\"${JAVA_HOME}\" && cd /usr/share/storm && storm $1" & } -if [[ "${USER}" == 'travis' ]]; then -start_storm_process dev-zookeeper -else -sudo bash "${SCRIPT_DIR}/config/install-zookeeper.sh" "$zookeeper_version" -fi +start_storm_process dev-zookeeper start_storm_process nimbus start_storm_process ui start_storm_process supervisor
[1/2] storm git commit: Deprecates the storm repl subcommand
Repository: storm Updated Branches: refs/heads/master 8e1627dde -> c52cedf9b Deprecates the storm repl subcommand Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/8533116d Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/8533116d Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/8533116d Branch: refs/heads/master Commit: 8533116d955506e5f295ac8e31bc83f408c26854 Parents: 2244074 Author: Derek Dagit Authored: Thu Oct 11 16:28:38 2018 -0500 Committer: Derek Dagit Committed: Thu Oct 11 16:28:38 2018 -0500 -- DEVELOPER.md| 8 bin/storm.py| 3 ++- docs/Command-line-client.md | 2 ++ 3 files changed, 4 insertions(+), 9 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/8533116d/DEVELOPER.md -- diff --git a/DEVELOPER.md b/DEVELOPER.md index 8b9ea9c..4e595eb 100644 --- a/DEVELOPER.md +++ b/DEVELOPER.md @@ -296,16 +296,8 @@ To run all unit tests and all integration tests execute one of the commands mvn -P all-tests integration-test -You can also run tests selectively via the Clojure REPL. The following example runs the tests in -[auth_test.clj](storm-core/test/clj/org/apache/storm/security/auth/auth_test.clj), which has the namespace -`org.apache.storm.security.auth.auth-test`. - You can also run tests selectively with `-Dtest=`. This works for both clojure and junit tests. -> Tip: IDEs such as IntelliJ IDEA support a built-in Clojure REPL, which you can also use to run tests selectively. -> Sometimes you may find that tests pass/fail depending on which REPL you use, which -- although frustrating -- -> can be helpful to narrow down errors. - Unfortunately you might experience failures in clojure tests which are wrapped in the `maven-clojure-plugin` and thus doesn't provide too much useful output at first sight - you might end up with a maven test failure with an error message as unhelpful as `Clojure failed.`. In this case it's recommended to look into `target/test-reports` of the failed project to see what actual tests have failed or scroll through the maven output looking for obvious issues like missing binaries. By default integration tests are not run in the test phase. To run Java and Clojure integration tests you must enable the profile http://git-wip-us.apache.org/repos/asf/storm/blob/8533116d/bin/storm.py -- diff --git a/bin/storm.py b/bin/storm.py index 2f2de6e..326b209 100755 --- a/bin/storm.py +++ b/bin/storm.py @@ -695,7 +695,8 @@ def shell(resourcesdir, command, *args): os.system("rm " + tmpjarpath) def repl(): -"""Syntax: [storm repl] +"""DEPRECATED: This subcommand may be removed in a future release. +Syntax: [storm repl] Opens up a Clojure REPL with the storm jars and configuration on the classpath. Useful for debugging. http://git-wip-us.apache.org/repos/asf/storm/blob/8533116d/docs/Command-line-client.md -- diff --git a/docs/Command-line-client.md b/docs/Command-line-client.md index 8243b25..2b22c4e 100644 --- a/docs/Command-line-client.md +++ b/docs/Command-line-client.md @@ -101,6 +101,8 @@ The rebalance command can also be used to change the parallelism of a running to ### repl +*DEPRECATED: This subcommand may be removed in a future release.* + Syntax: `storm repl` Opens up a Clojure REPL with the storm jars and configuration on the classpath. Useful for debugging.
[2/2] storm git commit: Merge branch 'storm-3253-deprecate-storm-repl' of https://github.com/d2r/storm
Merge branch 'storm-3253-deprecate-storm-repl' of https://github.com/d2r/storm STORM-3253: Deprecates the storm repl subcommand This closes #2875 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/c52cedf9 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/c52cedf9 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/c52cedf9 Branch: refs/heads/master Commit: c52cedf9bd7430ef7fb47279b232ca4d769e64f0 Parents: 8e1627d 8533116 Author: Robert Evans Authored: Mon Oct 15 08:51:22 2018 -0500 Committer: Robert Evans Committed: Mon Oct 15 08:51:22 2018 -0500 -- DEVELOPER.md| 8 bin/storm.py| 3 ++- docs/Command-line-client.md | 2 ++ 3 files changed, 4 insertions(+), 9 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/c52cedf9/DEVELOPER.md -- diff --cc DEVELOPER.md index e4da377,4e595eb..20ac51f --- a/DEVELOPER.md +++ b/DEVELOPER.md @@@ -292,20 -292,12 +292,12 @@@ To run all unit tests plus Clojure inte To run all unit tests and all integration tests execute one of the commands -mvn -P all-tests verify -mvn -P all-tests integration-test +mvn -P all-tests,examples,externals verify +mvn -P all-tests,examples,externals integration-test - You can also run tests selectively via the Clojure REPL. The following example runs the tests in - [auth_test.clj](storm-core/test/clj/org/apache/storm/security/auth/auth_test.clj), which has the namespace - `org.apache.storm.security.auth.auth-test`. - You can also run tests selectively with `-Dtest=`. This works for both clojure and junit tests. - > Tip: IDEs such as IntelliJ IDEA support a built-in Clojure REPL, which you can also use to run tests selectively. - > Sometimes you may find that tests pass/fail depending on which REPL you use, which -- although frustrating -- - > can be helpful to narrow down errors. - Unfortunately you might experience failures in clojure tests which are wrapped in the `maven-clojure-plugin` and thus doesn't provide too much useful output at first sight - you might end up with a maven test failure with an error message as unhelpful as `Clojure failed.`. In this case it's recommended to look into `target/test-reports` of the failed project to see what actual tests have failed or scroll through the maven output looking for obvious issues like missing binaries. By default integration tests are not run in the test phase. To run Java and Clojure integration tests you must enable the profile
[2/2] storm git commit: Merge branch 'dev-doc-clarify-maven-profiles' of https://github.com/d2r/storm
Merge branch 'dev-doc-clarify-maven-profiles' of https://github.com/d2r/storm Clarifies example mvn commands given new profiles this closes #2874 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/8e1627dd Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/8e1627dd Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/8e1627dd Branch: refs/heads/master Commit: 8e1627dde8d3fd4624a5ceecfccfd250cef1e407 Parents: 5ea90ce b89c56e Author: Robert Evans Authored: Mon Oct 15 08:50:29 2018 -0500 Committer: Robert Evans Committed: Mon Oct 15 08:50:29 2018 -0500 -- DEVELOPER.md | 10 +- 1 file changed, 5 insertions(+), 5 deletions(-) --
[1/2] storm git commit: Clarifies example mvn commands given new profiles
Repository: storm Updated Branches: refs/heads/master 5ea90ce0f -> 8e1627dde Clarifies example mvn commands given new profiles Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/b89c56ec Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/b89c56ec Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/b89c56ec Branch: refs/heads/master Commit: b89c56ec3ff599262588cf05fa50a27101912ee3 Parents: 2244074 Author: Derek Dagit Authored: Thu Oct 11 16:08:29 2018 -0500 Committer: Derek Dagit Committed: Thu Oct 11 16:08:29 2018 -0500 -- DEVELOPER.md | 10 +- 1 file changed, 5 insertions(+), 5 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/b89c56ec/DEVELOPER.md -- diff --git a/DEVELOPER.md b/DEVELOPER.md index 8b9ea9c..e4da377 100644 --- a/DEVELOPER.md +++ b/DEVELOPER.md @@ -283,17 +283,17 @@ Integration tests require that you activate the profile `integration-test` and t To run all Java and Clojure integration tests but no unit tests execute one of the commands -mvn -P integration-tests-only verify -mvn -P integration-tests-only integration-test +mvn -P integration-tests-only,examples,externals verify +mvn -P integration-tests-only,examples,externals integration-test To run all unit tests plus Clojure integration tests but no Java integration tests execute the command -mvn -P all-tests test +mvn -P all-tests,examples,externals test To run all unit tests and all integration tests execute one of the commands -mvn -P all-tests verify -mvn -P all-tests integration-test +mvn -P all-tests,examples,externals verify +mvn -P all-tests,examples,externals integration-test You can also run tests selectively via the Clojure REPL. The following example runs the tests in
[1/2] storm git commit: Removes and cleans up Clojure dependencies
Repository: storm Updated Branches: refs/heads/master 5a8378a34 -> 5ea90ce0f Removes and cleans up Clojure dependencies Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/1086857a Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/1086857a Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/1086857a Branch: refs/heads/master Commit: 1086857a24e824fde649be449135e8c5a89740f1 Parents: 2244074 Author: Derek Dagit Authored: Thu Oct 11 15:54:17 2018 -0500 Committer: Derek Dagit Committed: Thu Oct 11 15:54:17 2018 -0500 -- docs/STORM-UI-REST-API.md | 9 -- pom.xml | 107 --- storm-core/pom.xml | 97 ++--- .../src/clj/org/apache/storm/daemon_config.clj | 30 -- .../clj/org/apache/storm/internal/thrift.clj| 67 .../apache/storm/utils/ClojureTimerTask.java| 30 -- .../org/apache/storm/testing4j_test.clj | 1 - .../test/clj/org/apache/storm/cluster_test.clj | 3 +- .../test/clj/org/apache/storm/daemon_config.clj | 30 ++ 9 files changed, 37 insertions(+), 337 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/1086857a/docs/STORM-UI-REST-API.md -- diff --git a/docs/STORM-UI-REST-API.md b/docs/STORM-UI-REST-API.md index 2e2dd21..469855b 100644 --- a/docs/STORM-UI-REST-API.md +++ b/docs/STORM-UI-REST-API.md @@ -1572,15 +1572,6 @@ Sample Response: The API returns 500 HTTP status codes in case of any errors. -Sample response: - -```json -{ - "error": "Internal Server Error", - "errorMessage": "java.lang.NullPointerException\n\tat clojure.core$name.invoke(core.clj:1505)\n\tat org.apache.storm.ui.core$component_page.invoke(core.clj:752)\n\tat org.apache.storm.ui.core$fn__7766.invoke(core.clj:782)\n\tat compojure.core$make_route$fn__5755.invoke(core.clj:93)\n\tat compojure.core$if_route$fn__5743.invoke(core.clj:39)\n\tat compojure.core$if_method$fn__5736.invoke(core.clj:24)\n\tat compojure.core$routing$fn__5761.invoke(core.clj:106)\n\tat clojure.core$some.invoke(core.clj:2443)\n\tat compojure.core$routing.doInvoke(core.clj:106)\n\tat clojure.lang.RestFn.applyTo(RestFn.java:139)\n\tat clojure.core$apply.invoke(core.clj:619)\n\tat compojure.core$routes$fn__5765.invoke(core.clj:111)\n\tat ring.middleware.reload$wrap_reload$fn__6880.invoke(reload.clj:14)\n\tat org.apache.storm.ui.core$catch_errors$fn__7800.invoke(core.clj:836)\n\tat ring.middleware.keyword_params$wrap_keyword_params$fn__6319.invoke(keyword_params.clj:27)\n\tat ring.middleware.nested_params$wra p_nested_params$fn__6358.invoke(nested_params.clj:65)\n\tat ring.middleware.params$wrap_params$fn__6291.invoke(params.clj:55)\n\tat ring.middleware.multipart_params$wrap_multipart_params$fn__6386.invoke(multipart_params.clj:103)\n\tat ring.middleware.flash$wrap_flash$fn__6675.invoke(flash.clj:14)\n\tat ring.middleware.session$wrap_session$fn__6664.invoke(session.clj:43)\n\tat ring.middleware.cookies$wrap_cookies$fn__6595.invoke(cookies.clj:160)\n\tat ring.adapter.jetty$proxy_handler$fn__6112.invoke(jetty.clj:16)\n\tat ring.adapter.jetty.proxy$org.mortbay.jetty.handler.AbstractHandler$0.handle(Unknown Source)\n\tat org.mortbay.jetty.handler.HandlerWrapper.handle(HandlerWrapper.java:152)\n\tat org.mortbay.jetty.Server.handle(Server.java:326)\n\tat org.mortbay.jetty.HttpConnection.handleRequest(HttpConnection.java:542)\n\tat org.mortbay.jetty.HttpConnection$RequestHandler.headerComplete(HttpConnection.java:928)\n\tat org.mortbay.jetty.HttpParser.parseNext(HttpParser.java:549)\n\tat org .mortbay.jetty.HttpParser.parseAvailable(HttpParser.java:212)\n\tat org.mortbay.jetty.HttpConnection.handle(HttpConnection.java:404)\n\tat org.mortbay.jetty.bio.SocketConnector$Connection.run(SocketConnector.java:228)\n\tat org.mortbay.thread.QueuedThreadPool$PoolThread.run(QueuedThreadPool.java:582)\n" -} -``` - # DRPC REST API If DRPC is configured with either an http or https port it will expose a REST endpoint. (See [Setting up a Storm cluster](Setting-up-a-Storm-cluster.html) for how to do that) http://git-wip-us.apache.org/repos/asf/storm/blob/1086857a/pom.xml -- diff --git a/pom.xml b/pom.xml index 5817cd9..22fc32c 100644 --- a/pom.xml +++ b/pom.xml @@ -248,8 +248,6 @@ 1.7.0 -1.1.9 -0.3.6 1.16.1 2.6 2.6 @@ -258,19 +256,14 @@ 1.3.3 1.11 1.4 -0.8.0 4.0.1 2.12.0 1.1 -1.6.2 -0.4.0 9.4.7.v20170914 0.2.3 - 0.0.1 1.5.0 1.11
[2/2] storm git commit: Merge branch 'storm-3243-clojure-deps-removal' of https://github.com/d2r/storm into STORM-3243
Merge branch 'storm-3243-clojure-deps-removal' of https://github.com/d2r/storm into STORM-3243 STORM-3243: Removes and cleans up Clojure dependencies This closes #2873 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/5ea90ce0 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/5ea90ce0 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/5ea90ce0 Branch: refs/heads/master Commit: 5ea90ce0f26653770125358be0d036e411a1248d Parents: 5a8378a 1086857 Author: Robert Evans Authored: Mon Oct 15 07:30:00 2018 -0500 Committer: Robert Evans Committed: Mon Oct 15 07:30:00 2018 -0500 -- docs/STORM-UI-REST-API.md | 9 -- pom.xml | 107 --- storm-core/pom.xml | 97 ++--- .../src/clj/org/apache/storm/daemon_config.clj | 30 -- .../clj/org/apache/storm/internal/thrift.clj| 67 .../apache/storm/utils/ClojureTimerTask.java| 30 -- .../org/apache/storm/testing4j_test.clj | 1 - .../test/clj/org/apache/storm/cluster_test.clj | 3 +- .../test/clj/org/apache/storm/daemon_config.clj | 30 ++ 9 files changed, 37 insertions(+), 337 deletions(-) --
[2/2] storm git commit: Merge branch 'agresch_log_filter_fail' of https://github.com/agresch/storm into STORM-3251
Merge branch 'agresch_log_filter_fail' of https://github.com/agresch/storm into STORM-3251 STORM-3251: properly determine if logviewer filter is enabled This closes #2872 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/5a8378a3 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/5a8378a3 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/5a8378a3 Branch: refs/heads/master Commit: 5a8378a340467654f50d4ee827cdb5196e4e250d Parents: 63aabcb 1fa7699 Author: Robert Evans Authored: Fri Oct 12 16:19:16 2018 -0500 Committer: Robert Evans Committed: Fri Oct 12 16:19:16 2018 -0500 -- .../daemon/logviewer/utils/ResourceAuthorizer.java | 11 +-- .../logviewer/utils/ResourceAuthorizerTest.java | 16 2 files changed, 21 insertions(+), 6 deletions(-) --
[1/2] storm git commit: STORM-3251 properly determine if logviewer filter is enabled
Repository: storm Updated Branches: refs/heads/master 63aabcb9d -> 5a8378a34 STORM-3251 properly determine if logviewer filter is enabled Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/1fa76994 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/1fa76994 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/1fa76994 Branch: refs/heads/master Commit: 1fa76994608198dd50d0abcd0793ec4cf42e7bbf Parents: 2244074 Author: Aaron Gresch Authored: Thu Oct 11 13:20:59 2018 -0500 Committer: Aaron Gresch Committed: Thu Oct 11 13:20:59 2018 -0500 -- .../daemon/logviewer/utils/ResourceAuthorizer.java | 11 +-- .../logviewer/utils/ResourceAuthorizerTest.java | 16 2 files changed, 21 insertions(+), 6 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/1fa76994/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/ResourceAuthorizer.java -- diff --git a/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/ResourceAuthorizer.java b/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/ResourceAuthorizer.java index 3ee85bc..0854350 100644 --- a/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/ResourceAuthorizer.java +++ b/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/ResourceAuthorizer.java @@ -18,8 +18,6 @@ package org.apache.storm.daemon.logviewer.utils; -import static org.apache.storm.DaemonConfig.UI_FILTER; - import com.google.common.annotations.VisibleForTesting; import com.google.common.collect.Sets; @@ -59,13 +57,13 @@ public class ResourceAuthorizer { } /** - * Checks whether user is allowed to access file via UI. Always true when UI filter is not set. + * Checks whether user is allowed to access a Logviewer file via UI. Always true when the Logviewer filter is not configured. * * @param fileName file name to access * @param user username */ public boolean isUserAllowedToAccessFile(String user, String fileName) { -return isUiFilterNotSet() || isAuthorizedLogUser(user, fileName); +return !isLogviewerFilterConfigured() || isAuthorizedLogUser(user, fileName); } /** @@ -133,8 +131,9 @@ public class ResourceAuthorizer { } } -private boolean isUiFilterNotSet() { -return StringUtils.isBlank(ObjectReader.getString(stormConf.get(UI_FILTER), null)); +private boolean isLogviewerFilterConfigured() { +return StringUtils.isNotBlank(ObjectReader.getString(stormConf.get(DaemonConfig.LOGVIEWER_FILTER), null)) +|| StringUtils.isNotBlank(ObjectReader.getString(stormConf.get(DaemonConfig.UI_FILTER), null)); } public static class LogUserGroupWhitelist { http://git-wip-us.apache.org/repos/asf/storm/blob/1fa76994/storm-webapp/src/test/java/org/apache/storm/daemon/logviewer/utils/ResourceAuthorizerTest.java -- diff --git a/storm-webapp/src/test/java/org/apache/storm/daemon/logviewer/utils/ResourceAuthorizerTest.java b/storm-webapp/src/test/java/org/apache/storm/daemon/logviewer/utils/ResourceAuthorizerTest.java index 63363fa..e6aada6 100644 --- a/storm-webapp/src/test/java/org/apache/storm/daemon/logviewer/utils/ResourceAuthorizerTest.java +++ b/storm-webapp/src/test/java/org/apache/storm/daemon/logviewer/utils/ResourceAuthorizerTest.java @@ -34,9 +34,11 @@ import java.util.Collections; import java.util.HashMap; import java.util.Map; +import org.apache.storm.DaemonConfig; import org.apache.storm.daemon.logviewer.testsupport.ArgumentsVerifier; import org.apache.storm.utils.Utils; import org.junit.Test; +import org.mockito.Mockito; public class ResourceAuthorizerTest { @@ -179,4 +181,18 @@ public class ResourceAuthorizerTest { captor -> verify(authorizer).getUserGroups(captor.capture()), String.class, "alice"); } + +@Test +public void authorizationFailsWhenFilterConfigured() { +Map stormConf = Utils.readStormConfig(); +Map conf = new HashMap<>(stormConf); +ResourceAuthorizer authorizer = spy(new ResourceAuthorizer(conf)); +Mockito.when(authorizer.isAuthorizedLogUser(anyString(), anyString())).thenReturn(false); +boolean authorized = authorizer.isUserAllowedToAccessFile("bob", "anyfile"); +assertTrue(authorized); // no filter configured, allow anyone + +conf.put(DaemonConfig.LOGVIEWER_FILTER, "someFilter"); +authorized = authorizer.isUserAllowedToAccessFile("bob", "anyfile"); +assertFalse(authorized); // filter configured, should fail all users +} }
[3/3] storm git commit: Merge branch 'STORM-3252' of https://github.com/arunmahadevan/storm into STORM-3252
Merge branch 'STORM-3252' of https://github.com/arunmahadevan/storm into STORM-3252 STORM-3252: Bug fix for blobstore sync This closes #2871 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/63aabcb9 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/63aabcb9 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/63aabcb9 Branch: refs/heads/master Commit: 63aabcb9d6fdca2e48b16464863acea48676631b Parents: 2244074 a6fc2f9 Author: Robert Evans Authored: Fri Oct 12 15:56:17 2018 -0500 Committer: Robert Evans Committed: Fri Oct 12 15:56:17 2018 -0500 -- .../main/java/org/apache/storm/blobstore/BlobStoreUtils.java| 3 +++ .../main/java/org/apache/storm/blobstore/LocalFsBlobStore.java | 5 +++-- 2 files changed, 6 insertions(+), 2 deletions(-) --
[1/3] storm git commit: Bug fix for blobstore sync.
Repository: storm Updated Branches: refs/heads/master 22440747a -> 63aabcb9d Bug fix for blobstore sync. 1.Bug fix for blob sync frequency with time unit error. 2.Bug fix for blob sync delete file, add catch NoSuchFileException. 3.Bug fix for blob sync update blob flie, add catch FileNotFoundException Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/94e6bbcc Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/94e6bbcc Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/94e6bbcc Branch: refs/heads/master Commit: 94e6bbcc873c04034c55977653910b9921bc014c Parents: 2244074 Author: jiangzhileaf Authored: Thu Jul 19 16:37:22 2018 +0800 Committer: Arun Mahadevan Committed: Thu Oct 11 10:33:21 2018 -0700 -- .../main/java/org/apache/storm/blobstore/BlobStoreUtils.java| 3 +++ .../main/java/org/apache/storm/blobstore/LocalFsBlobStore.java | 5 +++-- 2 files changed, 6 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/94e6bbcc/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java -- diff --git a/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java b/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java index c61c0b5..0ba186f 100644 --- a/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java +++ b/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java @@ -12,6 +12,7 @@ package org.apache.storm.blobstore; +import java.io.FileNotFoundException; import java.io.IOException; import java.util.ArrayList; import java.util.HashSet; @@ -191,6 +192,8 @@ public class BlobStoreUtils { out.close(); } isSuccess = true; +} catch(FileNotFoundException fnf) { +LOG.warn("FileNotFoundException", fnf); } catch (IOException | AuthorizationException exception) { throw new RuntimeException(exception); } catch (KeyNotFoundException knf) { http://git-wip-us.apache.org/repos/asf/storm/blob/94e6bbcc/storm-server/src/main/java/org/apache/storm/blobstore/LocalFsBlobStore.java -- diff --git a/storm-server/src/main/java/org/apache/storm/blobstore/LocalFsBlobStore.java b/storm-server/src/main/java/org/apache/storm/blobstore/LocalFsBlobStore.java index 7724b31..4458758 100644 --- a/storm-server/src/main/java/org/apache/storm/blobstore/LocalFsBlobStore.java +++ b/storm-server/src/main/java/org/apache/storm/blobstore/LocalFsBlobStore.java @@ -19,6 +19,7 @@ import java.io.File; import java.io.FileNotFoundException; import java.io.IOException; import java.io.InputStream; +import java.nio.file.NoSuchFileException; import java.util.HashSet; import java.util.Iterator; import java.util.List; @@ -197,7 +198,7 @@ public class LocalFsBlobStore extends BlobStore { throw new RuntimeException(e); } } -}, 0, ObjectReader.getInt(conf.get(DaemonConfig.NIMBUS_CODE_SYNC_FREQ_SECS))); +}, 0, ObjectReader.getInt(conf.get(DaemonConfig.NIMBUS_CODE_SYNC_FREQ_SECS))*1000); } @@ -374,7 +375,7 @@ public class LocalFsBlobStore extends BlobStore { try { fbs.deleteKey(key); } catch (IOException e) { -if (e instanceof FileNotFoundException) { +if (e instanceof FileNotFoundException || e instanceof NoSuchFileException) { LOG.debug("Ignoring FileNotFoundException since we're about to delete such key... key: {}", key); } else { throw e;
[2/3] storm git commit: STORM-3252: update log message
STORM-3252: update log message Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/a6fc2f93 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/a6fc2f93 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/a6fc2f93 Branch: refs/heads/master Commit: a6fc2f933184c28f293fdd1d6aad562e983f21f8 Parents: 94e6bbc Author: Arun Mahadevan Authored: Thu Oct 11 11:33:36 2018 -0700 Committer: Arun Mahadevan Committed: Thu Oct 11 11:36:35 2018 -0700 -- .../src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/a6fc2f93/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java -- diff --git a/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java b/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java index 0ba186f..b9f93db 100644 --- a/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java +++ b/storm-server/src/main/java/org/apache/storm/blobstore/BlobStoreUtils.java @@ -193,7 +193,7 @@ public class BlobStoreUtils { } isSuccess = true; } catch(FileNotFoundException fnf) { -LOG.warn("FileNotFoundException", fnf); +LOG.warn("Blobstore file for key '{}' does not exist or got deleted before it could be downloaded.", key, fnf); } catch (IOException | AuthorizationException exception) { throw new RuntimeException(exception); } catch (KeyNotFoundException knf) {
[1/2] storm git commit: STORM-3249: Make sure times shut down and so does shut down thread
Repository: storm Updated Branches: refs/heads/master a0f3b1ace -> 22440747a STORM-3249: Make sure times shut down and so does shut down thread Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/bbfd7cd4 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/bbfd7cd4 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/bbfd7cd4 Branch: refs/heads/master Commit: bbfd7cd4050ae80d406daaed9cec882b00be1f98 Parents: 20d5581 Author: Robert (Bobby) Evans Authored: Mon Oct 8 13:25:17 2018 -0500 Committer: Robert (Bobby) Evans Committed: Mon Oct 8 13:25:17 2018 -0500 -- .../org/apache/storm/hdfs/spout/HdfsSpout.java | 2 +- .../org/apache/storm/hive/bolt/HiveBolt.java| 5 ++- .../apache/storm/hive/trident/HiveState.java| 3 +- .../src/jvm/org/apache/storm/utils/Utils.java | 43 .../storm/blobstore/LocalFsBlobStore.java | 5 ++- .../java/org/apache/storm/daemon/drpc/DRPC.java | 6 +-- 6 files changed, 40 insertions(+), 24 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/bbfd7cd4/external/storm-hdfs/src/main/java/org/apache/storm/hdfs/spout/HdfsSpout.java -- diff --git a/external/storm-hdfs/src/main/java/org/apache/storm/hdfs/spout/HdfsSpout.java b/external/storm-hdfs/src/main/java/org/apache/storm/hdfs/spout/HdfsSpout.java index 12debe5..bdf9da3 100644 --- a/external/storm-hdfs/src/main/java/org/apache/storm/hdfs/spout/HdfsSpout.java +++ b/external/storm-hdfs/src/main/java/org/apache/storm/hdfs/spout/HdfsSpout.java @@ -390,7 +390,7 @@ public class HdfsSpout extends BaseRichSpout { public void open(Map conf, TopologyContext context, SpoutOutputCollector collector) { LOG.info("Opening HDFS Spout"); this.conf = conf; -this.commitTimer = new Timer(); +this.commitTimer = new Timer(context.getThisTaskId() + "-commit-timer", true); this.tracker = new ProgressTracker(); this.hdfsConfig = new Configuration(); this.collector = collector; http://git-wip-us.apache.org/repos/asf/storm/blob/bbfd7cd4/external/storm-hive/src/main/java/org/apache/storm/hive/bolt/HiveBolt.java -- diff --git a/external/storm-hive/src/main/java/org/apache/storm/hive/bolt/HiveBolt.java b/external/storm-hive/src/main/java/org/apache/storm/hive/bolt/HiveBolt.java index cfabbd6..180f41b 100644 --- a/external/storm-hive/src/main/java/org/apache/storm/hive/bolt/HiveBolt.java +++ b/external/storm-hive/src/main/java/org/apache/storm/hive/bolt/HiveBolt.java @@ -79,7 +79,7 @@ public class HiveBolt extends BaseRichBolt { new ThreadFactoryBuilder().setNameFormat(timeoutName).build()); sendHeartBeat.set(true); -heartBeatTimer = new Timer(); +heartBeatTimer = new Timer(topologyContext.getThisTaskId() + "-hb-timer", true); setupHeartBeatTimer(); } catch (Exception e) { @@ -151,6 +151,9 @@ public class HiveBolt extends BaseRichBolt { } callTimeoutPool = null; +if (heartBeatTimer != null) { +heartBeatTimer.cancel(); +} super.cleanup(); LOG.info("Hive Bolt stopped"); } http://git-wip-us.apache.org/repos/asf/storm/blob/bbfd7cd4/external/storm-hive/src/main/java/org/apache/storm/hive/trident/HiveState.java -- diff --git a/external/storm-hive/src/main/java/org/apache/storm/hive/trident/HiveState.java b/external/storm-hive/src/main/java/org/apache/storm/hive/trident/HiveState.java index a698e24..6717329 100644 --- a/external/storm-hive/src/main/java/org/apache/storm/hive/trident/HiveState.java +++ b/external/storm-hive/src/main/java/org/apache/storm/hive/trident/HiveState.java @@ -80,7 +80,7 @@ public class HiveState implements State { String timeoutName = "hive-bolt-%d"; this.callTimeoutPool = Executors.newFixedThreadPool(1, new ThreadFactoryBuilder().setNameFormat(timeoutName).build()); -heartBeatTimer = new Timer(); +heartBeatTimer = new Timer("hive-hb-timer", true); setupHeartBeatTimer(); } catch (Exception e) { LOG.warn("unable to make connection to hive ", e); @@ -289,6 +289,7 @@ public class HiveState implements State { LOG.warn("shutdown interrupted on " + execService, ex); } } +heartBeatTimer.cancel(); callTimeoutPool = null;
[2/2] storm git commit: Merge branch 'STORM-3249' of https://github.com/revans2/incubator-storm into STORM-3249
Merge branch 'STORM-3249' of https://github.com/revans2/incubator-storm into STORM-3249 STORM-3249: Make sure times shut down and so does shut down thread This closes #2867 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/22440747 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/22440747 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/22440747 Branch: refs/heads/master Commit: 22440747aa0f2ed3f3d26baec9140f79bb82f644 Parents: a0f3b1a bbfd7cd Author: Robert Evans Authored: Tue Oct 9 14:35:30 2018 -0500 Committer: Robert Evans Committed: Tue Oct 9 14:35:30 2018 -0500 -- .../org/apache/storm/hdfs/spout/HdfsSpout.java | 2 +- .../org/apache/storm/hive/bolt/HiveBolt.java| 5 ++- .../apache/storm/hive/trident/HiveState.java| 3 +- .../src/jvm/org/apache/storm/utils/Utils.java | 43 .../storm/blobstore/LocalFsBlobStore.java | 5 ++- .../java/org/apache/storm/daemon/drpc/DRPC.java | 6 +-- 6 files changed, 40 insertions(+), 24 deletions(-) --
[1/2] storm git commit: STORM-1311: Fixing outputStats
Repository: storm Updated Branches: refs/heads/master 20d558127 -> d0a169978 STORM-1311: Fixing outputStats Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/fe6116a7 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/fe6116a7 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/fe6116a7 Branch: refs/heads/master Commit: fe6116a7119b1ee8ca55dca247b689ca173aeb53 Parents: 20d5581 Author: Govind Menon Authored: Mon Oct 8 09:45:05 2018 -0500 Committer: Govind Menon Committed: Mon Oct 8 09:46:53 2018 -0500 -- .../src/main/java/org/apache/storm/daemon/ui/UIHelpers.java | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/fe6116a7/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java -- diff --git a/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java b/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java index 1298b87..ff121ae 100644 --- a/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java +++ b/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java @@ -1226,7 +1226,7 @@ public class UIHelpers { * @return nullToZero */ private static Long nullToZero(Long value) { -return Objects.isNull(value) ? value : 0; +return !Objects.isNull(value) ? value : 0; } /** @@ -1235,7 +1235,7 @@ public class UIHelpers { * @return nullToZero */ private static Double nullToZero(Double value) { -return Objects.isNull(value) ? value : 0; +return !Objects.isNull(value) ? value : 0; } /**
[2/2] storm git commit: Merge branch 'STORM-1311-OutputStats' of https://github.com/govind-menon/storm into STORM-3248
Merge branch 'STORM-1311-OutputStats' of https://github.com/govind-menon/storm into STORM-3248 STORM-3248: Fixing outputStats This closes #2866 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/d0a16997 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/d0a16997 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/d0a16997 Branch: refs/heads/master Commit: d0a169978942736556fd0af1b8e4306c0291e107 Parents: 20d5581 fe6116a Author: Robert Evans Authored: Tue Oct 9 12:24:17 2018 -0500 Committer: Robert Evans Committed: Tue Oct 9 12:24:17 2018 -0500 -- .../src/main/java/org/apache/storm/daemon/ui/UIHelpers.java | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) --
[2/2] storm git commit: Merge branch 'agresch_blobstore_superuser' of https://github.com/agresch/storm into STORM-3247
Merge branch 'agresch_blobstore_superuser' of https://github.com/agresch/storm into STORM-3247 STORM-3247: remove BLOBSTORE_SUPERUSER This closes #2865 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/20d55812 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/20d55812 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/20d55812 Branch: refs/heads/master Commit: 20d5581270c9643f2764f7f5a4cab0e5ea84a673 Parents: 53d1399 e5e0dde Author: Robert Evans Authored: Mon Oct 8 08:16:34 2018 -0500 Committer: Robert Evans Committed: Mon Oct 8 08:16:34 2018 -0500 -- storm-client/src/jvm/org/apache/storm/Config.java | 6 +- storm-server/src/main/java/org/apache/storm/LocalCluster.java | 1 - 2 files changed, 1 insertion(+), 6 deletions(-) --
[1/2] storm git commit: STORM-3247 remove BLOBSTORE_SUPERUSER
Repository: storm Updated Branches: refs/heads/master 53d13997b -> 20d558127 STORM-3247 remove BLOBSTORE_SUPERUSER Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/e5e0ddec Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/e5e0ddec Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/e5e0ddec Branch: refs/heads/master Commit: e5e0ddec875534af34d8499523530f34ede086c8 Parents: e820523 Author: Aaron Gresch Authored: Fri Oct 5 14:23:44 2018 -0500 Committer: Aaron Gresch Committed: Fri Oct 5 14:23:44 2018 -0500 -- storm-client/src/jvm/org/apache/storm/Config.java | 6 +- storm-server/src/main/java/org/apache/storm/LocalCluster.java | 1 - 2 files changed, 1 insertion(+), 6 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/e5e0ddec/storm-client/src/jvm/org/apache/storm/Config.java -- diff --git a/storm-client/src/jvm/org/apache/storm/Config.java b/storm-client/src/jvm/org/apache/storm/Config.java index 8561eae..a1aea4c 100644 --- a/storm-client/src/jvm/org/apache/storm/Config.java +++ b/storm-client/src/jvm/org/apache/storm/Config.java @@ -1083,11 +1083,7 @@ public class Config extends HashMap { */ @isString public static final String CLIENT_BLOBSTORE = "client.blobstore.class"; -/** - * The blobstore super user has all read/write/admin permissions to all blobs - user running the blobstore. - */ -@isString -public static final String BLOBSTORE_SUPERUSER = "blobstore.superuser"; + /** * What directory to use for the blobstore. The directory is expected to be an absolute path when using HDFS blobstore, for * LocalFsBlobStore it could be either absolute or relative. If the setting is a relative directory, it is relative to root directory of http://git-wip-us.apache.org/repos/asf/storm/blob/e5e0ddec/storm-server/src/main/java/org/apache/storm/LocalCluster.java -- diff --git a/storm-server/src/main/java/org/apache/storm/LocalCluster.java b/storm-server/src/main/java/org/apache/storm/LocalCluster.java index 069bfb8..406bb56 100644 --- a/storm-server/src/main/java/org/apache/storm/LocalCluster.java +++ b/storm-server/src/main/java/org/apache/storm/LocalCluster.java @@ -208,7 +208,6 @@ public class LocalCluster implements ILocalClusterTrackedTopologyAware, Iface { conf.put(Config.TOPOLOGY_ENABLE_MESSAGE_TIMEOUTS, false); conf.put(Config.TOPOLOGY_TRIDENT_BATCH_EMIT_INTERVAL_MILLIS, 50); conf.put(Config.STORM_CLUSTER_MODE, "local"); -conf.put(Config.BLOBSTORE_SUPERUSER, System.getProperty("user.name")); conf.put(Config.BLOBSTORE_DIR, nimbusTmp.getPath()); conf.put(Config.TOPOLOGY_MIN_REPLICATION_COUNT, 1);
[1/2] storm git commit: Use utf-8 charset to write log files
Repository: storm Updated Branches: refs/heads/master 06a64949c -> 53d13997b Use utf-8 charset to write log files Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/c8ce783f Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/c8ce783f Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/c8ce783f Branch: refs/heads/master Commit: c8ce783f2fe5e444a97ea385a33ad300f99f67ff Parents: c367f9a Author: Kishor Patil Authored: Thu Oct 4 16:18:40 2018 -0400 Committer: Kishor Patil Committed: Thu Oct 4 16:18:40 2018 -0400 -- log4j2/cluster.xml | 2 +- log4j2/worker.xml | 2 +- 2 files changed, 2 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/c8ce783f/log4j2/cluster.xml -- diff --git a/log4j2/cluster.xml b/log4j2/cluster.xml index 2a12fba..3186836 100644 --- a/log4j2/cluster.xml +++ b/log4j2/cluster.xml @@ -24,7 +24,7 @@ - + ${pattern} http://git-wip-us.apache.org/repos/asf/storm/blob/c8ce783f/log4j2/worker.xml -- diff --git a/log4j2/worker.xml b/log4j2/worker.xml index cd785ec..6e73b44 100644 --- a/log4j2/worker.xml +++ b/log4j2/worker.xml @@ -26,7 +26,7 @@ - + ${pattern}
[2/2] storm git commit: Merge branch 'fix-logfile-charset' of https://github.com/kishorvpatil/incubator-storm
Merge branch 'fix-logfile-charset' of https://github.com/kishorvpatil/incubator-storm STORM-3246: Use utf-8 charset to write log files This closes #2864 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/53d13997 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/53d13997 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/53d13997 Branch: refs/heads/master Commit: 53d13997b3a8d0bc3a52e0ef45418e4308de8bb3 Parents: 06a6494 c8ce783 Author: Robert Evans Authored: Fri Oct 5 16:08:18 2018 -0500 Committer: Robert Evans Committed: Fri Oct 5 16:08:18 2018 -0500 -- log4j2/cluster.xml | 2 +- log4j2/worker.xml | 2 +- 2 files changed, 2 insertions(+), 2 deletions(-) --
[1/3] storm git commit: STORM-3244 allow logviewer to use independent filter settings from ui
Repository: storm Updated Branches: refs/heads/master bfcbf2c8e -> 06a64949c STORM-3244 allow logviewer to use independent filter settings from ui Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/9cad3415 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/9cad3415 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/9cad3415 Branch: refs/heads/master Commit: 9cad3415e597bd25534d64271109653676d9be46 Parents: a6f76df Author: Aaron Gresch Authored: Wed Oct 3 15:25:02 2018 -0500 Committer: Aaron Gresch Committed: Wed Oct 3 15:25:02 2018 -0500 -- SECURITY.md | 4 +++- conf/defaults.yaml| 2 ++ docs/SECURITY.md | 4 +++- .../src/main/java/org/apache/storm/DaemonConfig.java | 14 +- .../storm/daemon/logviewer/LogviewerServer.java | 7 ++- 5 files changed, 23 insertions(+), 8 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/9cad3415/SECURITY.md -- diff --git a/SECURITY.md b/SECURITY.md index 5aa3bd0..668ea8a 100644 --- a/SECURITY.md +++ b/SECURITY.md @@ -50,6 +50,8 @@ Some form of Authentication is typically required; e.g., by using java servlet f ```yaml ui.filter: "filter.class" ui.filter.params: "param1":"value1" +logviewer.filter: "filter.class" +logviewer.filter.params: "param1":"value1" ``` or by restricting the UI/log-viewers ports to only accept connections from localhost, and then front them with another web server, like Apache httpd, that can @@ -62,7 +64,7 @@ The servlet filters are preferred because they allow individual topologies to specify who is (and who is not) allowed to access the pages associated with each topology. -The Storm UI can be configured to use `AuthenticationFilter` from hadoop-auth. +The Storm UI (or logviewer) can be configured to use `AuthenticationFilter` from hadoop-auth. ```yaml ui.filter: "org.apache.hadoop.security.authentication.server.AuthenticationFilter" ui.filter.params: http://git-wip-us.apache.org/repos/asf/storm/blob/9cad3415/conf/defaults.yaml -- diff --git a/conf/defaults.yaml b/conf/defaults.yaml index 7f6a921..d9ad56c 100644 --- a/conf/defaults.yaml +++ b/conf/defaults.yaml @@ -106,6 +106,8 @@ logviewer.appender.name: "A1" logviewer.max.sum.worker.logs.size.mb: 4096 logviewer.max.per.worker.logs.size.mb: 2048 logviewer.disable.http.binding: true +logviewer.filter: null +logviewer.filter.params: null logs.users: null http://git-wip-us.apache.org/repos/asf/storm/blob/9cad3415/docs/SECURITY.md -- diff --git a/docs/SECURITY.md b/docs/SECURITY.md index 96aef73..38375a8 100644 --- a/docs/SECURITY.md +++ b/docs/SECURITY.md @@ -55,6 +55,8 @@ Some form of Authentication is typically required, with using java servlet filte ```yaml ui.filter: "filter.class" ui.filter.params: "param1":"value1" +logviewer.filter: "filter.class" +logviewer.filter.params: "param1":"value1" ``` or by restricting the UI/log viewers ports to only accept connections from local hosts, and then front them with another web server, like Apache httpd, that can @@ -67,7 +69,7 @@ The servlet filters are preferred because it allows individual topologies to specificy who is and who is not allowed to access the pages associated with them. -Storm UI can be configured to use AuthenticationFilter from hadoop-auth. +Storm UI (or logviewer) can be configured to use AuthenticationFilter from hadoop-auth. ```yaml ui.filter: "org.apache.hadoop.security.authentication.server.AuthenticationFilter" ui.filter.params: http://git-wip-us.apache.org/repos/asf/storm/blob/9cad3415/storm-server/src/main/java/org/apache/storm/DaemonConfig.java -- diff --git a/storm-server/src/main/java/org/apache/storm/DaemonConfig.java b/storm-server/src/main/java/org/apache/storm/DaemonConfig.java index 64c934a..6c12fbb 100644 --- a/storm-server/src/main/java/org/apache/storm/DaemonConfig.java +++ b/storm-server/src/main/java/org/apache/storm/DaemonConfig.java @@ -461,6 +461,18 @@ public class DaemonConfig implements Validated { public static final String LOGVIEWER_APPENDER_NAME = "logviewer.appender.name"; /** + * A class implementing javax.servlet.Filter for authenticating/filtering Logviewer requests. + */ +@isString +public static final String LOGVIEWER_FILTER = "logviewer.filter"; + +/** + * Initialization parameters for the javax.servlet.Filter for Logviewer. + */ +@isMapEntryType(keyType = String.
[3/3] storm git commit: Merge branch 'agresch_logviewer_filter' of https://github.com/agresch/storm into STORM-3244
Merge branch 'agresch_logviewer_filter' of https://github.com/agresch/storm into STORM-3244 STORM-3244: allow logviewer to use independent filter settings from ui This closes #2861 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/06a64949 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/06a64949 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/06a64949 Branch: refs/heads/master Commit: 06a64949c8c5b764a33a10beb6088cdd8f182aa0 Parents: bfcbf2c e63dee6 Author: Robert Evans Authored: Fri Oct 5 15:43:29 2018 -0500 Committer: Robert Evans Committed: Fri Oct 5 15:43:29 2018 -0500 -- SECURITY.md | 4 +++- conf/defaults.yaml | 2 ++ docs/SECURITY.md | 4 +++- .../src/main/java/org/apache/storm/DaemonConfig.java | 14 +- .../storm/daemon/logviewer/LogviewerServer.java | 15 --- 5 files changed, 29 insertions(+), 10 deletions(-) --
[2/3] storm git commit: STORM-3244 default to ui filter if logviewer filter is not set
STORM-3244 default to ui filter if logviewer filter is not set Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/e63dee61 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/e63dee61 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/e63dee61 Branch: refs/heads/master Commit: e63dee617bdbdb8294978ab8442b2dd1e007b2b9 Parents: 9cad341 Author: Aaron Gresch Authored: Fri Oct 5 13:52:53 2018 -0500 Committer: Aaron Gresch Committed: Fri Oct 5 13:52:53 2018 -0500 -- .../apache/storm/daemon/logviewer/LogviewerServer.java| 10 +++--- 1 file changed, 7 insertions(+), 3 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/e63dee61/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/LogviewerServer.java -- diff --git a/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/LogviewerServer.java b/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/LogviewerServer.java index b772aa5..daa53bc 100644 --- a/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/LogviewerServer.java +++ b/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/LogviewerServer.java @@ -18,8 +18,6 @@ package org.apache.storm.daemon.logviewer; -import static org.apache.storm.DaemonConfig.UI_HEADER_BUFFER_BYTES; - import com.codahale.metrics.Meter; import com.google.common.annotations.VisibleForTesting; @@ -29,6 +27,7 @@ import java.util.Arrays; import java.util.List; import java.util.Map; +import org.apache.commons.lang.StringUtils; import org.apache.storm.DaemonConfig; import org.apache.storm.daemon.logviewer.utils.DirectoryCleaner; import org.apache.storm.daemon.logviewer.utils.ExceptionMeterNames; @@ -64,9 +63,14 @@ public class LogviewerServer implements AutoCloseable { Server ret = null; if (logviewerHttpPort != null && logviewerHttpPort >= 0) { LOG.info("Starting Logviewer HTTP servers..."); +String filterParamKey = DaemonConfig.LOGVIEWER_FILTER_PARAMS; String filterClass = (String) (conf.get(DaemonConfig.LOGVIEWER_FILTER)); +if (StringUtils.isBlank(filterClass)) { +filterClass = (String) (conf.get(DaemonConfig.UI_FILTER)); +filterParamKey = DaemonConfig.UI_FILTER_PARAMS; +} @SuppressWarnings("unchecked") -Map filterParams = (Map) (conf.get(DaemonConfig.LOGVIEWER_FILTER_PARAMS)); +Map filterParams = (Map) (conf.get(filterParamKey)); FilterConfiguration filterConfiguration = new FilterConfiguration(filterClass, filterParams); final List filterConfigurations = Arrays.asList(filterConfiguration);
[1/2] storm git commit: STORM-3245: Don't blow up if empty log dirs exist
Repository: storm Updated Branches: refs/heads/master e82052339 -> bfcbf2c8e STORM-3245: Don't blow up if empty log dirs exist Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/d012f6c3 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/d012f6c3 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/d012f6c3 Branch: refs/heads/master Commit: d012f6c323d29a48ded29ff2ef86d77970a5d080 Parents: dad7410 Author: Robert (Bobby) Evans Authored: Thu Oct 4 15:12:18 2018 -0500 Committer: Robert (Bobby) Evans Committed: Thu Oct 4 15:12:18 2018 -0500 -- .../logviewer/utils/DirectoryCleaner.java | 6 +-- .../daemon/logviewer/utils/LogCleaner.java | 9 +--- .../daemon/logviewer/utils/WorkerLogs.java | 47 .../daemon/logviewer/utils/LogCleanerTest.java | 24 ++ .../daemon/logviewer/utils/WorkerLogsTest.java | 7 ++- 5 files changed, 53 insertions(+), 40 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/d012f6c3/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/DirectoryCleaner.java -- diff --git a/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/DirectoryCleaner.java b/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/DirectoryCleaner.java index 0b17c84..2509fc8 100644 --- a/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/DirectoryCleaner.java +++ b/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/DirectoryCleaner.java @@ -86,7 +86,7 @@ public class DirectoryCleaner { * @return number of files deleted */ public DeletionMeta deleteOldestWhileTooLarge(List dirs, - long quota, boolean forPerDir, Set activeDirs) throws IOException { + long quota, boolean forPerDir, Set activeDirs) throws IOException { long totalSize = 0; for (File dir : dirs) { try (DirectoryStream stream = getStreamForDirectory(dir)) { @@ -171,12 +171,12 @@ public class DirectoryCleaner { return new DeletionMeta(deletedSize, deletedFiles); } -private boolean isFileEligibleToSkipDelete(boolean forPerDir, Set activeDirs, File dir, File file) throws IOException { +private boolean isFileEligibleToSkipDelete(boolean forPerDir, Set activeDirs, File dir, File file) throws IOException { if (forPerDir) { return ACTIVE_LOG_PATTERN.matcher(file.getName()).matches(); } else { // for global cleanup // for an active worker's dir, make sure for the last "/" -return activeDirs.contains(dir.getCanonicalPath()) ? ACTIVE_LOG_PATTERN.matcher(file.getName()).matches() : +return activeDirs.contains(dir) ? ACTIVE_LOG_PATTERN.matcher(file.getName()).matches() : META_LOG_PATTERN.matcher(file.getName()).matches(); } } http://git-wip-us.apache.org/repos/asf/storm/blob/d012f6c3/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/LogCleaner.java -- diff --git a/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/LogCleaner.java b/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/LogCleaner.java index 035fe3b..036d224 100644 --- a/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/LogCleaner.java +++ b/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/utils/LogCleaner.java @@ -210,7 +210,7 @@ public class LogCleaner implements Runnable, Closeable { @VisibleForTesting DeletionMeta globalLogCleanup(long size) throws Exception { List workerDirs = new ArrayList<>(workerLogs.getAllWorkerDirs()); -Set aliveWorkerDirs = new HashSet<>(workerLogs.getAliveWorkerDirs()); +Set aliveWorkerDirs = workerLogs.getAliveWorkerDirs(); return directoryCleaner.deleteOldestWhileTooLarge(workerDirs, size, false, aliveWorkerDirs); } @@ -235,12 +235,7 @@ public class LogCleaner implements Runnable, Closeable { return new TreeSet<>(); } else { Set aliveIds = workerLogs.getAliveIds(nowSecs); -Map idToDir = workerLogs.identifyWorkerLogDirs(logDirs); - -return idToDir.entrySet().stream() -.filter(entry -> !aliveIds.contains(entry.getKey())) -.map(Map.Entry::getValue) -.collect(toCollection(TreeSet::new)); +return workerLogs.getLogDirs(logDirs, (wid) -> !aliveIds.contains(wid)); } } http://git-wip-us.apache.org
[2/2] storm git commit: Merge branch 'STORM-3245' of https://github.com/revans2/incubator-storm into STORM-3245
Merge branch 'STORM-3245' of https://github.com/revans2/incubator-storm into STORM-3245 STORM-3245: Don't blow up if empty log dirs exist This closes #2863 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/bfcbf2c8 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/bfcbf2c8 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/bfcbf2c8 Branch: refs/heads/master Commit: bfcbf2c8e35518de02f6326cd10aad6c7cf7ad92 Parents: e820523 d012f6c Author: Robert Evans Authored: Fri Oct 5 15:18:10 2018 -0500 Committer: Robert Evans Committed: Fri Oct 5 15:18:10 2018 -0500 -- .../logviewer/utils/DirectoryCleaner.java | 6 +-- .../daemon/logviewer/utils/LogCleaner.java | 9 +--- .../daemon/logviewer/utils/WorkerLogs.java | 47 .../daemon/logviewer/utils/LogCleanerTest.java | 24 ++ .../daemon/logviewer/utils/WorkerLogsTest.java | 7 ++- 5 files changed, 53 insertions(+), 40 deletions(-) --
[1/2] storm git commit: Fixes & ports shell submission to Java
Repository: storm Updated Branches: refs/heads/master b6b2b7d14 -> e82052339 Fixes & ports shell submission to Java Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/d1b6f763 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/d1b6f763 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/d1b6f763 Branch: refs/heads/master Commit: d1b6f76381ec8d8aca4d28fdec5e82711ed06176 Parents: ad63cc2 Author: Derek Dagit Authored: Wed Oct 3 11:06:31 2018 -0500 Committer: Derek Dagit Committed: Wed Oct 3 11:06:31 2018 -0500 -- bin/storm.py| 2 +- .../apache/storm/command/shell_submission.clj | 33 -- .../apache/storm/command/ShellSubmission.java | 46 3 files changed, 47 insertions(+), 34 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/d1b6f763/bin/storm.py -- diff --git a/bin/storm.py b/bin/storm.py index 1183f9d..2f2de6e 100755 --- a/bin/storm.py +++ b/bin/storm.py @@ -687,7 +687,7 @@ def shell(resourcesdir, command, *args): runnerargs = [tmpjarpath, command] runnerargs.extend(args) exec_storm_class( -"org.apache.storm.command.shell_submission", +"org.apache.storm.command.ShellSubmission", args=runnerargs, jvmtype="-client", extrajars=[USER_CONF_DIR], http://git-wip-us.apache.org/repos/asf/storm/blob/d1b6f763/storm-core/src/clj/org/apache/storm/command/shell_submission.clj -- diff --git a/storm-core/src/clj/org/apache/storm/command/shell_submission.clj b/storm-core/src/clj/org/apache/storm/command/shell_submission.clj deleted file mode 100644 index 6bd5570..000 --- a/storm-core/src/clj/org/apache/storm/command/shell_submission.clj +++ /dev/null @@ -1,33 +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. -(ns org.apache.storm.command.shell-submission - (:import [org.apache.storm Config StormSubmitter] - [org.apache.storm.utils ServerUtils]) - (:use [org.apache.storm util config log]) - (:import [org.apache.storm.utils ConfigUtils NimbusClient]) - (:gen-class)) - - -(defn -main [^String tmpjarpath & args] - (let [conf (clojurify-structure (ConfigUtils/readStormConfig))] -(with-open [client (NimbusClient/getConfiguredClient conf)] - (let [c (.getClient client) -ns (.getLeader c) -host (.get_host ns) -port (.get_port ns) -jarpath (StormSubmitter/submitJar conf tmpjarpath) -args (concat args [host port jarpath])] -(ServerUtils/execCommand args) http://git-wip-us.apache.org/repos/asf/storm/blob/d1b6f763/storm-core/src/jvm/org/apache/storm/command/ShellSubmission.java -- diff --git a/storm-core/src/jvm/org/apache/storm/command/ShellSubmission.java b/storm-core/src/jvm/org/apache/storm/command/ShellSubmission.java new file mode 100644 index 000..895a266 --- /dev/null +++ b/storm-core/src/jvm/org/apache/storm/command/ShellSubmission.java @@ -0,0 +1,46 @@ +/** + * 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.storm.command; + +import org.apache.common
[2/2] storm git commit: Merge branch 'port-shellsubmission-to-java' of https://github.com/d2r/storm into STORM-1318
Merge branch 'port-shellsubmission-to-java' of https://github.com/d2r/storm into STORM-1318 STORM-1318: Fixes & ports shell submission to Java This closes #2860 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/e8205233 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/e8205233 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/e8205233 Branch: refs/heads/master Commit: e82052339bd0567d132017ace4e303be3056c82f Parents: b6b2b7d d1b6f76 Author: Robert Evans Authored: Fri Oct 5 09:30:14 2018 -0500 Committer: Robert Evans Committed: Fri Oct 5 09:30:14 2018 -0500 -- bin/storm.py| 2 +- .../apache/storm/command/shell_submission.clj | 33 -- .../apache/storm/command/ShellSubmission.java | 46 3 files changed, 47 insertions(+), 34 deletions(-) --
[1/2] storm git commit: Replace docs reference to Flume with HiveBolt
Repository: storm Updated Branches: refs/heads/master 7b3cf31d0 -> b6b2b7d14 Replace docs reference to Flume with HiveBolt Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/a930e078 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/a930e078 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/a930e078 Branch: refs/heads/master Commit: a930e078db453bc933e775cb16cd9733a1e49e73 Parents: ad63cc2 Author: Edward Samson Authored: Wed Oct 3 14:18:41 2018 +0800 Committer: Edward Samson Committed: Wed Oct 3 14:18:41 2018 +0800 -- docs/storm-hive.md| 2 +- external/storm-hive/README.md | 2 +- 2 files changed, 2 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/a930e078/docs/storm-hive.md -- diff --git a/docs/storm-hive.md b/docs/storm-hive.md index e2dd657..9da4a8e 100644 --- a/docs/storm-hive.md +++ b/docs/storm-hive.md @@ -75,7 +75,7 @@ HiveOptions params |dbName | database name | String (required) | |tblName | table name | String (required) | |mapper| Mapper class to map Tuple field names to Table column names | DelimitedRecordHiveMapper or JsonRecordHiveMapper (required) | -|withTxnsPerBatch | Hive grants a *batch of transactions* instead of single transactions to streaming clients like HiveBolt.This setting configures the number of desired transactions per Transaction Batch. Data from all transactions in a single batch end up in a single file. Flume will write a maximum of batchSize events in each transaction in the batch. This setting in conjunction with batchSize provides control over the size of each file. Note that eventually Hive will transparently compact these files into larger files.| Integer . default 100 | +|withTxnsPerBatch | Hive grants a *batch of transactions* instead of single transactions to streaming clients like HiveBolt.This setting configures the number of desired transactions per Transaction Batch. Data from all transactions in a single batch end up in a single file. HiveBolt will write a maximum of batchSize events in each transaction in the batch. This setting in conjunction with batchSize provides control over the size of each file. Note that eventually Hive will transparently compact these files into larger files.| Integer . default 100 | |withMaxOpenConnections| Allow only this number of open connections. If this number is exceeded, the least recently used connection is closed.| Integer . default 100| |withBatchSize| Max number of events written to Hive in a single Hive transaction| Integer. default 15000| |withCallTimeout| (In milliseconds) Timeout for Hive & HDFS I/O operations, such as openTxn, write, commit, abort. | Integer. default 1| http://git-wip-us.apache.org/repos/asf/storm/blob/a930e078/external/storm-hive/README.md -- diff --git a/external/storm-hive/README.md b/external/storm-hive/README.md index 6e9970c..5e5df1f 100644 --- a/external/storm-hive/README.md +++ b/external/storm-hive/README.md @@ -71,7 +71,7 @@ HiveOptions params |dbName | database name | String (required) | |tblName | table name | String (required) | |mapper| Mapper class to map Tuple field names to Table column names | DelimitedRecordHiveMapper or JsonRecordHiveMapper (required) | -|withTxnsPerBatch | Hive grants a *batch of transactions* instead of single transactions to streaming clients like HiveBolt.This setting configures the number of desired transactions per Transaction Batch. Data from all transactions in a single batch end up in a single file. Flume will write a maximum of batchSize events in each transaction in the batch. This setting in conjunction with batchSize provides control over the size of each file. Note that eventually Hive will transparently compact these files into larger files.| Integer . default 100 | +|withTxnsPerBatch | Hive grants a *batch of transactions* instead of single transactions to streaming clients like HiveBolt.This setting configures the number of desired transactions per Transaction Batch. Data from all transactions in a single batch end up in a single file. HiveBolt will write a maximum of batchSize events in each transaction in the batch. This setting in conjunction with batchSize provides control over the size of each file. Note that eventually Hive will transparently compact these files into larger files.| Integer . default 100 | |withMaxOpenConnections| Allow only this number of open connections. If this number is exceeded, the least recently used connection is closed.| Integer . default 100| |withBatchSize| Max number of events written to Hive in a single Hive transaction| Integer. default 15000| |withCallTim
[2/2] storm git commit: Merge branch 'hive-doc-fix' of https://github.com/esamson/storm
Merge branch 'hive-doc-fix' of https://github.com/esamson/storm Replace docs reference to Flume with HiveBolt This closes #2859 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/b6b2b7d1 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/b6b2b7d1 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/b6b2b7d1 Branch: refs/heads/master Commit: b6b2b7d143642b5cbcbfce86b9da8e77292368a9 Parents: 7b3cf31 a930e07 Author: Robert Evans Authored: Fri Oct 5 09:25:54 2018 -0500 Committer: Robert Evans Committed: Fri Oct 5 09:25:54 2018 -0500 -- docs/storm-hive.md| 2 +- external/storm-hive/README.md | 2 +- 2 files changed, 2 insertions(+), 2 deletions(-) --
[2/2] storm git commit: Merge branch 'STORM-3239' of https://github.com/govind-menon/storm into STORM-3239
Merge branch 'STORM-3239' of https://github.com/govind-menon/storm into STORM-3239 STORM-3239: Adding dumpjstack action and removing wrong timeout param This closes #2854 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/2bc519f0 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/2bc519f0 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/2bc519f0 Branch: refs/heads/master Commit: 2bc519f083832bd1b5a458c6b6529c6c16fd69ce Parents: a6f76df 954c810 Author: Robert Evans Authored: Wed Oct 3 16:27:29 2018 -0500 Committer: Robert Evans Committed: Wed Oct 3 16:27:29 2018 -0500 -- .../org/apache/storm/daemon/ui/UIHelpers.java | 48 +++--- .../daemon/ui/resources/StormApiResource.java | 52 +--- 2 files changed, 64 insertions(+), 36 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/2bc519f0/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java --
[1/2] storm git commit: STORM-3239: Adding dumpjstack action and removing wrong timeout parameter
Repository: storm Updated Branches: refs/heads/master a6f76df3f -> 2bc519f08 STORM-3239: Adding dumpjstack action and removing wrong timeout parameter Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/954c810e Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/954c810e Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/954c810e Branch: refs/heads/master Commit: 954c810e150971e94219ec192f16fdd7d772d8df Parents: c367f9a Author: Govind Menon Authored: Sun Sep 30 19:30:23 2018 -0500 Committer: Govind Menon Committed: Sun Sep 30 19:30:23 2018 -0500 -- .../org/apache/storm/daemon/ui/UIHelpers.java | 48 +++--- .../daemon/ui/resources/StormApiResource.java | 52 +--- 2 files changed, 64 insertions(+), 36 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/954c810e/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java -- diff --git a/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java b/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java index 8237009..8192311 100644 --- a/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java +++ b/storm-webapp/src/main/java/org/apache/storm/daemon/ui/UIHelpers.java @@ -1846,11 +1846,11 @@ public class UIHelpers { public static Map getActiveAction(ProfileRequest profileRequest, Map config, String topologyId) { Map result = new HashMap(); result.put("host", profileRequest.get_nodeInfo().get_node()); -result.put("port", String.valueOf(profileRequest.get_nodeInfo().get_port().iterator().next())); +result.put("port", String.valueOf(profileRequest.get_nodeInfo().get_port().toArray()[0])); result.put("dumplink", getWorkerDumpLink( profileRequest.get_nodeInfo().get_node(), - profileRequest.get_nodeInfo().get_port().iterator().next(), topologyId, config +(Long) profileRequest.get_nodeInfo().get_port().toArray()[0], topologyId, config )); result.put("timestamp", System.currentTimeMillis() - profileRequest.get_time_stamp()); return result; @@ -2165,21 +2165,20 @@ public class UIHelpers { * @param client client * @param id id * @param hostPort hostPort - * @param timeout timeout + * @param timestamp timestamp * @param config config * @param profileAction profileAction * @throws TException TException */ public static void getTopologyProfilingAction( Nimbus.Iface client, String id, -String hostPort, String timeout, Map config, ProfileAction profileAction) throws TException { String host = hostPort.split(":")[0]; Set ports = new HashSet(); String port = hostPort.split(":")[1]; ports.add(Long.valueOf(port)); NodeInfo nodeInfo = new NodeInfo(host, ports); -Long timestamp = System.currentTimeMillis() + Long.valueOf(timeout); ProfileRequest profileRequest = new ProfileRequest(nodeInfo, profileAction); profileRequest.set_time_stamp(timestamp); client.setWorkerProfiler(id, profileRequest); @@ -2198,7 +2197,9 @@ public class UIHelpers { public static Map getTopologyProfilingStart(Nimbus.Iface client, String id, String hostPort, String timeout, Map config) throws TException { -getTopologyProfilingAction(client, id , hostPort, timeout, config, ProfileAction.JPROFILE_START); +getTopologyProfilingAction( +client, id , hostPort, System.currentTimeMillis() + Long.valueOf(timeout), +config, ProfileAction.JPROFILE_START); Map result = new HashMap(); String host = hostPort.split(":")[0]; String port = hostPort.split(":")[1]; @@ -2214,15 +2215,14 @@ public class UIHelpers { * @param client client * @param id id * @param hostPort hostPort - * @param timeout timeout * @param config config * @return getTopologyProfilingStop * @throws TException TException */ public static Map getTopologyProfilingStop(Nimbus.Iface client, String id, - String hostPort, String timeout, + String hostPort, Map config) throws TException { -getTopologyProfilingAction(client, id , hostPort, timeout, config, ProfileAction.JPROFILE_STOP); +
[2/2] storm git commit: Merge branch 'patch-2' of https://github.com/jacobtolar/storm
Merge branch 'patch-2' of https://github.com/jacobtolar/storm Fix typo: indifinitely => indefinitely This closes #2857 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/a6f76df3 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/a6f76df3 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/a6f76df3 Branch: refs/heads/master Commit: a6f76df3f2a15bc1f8b88e7f432efaa0cdf8eed9 Parents: 353399a e1815c1 Author: Robert Evans Authored: Wed Oct 3 13:48:47 2018 -0500 Committer: Robert Evans Committed: Wed Oct 3 13:48:47 2018 -0500 -- docs/Command-line-client.md | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) --
[1/2] storm git commit: indifinitely => indefinitely
Repository: storm Updated Branches: refs/heads/master 353399a1f -> a6f76df3f indifinitely => indefinitely Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/e1815c1e Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/e1815c1e Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/e1815c1e Branch: refs/heads/master Commit: e1815c1e8aa5280838eae32d9a0cb914b8fef57c Parents: c47c23c Author: jacobtolar Authored: Mon Oct 1 17:21:02 2018 -0500 Committer: GitHub Committed: Mon Oct 1 17:21:02 2018 -0500 -- docs/Command-line-client.md | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/e1815c1e/docs/Command-line-client.md -- diff --git a/docs/Command-line-client.md b/docs/Command-line-client.md index 8bbf65e..8243b25 100644 --- a/docs/Command-line-client.md +++ b/docs/Command-line-client.md @@ -299,7 +299,7 @@ e.g. ./bin/storm set_log_level -l com.myapp=WARN -l com.myOtherLogger=ERROR:123 topology-name - Set the com.myapp logger's level to WARN indifinitely, and com.myOtherLogger to ERROR for 123 seconds + Set the com.myapp logger's level to WARN indefinitely, and com.myOtherLogger to ERROR for 123 seconds ./bin/storm set_log_level -r com.myOtherLogger topology-name
[1/5] storm git commit: STORM-3240 health checks should succeed on exit code 0
Repository: storm Updated Branches: refs/heads/master ad63cc20f -> fe3cfdf0a STORM-3240 health checks should succeed on exit code 0 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/39d28387 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/39d28387 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/39d28387 Branch: refs/heads/master Commit: 39d28387296a57835947da45f4c5c1f92eb676e6 Parents: c367f9a Author: Aaron Gresch Authored: Mon Oct 1 11:12:28 2018 -0500 Committer: Aaron Gresch Committed: Mon Oct 1 11:12:28 2018 -0500 -- .../main/java/org/apache/storm/healthcheck/HealthChecker.java| 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/39d28387/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java -- diff --git a/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java b/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java index 83faf13..5f877fc 100644 --- a/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java +++ b/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java @@ -106,13 +106,13 @@ public class HealthChecker { BufferedReader reader = new BufferedReader(new InputStreamReader(stdin)); while ((str = reader.readLine()) != null) { if (str.startsWith("ERROR")) { +LOG.warn("The healthcheck process {} exited with code {}", script, process.exitValue()); return FAILED; } } return SUCCESS; } -LOG.warn("The healthcheck process {} exited with code {}", script, process.exitValue()); -return FAILED_WITH_EXIT_CODE; +return SUCCESS; } catch (InterruptedException | ClosedByInterruptException e) { LOG.warn("Script: {} timed out.", script); return TIMEOUT;
[4/5] storm git commit: STORM-3240 any non-zero exit code causes health check failure
STORM-3240 any non-zero exit code causes health check failure Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/0b32a295 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/0b32a295 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/0b32a295 Branch: refs/heads/master Commit: 0b32a2950c61814ec6f9a9d73a82242559bb003f Parents: 9e84142 Author: Aaron Gresch Authored: Tue Oct 2 15:35:59 2018 -0500 Committer: Aaron Gresch Committed: Tue Oct 2 15:35:59 2018 -0500 -- docs/Setting-up-a-Storm-cluster.md | 2 +- .../main/java/org/apache/storm/healthcheck/HealthChecker.java| 4 ++-- 2 files changed, 3 insertions(+), 3 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/0b32a295/docs/Setting-up-a-Storm-cluster.md -- diff --git a/docs/Setting-up-a-Storm-cluster.md b/docs/Setting-up-a-Storm-cluster.md index c4a637c..d770a58 100644 --- a/docs/Setting-up-a-Storm-cluster.md +++ b/docs/Setting-up-a-Storm-cluster.md @@ -92,7 +92,7 @@ drpc.servers: ["111.222.333.44"] ### Monitoring Health of Supervisors -Storm provides a mechanism by which administrators can configure the supervisor to run administrator supplied scripts periodically to determine if a node is healthy or not. Administrators can have the supervisor determine if the node is in a healthy state by performing any checks of their choice in scripts located in storm.health.check.dir. If a script detects the node to be in an unhealthy state, it must print a line to standard output beginning with the string ERROR and return a non-zero exit code. In pre-Storm 2.x releases, a bug considered a script exit value of 0 to be a failure. This has now been fixed. The supervisor will periodically run the scripts in the health check dir and check the output. If the scriptâs output contains the string ERROR, as described above, the supervisor will shut down any workers and exit. +Storm provides a mechanism by which administrators can configure the supervisor to run administrator supplied scripts periodically to determine if a node is healthy or not. Administrators can have the supervisor determine if the node is in a healthy state by performing any checks of their choice in scripts located in storm.health.check.dir. If a script detects the node to be in an unhealthy state, it must return a non-zero exit code. In pre-Storm 2.x releases, a bug considered a script exit value of 0 to be a failure. This has now been fixed. The supervisor will periodically run the scripts in the health check dir and check the output. If the scriptâs output contains the string ERROR, as described above, the supervisor will shut down any workers and exit. If the supervisor is running with supervision "/bin/storm node-health-check" can be called to determine if the supervisor should be launched or if the node is unhealthy. http://git-wip-us.apache.org/repos/asf/storm/blob/0b32a295/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java -- diff --git a/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java b/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java index 38bcf64..b5f3655 100644 --- a/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java +++ b/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java @@ -107,10 +107,10 @@ public class HealthChecker { while ((str = reader.readLine()) != null) { if (str.startsWith("ERROR")) { LOG.warn("The healthcheck process {} exited with code {}", script, process.exitValue()); -return FAILED_WITH_EXIT_CODE; +return FAILED; } } -return SUCCESS; +return FAILED_WITH_EXIT_CODE; } return SUCCESS; } catch (InterruptedException | ClosedByInterruptException e) {
[3/5] storm git commit: STORM-3240 swap failure codes to better much functionality
STORM-3240 swap failure codes to better much functionality Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/9e84142c Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/9e84142c Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/9e84142c Branch: refs/heads/master Commit: 9e84142c84ea4975d6b446c1aaea19e04daa120c Parents: 7811fd0 Author: Aaron Gresch Authored: Tue Oct 2 15:16:40 2018 -0500 Committer: Aaron Gresch Committed: Tue Oct 2 15:16:40 2018 -0500 -- .../main/java/org/apache/storm/healthcheck/HealthChecker.java| 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/9e84142c/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java -- diff --git a/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java b/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java index 5f877fc..38bcf64 100644 --- a/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java +++ b/storm-server/src/main/java/org/apache/storm/healthcheck/HealthChecker.java @@ -107,7 +107,7 @@ public class HealthChecker { while ((str = reader.readLine()) != null) { if (str.startsWith("ERROR")) { LOG.warn("The healthcheck process {} exited with code {}", script, process.exitValue()); -return FAILED; +return FAILED_WITH_EXIT_CODE; } } return SUCCESS; @@ -118,7 +118,7 @@ public class HealthChecker { return TIMEOUT; } catch (Exception e) { LOG.warn("Script failed with exception: ", e); -return FAILED_WITH_EXIT_CODE; +return FAILED; } finally { if (interruptThread != null) { interruptThread.interrupt();
[5/5] storm git commit: Merge branch 'agresch_healthcheck' of https://github.com/agresch/storm into STORM-3240
Merge branch 'agresch_healthcheck' of https://github.com/agresch/storm into STORM-3240 STORM-3240: health checks should succeed on exit code 0 This clsoes #2855 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/fe3cfdf0 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/fe3cfdf0 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/fe3cfdf0 Branch: refs/heads/master Commit: fe3cfdf0afb111ac16479a059e8aae54fd0a0321 Parents: ad63cc2 0b32a29 Author: Robert Evans Authored: Wed Oct 3 12:49:39 2018 -0500 Committer: Robert Evans Committed: Wed Oct 3 12:49:39 2018 -0500 -- docs/Setting-up-a-Storm-cluster.md | 2 +- .../java/org/apache/storm/healthcheck/HealthChecker.java | 8 2 files changed, 5 insertions(+), 5 deletions(-) --
[2/5] storm git commit: STORM-3240 update health check documentation with behavior change
STORM-3240 update health check documentation with behavior change Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/7811fd02 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/7811fd02 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/7811fd02 Branch: refs/heads/master Commit: 7811fd02c5880067593a809024893268ea629db4 Parents: 39d2838 Author: Aaron Gresch Authored: Tue Oct 2 14:00:52 2018 -0500 Committer: Aaron Gresch Committed: Tue Oct 2 14:00:52 2018 -0500 -- docs/Setting-up-a-Storm-cluster.md | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/7811fd02/docs/Setting-up-a-Storm-cluster.md -- diff --git a/docs/Setting-up-a-Storm-cluster.md b/docs/Setting-up-a-Storm-cluster.md index 67ad727..c4a637c 100644 --- a/docs/Setting-up-a-Storm-cluster.md +++ b/docs/Setting-up-a-Storm-cluster.md @@ -92,7 +92,7 @@ drpc.servers: ["111.222.333.44"] ### Monitoring Health of Supervisors -Storm provides a mechanism by which administrators can configure the supervisor to run administrator supplied scripts periodically to determine if a node is healthy or not. Administrators can have the supervisor determine if the node is in a healthy state by performing any checks of their choice in scripts located in storm.health.check.dir. If a script detects the node to be in an unhealthy state, it must print a line to standard output beginning with the string ERROR. The supervisor will periodically run the scripts in the health check dir and check the output. If the scriptâs output contains the string ERROR, as described above, the supervisor will shut down any workers and exit. +Storm provides a mechanism by which administrators can configure the supervisor to run administrator supplied scripts periodically to determine if a node is healthy or not. Administrators can have the supervisor determine if the node is in a healthy state by performing any checks of their choice in scripts located in storm.health.check.dir. If a script detects the node to be in an unhealthy state, it must print a line to standard output beginning with the string ERROR and return a non-zero exit code. In pre-Storm 2.x releases, a bug considered a script exit value of 0 to be a failure. This has now been fixed. The supervisor will periodically run the scripts in the health check dir and check the output. If the scriptâs output contains the string ERROR, as described above, the supervisor will shut down any workers and exit. If the supervisor is running with supervision "/bin/storm node-health-check" can be called to determine if the supervisor should be launched or if the node is unhealthy.
[2/2] storm git commit: Merge branch 'STORM-2963' of https://github.com/roshannaik/storm
Merge branch 'STORM-2963' of https://github.com/roshannaik/storm STORM-2963: Adding notes on GC to Performance.md This closes #2853 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/ad63cc20 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/ad63cc20 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/ad63cc20 Branch: refs/heads/master Commit: ad63cc20ff0df278eaf55b3df28de62d6584790a Parents: c47c23c 50bcb3c Author: Robert Evans Authored: Tue Oct 2 11:25:29 2018 -0500 Committer: Robert Evans Committed: Tue Oct 2 11:25:29 2018 -0500 -- docs/Performance.md | 14 ++ 1 file changed, 14 insertions(+) --
[1/2] storm git commit: STORM-2963 Adding notes on GC & single worker mode to Performance.md
Repository: storm Updated Branches: refs/heads/master c47c23cf8 -> ad63cc20f STORM-2963 Adding notes on GC & single worker mode to Performance.md Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/50bcb3c2 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/50bcb3c2 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/50bcb3c2 Branch: refs/heads/master Commit: 50bcb3c20a2c7ef77fbeac47ed8354b586b43d99 Parents: c367f9a Author: roshannaik Authored: Sun Sep 30 03:21:02 2018 -0700 Committer: roshannaik Committed: Sun Sep 30 03:22:09 2018 -0700 -- docs/Performance.md | 14 ++ 1 file changed, 14 insertions(+) -- http://git-wip-us.apache.org/repos/asf/storm/blob/50bcb3c2/docs/Performance.md -- diff --git a/docs/Performance.md b/docs/Performance.md index 20acb32..9738fef 100644 --- a/docs/Performance.md +++ b/docs/Performance.md @@ -163,3 +163,17 @@ Executors that are not expected to be busy can be allocated a smaller fraction o core for executors that are not likely to saturate the CPU. The *system bolt* generally processes very few messages per second, and so requires very little cpu (typically less than 10% of a physical core). + + +## 9. Garbage Collection +Choice of GC is an important concern for topologies that are latency or throughput sensitive. It is recommended to try the both the CMS and G1 collectors. Performance characteristics +of the collectors can change between single and multiworker modes and is dependent on hardware characteristics such as number of CPUs and memory localities. Number of GC threads can +also affect performance. Sometimes fewer GC threads can yield better performance. It is advisable to select a collector and tune it by mimicking anticipated peak data rates on hardware +similar to what is used in production. + + +## 10. Scaling out with Single Worker mode +Communication between executors within a worker process is very fast as there is neither a need to serialize and deserialize messages nor does it involve communicating over the network +stack. In multiworker mode, messages often cross worker process boundaries. For performance sensitive cases, if it is possible to configure a topology to run as many single-worker +instances (for ex. one worker per input partition) rather than one multiworker instance, it may yield significantly better throughput and latency on the same hardware. +The downside to this approach is that it adds the overhead of monitoring and managing many instances rather than one multiworker instance.
[2/2] storm git commit: Merge branch 'agresch_mkAssignments_fail' of https://github.com/agresch/storm into STORM-3237
Merge branch 'agresch_mkAssignments_fail' of https://github.com/agresch/storm into STORM-3237 STORM-3237: track Nimbus mkAssignment failures This closes #2852 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/c47c23cf Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/c47c23cf Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/c47c23cf Branch: refs/heads/master Commit: c47c23cf8140db59c2ad76bd3ac5ef17ceb8fad4 Parents: c367f9a d7d2c62 Author: Robert Evans Authored: Mon Oct 1 09:27:44 2018 -0500 Committer: Robert Evans Committed: Mon Oct 1 09:27:44 2018 -0500 -- docs/ClusterMetrics.md | 1 + .../org/apache/storm/daemon/nimbus/Nimbus.java | 108 +++ 2 files changed, 63 insertions(+), 46 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/c47c23cf/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java --
[1/2] storm git commit: STORM-3237 track Nimbus mkAssignment failures
Repository: storm Updated Branches: refs/heads/master c367f9af7 -> c47c23cf8 STORM-3237 track Nimbus mkAssignment failures Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/d7d2c623 Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/d7d2c623 Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/d7d2c623 Branch: refs/heads/master Commit: d7d2c62312d2e3ed73273a88359d1e2f35a0ecbc Parents: 4c3a9bd Author: Aaron Gresch Authored: Thu Sep 27 13:30:59 2018 -0500 Committer: Aaron Gresch Committed: Fri Sep 28 15:23:58 2018 -0500 -- docs/ClusterMetrics.md | 1 + .../org/apache/storm/daemon/nimbus/Nimbus.java | 108 +++ 2 files changed, 63 insertions(+), 46 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/d7d2c623/docs/ClusterMetrics.md -- diff --git a/docs/ClusterMetrics.md b/docs/ClusterMetrics.md index 17c4ca1..4e4d0f1 100644 --- a/docs/ClusterMetrics.md +++ b/docs/ClusterMetrics.md @@ -101,6 +101,7 @@ These are metrics that are specific to a nimbus instance. In many instances onl | nimbus:num-uploadChunk-calls | meter | calls to uploadChunk thrift method. | | nimbus:num-uploadNewCredentials-calls | meter | calls to uploadNewCredentials thrift method. | | nimbus:process-worker-metric-calls | meter | calls to processWorkerMetrics thrift method. | +| nimbus:mkAssignments-Errors | meter | tracks exceptions from mkAssignments | | nimbus:topology-scheduling-duration-ms | timer | time it takes to do a scheduling run. | | nimbus:total-available-memory-non-negative | gauge | available memory on the cluster MB | | nimbuses:uptime-secs | histogram | uptime of nimbuses | http://git-wip-us.apache.org/repos/asf/storm/blob/d7d2c623/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java -- diff --git a/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java b/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java index d66c0ef..98ecfe3 100644 --- a/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java +++ b/storm-server/src/main/java/org/apache/storm/daemon/nimbus/Nimbus.java @@ -265,6 +265,8 @@ public class Nimbus implements Iface, Shutdownable, DaemonCommon { private final Meter getOwnerResourceSummariesCalls; private final Meter shutdownCalls; private final Meter processWorkerMetricsCalls; +private final Meter mkAssignmentsErrors; + //Timer private final Timer fileUploadDuration; private final Timer schedulingDuration; @@ -511,6 +513,7 @@ public class Nimbus implements Iface, Shutdownable, DaemonCommon { "nimbus:num-getOwnerResourceSummaries-calls"); this.shutdownCalls = metricsRegistry.registerMeter("nimbus:num-shutdown-calls"); this.processWorkerMetricsCalls = metricsRegistry.registerMeter("nimbus:process-worker-metric-calls"); +this.mkAssignmentsErrors = metricsRegistry.registerMeter("nimbus:mkAssignments-Errors"); this.fileUploadDuration = metricsRegistry.registerTimer("nimbus:files-upload-duration-ms"); this.schedulingDuration = metricsRegistry.registerTimer("nimbus:topology-scheduling-duration-ms"); this.numAddedExecPerScheduling = metricsRegistry.registerHistogram("nimbus:num-added-executors-per-scheduling"); @@ -2166,64 +2169,77 @@ public class Nimbus implements Iface, Shutdownable, DaemonCommon { } private void mkAssignments(String scratchTopoId) throws Exception { -if (!isReadyForMKAssignments()) { -return; -} -// get existing assignment (just the topologyToExecutorToNodePort map) -> default to {} -// filter out ones which have a executor timeout -// figure out available slots on cluster. add to that the used valid slots to get total slots. figure out how many executors -// should be in each slot (e.g., 4, 4, 4, 5) -// only keep existing slots that satisfy one of those slots. for rest, reassign them across remaining slots -// edge case for slots with no executor timeout but with supervisor timeout... just treat these as valid slots that can be -// reassigned to. worst comes to worse the executor will timeout and won't assign here next time around +try { +if (!isReadyForMKAssignments()) { +return; +} +// get existing assignment (just the topologyToExecutorToNodePort map) -> default to {} +// filter out ones which have a executor timeout +// figure out available slots on cluster. add to that the used valid slots to get total slots. figure ou
[1/2] storm git commit: STORM-3236 mark shutdown meters before stopping metric reporting
Repository: storm Updated Branches: refs/heads/master 53434b7eb -> c367f9af7 STORM-3236 mark shutdown meters before stopping metric reporting Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/2b9f404c Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/2b9f404c Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/2b9f404c Branch: refs/heads/master Commit: 2b9f404cc29f1b46008cdf7ae0d2337dc905a47b Parents: 4c3a9bd Author: Aaron Gresch Authored: Wed Sep 26 16:37:03 2018 -0500 Committer: Aaron Gresch Committed: Wed Sep 26 16:37:03 2018 -0500 -- .../src/main/java/org/apache/storm/daemon/drpc/DRPCServer.java | 4 +--- .../java/org/apache/storm/daemon/logviewer/LogviewerServer.java | 4 +--- 2 files changed, 2 insertions(+), 6 deletions(-) -- http://git-wip-us.apache.org/repos/asf/storm/blob/2b9f404c/storm-webapp/src/main/java/org/apache/storm/daemon/drpc/DRPCServer.java -- diff --git a/storm-webapp/src/main/java/org/apache/storm/daemon/drpc/DRPCServer.java b/storm-webapp/src/main/java/org/apache/storm/daemon/drpc/DRPCServer.java index 66653d1..9c24f06 100644 --- a/storm-webapp/src/main/java/org/apache/storm/daemon/drpc/DRPCServer.java +++ b/storm-webapp/src/main/java/org/apache/storm/daemon/drpc/DRPCServer.java @@ -172,9 +172,6 @@ public class DRPCServer implements AutoCloseable { @Override public synchronized void close() { if (!closed) { -//This is kind of useless... -meterShutdownCalls.mark(); - if (handlerServer != null) { handlerServer.stop(); } @@ -229,6 +226,7 @@ public class DRPCServer implements AutoCloseable { try (DRPCServer server = new DRPCServer(conf, metricsRegistry)) { metricsRegistry.startMetricsReporters(conf); Utils.addShutdownHookWithForceKillIn1Sec(() -> { +server.meterShutdownCalls.mark(); metricsRegistry.stopMetricsReporters(); server.close(); }); http://git-wip-us.apache.org/repos/asf/storm/blob/2b9f404c/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/LogviewerServer.java -- diff --git a/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/LogviewerServer.java b/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/LogviewerServer.java index 6eba28d..811f1ef 100644 --- a/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/LogviewerServer.java +++ b/storm-webapp/src/main/java/org/apache/storm/daemon/logviewer/LogviewerServer.java @@ -145,9 +145,6 @@ public class LogviewerServer implements AutoCloseable { @Override public synchronized void close() { if (!closed) { -//This is kind of useless... -meterShutdownCalls.mark(); - //TODO this is causing issues... //if (httpServer != null) { //httpServer.destroy(); @@ -175,6 +172,7 @@ public class LogviewerServer implements AutoCloseable { LogCleaner logCleaner = new LogCleaner(conf, workerLogs, directoryCleaner, logRootDir, metricsRegistry)) { metricsRegistry.startMetricsReporters(conf); Utils.addShutdownHookWithForceKillIn1Sec(() -> { +server.meterShutdownCalls.mark(); metricsRegistry.stopMetricsReporters(); server.close(); });
[2/2] storm git commit: Merge branch 'agresch_storm-3236' of https://github.com/agresch/storm into STORM-3236
Merge branch 'agresch_storm-3236' of https://github.com/agresch/storm into STORM-3236 STORM-3236: mark shutdown meters before stopping metric reporting This closes #2851 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/c367f9af Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/c367f9af Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/c367f9af Branch: refs/heads/master Commit: c367f9af74669ae5a1f314b826fa9057cdfd546f Parents: 53434b7 2b9f404 Author: Robert Evans Authored: Fri Sep 28 12:45:55 2018 -0500 Committer: Robert Evans Committed: Fri Sep 28 12:45:55 2018 -0500 -- .../src/main/java/org/apache/storm/daemon/drpc/DRPCServer.java | 4 +--- .../java/org/apache/storm/daemon/logviewer/LogviewerServer.java | 4 +--- 2 files changed, 2 insertions(+), 6 deletions(-) --
[2/2] storm git commit: Merge branch 'storm3235' of https://github.com/kishorvpatil/incubator-storm into STORM-3235
Merge branch 'storm3235' of https://github.com/kishorvpatil/incubator-storm into STORM-3235 STORM-3235: Fix WorkerToken renewal criteria and refactor This closes #2850 Project: http://git-wip-us.apache.org/repos/asf/storm/repo Commit: http://git-wip-us.apache.org/repos/asf/storm/commit/53434b7e Tree: http://git-wip-us.apache.org/repos/asf/storm/tree/53434b7e Diff: http://git-wip-us.apache.org/repos/asf/storm/diff/53434b7e Branch: refs/heads/master Commit: 53434b7eb8388455f43826519ba1a40bff1493d2 Parents: 4c3a9bd 24a28ea Author: Robert Evans Authored: Fri Sep 28 12:09:18 2018 -0500 Committer: Robert Evans Committed: Fri Sep 28 12:09:18 2018 -0500 -- .../org/apache/storm/daemon/nimbus/Nimbus.java | 26 ++ .../auth/workertoken/WorkerTokenManager.java| 37 +--- .../auth/workertoken/WorkerTokenTest.java | 16 +++-- 3 files changed, 41 insertions(+), 38 deletions(-) --