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 2016/04/18 18:16:13 UTC

[3/4] camel git commit: CAMEL-9879: Circuit Breaker EIP - That is using hystrix

CAMEL-9879: Circuit Breaker EIP - That is using hystrix


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

Branch: refs/heads/hys
Commit: 285cd7ef96260150ff62c3425d34060f063889ef
Parents: 7e2b719
Author: Claus Ibsen <da...@apache.org>
Authored: Mon Apr 18 18:11:28 2016 +0200
Committer: Claus Ibsen <da...@apache.org>
Committed: Mon Apr 18 18:11:28 2016 +0200

----------------------------------------------------------------------
 .../model/HystrixCircuitBreakerDefinition.java  |   9 ++
 .../apache/camel/model/ProcessorDefinition.java |   3 +
 .../hystrix/HystrixDummyProcessor.java          |  28 -----
 .../component/hystrix/HystrixProcessor.java     | 107 +++++++++++++++++++
 .../hystrix/HystrixProcessorCommand.java        |  87 +++++++++++++++
 .../hystrix/HystrixProcessorFactory.java        |  18 +++-
 .../hystrix/HystrixCircuitBreakerTest.java      |   7 +-
 7 files changed, 225 insertions(+), 34 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/camel/blob/285cd7ef/camel-core/src/main/java/org/apache/camel/model/HystrixCircuitBreakerDefinition.java
----------------------------------------------------------------------
diff --git a/camel-core/src/main/java/org/apache/camel/model/HystrixCircuitBreakerDefinition.java b/camel-core/src/main/java/org/apache/camel/model/HystrixCircuitBreakerDefinition.java
index c7b4cd4..3228596 100644
--- a/camel-core/src/main/java/org/apache/camel/model/HystrixCircuitBreakerDefinition.java
+++ b/camel-core/src/main/java/org/apache/camel/model/HystrixCircuitBreakerDefinition.java
@@ -80,6 +80,14 @@ public class HystrixCircuitBreakerDefinition extends OutputDefinition<HystrixCir
         return super.end();
     }
 
+    public FallbackDefinition getFallback() {
+        return fallback;
+    }
+
+    public void setFallback(FallbackDefinition fallback) {
+        this.fallback = fallback;
+    }
+
     // Fluent API
     // -------------------------------------------------------------------------
 
@@ -90,6 +98,7 @@ public class HystrixCircuitBreakerDefinition extends OutputDefinition<HystrixCir
      */
     public HystrixCircuitBreakerDefinition fallback() {
         fallback = new FallbackDefinition();
+        fallback.setParent(this);
         return this;
     }
 

http://git-wip-us.apache.org/repos/asf/camel/blob/285cd7ef/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 aa82e88..2cdfa02 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
@@ -287,6 +287,9 @@ public abstract class ProcessorDefinition<Type extends ProcessorDefinition<Type>
         } else if (defn instanceof OnExceptionDefinition || ProcessorDefinitionHelper.isParentOfType(OnExceptionDefinition.class, defn, true)) {
             log.trace("{} is part of OnException so no error handler is applied", defn);
             // do not use error handler for onExceptions blocks as it will handle errors itself
+        } else if (defn instanceof HystrixCircuitBreakerDefinition || ProcessorDefinitionHelper.isParentOfType(HystrixCircuitBreakerDefinition.class, defn, true)) {
+            log.trace("{} is part of HystrixCircuitBreaker so no error handler is applied", defn);
+            // do not use error handler for hystrixCircuitBreaker blocks as it will handle errors itself
         } else if (defn instanceof MulticastDefinition) {
             // do not use error handler for multicast as it offers fine grained error handlers for its outputs
             // however if share unit of work is enabled, we need to wrap an error handler on the multicast parent

http://git-wip-us.apache.org/repos/asf/camel/blob/285cd7ef/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixDummyProcessor.java
----------------------------------------------------------------------
diff --git a/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixDummyProcessor.java b/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixDummyProcessor.java
deleted file mode 100644
index 48d52f5..0000000
--- a/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixDummyProcessor.java
+++ /dev/null
@@ -1,28 +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
- * <p/>
- * http://www.apache.org/licenses/LICENSE-2.0
- * <p/>
- * 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.component.hystrix;
-
-import org.apache.camel.Exchange;
-import org.apache.camel.Processor;
-
-public class HystrixDummyProcessor implements Processor {
-
-    @Override
-    public void process(Exchange exchange) throws Exception {
-        System.out.println("Dummy processor");
-    }
-}

http://git-wip-us.apache.org/repos/asf/camel/blob/285cd7ef/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessor.java
----------------------------------------------------------------------
diff --git a/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessor.java b/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessor.java
new file mode 100644
index 0000000..6690234
--- /dev/null
+++ b/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessor.java
@@ -0,0 +1,107 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.component.hystrix;
+
+import java.util.ArrayList;
+import java.util.List;
+
+import com.netflix.hystrix.HystrixCommandGroupKey;
+import org.apache.camel.AsyncCallback;
+import org.apache.camel.AsyncProcessor;
+import org.apache.camel.Exchange;
+import org.apache.camel.Navigate;
+import org.apache.camel.Processor;
+import org.apache.camel.spi.IdAware;
+import org.apache.camel.support.ServiceSupport;
+import org.apache.camel.util.AsyncProcessorConverterHelper;
+import org.apache.camel.util.AsyncProcessorHelper;
+
+public class HystrixProcessor extends ServiceSupport implements AsyncProcessor, Navigate<Processor>, org.apache.camel.Traceable, IdAware {
+
+    private String id;
+    private final AsyncProcessor processor;
+    private final AsyncProcessor fallback;
+
+    public HystrixProcessor(String id, Processor processor, Processor fallback) {
+        this.id = id;
+        this.processor = AsyncProcessorConverterHelper.convert(processor);
+        this.fallback = AsyncProcessorConverterHelper.convert(fallback);
+    }
+
+    @Override
+    public String getId() {
+        return id;
+    }
+
+    @Override
+    public void setId(String id) {
+        this.id = id;
+    }
+
+    @Override
+    public String getTraceLabel() {
+        return "hystrixCircuitBreaker";
+    }
+
+    @Override
+    public List<Processor> next() {
+        if (!hasNext()) {
+            return null;
+        }
+        List<Processor> answer = new ArrayList<Processor>();
+        answer.add(processor);
+        if (fallback != null) {
+            answer.add(fallback);
+        }
+        return answer;
+    }
+
+    @Override
+    public boolean hasNext() {
+        return true;
+    }
+
+    @Override
+    public void process(Exchange exchange) throws Exception {
+        AsyncProcessorHelper.process(this, exchange);
+    }
+
+    @Override
+    public boolean process(Exchange exchange, AsyncCallback callback) {
+        HystrixCommandGroupKey key = HystrixCommandGroupKey.Factory.asKey(id);
+
+        HystrixProcessorCommand command = new HystrixProcessorCommand(key, exchange, callback, processor, fallback);
+        try {
+            command.execute();
+        } catch (Throwable e) {
+            exchange.setException(e);
+        }
+
+        callback.done(true);
+        return true;
+    }
+
+    @Override
+    protected void doStart() throws Exception {
+        // noop
+    }
+
+    @Override
+    protected void doStop() throws Exception {
+        // noop
+    }
+}

http://git-wip-us.apache.org/repos/asf/camel/blob/285cd7ef/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessorCommand.java
----------------------------------------------------------------------
diff --git a/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessorCommand.java b/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessorCommand.java
new file mode 100644
index 0000000..39d64d1
--- /dev/null
+++ b/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessorCommand.java
@@ -0,0 +1,87 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.component.hystrix;
+
+import com.netflix.hystrix.HystrixCommand;
+import com.netflix.hystrix.HystrixCommandGroupKey;
+import org.apache.camel.AsyncCallback;
+import org.apache.camel.AsyncProcessor;
+import org.apache.camel.Exchange;
+import org.apache.camel.Processor;
+
+public class HystrixProcessorCommand extends HystrixCommand<Exchange> {
+
+    private final Exchange exchange;
+    private final AsyncCallback callback;
+    private final AsyncProcessor processor;
+    private final AsyncProcessor fallback;
+
+    public HystrixProcessorCommand(HystrixCommandGroupKey group, Exchange exchange, AsyncCallback callback,
+                                   AsyncProcessor processor, AsyncProcessor fallback) {
+        super(group);
+        this.exchange = exchange;
+        this.callback = callback;
+        this.processor = processor;
+        this.fallback = fallback;
+    }
+
+    @Override
+    protected Exchange getFallback() {
+        if (fallback != null) {
+            try {
+                Exception e = exchange.getException();
+                // store the last to endpoint as the failure endpoint
+                if (exchange.getProperty(Exchange.FAILURE_ENDPOINT) == null) {
+                    exchange.setProperty(Exchange.FAILURE_ENDPOINT, exchange.getProperty(Exchange.TO_ENDPOINT));
+                }
+                // give the rest of the pipeline another chance
+                exchange.setProperty(Exchange.EXCEPTION_HANDLED, true);
+                exchange.setProperty(Exchange.EXCEPTION_CAUGHT, e);
+                exchange.setException(null);
+                // and we should not be regarded as exhausted as we are in a try .. catch block
+                exchange.removeProperty(Exchange.REDELIVERY_EXHAUSTED);
+
+                fallback.process(exchange, callback);
+            } catch (Exception e) {
+                exchange.setException(e);
+            } finally {
+                callback.done(true);
+            }
+            return exchange;
+        } else {
+            return null;
+        }
+    }
+
+    @Override
+    protected Exchange run() throws Exception {
+        try {
+            processor.process(exchange, callback);
+        } catch (Exception e) {
+            exchange.setException(e);
+        } finally {
+            callback.done(true);
+        }
+
+        // if we failed then throw an exception
+        if (exchange.getException() != null) {
+            throw exchange.getException();
+        }
+
+        return exchange;
+    }
+}

http://git-wip-us.apache.org/repos/asf/camel/blob/285cd7ef/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessorFactory.java
----------------------------------------------------------------------
diff --git a/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessorFactory.java b/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessorFactory.java
index c0924b1..40b5eb0 100644
--- a/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessorFactory.java
+++ b/components/camel-hystrix/src/main/java/org/apache/camel/component/hystrix/HystrixProcessorFactory.java
@@ -17,6 +17,7 @@
 package org.apache.camel.component.hystrix;
 
 import org.apache.camel.Processor;
+import org.apache.camel.model.HystrixCircuitBreakerDefinition;
 import org.apache.camel.model.ProcessorDefinition;
 import org.apache.camel.spi.ProcessorFactory;
 import org.apache.camel.spi.RouteContext;
@@ -25,11 +26,26 @@ public class HystrixProcessorFactory implements ProcessorFactory {
 
     @Override
     public Processor createChildProcessor(RouteContext routeContext, ProcessorDefinition<?> definition, boolean mandatory) throws Exception {
+        // not in use
         return null;
     }
 
     @Override
     public Processor createProcessor(RouteContext routeContext, ProcessorDefinition<?> definition) throws Exception {
-        return new HystrixDummyProcessor();
+        if (definition instanceof HystrixCircuitBreakerDefinition) {
+            HystrixCircuitBreakerDefinition cb = (HystrixCircuitBreakerDefinition) definition;
+            String id = cb.idOrCreate(routeContext.getCamelContext().getNodeIdFactory());
+
+            // create the regular processor
+            Processor processor = cb.createChildProcessor(routeContext, true);
+            Processor fallback = null;
+            if (cb.getFallback() != null) {
+                fallback = cb.getFallback().createProcessor(routeContext);
+            }
+
+            return new HystrixProcessor(id, processor, fallback);
+        } else {
+            return null;
+        }
     }
 }

http://git-wip-us.apache.org/repos/asf/camel/blob/285cd7ef/components/camel-hystrix/src/test/java/org/apache/camel/component/hystrix/HystrixCircuitBreakerTest.java
----------------------------------------------------------------------
diff --git a/components/camel-hystrix/src/test/java/org/apache/camel/component/hystrix/HystrixCircuitBreakerTest.java b/components/camel-hystrix/src/test/java/org/apache/camel/component/hystrix/HystrixCircuitBreakerTest.java
index d714435..f4c035d 100644
--- a/components/camel-hystrix/src/test/java/org/apache/camel/component/hystrix/HystrixCircuitBreakerTest.java
+++ b/components/camel-hystrix/src/test/java/org/apache/camel/component/hystrix/HystrixCircuitBreakerTest.java
@@ -24,7 +24,7 @@ public class HystrixCircuitBreakerTest extends CamelTestSupport {
 
     @Test
     public void testHystrixCircuitBreaker() throws Exception {
-        getMockEndpoint("mock:result").expectedBodiesReceived("Hello World");
+        getMockEndpoint("mock:result").expectedBodiesReceived("Fallback message");
 
         template.sendBody("direct:start", "Hello World");
 
@@ -38,14 +38,11 @@ public class HystrixCircuitBreakerTest extends CamelTestSupport {
             public void configure() throws Exception {
                 from("direct:start")
                     .hystrixCircuitBreaker()
-                        .to("direct:foo")
+                        .throwException(new IllegalArgumentException("Forced"))
                     .fallback()
                         .transform().constant("Fallback message")
                     .end()
                     .to("mock:result");
-
-                from("direct:foo")
-                        .throwException(new IllegalArgumentException("Forced"));
             }
         };
     }