Repository: incubator-samza
Updated Branches:
  refs/heads/master 89939a6c9 -> decfa3ab5


SAMZA-176: Improve docs for MessageCollector. Reviewed by Jakob Homan.


Project: http://git-wip-us.apache.org/repos/asf/incubator-samza/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-samza/commit/decfa3ab
Tree: http://git-wip-us.apache.org/repos/asf/incubator-samza/tree/decfa3ab
Diff: http://git-wip-us.apache.org/repos/asf/incubator-samza/diff/decfa3ab

Branch: refs/heads/master
Commit: decfa3ab511d77d626f797f6743f49fde7178b30
Parents: 89939a6
Author: Martin Kleppmann <[email protected]>
Authored: Fri Mar 21 15:53:23 2014 +0000
Committer: Martin Kleppmann <[email protected]>
Committed: Sat Mar 22 14:27:17 2014 +0000

----------------------------------------------------------------------
 docs/learn/documentation/0.7.0/api/overview.md                 | 2 ++
 docs/learn/documentation/0.7.0/container/windowing.md          | 2 ++
 .../src/main/java/org/apache/samza/task/MessageCollector.java  | 6 +++++-
 samza-api/src/main/java/org/apache/samza/task/StreamTask.java  | 4 +++-
 .../src/main/java/org/apache/samza/task/WindowableTask.java    | 4 +++-
 5 files changed, 15 insertions(+), 3 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-samza/blob/decfa3ab/docs/learn/documentation/0.7.0/api/overview.md
----------------------------------------------------------------------
diff --git a/docs/learn/documentation/0.7.0/api/overview.md 
b/docs/learn/documentation/0.7.0/api/overview.md
index 109d604..2d03ec3 100644
--- a/docs/learn/documentation/0.7.0/api/overview.md
+++ b/docs/learn/documentation/0.7.0/api/overview.md
@@ -74,6 +74,8 @@ public interface MessageCollector {
 
 The collector takes OutgoingMessageEnvelope, which allows tasks to supply a 
partition key when sending the message. The partition key, if supplied, is used 
to determine which partition of a stream a message is destined for.
 
+Please only use the MessageCollector object within the process() method. If 
you hold onto a MessageCollector instance and use it again later, your messages 
may not be sent correctly.
+
 ```
 /** A message envelope that has a key. */
 public class OutgoingMessageEnvelope {

http://git-wip-us.apache.org/repos/asf/incubator-samza/blob/decfa3ab/docs/learn/documentation/0.7.0/container/windowing.md
----------------------------------------------------------------------
diff --git a/docs/learn/documentation/0.7.0/container/windowing.md 
b/docs/learn/documentation/0.7.0/container/windowing.md
index caf14e5..6a24378 100644
--- a/docs/learn/documentation/0.7.0/container/windowing.md
+++ b/docs/learn/documentation/0.7.0/container/windowing.md
@@ -15,4 +15,6 @@ public interface WindowableTask {
 
 If you choose to implement the WindowableTask interface, you can use the Samza 
job's configuration to define how often the TaskRunner should call your 
window() method. In the PageViewEvent example (above), you would define it to 
flush every 60000 milliseconds (60 seconds).
 
+If you need to send messages to output streams, you can use the 
MessageCollector object passed to the window() method. Please only use that 
MessageCollector object for sending messages, and don't use it outside of the 
call to window().
+
 ## [Event Loop &raquo;](event-loop.html)

http://git-wip-us.apache.org/repos/asf/incubator-samza/blob/decfa3ab/samza-api/src/main/java/org/apache/samza/task/MessageCollector.java
----------------------------------------------------------------------
diff --git 
a/samza-api/src/main/java/org/apache/samza/task/MessageCollector.java 
b/samza-api/src/main/java/org/apache/samza/task/MessageCollector.java
index 0da816e..3e58cdf 100644
--- a/samza-api/src/main/java/org/apache/samza/task/MessageCollector.java
+++ b/samza-api/src/main/java/org/apache/samza/task/MessageCollector.java
@@ -22,7 +22,11 @@ package org.apache.samza.task;
 import org.apache.samza.system.OutgoingMessageEnvelope;
 
 /**
- * Used as an interface for the means of sending message envelopes.
+ * Used as an interface for the means of sending message envelopes to an 
output stream.
+ *
+ * <p>A MessageCollector is provided on every call to {@link 
StreamTask#process} and
+ * {@link WindowableTask#window}. You must use those MessageCollector objects 
only within
+ * those method calls, and not hold on to a reference for use at any other 
time.
  */
 public interface MessageCollector {
   /**

http://git-wip-us.apache.org/repos/asf/incubator-samza/blob/decfa3ab/samza-api/src/main/java/org/apache/samza/task/StreamTask.java
----------------------------------------------------------------------
diff --git a/samza-api/src/main/java/org/apache/samza/task/StreamTask.java 
b/samza-api/src/main/java/org/apache/samza/task/StreamTask.java
index 3d87fbb..00d5efd 100644
--- a/samza-api/src/main/java/org/apache/samza/task/StreamTask.java
+++ b/samza-api/src/main/java/org/apache/samza/task/StreamTask.java
@@ -30,7 +30,9 @@ public interface StreamTask {
    * Called once for each message that this StreamTask receives.
    * @param envelope Contains the received deserialized message and key, and 
also information regarding the stream and
    * partition of which the message was received from.
-   * @param collector Contains the means of sending message envelopes to the 
output stream.
+   * @param collector Contains the means of sending message envelopes to the 
output stream. The collector must only
+   * be used during the current call to the process method; you should not 
reuse the collector between invocations
+   * of this method.
    * @param coordinator Manages execution of tasks.
    * @throws Exception Any exception types encountered during the execution of 
the processing task.
    */

http://git-wip-us.apache.org/repos/asf/incubator-samza/blob/decfa3ab/samza-api/src/main/java/org/apache/samza/task/WindowableTask.java
----------------------------------------------------------------------
diff --git a/samza-api/src/main/java/org/apache/samza/task/WindowableTask.java 
b/samza-api/src/main/java/org/apache/samza/task/WindowableTask.java
index be22f24..1f48eec 100644
--- a/samza-api/src/main/java/org/apache/samza/task/WindowableTask.java
+++ b/samza-api/src/main/java/org/apache/samza/task/WindowableTask.java
@@ -25,7 +25,9 @@ package org.apache.samza.task;
 public interface WindowableTask {
   /**
    * Called by TaskRunner for each implementing task at the end of every 
specified window.
-   * @param collector Contains the means of sending message envelopes to the 
output stream.
+   * @param collector Contains the means of sending message envelopes to the 
output stream. The collector must only
+   * be used during the current call to the window method; you should not 
reuse the collector between invocations
+   * of this method.
    * @param coordinator Manages execution of tasks.
    * @throws Exception Any exception types encountered during the execution of 
the processing task.
    */

Reply via email to