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 2019/12/06 09:27:47 UTC

[camel] branch camel-3.0.x updated: CAMEL-14263: camel-avro should use source code generated configurer to avoid reflection configuration.

This is an automated email from the ASF dual-hosted git repository.

davsclaus pushed a commit to branch camel-3.0.x
in repository https://gitbox.apache.org/repos/asf/camel.git


The following commit(s) were added to refs/heads/camel-3.0.x by this push:
     new d699fd5  CAMEL-14263: camel-avro should use source code generated configurer to avoid reflection configuration.
d699fd5 is described below

commit d699fd561f76efe25c7c21ab861c254551b98950
Author: Claus Ibsen <cl...@gmail.com>
AuthorDate: Fri Dec 6 10:26:46 2019 +0100

    CAMEL-14263: camel-avro should use source code generated configurer to avoid reflection configuration.
---
 .../apache/camel/component/avro/AvroComponent.java | 63 ++---------------
 .../camel/component/avro/AvroConfiguration.java    |  3 +-
 .../apache/camel/component/avro/AvroEndpoint.java  | 79 ++++++++++++++++------
 3 files changed, 68 insertions(+), 77 deletions(-)

diff --git a/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroComponent.java b/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroComponent.java
index 8ca0798..b12ed49 100644
--- a/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroComponent.java
+++ b/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroComponent.java
@@ -16,15 +16,11 @@
  */
 package org.apache.camel.component.avro;
 
-import java.lang.reflect.Field;
 import java.net.URI;
-import java.util.Collections;
 import java.util.Map;
 import java.util.concurrent.ConcurrentHashMap;
 import java.util.concurrent.ConcurrentMap;
 
-import org.apache.avro.Protocol;
-import org.apache.avro.reflect.ReflectData;
 import org.apache.camel.CamelContext;
 import org.apache.camel.Endpoint;
 import org.apache.camel.spi.Metadata;
@@ -69,66 +65,21 @@ public class AvroComponent extends DefaultComponent {
         }
 
         URI endpointUri = new URI(URISupport.normalizeUri(remaining));
-        applyToConfiguration(config, endpointUri, parameters);
+        config.parseURI(endpointUri);
 
+        Endpoint answer;
         if (AvroConstants.AVRO_NETTY_TRANSPORT.equals(endpointUri.getScheme())) {
-            return new AvroNettyEndpoint(remaining, this, config);
+            answer = new AvroNettyEndpoint(remaining, this, config);
         } else if (AvroConstants.AVRO_HTTP_TRANSPORT.equals(endpointUri.getScheme())) {
-            return new AvroHttpEndpoint(remaining, this, config);
+            answer = new AvroHttpEndpoint(remaining, this, config);
         } else {
             throw new IllegalArgumentException("Unknown avro scheme. Should use either netty or http.");
         }
+        setProperties(answer, parameters);
+        return answer;
     }
 
-    /**
-     * Applies endpoint parameters to configuration & resolves protocol and other required configuration properties.
-     */
-    private void applyToConfiguration(AvroConfiguration config, URI endpointUri, Map<String, Object> parameters) throws Exception {
-        config.parseURI(endpointUri, parameters, this);
-        setProperties(config, parameters);
-
-        if (config.getProtocol() == null && config.getProtocolClassName() != null) {
-            Class<?> protocolClass = getCamelContext().getClassResolver().resolveClass(config.getProtocolClassName());
-            if (protocolClass != null) {
-                try {
-                    Field f = protocolClass.getField("PROTOCOL");
-                    if (f != null) {
-                        Protocol protocol = (Protocol)f.get(null);
-                        config.setProtocol(protocol);
-                    }
-                } catch (NoSuchFieldException e) {
-                    ReflectData reflectData = ReflectData.get();
-                    config.setProtocol(reflectData.getProtocol(protocolClass));
-                    config.setReflectionProtocol(true);
-                }
-            }
-        }
-
-        if (config.getProtocol() == null) {
-            throw new IllegalArgumentException("Avro configuration does not contain protocol");
-        }
-
-        if (config.getMessageName() != null && !config.getProtocol().getMessages().containsKey(config.getMessageName())) {
-            throw new IllegalArgumentException("Message " + config.getMessageName() + " is not defined in protocol");
-        }
-
-        if (config.isSingleParameter()) {
-            Map<String, Protocol.Message> messageMap = config.getProtocol().getMessages();
-            Iterable<Protocol.Message> messagesToCheck = config.getMessageName() == null 
-                ? messageMap.values() 
-                : Collections.singleton(messageMap.get(config.getMessageName()));
-            for (Protocol.Message message : messagesToCheck) {
-                if (message.getRequest().getFields().size() != 1) {
-                    throw new IllegalArgumentException("Single parameter option can't be used with message "
-                            + message.getName() + " because it has " + message.getRequest().getFields().size()
-                            + " parameters defined"
-                    );
-                }
-            }
-        }
-    }
-    
-    /**
+   /**
      * Registers new responder with uri as key. Registers consumer in responder.
      * In case if responder is already registered by this uri then just
      * registers consumer.
diff --git a/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroConfiguration.java b/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroConfiguration.java
index c68a5f5..e017e79 100644
--- a/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroConfiguration.java
+++ b/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroConfiguration.java
@@ -17,7 +17,6 @@
 package org.apache.camel.component.avro;
 
 import java.net.URI;
-import java.util.Map;
 
 import org.apache.avro.Protocol;
 import org.apache.camel.RuntimeCamelException;
@@ -62,7 +61,7 @@ public class AvroConfiguration implements Cloneable {
         }
     }
 
-    public void parseURI(URI uri, Map<String, Object> parameters, AvroComponent component) throws Exception {
+    public void parseURI(URI uri) throws Exception {
         transport = AvroTransport.valueOf(uri.getScheme());
 
         setHost(uri.getHost());
diff --git a/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroEndpoint.java b/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroEndpoint.java
index 730dfe5..828d8f1 100644
--- a/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroEndpoint.java
+++ b/components/camel-avro/src/main/java/org/apache/camel/component/avro/AvroEndpoint.java
@@ -16,8 +16,13 @@
  */
 package org.apache.camel.component.avro;
 
+import java.lang.reflect.Field;
+import java.util.Collections;
+import java.util.Map;
+
 import org.apache.avro.Protocol;
 import org.apache.avro.Schema;
+import org.apache.avro.reflect.ReflectData;
 import org.apache.camel.AsyncEndpoint;
 import org.apache.camel.Component;
 import org.apache.camel.Consumer;
@@ -37,14 +42,6 @@ public abstract class AvroEndpoint extends DefaultEndpoint implements AsyncEndpo
     @UriParam
     private AvroConfiguration configuration;
 
-    /**
-     * Constructs a fully-initialized DefaultEndpoint instance. This is the
-     * preferred method of constructing an object from Java code (as opposed to
-     * Spring beans, etc.).
-     *
-     * @param endpointUri the full URI used to create this endpoint
-     * @param component   the component that created this endpoint
-     */
     public AvroEndpoint(String endpointUri, Component component, AvroConfiguration configuration) {
         super(endpointUri, component);
         this.configuration = configuration;
@@ -61,23 +58,67 @@ public abstract class AvroEndpoint extends DefaultEndpoint implements AsyncEndpo
         return exchange;
     }
     
-    /**
-     * Creates a new <a
-     * href="http://camel.apache.org/event-driven-consumer.html">Event
-     * Driven Consumer</a> which consumes messages from the endpoint using the
-     * given processor
-     *
-     * @param processor the given processor
-     * @return a newly created consumer
-     * @throws Exception can be thrown
-     */
     @Override
     public Consumer createConsumer(Processor processor) throws Exception {
-        return new AvroConsumer(this, processor);
+        AvroConsumer consumer = new AvroConsumer(this, processor);
+        configureConsumer(consumer);
+        return consumer;
     }
 
     public AvroConfiguration getConfiguration() {
         return configuration;
     }
 
+    @Override
+    protected void doStart() throws Exception {
+        super.doStart();
+
+        validateConfiguration(configuration);
+
+    }
+
+    /**
+     * Validates configuration
+     */
+    private void validateConfiguration(AvroConfiguration config) throws Exception {
+        if (config.getProtocol() == null && config.getProtocolClassName() != null) {
+            Class<?> protocolClass = getCamelContext().getClassResolver().resolveClass(config.getProtocolClassName());
+            if (protocolClass != null) {
+                try {
+                    Field f = protocolClass.getField("PROTOCOL");
+                    if (f != null) {
+                        Protocol protocol = (Protocol)f.get(null);
+                        config.setProtocol(protocol);
+                    }
+                } catch (NoSuchFieldException e) {
+                    ReflectData reflectData = ReflectData.get();
+                    config.setProtocol(reflectData.getProtocol(protocolClass));
+                    config.setReflectionProtocol(true);
+                }
+            }
+        }
+
+        if (config.getProtocol() == null) {
+            throw new IllegalArgumentException("Avro configuration does not contain protocol");
+        }
+
+        if (config.getMessageName() != null && !config.getProtocol().getMessages().containsKey(config.getMessageName())) {
+            throw new IllegalArgumentException("Message " + config.getMessageName() + " is not defined in protocol");
+        }
+
+        if (config.isSingleParameter()) {
+            Map<String, Protocol.Message> messageMap = config.getProtocol().getMessages();
+            Iterable<Protocol.Message> messagesToCheck = config.getMessageName() == null
+                    ? messageMap.values()
+                    : Collections.singleton(messageMap.get(config.getMessageName()));
+            for (Protocol.Message message : messagesToCheck) {
+                if (message.getRequest().getFields().size() != 1) {
+                    throw new IllegalArgumentException("Single parameter option can't be used with message "
+                            + message.getName() + " because it has " + message.getRequest().getFields().size()
+                            + " parameters defined"
+                    );
+                }
+            }
+        }
+    }
 }