You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@camel.apache.org by da...@apache.org on 2014/09/01 11:18:18 UTC

[1/3] git commit: CAMEL-7581: Added aggregateOnException option to enrich/pollEnrich so end users can handle exception in their aggregte method.

Repository: camel
Updated Branches:
  refs/heads/master 509f4ea5d -> 4ffc5e258


CAMEL-7581: Added aggregateOnException option to enrich/pollEnrich so end users can handle exception in their aggregte method.


Project: http://git-wip-us.apache.org/repos/asf/camel/repo
Commit: http://git-wip-us.apache.org/repos/asf/camel/commit/16e193c8
Tree: http://git-wip-us.apache.org/repos/asf/camel/tree/16e193c8
Diff: http://git-wip-us.apache.org/repos/asf/camel/diff/16e193c8

Branch: refs/heads/master
Commit: 16e193c85a68e1c61be9eb0354eb6d962a31306c
Parents: 509f4ea
Author: Claus Ibsen <da...@apache.org>
Authored: Mon Sep 1 10:28:57 2014 +0200
Committer: Claus Ibsen <da...@apache.org>
Committed: Mon Sep 1 10:28:57 2014 +0200

----------------------------------------------------------------------
 .../apache/camel/model/EnrichDefinition.java    |  13 ++
 .../camel/model/PollEnrichDefinition.java       |  13 ++
 .../apache/camel/model/ProcessorDefinition.java | 100 +++++++++++++++
 .../org/apache/camel/processor/Enricher.java    |  21 +++-
 .../apache/camel/processor/PollEnricher.java    |  29 ++++-
 .../EnricherAggregateOnExceptionTest.java       | 116 ++++++++++++++++++
 .../PollEnricherAggregateOnExceptionTest.java   | 122 +++++++++++++++++++
 7 files changed, 406 insertions(+), 8 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/camel/blob/16e193c8/camel-core/src/main/java/org/apache/camel/model/EnrichDefinition.java
----------------------------------------------------------------------
diff --git a/camel-core/src/main/java/org/apache/camel/model/EnrichDefinition.java b/camel-core/src/main/java/org/apache/camel/model/EnrichDefinition.java
index 64935bb..789f84b 100644
--- a/camel-core/src/main/java/org/apache/camel/model/EnrichDefinition.java
+++ b/camel-core/src/main/java/org/apache/camel/model/EnrichDefinition.java
@@ -50,6 +50,8 @@ public class EnrichDefinition extends NoOutputDefinition<EnrichDefinition> imple
     private String aggregationStrategyMethodName;
     @XmlAttribute(name = "strategyMethodAllowNull")
     private Boolean aggregationStrategyMethodAllowNull;
+    @XmlAttribute
+    private Boolean aggregateOnException;
     @XmlTransient
     private AggregationStrategy aggregationStrategy;
     
@@ -115,6 +117,9 @@ public class EnrichDefinition extends NoOutputDefinition<EnrichDefinition> imple
         } else {
             enricher.setAggregationStrategy(strategy);
         }
+        if (getAggregateOnException() != null) {
+            enricher.setAggregateOnException(getAggregateOnException());
+        }
         return enricher;
     }
 
@@ -190,4 +195,12 @@ public class EnrichDefinition extends NoOutputDefinition<EnrichDefinition> imple
     public void setAggregationStrategy(AggregationStrategy aggregationStrategy) {
         this.aggregationStrategy = aggregationStrategy;
     }
+
+    public Boolean getAggregateOnException() {
+        return aggregateOnException;
+    }
+
+    public void setAggregateOnException(Boolean aggregateOnException) {
+        this.aggregateOnException = aggregateOnException;
+    }
 }

http://git-wip-us.apache.org/repos/asf/camel/blob/16e193c8/camel-core/src/main/java/org/apache/camel/model/PollEnrichDefinition.java
----------------------------------------------------------------------
diff --git a/camel-core/src/main/java/org/apache/camel/model/PollEnrichDefinition.java b/camel-core/src/main/java/org/apache/camel/model/PollEnrichDefinition.java
index 3ddec0c..ac5a323 100644
--- a/camel-core/src/main/java/org/apache/camel/model/PollEnrichDefinition.java
+++ b/camel-core/src/main/java/org/apache/camel/model/PollEnrichDefinition.java
@@ -52,6 +52,8 @@ public class PollEnrichDefinition extends NoOutputDefinition<PollEnrichDefinitio
     private String aggregationStrategyMethodName;
     @XmlAttribute(name = "strategyMethodAllowNull")
     private Boolean aggregationStrategyMethodAllowNull;
+    @XmlAttribute
+    private Boolean aggregateOnException;
     @XmlTransient
     private AggregationStrategy aggregationStrategy;
 
@@ -120,6 +122,9 @@ public class PollEnrichDefinition extends NoOutputDefinition<PollEnrichDefinitio
         } else {
             enricher.setAggregationStrategy(strategy);
         }
+        if (getAggregateOnException() != null) {
+            enricher.setAggregateOnException(getAggregateOnException());
+        }
 
         return enricher;
     }
@@ -204,4 +209,12 @@ public class PollEnrichDefinition extends NoOutputDefinition<PollEnrichDefinitio
     public void setAggregationStrategy(AggregationStrategy aggregationStrategy) {
         this.aggregationStrategy = aggregationStrategy;
     }
+
+    public Boolean getAggregateOnException() {
+        return aggregateOnException;
+    }
+
+    public void setAggregateOnException(Boolean aggregateOnException) {
+        this.aggregateOnException = aggregateOnException;
+    }
 }
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/camel/blob/16e193c8/camel-core/src/main/java/org/apache/camel/model/ProcessorDefinition.java
----------------------------------------------------------------------
diff --git a/camel-core/src/main/java/org/apache/camel/model/ProcessorDefinition.java b/camel-core/src/main/java/org/apache/camel/model/ProcessorDefinition.java
index 6af6ca6..af44b18 100644
--- a/camel-core/src/main/java/org/apache/camel/model/ProcessorDefinition.java
+++ b/camel-core/src/main/java/org/apache/camel/model/ProcessorDefinition.java
@@ -2920,6 +2920,23 @@ public abstract class ProcessorDefinition<Type extends ProcessorDefinition<Type>
     /**
      * The <a href="http://camel.apache.org/content-enricher.html">Content Enricher EIP</a>
      * enriches an exchange with additional data obtained from a <code>resourceUri</code>.
+     *
+     * @param resourceUri           URI of resource endpoint for obtaining additional data.
+     * @param aggregationStrategy   aggregation strategy to aggregate input data and additional data.
+     * @return the builder
+     * @see org.apache.camel.processor.Enricher
+     */
+    @SuppressWarnings("unchecked")
+    public Type enrich(String resourceUri, AggregationStrategy aggregationStrategy, boolean aggregateOnException) {
+        EnrichDefinition enrich = new EnrichDefinition(aggregationStrategy, resourceUri);
+        enrich.setAggregateOnException(aggregateOnException);
+        addOutput(enrich);
+        return (Type) this;
+    }
+
+    /**
+     * The <a href="http://camel.apache.org/content-enricher.html">Content Enricher EIP</a>
+     * enriches an exchange with additional data obtained from a <code>resourceUri</code>.
      * <p/>
      * The difference between this and {@link #pollEnrich(String)} is that this uses a producer
      * to obatin the additional data, where as pollEnrich uses a polling consumer.
@@ -2957,6 +2974,30 @@ public abstract class ProcessorDefinition<Type extends ProcessorDefinition<Type>
 
     /**
      * The <a href="http://camel.apache.org/content-enricher.html">Content Enricher EIP</a>
+     * enriches an exchange with additional data obtained from a <code>resourceUri</code>.
+     * <p/>
+     * The difference between this and {@link #pollEnrich(String)} is that this uses a producer
+     * to obtain the additional data, where as pollEnrich uses a polling consumer.
+     *
+     * @param resourceRef            Reference of resource endpoint for obtaining additional data.
+     * @param aggregationStrategyRef Reference of aggregation strategy to aggregate input data and additional data.
+     * @param aggregateOnException   whether to call {@link org.apache.camel.processor.aggregate.AggregationStrategy#aggregate(org.apache.camel.Exchange, org.apache.camel.Exchange)} if
+     *                               an exception was thrown.
+     * @return the builder
+     * @see org.apache.camel.processor.Enricher
+     */
+    @SuppressWarnings("unchecked")
+    public Type enrichRef(String resourceRef, String aggregationStrategyRef, boolean aggregateOnException) {
+        EnrichDefinition enrich = new EnrichDefinition();
+        enrich.setResourceRef(resourceRef);
+        enrich.setAggregationStrategyRef(aggregationStrategyRef);
+        enrich.setAggregateOnException(aggregateOnException);
+        addOutput(enrich);
+        return (Type) this;
+    }
+
+    /**
+     * The <a href="http://camel.apache.org/content-enricher.html">Content Enricher EIP</a>
      * enriches an exchange with additional data obtained from a <code>resourceUri</code>
      * using a {@link org.apache.camel.PollingConsumer} to poll the endpoint.
      * <p/>
@@ -3036,6 +3077,34 @@ public abstract class ProcessorDefinition<Type extends ProcessorDefinition<Type>
      *
      * @param resourceUri           URI of resource endpoint for obtaining additional data.
      * @param timeout               timeout in millis to wait at most for data to be available.
+     * @param aggregationStrategy   aggregation strategy to aggregate input data and additional data.
+     * @param aggregateOnException   whether to call {@link org.apache.camel.processor.aggregate.AggregationStrategy#aggregate(org.apache.camel.Exchange, org.apache.camel.Exchange)} if
+     *                               an exception was thrown.
+     * @return the builder
+     * @see org.apache.camel.processor.PollEnricher
+     */
+    @SuppressWarnings("unchecked")
+    public Type pollEnrich(String resourceUri, long timeout, AggregationStrategy aggregationStrategy, boolean aggregateOnException) {
+        PollEnrichDefinition pollEnrich = new PollEnrichDefinition(aggregationStrategy, resourceUri, timeout);
+        pollEnrich.setAggregateOnException(aggregateOnException);
+        addOutput(pollEnrich);
+        return (Type) this;
+    }
+
+    /**
+     * The <a href="http://camel.apache.org/content-enricher.html">Content Enricher EIP</a>
+     * enriches an exchange with additional data obtained from a <code>resourceUri</code>
+     * using a {@link org.apache.camel.PollingConsumer} to poll the endpoint.
+     * <p/>
+     * The difference between this and {@link #enrich(String)} is that this uses a consumer
+     * to obtain the additional data, where as enrich uses a producer.
+     * <p/>
+     * The timeout controls which operation to use on {@link org.apache.camel.PollingConsumer}.
+     * If timeout is negative, we use <tt>receive</tt>. If timeout is 0 then we use <tt>receiveNoWait</tt>
+     * otherwise we use <tt>receive(timeout)</tt>.
+     *
+     * @param resourceUri           URI of resource endpoint for obtaining additional data.
+     * @param timeout               timeout in millis to wait at most for data to be available.
      * @return the builder
      * @see org.apache.camel.processor.PollEnricher
      */
@@ -3074,6 +3143,37 @@ public abstract class ProcessorDefinition<Type extends ProcessorDefinition<Type>
     }
 
     /**
+     * The <a href="http://camel.apache.org/content-enricher.html">Content Enricher EIP</a>
+     * enriches an exchange with additional data obtained from a <code>resourceUri</code>
+     * using a {@link org.apache.camel.PollingConsumer} to poll the endpoint.
+     * <p/>
+     * The difference between this and {@link #enrich(String)} is that this uses a consumer
+     * to obtain the additional data, where as enrich uses a producer.
+     * <p/>
+     * The timeout controls which operation to use on {@link org.apache.camel.PollingConsumer}.
+     * If timeout is negative, we use <tt>receive</tt>. If timeout is 0 then we use <tt>receiveNoWait</tt>
+     * otherwise we use <tt>receive(timeout)</tt>.
+     *
+     * @param resourceRef            Reference of resource endpoint for obtaining additional data.
+     * @param timeout                timeout in millis to wait at most for data to be available.
+     * @param aggregationStrategyRef Reference of aggregation strategy to aggregate input data and additional data.
+     * @param aggregateOnException   whether to call {@link org.apache.camel.processor.aggregate.AggregationStrategy#aggregate(org.apache.camel.Exchange, org.apache.camel.Exchange)} if
+     *                               an exception was thrown.
+     * @return the builder
+     * @see org.apache.camel.processor.PollEnricher
+     */
+    @SuppressWarnings("unchecked")
+    public Type pollEnrichRef(String resourceRef, long timeout, String aggregationStrategyRef, boolean aggregateOnException) {
+        PollEnrichDefinition pollEnrich = new PollEnrichDefinition();
+        pollEnrich.setResourceRef(resourceRef);
+        pollEnrich.setTimeout(timeout);
+        pollEnrich.setAggregationStrategyRef(aggregationStrategyRef);
+        pollEnrich.setAggregateOnException(aggregateOnException);
+        addOutput(pollEnrich);
+        return (Type) this;
+    }
+
+    /**
      * Adds a onComplection {@link org.apache.camel.spi.Synchronization} hook that invoke this route as
      * a callback when the {@link org.apache.camel.Exchange} has finished being processed.
      * The hook invoke callbacks for either onComplete or onFailure.

http://git-wip-us.apache.org/repos/asf/camel/blob/16e193c8/camel-core/src/main/java/org/apache/camel/processor/Enricher.java
----------------------------------------------------------------------
diff --git a/camel-core/src/main/java/org/apache/camel/processor/Enricher.java b/camel-core/src/main/java/org/apache/camel/processor/Enricher.java
index 72faa12..9ab0aa9 100644
--- a/camel-core/src/main/java/org/apache/camel/processor/Enricher.java
+++ b/camel-core/src/main/java/org/apache/camel/processor/Enricher.java
@@ -53,6 +53,7 @@ public class Enricher extends ServiceSupport implements AsyncProcessor {
     private static final Logger LOG = LoggerFactory.getLogger(Enricher.class);
     private AggregationStrategy aggregationStrategy;
     private Producer producer;
+    private boolean aggregateOnException;
 
     /**
      * Creates a new {@link Enricher}. The default aggregation strategy is to
@@ -86,6 +87,22 @@ public class Enricher extends ServiceSupport implements AsyncProcessor {
         this.aggregationStrategy = aggregationStrategy;
     }
 
+    public AggregationStrategy getAggregationStrategy() {
+        return aggregationStrategy;
+    }
+
+    public boolean isAggregateOnException() {
+        return aggregateOnException;
+    }
+
+    /**
+     * Whether to call {@link org.apache.camel.processor.aggregate.AggregationStrategy#aggregate(org.apache.camel.Exchange, org.apache.camel.Exchange)} if
+     * an exception was thrown.
+     */
+    public void setAggregateOnException(boolean aggregateOnException) {
+        this.aggregateOnException = aggregateOnException;
+    }
+
     /**
      * Sets the default aggregation strategy for this enricher.
      */
@@ -128,7 +145,7 @@ public class Enricher extends ServiceSupport implements AsyncProcessor {
                 long timeTaken = watch.stop();
                 EventHelper.notifyExchangeSent(resourceExchange.getContext(), resourceExchange, destination, timeTaken);
                 
-                if (resourceExchange.isFailed()) {
+                if (!isAggregateOnException() && resourceExchange.isFailed()) {
                     // copy resource exchange onto original exchange (preserving pattern)
                     copyResultsPreservePattern(exchange, resourceExchange);
                 } else {
@@ -171,7 +188,7 @@ public class Enricher extends ServiceSupport implements AsyncProcessor {
         long timeTaken = watch.stop();
         EventHelper.notifyExchangeSent(resourceExchange.getContext(), resourceExchange, destination, timeTaken);
         
-        if (resourceExchange.isFailed()) {
+        if (!isAggregateOnException() && resourceExchange.isFailed()) {
             // copy resource exchange onto original exchange (preserving pattern)
             copyResultsPreservePattern(exchange, resourceExchange);
         } else {

http://git-wip-us.apache.org/repos/asf/camel/blob/16e193c8/camel-core/src/main/java/org/apache/camel/processor/PollEnricher.java
----------------------------------------------------------------------
diff --git a/camel-core/src/main/java/org/apache/camel/processor/PollEnricher.java b/camel-core/src/main/java/org/apache/camel/processor/PollEnricher.java
index 34d445a..7960d3d 100644
--- a/camel-core/src/main/java/org/apache/camel/processor/PollEnricher.java
+++ b/camel-core/src/main/java/org/apache/camel/processor/PollEnricher.java
@@ -49,6 +49,7 @@ public class PollEnricher extends ServiceSupport implements AsyncProcessor {
     private AggregationStrategy aggregationStrategy;
     private PollingConsumer consumer;
     private long timeout;
+    private boolean aggregateOnException;
 
     /**
      * Creates a new {@link PollEnricher}. The default aggregation strategy is to
@@ -75,6 +76,10 @@ public class PollEnricher extends ServiceSupport implements AsyncProcessor {
         this.timeout = timeout;
     }
 
+    public AggregationStrategy getAggregationStrategy() {
+        return aggregationStrategy;
+    }
+
     /**
      * Sets the aggregation strategy for this poll enricher.
      *
@@ -84,11 +89,8 @@ public class PollEnricher extends ServiceSupport implements AsyncProcessor {
         this.aggregationStrategy = aggregationStrategy;
     }
 
-    /**
-     * Sets the default aggregation strategy for this poll enricher.
-     */
-    public void setDefaultAggregationStrategy() {
-        this.aggregationStrategy = defaultAggregationStrategy();
+    public long getTimeout() {
+        return timeout;
     }
 
     /**
@@ -103,6 +105,21 @@ public class PollEnricher extends ServiceSupport implements AsyncProcessor {
         this.timeout = timeout;
     }
 
+    public boolean isAggregateOnException() {
+        return aggregateOnException;
+    }
+
+    public void setAggregateOnException(boolean aggregateOnException) {
+        this.aggregateOnException = aggregateOnException;
+    }
+
+    /**
+     * Sets the default aggregation strategy for this poll enricher.
+     */
+    public void setDefaultAggregationStrategy() {
+        this.aggregationStrategy = defaultAggregationStrategy();
+    }
+
     public void process(Exchange exchange) throws Exception {
         AsyncProcessorHelper.process(this, exchange);
     }
@@ -147,7 +164,7 @@ public class PollEnricher extends ServiceSupport implements AsyncProcessor {
             LOG.debug("Consumer received: {}", resourceExchange);
         }
 
-        if (resourceExchange != null && resourceExchange.isFailed()) {
+        if (!isAggregateOnException() && (resourceExchange != null && resourceExchange.isFailed())) {
             // copy resource exchange onto original exchange (preserving pattern)
             copyResultsPreservePattern(exchange, resourceExchange);
         } else {

http://git-wip-us.apache.org/repos/asf/camel/blob/16e193c8/camel-core/src/test/java/org/apache/camel/processor/enricher/EnricherAggregateOnExceptionTest.java
----------------------------------------------------------------------
diff --git a/camel-core/src/test/java/org/apache/camel/processor/enricher/EnricherAggregateOnExceptionTest.java b/camel-core/src/test/java/org/apache/camel/processor/enricher/EnricherAggregateOnExceptionTest.java
new file mode 100644
index 0000000..4353a66
--- /dev/null
+++ b/camel-core/src/test/java/org/apache/camel/processor/enricher/EnricherAggregateOnExceptionTest.java
@@ -0,0 +1,116 @@
+/**
+ * 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.camel.processor.enricher;
+
+import org.apache.camel.CamelExecutionException;
+import org.apache.camel.ContextTestSupport;
+import org.apache.camel.Exchange;
+import org.apache.camel.Processor;
+import org.apache.camel.builder.RouteBuilder;
+import org.apache.camel.component.mock.MockEndpoint;
+import org.apache.camel.processor.aggregate.AggregationStrategy;
+
+/**
+ * @version 
+ */
+public class EnricherAggregateOnExceptionTest extends ContextTestSupport {
+
+    public void testEnrichTrueOk() throws Exception {
+        MockEndpoint mock = getMockEndpoint("mock:result");
+        mock.expectedBodiesReceived("Hello World");
+
+        template.sendBody("direct:start", "World");
+
+        assertMockEndpointsSatisfied();
+    }
+
+    public void testEnrichTrueKaboom() throws Exception {
+        MockEndpoint mock = getMockEndpoint("mock:result");
+        mock.expectedBodiesReceived("I cannot do this");
+
+        template.sendBody("direct:start", "Kaboom");
+
+        assertMockEndpointsSatisfied();
+    }
+
+    public void testEnrichFalseOk() throws Exception {
+        MockEndpoint mock = getMockEndpoint("mock:result");
+        mock.expectedBodiesReceived("Hello World");
+
+        template.sendBody("direct:start2", "World");
+
+        assertMockEndpointsSatisfied();
+    }
+
+    public void testEnrichFalseKaboom() throws Exception {
+        MockEndpoint mock = getMockEndpoint("mock:result");
+        mock.expectedMessageCount(0);
+
+        try {
+            template.sendBody("direct:start2", "Kaboom");
+            fail("Should have thrown exception");
+        } catch (CamelExecutionException e) {
+            assertIsInstanceOf(IllegalArgumentException.class, e.getCause());
+            assertEquals("I cannot do this", e.getCause().getMessage());
+        }
+
+        assertMockEndpointsSatisfied();
+    }
+
+    @Override
+    protected RouteBuilder createRouteBuilder() throws Exception {
+        return new RouteBuilder() {
+            @Override
+            public void configure() throws Exception {
+                from("direct:start")
+                    .enrich("direct:foo", new MyAggregationStrategy(), true)
+                    .to("mock:result");
+
+                from("direct:start2")
+                    .enrich("direct:foo", new MyAggregationStrategy(), false)
+                    .to("mock:result");
+
+                from("direct:foo")
+                    .process(new Processor() {
+                        @Override
+                        public void process(Exchange exchange) throws Exception {
+                            String body = exchange.getIn().getBody(String.class);
+                            if (body.startsWith("Kaboom")) {
+                                throw new IllegalArgumentException("I cannot do this");
+                            }
+                            exchange.getIn().setBody("Hello " + body);
+                        }
+                    });
+            }
+        };
+    }
+
+    private class MyAggregationStrategy implements AggregationStrategy {
+
+        @Override
+        public Exchange aggregate(Exchange oldExchange, Exchange newExchange) {
+            if (newExchange.getException() != null) {
+                oldExchange.getIn().setBody(newExchange.getException().getMessage());
+                return oldExchange;
+            }
+
+            // replace body
+            oldExchange.getIn().setBody(newExchange.getIn().getBody());
+            return oldExchange;
+        }
+    }
+}

http://git-wip-us.apache.org/repos/asf/camel/blob/16e193c8/camel-core/src/test/java/org/apache/camel/processor/enricher/PollEnricherAggregateOnExceptionTest.java
----------------------------------------------------------------------
diff --git a/camel-core/src/test/java/org/apache/camel/processor/enricher/PollEnricherAggregateOnExceptionTest.java b/camel-core/src/test/java/org/apache/camel/processor/enricher/PollEnricherAggregateOnExceptionTest.java
new file mode 100644
index 0000000..a53ce27
--- /dev/null
+++ b/camel-core/src/test/java/org/apache/camel/processor/enricher/PollEnricherAggregateOnExceptionTest.java
@@ -0,0 +1,122 @@
+/**
+ * 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.camel.processor.enricher;
+
+import org.apache.camel.CamelExecutionException;
+import org.apache.camel.ContextTestSupport;
+import org.apache.camel.Exchange;
+import org.apache.camel.Processor;
+import org.apache.camel.builder.RouteBuilder;
+import org.apache.camel.component.mock.MockEndpoint;
+import org.apache.camel.processor.aggregate.AggregationStrategy;
+
+/**
+ * @version 
+ */
+public class PollEnricherAggregateOnExceptionTest extends ContextTestSupport {
+
+    public void testEnrichTrueOk() throws Exception {
+        template.sendBody("seda:foo", "Hello World");
+
+        MockEndpoint mock = getMockEndpoint("mock:result");
+        mock.expectedBodiesReceived("Hello World");
+
+        template.sendBody("direct:start", "World");
+
+        assertMockEndpointsSatisfied();
+    }
+
+    public void testEnrichTrueKaboom() throws Exception {
+        template.send("seda:foo", new Processor() {
+            @Override
+            public void process(Exchange exchange) throws Exception {
+                exchange.setException(new IllegalArgumentException("I cannot do this"));
+            }
+        });
+
+        MockEndpoint mock = getMockEndpoint("mock:result");
+        mock.expectedBodiesReceived("I cannot do this");
+
+        template.sendBody("direct:start", "Kaboom");
+
+        assertMockEndpointsSatisfied();
+    }
+
+    public void testEnrichFalseOk() throws Exception {
+        template.sendBody("seda:foo", "Hello World");
+
+        MockEndpoint mock = getMockEndpoint("mock:result");
+        mock.expectedBodiesReceived("Hello World");
+
+        template.sendBody("direct:start2", "World");
+
+        assertMockEndpointsSatisfied();
+    }
+
+    public void testEnrichFalseKaboom() throws Exception {
+        template.send("seda:foo", new Processor() {
+            @Override
+            public void process(Exchange exchange) throws Exception {
+                exchange.setException(new IllegalArgumentException("I cannot do this"));
+            }
+        });
+
+        MockEndpoint mock = getMockEndpoint("mock:result");
+        mock.expectedMessageCount(0);
+
+        try {
+            template.sendBody("direct:start2", "Kaboom");
+            fail("Should have thrown exception");
+        } catch (CamelExecutionException e) {
+            assertIsInstanceOf(IllegalArgumentException.class, e.getCause());
+            assertEquals("I cannot do this", e.getCause().getMessage());
+        }
+
+        assertMockEndpointsSatisfied();
+    }
+
+    @Override
+    protected RouteBuilder createRouteBuilder() throws Exception {
+        return new RouteBuilder() {
+            @Override
+            public void configure() throws Exception {
+                from("direct:start")
+                    .pollEnrich("seda:foo", 5000, new MyAggregationStrategy(), true)
+                    .to("mock:result");
+
+                from("direct:start2")
+                    .pollEnrich("seda:foo", 5000, new MyAggregationStrategy(), false)
+                    .to("mock:result");
+            }
+        };
+    }
+
+    private class MyAggregationStrategy implements AggregationStrategy {
+
+        @Override
+        public Exchange aggregate(Exchange oldExchange, Exchange newExchange) {
+            if (newExchange.getException() != null) {
+                oldExchange.getIn().setBody(newExchange.getException().getMessage());
+                return oldExchange;
+            }
+
+            // replace body
+            oldExchange.getIn().setBody(newExchange.getIn().getBody());
+            return oldExchange;
+        }
+    }
+}


[3/3] git commit: CAMEL-7581: Added aggregateOnException option to enrich/pollEnrich so end users can handle exception in their aggregte method.

Posted by da...@apache.org.
CAMEL-7581: Added aggregateOnException option to enrich/pollEnrich so end users can handle exception in their aggregte method.


Project: http://git-wip-us.apache.org/repos/asf/camel/repo
Commit: http://git-wip-us.apache.org/repos/asf/camel/commit/4ffc5e25
Tree: http://git-wip-us.apache.org/repos/asf/camel/tree/4ffc5e25
Diff: http://git-wip-us.apache.org/repos/asf/camel/diff/4ffc5e25

Branch: refs/heads/master
Commit: 4ffc5e258c0d0e3ebba2bfe08c0c48416e4099a0
Parents: 5ad6cb8
Author: Claus Ibsen <da...@apache.org>
Authored: Mon Sep 1 11:18:06 2014 +0200
Committer: Claus Ibsen <da...@apache.org>
Committed: Mon Sep 1 11:18:06 2014 +0200

----------------------------------------------------------------------
 .../EnricherAggregateOnExceptionTest.java       | 25 ++++++-----
 .../scala/org/apache/camel/scala/dsl/DSL.scala  |  3 ++
 .../camel/scala/dsl/SAbstractDefinition.scala   |  3 ++
 .../camel/scala/dsl/builder/RouteBuilder.scala  |  3 ++
 .../dsl/SEnricherAggregateOnExceptionTest.scala | 41 +++++++++++++++++
 .../SpringEnricherAggregateOnExceptionTest.java | 30 +++++++++++++
 .../SpringEnricherAggregateOnExceptionTest.xml  | 47 ++++++++++++++++++++
 7 files changed, 141 insertions(+), 11 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/camel/blob/4ffc5e25/camel-core/src/test/java/org/apache/camel/processor/enricher/EnricherAggregateOnExceptionTest.java
----------------------------------------------------------------------
diff --git a/camel-core/src/test/java/org/apache/camel/processor/enricher/EnricherAggregateOnExceptionTest.java b/camel-core/src/test/java/org/apache/camel/processor/enricher/EnricherAggregateOnExceptionTest.java
index 4353a66..844b598 100644
--- a/camel-core/src/test/java/org/apache/camel/processor/enricher/EnricherAggregateOnExceptionTest.java
+++ b/camel-core/src/test/java/org/apache/camel/processor/enricher/EnricherAggregateOnExceptionTest.java
@@ -85,21 +85,24 @@ public class EnricherAggregateOnExceptionTest extends ContextTestSupport {
                     .to("mock:result");
 
                 from("direct:foo")
-                    .process(new Processor() {
-                        @Override
-                        public void process(Exchange exchange) throws Exception {
-                            String body = exchange.getIn().getBody(String.class);
-                            if (body.startsWith("Kaboom")) {
-                                throw new IllegalArgumentException("I cannot do this");
-                            }
-                            exchange.getIn().setBody("Hello " + body);
-                        }
-                    });
+                    .process(new MyProcessor());
             }
         };
     }
 
-    private class MyAggregationStrategy implements AggregationStrategy {
+    public static class MyProcessor implements Processor {
+
+        @Override
+        public void process(Exchange exchange) throws Exception {
+            String body = exchange.getIn().getBody(String.class);
+            if (body.startsWith("Kaboom")) {
+                throw new IllegalArgumentException("I cannot do this");
+            }
+            exchange.getIn().setBody("Hello " + body);
+        }
+    }
+
+    public static class MyAggregationStrategy implements AggregationStrategy {
 
         @Override
         public Exchange aggregate(Exchange oldExchange, Exchange newExchange) {

http://git-wip-us.apache.org/repos/asf/camel/blob/4ffc5e25/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/DSL.scala
----------------------------------------------------------------------
diff --git a/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/DSL.scala b/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/DSL.scala
index a94416c..9900246 100644
--- a/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/DSL.scala
+++ b/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/DSL.scala
@@ -43,6 +43,7 @@ trait DSL {
   def dynamicRouter(expression: Exchange => Any) : DSL
 
   def enrich(uri:String, strategy: AggregationStrategy) : DSL
+  def enrich(uri:String, strategy: AggregationStrategy, aggregateOnException: Boolean) : DSL
 
   def filter(predicate: Exchange => Any) : SFilterDefinition
 
@@ -71,6 +72,7 @@ trait DSL {
   def pipeline : SPipelineDefinition
   def policy(policy: Policy) : DSL
   def pollEnrich(uri: String, strategy: AggregationStrategy = null, timeout: Long = 0) : DSL
+  def pollEnrich(uri: String, strategy: AggregationStrategy, timeout: Long, aggregateOnException: Boolean) : DSL
   def process(function: Exchange => Unit) : DSL
   def process(processor: Processor) : DSL
 
@@ -85,6 +87,7 @@ trait DSL {
   def setBody(expression: Exchange => Any) : DSL
   def setFaultBody(expression: Exchange => Any) : DSL
   def setHeader(header: String, expression: Exchange => Any) : DSL
+  def setExchangePattern(mep: ExchangePattern) : DSL
   def setProperty(header: String, expression: Exchange => Any) : DSL
   def sort[T](expression: Exchange => Any, comparator: Comparator[T] = null) : DSL
   def split(expression: Exchange => Any) : SSplitDefinition

http://git-wip-us.apache.org/repos/asf/camel/blob/4ffc5e25/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/SAbstractDefinition.scala
----------------------------------------------------------------------
diff --git a/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/SAbstractDefinition.scala b/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/SAbstractDefinition.scala
index 16005c5..275b6d0 100644
--- a/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/SAbstractDefinition.scala
+++ b/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/SAbstractDefinition.scala
@@ -71,6 +71,7 @@ abstract class SAbstractDefinition[P <: ProcessorDefinition[_]] extends DSL with
   def dynamicRouter(expression: Exchange => Any) = wrap(target.dynamicRouter(expression))
 
   def enrich(uri: String, strategy: AggregationStrategy) = wrap(target.enrich(uri, strategy))
+  def enrich(uri: String, strategy: AggregationStrategy, aggregateOnException: Boolean) = wrap(target.enrich(uri, strategy, aggregateOnException))
 
   def filter(predicate: Exchange => Any) = SFilterDefinition(target.filter(predicateBuilder(predicate)))
 
@@ -110,6 +111,8 @@ abstract class SAbstractDefinition[P <: ProcessorDefinition[_]] extends DSL with
   def policy(policy: Policy) = wrap(target.policy(policy))
   def pollEnrich(uri: String, strategy: AggregationStrategy = null, timeout: Long = -1) =
     wrap(target.pollEnrich(uri, timeout, strategy))
+  def pollEnrich(uri: String, strategy: AggregationStrategy, timeout: Long, aggregateOnException: Boolean) =
+    wrap(target.pollEnrich(uri, timeout, strategy, aggregateOnException))
   def process(function: Exchange => Unit) = wrap(target.process(new ScalaProcessor(function)))
   def process(processor: Processor) = wrap(target.process(processor))
 

http://git-wip-us.apache.org/repos/asf/camel/blob/4ffc5e25/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/builder/RouteBuilder.scala
----------------------------------------------------------------------
diff --git a/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/builder/RouteBuilder.scala b/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/builder/RouteBuilder.scala
index dead968..5d68b63 100644
--- a/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/builder/RouteBuilder.scala
+++ b/components/camel-scala/src/main/scala/org/apache/camel/scala/dsl/builder/RouteBuilder.scala
@@ -110,6 +110,7 @@ class RouteBuilder extends Preamble with DSL with RoutesBuilder with Languages w
   def dynamicRouter(expression: Exchange => Any) = stack.top.dynamicRouter(expression)
 
   def enrich(uri: String, strategy: AggregationStrategy) = stack.top.enrich(uri, strategy)
+  def enrich(uri: String, strategy: AggregationStrategy, aggregateOnException: Boolean) = stack.top.enrich(uri, strategy, aggregateOnException)
   def errorHandler(error: ErrorHandlerBuilder) { builder.setErrorHandlerBuilder(error) }
   def deadLetterChannel(uri: String) = {
     val dlc = new DeadLetterChannelBuilder
@@ -163,6 +164,7 @@ class RouteBuilder extends Preamble with DSL with RoutesBuilder with Languages w
 
   def pipeline = stack.top.pipeline
   def pollEnrich(uri: String, strategy: AggregationStrategy = null, timeout: Long = 0) = stack.top.pollEnrich(uri, strategy, timeout)
+  def pollEnrich(uri: String, strategy: AggregationStrategy, timeout: Long, aggregateOnException: Boolean) = stack.top.pollEnrich(uri, strategy, timeout, aggregateOnException)
   def policy(policy: Policy) = stack.top.policy(policy)
   def process(function: Exchange => Unit) = stack.top.process(function)
   def process(processor: Processor) = stack.top.process(processor)
@@ -178,6 +180,7 @@ class RouteBuilder extends Preamble with DSL with RoutesBuilder with Languages w
   def setBody(expression : Exchange => Any) = stack.top.setBody(expression)
   def setFaultBody(expression: Exchange => Any) = stack.top.setFaultBody(expression)
   def setHeader(name: String, expression: Exchange => Any) = stack.top.setHeader(name, expression)
+  def setExchangePattern(mep: ExchangePattern) = stack.top.setExchangePattern(mep)
   def setProperty(name: String, expression: Exchange => Any) = stack.top.setProperty(name, expression)
   def sort[T](expression: (Exchange) => Any, comparator: Comparator[T] = null) = stack.top.sort(expression, comparator)
   def split(expression: Exchange => Any) = stack.top.split(expression)

http://git-wip-us.apache.org/repos/asf/camel/blob/4ffc5e25/components/camel-scala/src/test/scala/org/apache/camel/scala/dsl/SEnricherAggregateOnExceptionTest.scala
----------------------------------------------------------------------
diff --git a/components/camel-scala/src/test/scala/org/apache/camel/scala/dsl/SEnricherAggregateOnExceptionTest.scala b/components/camel-scala/src/test/scala/org/apache/camel/scala/dsl/SEnricherAggregateOnExceptionTest.scala
new file mode 100644
index 0000000..e8d4a8b
--- /dev/null
+++ b/components/camel-scala/src/test/scala/org/apache/camel/scala/dsl/SEnricherAggregateOnExceptionTest.scala
@@ -0,0 +1,41 @@
+/**
+ * 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.camel.scala.dsl
+
+import org.apache.camel.scala.dsl.builder.{RouteBuilderSupport, RouteBuilder}
+import org.apache.camel.processor.enricher.EnricherAggregateOnExceptionTest
+
+class SEnricherAggregateOnExceptionTest extends EnricherAggregateOnExceptionTest with RouteBuilderSupport {
+
+  override def createRouteBuilder = new RouteBuilder {
+    "direct:start" ==> {
+      enrich("direct:foo", new EnricherAggregateOnExceptionTest.MyAggregationStrategy(), aggregateOnException = true)
+      to("mock:result")
+    }
+
+    "direct:start2" ==> {
+      enrich("direct:foo", new EnricherAggregateOnExceptionTest.MyAggregationStrategy(), aggregateOnException = false)
+      to("mock:result")
+    }
+
+    "direct:foo" ==> {
+      process(new EnricherAggregateOnExceptionTest.MyProcessor())
+    }
+
+  }
+
+}
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/camel/blob/4ffc5e25/components/camel-spring/src/test/java/org/apache/camel/spring/processor/SpringEnricherAggregateOnExceptionTest.java
----------------------------------------------------------------------
diff --git a/components/camel-spring/src/test/java/org/apache/camel/spring/processor/SpringEnricherAggregateOnExceptionTest.java b/components/camel-spring/src/test/java/org/apache/camel/spring/processor/SpringEnricherAggregateOnExceptionTest.java
new file mode 100644
index 0000000..d585d31
--- /dev/null
+++ b/components/camel-spring/src/test/java/org/apache/camel/spring/processor/SpringEnricherAggregateOnExceptionTest.java
@@ -0,0 +1,30 @@
+/**
+ * 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.camel.spring.processor;
+
+import org.apache.camel.CamelContext;
+import org.apache.camel.processor.enricher.EnricherAggregateOnExceptionTest;
+
+import static org.apache.camel.spring.processor.SpringTestHelper.createSpringCamelContext;
+
+public class SpringEnricherAggregateOnExceptionTest extends EnricherAggregateOnExceptionTest {
+
+    protected CamelContext createCamelContext() throws Exception {
+        return createSpringCamelContext(this, "org/apache/camel/spring/processor/SpringEnricherAggregateOnExceptionTest.xml");
+    }
+
+}

http://git-wip-us.apache.org/repos/asf/camel/blob/4ffc5e25/components/camel-spring/src/test/resources/org/apache/camel/spring/processor/SpringEnricherAggregateOnExceptionTest.xml
----------------------------------------------------------------------
diff --git a/components/camel-spring/src/test/resources/org/apache/camel/spring/processor/SpringEnricherAggregateOnExceptionTest.xml b/components/camel-spring/src/test/resources/org/apache/camel/spring/processor/SpringEnricherAggregateOnExceptionTest.xml
new file mode 100644
index 0000000..8084d3a
--- /dev/null
+++ b/components/camel-spring/src/test/resources/org/apache/camel/spring/processor/SpringEnricherAggregateOnExceptionTest.xml
@@ -0,0 +1,47 @@
+<?xml version="1.0" encoding="UTF-8"?>
+<!--
+    Licensed to the Apache Software Foundation (ASF) under one or more
+    contributor license agreements.  See the NOTICE file distributed with
+    this work for additional information regarding copyright ownership.
+    The ASF licenses this file to You under the Apache License, Version 2.0
+    (the "License"); you may not use this file except in compliance with
+    the License.  You may obtain a copy of the License at
+
+    http://www.apache.org/licenses/LICENSE-2.0
+
+    Unless required by applicable law or agreed to in writing, software
+    distributed under the License is distributed on an "AS IS" BASIS,
+    WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+    See the License for the specific language governing permissions and
+    limitations under the License.
+-->
+<beans xmlns="http://www.springframework.org/schema/beans"
+       xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
+       xsi:schemaLocation="
+       http://www.springframework.org/schema/beans http://www.springframework.org/schema/beans/spring-beans.xsd
+       http://camel.apache.org/schema/spring http://camel.apache.org/schema/spring/camel-spring.xsd
+    ">
+
+  <camelContext xmlns="http://camel.apache.org/schema/spring">
+    <route>
+      <from uri="direct:start"/>
+      <enrich uri="direct:foo" strategyRef="myAggregator" aggregateOnException="true"/>
+      <to uri="mock:result"/>
+    </route>
+    <route>
+      <from uri="direct:start2"/>
+      <enrich uri="direct:foo" strategyRef="myAggregator" aggregateOnException="false"/>
+      <to uri="mock:result"/>
+    </route>
+    <route>
+      <from uri="direct:foo"/>
+      <process ref="myProcessor"/>
+    </route>
+  </camelContext>
+
+  <bean id="myProcessor" class="org.apache.camel.processor.enricher.EnricherAggregateOnExceptionTest.MyProcessor"/>
+
+  <bean id="myAggregator"
+        class="org.apache.camel.processor.enricher.EnricherAggregateOnExceptionTest.MyAggregationStrategy"/>
+
+</beans>


[2/3] git commit: CAMEL-7773: Fixed pom.xml

Posted by da...@apache.org.
CAMEL-7773: Fixed pom.xml


Project: http://git-wip-us.apache.org/repos/asf/camel/repo
Commit: http://git-wip-us.apache.org/repos/asf/camel/commit/5ad6cb84
Tree: http://git-wip-us.apache.org/repos/asf/camel/tree/5ad6cb84
Diff: http://git-wip-us.apache.org/repos/asf/camel/diff/5ad6cb84

Branch: refs/heads/master
Commit: 5ad6cb84a4c8797726b3c038d86b9ebd236676c3
Parents: 16e193c
Author: Claus Ibsen <da...@apache.org>
Authored: Mon Sep 1 10:29:49 2014 +0200
Committer: Claus Ibsen <da...@apache.org>
Committed: Mon Sep 1 10:29:49 2014 +0200

----------------------------------------------------------------------
 components/pom.xml | 2 +-
 1 file changed, 1 insertion(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/camel/blob/5ad6cb84/components/pom.xml
----------------------------------------------------------------------
diff --git a/components/pom.xml b/components/pom.xml
index 962c7a6..553f753 100644
--- a/components/pom.xml
+++ b/components/pom.xml
@@ -146,7 +146,7 @@
     <module>camel-netty4</module>
     <module>camel-netty-http</module>
     <module>camel-ognl</module>
-    <module>camel-olingo</module>
+    <module>camel-olingo2</module>
     <module>camel-openshift</module>
     <module>camel-optaplanner</module>
     <module>camel-paxlogging</module>