[
https://issues.apache.org/jira/browse/STORM-495?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14129222#comment-14129222
]
ASF GitHub Bot commented on STORM-495:
--------------------------------------
Github user Parth-Brahmbhatt commented on a diff in the pull request:
https://github.com/apache/incubator-storm/pull/254#discussion_r17392468
--- Diff: external/storm-kafka/src/jvm/storm/kafka/PartitionManager.java ---
@@ -191,6 +208,7 @@ public void ack(Long offset) {
_pending.headSet(offset -
_spoutConfig.maxOffsetBehind).clear();
}
_pending.remove(offset);
+ retryRecords.remove(offset);
--- End diff --
This is the only place where retryRecords.remove() is called, so if a
message keeps failing it will never be removed from this Map in other words we
seem to be leaking instances here.
> 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)