[
https://issues.apache.org/jira/browse/STORM-495?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14129234#comment-14129234
]
ASF GitHub Bot commented on STORM-495:
--------------------------------------
Github user ptgoetz commented on a diff in the pull request:
https://github.com/apache/incubator-storm/pull/254#discussion_r17392894
--- Diff: external/storm-kafka/src/jvm/storm/kafka/PartitionManager.java ---
@@ -261,4 +283,45 @@ public KafkaMessageId(Partition partition, long
offset) {
this.offset = offset;
}
}
+
+ /**
+ * A MessageRetryRecord holds the data of how many times a message has
+ * failed and been retried, and when the last failure occurred. It can
+ * determine whether it is ready to be retried by employing an
exponential
+ * back-off calculation using config values stored in SpoutConfig:
+ * <ul>
+ * <li>retryInitialDelayMs - time to delay before the first retry</li>
+ * <li>retryDelayMultiplier - multiplier by which to increase the
delay for each subsequent retry</li>
+ * <li>retryDelayMaxMs - maximum retry delay (once this delay time is
reached, subsequent retries will
+ * delay for this amount of time every time)
+ * </li>
+ * </ul>
+ */
+ class MessageRetryRecord {
+ private final int retryNum;
+ private final long retryTimeUTC;
+
+ public MessageRetryRecord() {
+ this(1);
+ }
+
+ private MessageRetryRecord(int retryNum) {
+ this.retryNum = retryNum;
+ this.retryTimeUTC = new Date().getTime() +
calculateRetryDelay();
--- End diff --
System.currentTimeMillis()?
> Add delayed retries to KafkaSpout
> ---------------------------------
>
> Key: STORM-495
> URL: https://issues.apache.org/jira/browse/STORM-495
> Project: Apache Storm (Incubating)
> Issue Type: Improvement
> Affects Versions: 0.9.3-incubating
> Environment: all environments
> Reporter: Rick Kilgore
> Priority: Minor
> Labels: kafka, retry
>
> If a tuple in the topology originates from the KafkaSpout from the
> external/storm-kafka sources, and if a bolt in the topology indicates a
> failure by calling fail() on its OutputCollector, the KafkaSpout will
> immediately retry the message.
> We wish to use this failure and retry behavior in our ingestion system
> whenever we experience a recoverable error from a downstream system, such as
> a 500 or 503 error from a service we depend on. But with the current
> KafkaSpout behavior, doing so results in a tight loop where we retry several
> times over a few seconds and then give up. I want to be able to delay retry
> to give the downstream service some time to recover. Ideally, I would like
> to have configurable, exponential backoff retry.
--
This message was sent by Atlassian JIRA
(v6.3.4#6332)