You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@qpid.apache.org by ta...@apache.org on 2018/10/19 18:06:00 UTC

qpid-jms git commit: QPIDJMS-421 Cache common message annotation encodings for sends

Repository: qpid-jms
Updated Branches:
  refs/heads/master add459362 -> 0089ed185


QPIDJMS-421 Cache common message annotation encodings for sends

Use a cache to store the msot commonly used message annotations that
are applied to outbound messages in order to reduce overhead of sends
that would otherwise be encoding the same annotations map each time.

Only when additional JMS features like scheduled delivery time are
used will we need to perform a full encode on the MessageAnnotations
section of the outbound message.


Project: http://git-wip-us.apache.org/repos/asf/qpid-jms/repo
Commit: http://git-wip-us.apache.org/repos/asf/qpid-jms/commit/0089ed18
Tree: http://git-wip-us.apache.org/repos/asf/qpid-jms/tree/0089ed18
Diff: http://git-wip-us.apache.org/repos/asf/qpid-jms/diff/0089ed18

Branch: refs/heads/master
Commit: 0089ed185c8e98c3a05e5724becf8091076134b8
Parents: add4593
Author: Timothy Bish <ta...@gmail.com>
Authored: Fri Oct 19 13:48:03 2018 -0400
Committer: Timothy Bish <ta...@gmail.com>
Committed: Fri Oct 19 13:49:41 2018 -0400

----------------------------------------------------------------------
 .../jms/provider/amqp/message/AmqpCodec.java    |  95 ++++++++++++-
 .../amqp/message/AmqpDestinationHelper.java     |  55 +++----
 .../amqp/message/AmqpJmsMessageFacade.java      |  18 ++-
 .../provider/amqp/message/AmqpCodecTest.java    | 142 ++++++++++++++++++-
 .../amqp/message/AmqpDestinationHelperTest.java |  34 +++--
 .../message/AmqpJmsBytesMessageFacadeTest.java  |  12 +-
 .../message/AmqpJmsMapMessageFacadeTest.java    |  11 +-
 .../amqp/message/AmqpJmsMessageFacadeTest.java  |   9 +-
 .../message/AmqpJmsObjectMessageFacadeTest.java |  11 +-
 .../message/AmqpJmsStreamMessageFacadeTest.java |  14 +-
 .../message/AmqpJmsTextMessageFacadeTest.java   |  14 +-
 11 files changed, 313 insertions(+), 102 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/qpid-jms/blob/0089ed18/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpCodec.java
----------------------------------------------------------------------
diff --git a/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpCodec.java b/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpCodec.java
index 733294f..fc68747 100644
--- a/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpCodec.java
+++ b/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpCodec.java
@@ -28,8 +28,13 @@ import static org.apache.qpid.jms.provider.amqp.message.AmqpMessageSupport.SERIA
 import static org.apache.qpid.jms.provider.amqp.message.AmqpMessageSupport.isContentType;
 
 import java.io.IOException;
+import java.nio.ByteBuffer;
 import java.nio.charset.Charset;
 import java.nio.charset.StandardCharsets;
+import java.util.HashMap;
+import java.util.Map;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.ConcurrentMap;
 
 import org.apache.qpid.jms.provider.amqp.AmqpConsumer;
 import org.apache.qpid.jms.util.ContentTypeSupport;
@@ -59,22 +64,33 @@ import io.netty.buffer.ByteBuf;
  */
 public final class AmqpCodec {
 
-    private static class EncoderDecoderPair {
+    private static class EncoderDecoderContext {
         DecoderImpl decoder = new DecoderImpl();
         EncoderImpl encoder = new EncoderImpl(decoder);
         {
             AMQPDefinedTypes.registerAllTypes(decoder, encoder);
         }
+
+        // Store local duplicates from the global cache for thread safety.
+        Map<Integer, ReadableBuffer> messageAnnotationsCache = new HashMap<>();
     }
 
-    private static final ThreadLocal<EncoderDecoderPair> TLS_CODEC = new ThreadLocal<EncoderDecoderPair>() {
+    private static final ThreadLocal<EncoderDecoderContext> TLS_CODEC = new ThreadLocal<EncoderDecoderContext>() {
         @Override
-        protected EncoderDecoderPair initialValue() {
-            return new EncoderDecoderPair();
+        protected EncoderDecoderContext initialValue() {
+            return new EncoderDecoderContext();
         }
     };
 
     /**
+     * Static cache for all cached MessageAnnotation data which is used to populate the
+     * duplicate values stored in the TLS Encoder Decoder contexts.  This Map instance must
+     * be thread safe as many different producers on different threads can be passing data
+     * through this codec and accessing the cache if a TLS duplicate isn't populated yet.
+     */
+    private static ConcurrentMap<Integer, ReadableBuffer> GLOBAL_ANNOTATIONS_CACHE = new ConcurrentHashMap<>();
+
+    /**
      * @return a Encoder instance.
      */
     public static EncoderImpl getEncoder() {
@@ -143,9 +159,11 @@ public final class AmqpCodec {
      * @return a buffer containing the wire level representation of the input Message.
      */
     public static ByteBuf encodeMessage(AmqpJmsMessageFacade message) {
+        EncoderDecoderContext context = TLS_CODEC.get();
+
         AmqpWritableBuffer buffer = new AmqpWritableBuffer();
 
-        EncoderImpl encoder = getEncoder();
+        EncoderImpl encoder = context.encoder;
         encoder.setByteBuffer(buffer);
 
         Header header = message.getHeader();
@@ -163,7 +181,13 @@ public final class AmqpCodec {
             encoder.writeObject(deliveryAnnotations);
         }
         if (messageAnnotations != null) {
+            // Ensure annotations contain required message type and destination type data
+            AmqpDestinationHelper.setReplyToAnnotationFromDestination(message.getReplyTo(), messageAnnotations);
+            AmqpDestinationHelper.setToAnnotationFromDestination(message.getDestination(), messageAnnotations);
+            messageAnnotations.getValue().put(AmqpMessageSupport.JMS_MSG_TYPE, message.getJmsMsgType());
             encoder.writeObject(messageAnnotations);
+        } else {
+            buffer.put(getCachedMessageAnnotationsBuffer(message, context));
         }
         if (properties != null) {
             encoder.writeObject(properties);
@@ -183,6 +207,67 @@ public final class AmqpCodec {
         return buffer.getBuffer();
     }
 
+    private static ReadableBuffer getCachedMessageAnnotationsBuffer(AmqpJmsMessageFacade message, EncoderDecoderContext context) {
+        byte msgType = message.getJmsMsgType();
+        byte toType = AmqpDestinationHelper.toTypeAnnotation(message.getDestination());
+        byte replyToType = AmqpDestinationHelper.toTypeAnnotation(message.getReplyTo());
+
+        Integer entryKey = Integer.valueOf((replyToType << 16) | (toType << 8) | msgType);
+
+        ReadableBuffer result = context.messageAnnotationsCache.get(entryKey);
+        if (result == null) {
+            result = populateMessageAnnotationsCacheEntry(message, entryKey, context);
+        }
+
+        return result.rewind();
+    }
+
+    private static ReadableBuffer populateMessageAnnotationsCacheEntry(AmqpJmsMessageFacade message, Integer entryKey, EncoderDecoderContext context) {
+        ReadableBuffer result = GLOBAL_ANNOTATIONS_CACHE.get(entryKey);
+        if (result == null) {
+            MessageAnnotations messageAnnotations = new MessageAnnotations(new HashMap<>());
+
+            // Sets the Reply To annotation which will likely not be present most of the time so do it first
+            // to avoid extra work within the map operations.
+            AmqpDestinationHelper.setReplyToAnnotationFromDestination(message.getReplyTo(), messageAnnotations);
+            // Sets the To value's destination annotation set and a known JMS destination type likely to always
+            // be present but we do allow of edge case of unknown types which won't encode an annotation.
+            AmqpDestinationHelper.setToAnnotationFromDestination(message.getDestination(), messageAnnotations);
+            // Now store the message type which we know will always be present so do it last to ensure
+            // the previous calls don't need to compare anything to this value in the map during add or remove
+            messageAnnotations.getValue().put(AmqpMessageSupport.JMS_MSG_TYPE, message.getJmsMsgType());
+
+            // This is the maximum possible encoding size that could appear for all the possible data we
+            // store in the cached buffer if the codec was to do the worst possible encode of these types.
+            // We could do a custom encoding to make it minimal which would result in a max of 70 bytes.
+            ByteBuffer buffer = ByteBuffer.allocate(124);
+
+            WritableBuffer oldBuffer = context.encoder.getBuffer();
+            context.encoder.setByteBuffer(buffer);
+            context.encoder.writeObject(messageAnnotations);
+            context.encoder.setByteBuffer(oldBuffer);
+
+            buffer.flip();
+
+            result = ReadableBuffer.ByteBufferReader.wrap(buffer);
+
+            // Race on populating the global cache could duplicate work but we should avoid keeping
+            // both copies around in memory.
+            ReadableBuffer previous = GLOBAL_ANNOTATIONS_CACHE.putIfAbsent(entryKey, result);
+            if (previous != null) {
+                result = previous.duplicate();
+            } else {
+                result = result.duplicate();
+            }
+        } else {
+            result = result.duplicate();
+        }
+
+        context.messageAnnotationsCache.put(entryKey, result);
+
+        return result;
+    }
+
     /**
      * Create a new JmsMessage and underlying JmsMessageFacade that represents the proper
      * message type for the incoming AMQP message.

http://git-wip-us.apache.org/repos/asf/qpid-jms/blob/0089ed18/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpDestinationHelper.java
----------------------------------------------------------------------
diff --git a/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpDestinationHelper.java b/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpDestinationHelper.java
index d20b789..3f6dc2c 100644
--- a/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpDestinationHelper.java
+++ b/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpDestinationHelper.java
@@ -26,6 +26,7 @@ import org.apache.qpid.jms.JmsTemporaryTopic;
 import org.apache.qpid.jms.JmsTopic;
 import org.apache.qpid.jms.provider.amqp.AmqpConnection;
 import org.apache.qpid.proton.amqp.Symbol;
+import org.apache.qpid.proton.amqp.messaging.MessageAnnotations;
 
 /**
  * A set of static utility method useful when mapping JmsDestination types to / from the AMQP
@@ -43,7 +44,7 @@ public class AmqpDestinationHelper {
     public static final Symbol TOPIC_CAPABILITY = Symbol.valueOf("topic");
     public static final Symbol TEMP_QUEUE_CAPABILITY = Symbol.valueOf("temporary-queue");
     public static final Symbol TEMP_TOPIC_CAPABILITY = Symbol.valueOf("temporary-topic");
-    private static final byte UNKNOWN_TYPE = -1;
+    public static final byte UNKNOWN_TYPE = -1;
 
     /**
      * Decode the provided To address, type description, and consumer destination
@@ -157,42 +158,46 @@ public class AmqpDestinationHelper {
         return new JmsQueue(address);
     }
 
-    public static byte setToAddressFromDestination(AmqpJmsMessageFacade message, JmsDestination destination) {
-        String address = getDestinationAddress(destination, message.getConnection());
-        byte typeValue = toTypeAnnotation(destination);
-
-        message.setToAddress(address);
+    public static void setToAddressFromDestination(AmqpJmsMessageFacade message, JmsDestination destination) {
+        message.setToAddress(getDestinationAddress(destination, message.getConnection()));
 
-        // Set or clear the new byte type annotation as appropriate
-        if (address == null || typeValue == UNKNOWN_TYPE) {
-            message.removeMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL);
-        } else {
-            message.setMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL, typeValue);
-        }
+        // Clear any previous annotations about destination type, we will add proper annotations on send
+        message.removeMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL);
 
         // Always clear the legacy string type annotation
         message.removeMessageAnnotation(AmqpMessageSupport.LEGACY_TO_TYPE_MSG_ANNOTATION_SYMBOL);
+    }
+
+    public static void setReplyToAddressFromDestination(AmqpJmsMessageFacade message, JmsDestination destination) {
+        message.setReplyToAddress(getDestinationAddress(destination, message.getConnection()));
 
-        return typeValue;
+        // Clear any previous annotations about destination type, we will add proper annotations on send
+        message.removeMessageAnnotation(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL);
+
+        // Always clear the legacy string type annotation
+        message.removeMessageAnnotation(AmqpMessageSupport.LEGACY_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL);
     }
 
-    public static byte setReplyToAddressFromDestination(AmqpJmsMessageFacade message, JmsDestination destination) {
-        String replyToAddress = getDestinationAddress(destination, message.getConnection());
+    public static void setToAnnotationFromDestination(JmsDestination destination, MessageAnnotations annotations) {
         byte typeValue = toTypeAnnotation(destination);
 
-        message.setReplyToAddress(replyToAddress);
-
         // Set or clear the new byte type annotation as appropriate
-        if (replyToAddress == null || typeValue == UNKNOWN_TYPE) {
-            message.removeMessageAnnotation(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL);
+        if (destination == null || typeValue == UNKNOWN_TYPE) {
+            annotations.getValue().remove(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL);
         } else {
-            message.setMessageAnnotation(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL, typeValue);
+            annotations.getValue().put(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL, typeValue);
         }
+    }
 
-        // Always clear the legacy string type annotation
-        message.removeMessageAnnotation(AmqpMessageSupport.LEGACY_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL);
+    public static void setReplyToAnnotationFromDestination(JmsDestination destination, MessageAnnotations annotations) {
+        byte typeValue = toTypeAnnotation(destination);
 
-        return typeValue;
+        // Set or clear the new byte type annotation as appropriate
+        if (destination == null || typeValue == UNKNOWN_TYPE) {
+            annotations.getValue().remove(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL);
+        } else {
+            annotations.getValue().put(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL, typeValue);
+        }
     }
 
     public static String getDestinationAddress(JmsDestination destination, AmqpConnection connection) {
@@ -226,7 +231,7 @@ public class AmqpDestinationHelper {
      * @return the annotation type value, or {@value AmqpDestinationHelper#UNKNOWN_TYPE} if the
      *         supplied destination null or can't be classified
      */
-    private static byte toTypeAnnotation(JmsDestination destination) {
+    static byte toTypeAnnotation(JmsDestination destination) {
         if (destination == null) {
             return UNKNOWN_TYPE;
         }
@@ -266,7 +271,7 @@ public class AmqpDestinationHelper {
         return typeSet;
     }
 
-    private static byte getTypeByte(AmqpJmsMessageFacade message, Symbol annotationName) {
+    static byte getTypeByte(AmqpJmsMessageFacade message, Symbol annotationName) {
         Object typeAnnotation = message.getMessageAnnotation(annotationName);
 
         if (typeAnnotation == null) {

http://git-wip-us.apache.org/repos/asf/qpid-jms/blob/0089ed18/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMessageFacade.java
----------------------------------------------------------------------
diff --git a/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMessageFacade.java b/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMessageFacade.java
index 154b74e..c26fb00 100644
--- a/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMessageFacade.java
+++ b/qpid-jms-client/src/main/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMessageFacade.java
@@ -86,7 +86,6 @@ public class AmqpJmsMessageFacade implements JmsMessageFacade {
     public void initialize(AmqpConnection connection) {
         this.connection = connection;
 
-        setMessageAnnotation(AmqpMessageSupport.JMS_MSG_TYPE, getJmsMsgType());
         setPersistent(true);
         initializeEmptyBody();
     }
@@ -110,6 +109,10 @@ public class AmqpJmsMessageFacade implements JmsMessageFacade {
         if (getMessageAnnotation(JMS_DELIVERY_TIME) == null) {
             syntheticDeliveryTime = getTimestamp();
         }
+
+        // We now know what type of message this is, so remove this so if resent the
+        // annotations can come from the cache if possible.
+        removeMessageAnnotation(AmqpMessageSupport.JMS_MSG_TYPE);
     }
 
     /**
@@ -214,7 +217,6 @@ public class AmqpJmsMessageFacade implements JmsMessageFacade {
 
     @Override
     public void onSend(long producerTtl) throws JMSException {
-
         // Set the ttl field of the Header field if needed, complementing the expiration
         // field of Properties for any peers that only inspect the mutable ttl field.
         long ttl = 0;
@@ -225,8 +227,6 @@ public class AmqpJmsMessageFacade implements JmsMessageFacade {
         }
 
         header.setTimeToLive(ttl);
-
-        setMessageAnnotation(AmqpMessageSupport.JMS_MSG_TYPE, getJmsMsgType());
     }
 
     @Override
@@ -273,22 +273,22 @@ public class AmqpJmsMessageFacade implements JmsMessageFacade {
 
         target.setBody(body);
 
-        if (deliveryAnnotationsMap != null) {
+        if (deliveryAnnotationsMap != null && !deliveryAnnotationsMap.isEmpty()) {
             target.lazyCreateDeliveryAnnotations();
             target.deliveryAnnotationsMap.putAll(deliveryAnnotationsMap);
         }
 
-        if (applicationPropertiesMap != null) {
+        if (applicationPropertiesMap != null && !applicationPropertiesMap.isEmpty()) {
             target.lazyCreateApplicationProperties();
             target.applicationPropertiesMap.putAll(applicationPropertiesMap);
         }
 
-        if (messageAnnotationsMap != null) {
+        if (messageAnnotationsMap != null && !messageAnnotationsMap.isEmpty()) {
             target.lazyCreateMessageAnnotations();
             target.messageAnnotationsMap.putAll(messageAnnotationsMap);
         }
 
-        if (footerMap != null) {
+        if (footerMap != null && !footerMap.isEmpty()) {
             target.lazyCreateFooter();
             target.footerMap.putAll(footerMap);
         }
@@ -614,7 +614,6 @@ public class AmqpJmsMessageFacade implements JmsMessageFacade {
     @Override
     public void setDestination(JmsDestination destination) {
         this.destination = destination;
-        lazyCreateMessageAnnotations();
         AmqpDestinationHelper.setToAddressFromDestination(this, destination);
     }
 
@@ -630,7 +629,6 @@ public class AmqpJmsMessageFacade implements JmsMessageFacade {
     @Override
     public void setReplyTo(JmsDestination replyTo) {
         this.replyTo = replyTo;
-        lazyCreateMessageAnnotations();
         AmqpDestinationHelper.setReplyToAddressFromDestination(this, replyTo);
     }
 

http://git-wip-us.apache.org/repos/asf/qpid-jms/blob/0089ed18/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpCodecTest.java
----------------------------------------------------------------------
diff --git a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpCodecTest.java b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpCodecTest.java
index 9d72d27..472d756 100644
--- a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpCodecTest.java
+++ b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpCodecTest.java
@@ -38,6 +38,11 @@ import java.util.UUID;
 
 import javax.jms.DeliveryMode;
 
+import org.apache.qpid.jms.JmsDestination;
+import org.apache.qpid.jms.JmsQueue;
+import org.apache.qpid.jms.JmsTemporaryQueue;
+import org.apache.qpid.jms.JmsTemporaryTopic;
+import org.apache.qpid.jms.JmsTopic;
 import org.apache.qpid.jms.message.JmsBytesMessage;
 import org.apache.qpid.jms.message.JmsMessage;
 import org.apache.qpid.jms.message.JmsObjectMessage;
@@ -46,6 +51,7 @@ import org.apache.qpid.jms.message.JmsTextMessage;
 import org.apache.qpid.jms.message.facade.JmsMessageFacade;
 import org.apache.qpid.jms.meta.JmsConsumerId;
 import org.apache.qpid.jms.meta.JmsConsumerInfo;
+import org.apache.qpid.jms.provider.amqp.AmqpConnection;
 import org.apache.qpid.jms.provider.amqp.AmqpConsumer;
 import org.apache.qpid.jms.test.QpidJmsTestCase;
 import org.apache.qpid.proton.Proton;
@@ -66,7 +72,9 @@ import org.mockito.Mockito;
 import io.netty.buffer.ByteBuf;
 
 public class AmqpCodecTest extends QpidJmsTestCase {
+
     private AmqpConsumer mockConsumer;
+    private AmqpConnection mockConnection;
 
     @Before
     @Override
@@ -74,6 +82,7 @@ public class AmqpCodecTest extends QpidJmsTestCase {
         super.setUp();
 
         JmsConsumerId consumerId = new JmsConsumerId("ID:MOCK:1", 1, 1);
+        mockConnection = Mockito.mock(AmqpConnection.class);
         mockConsumer = Mockito.mock(AmqpConsumer.class);
         Mockito.when(mockConsumer.getResourceInfo()).thenReturn(new JmsConsumerInfo(consumerId, null));
     }
@@ -400,7 +409,6 @@ public class AmqpCodecTest extends QpidJmsTestCase {
      */
     @Test
     public void testCreateStreamMessageFromMessageTypeAnnotation() throws Exception {
-
         Message message = Proton.message();
 
         Map<Symbol, Object> map = new HashMap<Symbol, Object>();
@@ -876,8 +884,7 @@ public class AmqpCodecTest extends QpidJmsTestCase {
      * @throws Exception if an error occurs during the test.
      */
     @Test
-    public void testCreateObjectMessageMessageFromAmqpSequence() throws Exception
-    {
+    public void testCreateObjectMessageMessageFromAmqpSequence() throws Exception {
         Message message = Proton.message();
         List<String> list = new ArrayList<String>();
         message.setBody(new AmqpSequence(list));
@@ -893,4 +900,133 @@ public class AmqpCodecTest extends QpidJmsTestCase {
         AmqpObjectTypeDelegate delegate = ((AmqpJmsObjectMessageFacade) facade).getDelegate();
         assertTrue("Unexpected delegate type: " + delegate, delegate instanceof AmqpTypedObjectDelegate);
     }
+
+    //----- Message Annotation Handling --------------------------------------//
+
+    public void testJMSMessageWithNoToMessageAnnotationValidity() throws Exception {
+        doTestJMSMessageEncodingAddsProperMessageAnnotations(AmqpMessageSupport.JMS_MESSAGE, AmqpDestinationHelper.UNKNOWN_TYPE, AmqpDestinationHelper.UNKNOWN_TYPE);
+    }
+
+    public void testJMSMessageToQueueMessageAnnotationValidity() throws Exception {
+        doTestJMSMessageEncodingAddsProperMessageAnnotations(AmqpMessageSupport.JMS_BYTES_MESSAGE, AmqpDestinationHelper.QUEUE_TYPE, AmqpDestinationHelper.UNKNOWN_TYPE);
+    }
+
+    public void testJMSMessageToTemporaryQueueMessageAnnotationValidity() throws Exception {
+        doTestJMSMessageEncodingAddsProperMessageAnnotations(AmqpMessageSupport.JMS_MAP_MESSAGE, AmqpDestinationHelper.TEMP_QUEUE_TYPE, AmqpDestinationHelper.UNKNOWN_TYPE);
+    }
+
+    public void testJMSMessageToTopicMessageAnnotationValidity() throws Exception {
+        doTestJMSMessageEncodingAddsProperMessageAnnotations(AmqpMessageSupport.JMS_STREAM_MESSAGE, AmqpDestinationHelper.TOPIC_TYPE, AmqpDestinationHelper.UNKNOWN_TYPE);
+    }
+
+    public void testJMSMessageToTemporaryTopicMessageAnnotationValidity() throws Exception {
+        doTestJMSMessageEncodingAddsProperMessageAnnotations(AmqpMessageSupport.JMS_TEXT_MESSAGE, AmqpDestinationHelper.TEMP_TOPIC_TYPE, AmqpDestinationHelper.UNKNOWN_TYPE);
+    }
+
+    public void testJMSMessageToQueueWithReplyToMessageAnnotationValidity() throws Exception {
+        doTestJMSMessageEncodingAddsProperMessageAnnotations(AmqpMessageSupport.JMS_OBJECT_MESSAGE, AmqpDestinationHelper.QUEUE_TYPE, AmqpDestinationHelper.TEMP_TOPIC_TYPE);
+    }
+
+    public void testJMSMessageToTemporaryQueueWithReplyToMessageAnnotationValidity() throws Exception {
+        doTestJMSMessageEncodingAddsProperMessageAnnotations(AmqpMessageSupport.JMS_TEXT_MESSAGE, AmqpDestinationHelper.TEMP_QUEUE_TYPE, AmqpDestinationHelper.TOPIC_TYPE);
+    }
+
+    public void testJMSMessageToTopicWithReplyToMessageAnnotationValidity() throws Exception {
+        doTestJMSMessageEncodingAddsProperMessageAnnotations(AmqpMessageSupport.JMS_STREAM_MESSAGE, AmqpDestinationHelper.TOPIC_TYPE, AmqpDestinationHelper.TEMP_QUEUE_TYPE);
+    }
+
+    public void testJMSMessageToTemporaryTopicWithReplyToMessageAnnotationValidity() throws Exception {
+        doTestJMSMessageEncodingAddsProperMessageAnnotations(AmqpMessageSupport.JMS_MESSAGE, AmqpDestinationHelper.TEMP_TOPIC_TYPE, AmqpDestinationHelper.QUEUE_TYPE);
+    }
+
+    private void doTestJMSMessageEncodingAddsProperMessageAnnotations(byte msgType, byte toType, byte replyToType) throws Exception {
+        final AmqpJmsMessageFacade message = createMessageFacadeFromTypeId(msgType);
+        final JmsDestination to = createDestinationFromTypeId(toType);
+        final JmsDestination replyTo = createDestinationFromTypeId(replyToType);
+
+        message.setDestination(to);
+        message.setReplyTo(replyTo);
+
+        // Allows the code to run through what should be cached in the TLS portion of the codec
+        // and not be using the globally cached bits, this checks that nothing NPEs or otherwise
+        // fails and should show in test coverage that the cache fill + cache use is exercised.
+        for (int i = 0; i <= 2; ++i) {
+            MessageImpl amqpMessage = (MessageImpl) AmqpMessageSupport.decodeMessage(AmqpCodec.encodeMessage(message));
+
+            MessageAnnotations messageAnnotations = amqpMessage.getMessageAnnotations();
+            assertNotNull(messageAnnotations);
+            assertNotNull(messageAnnotations.getValue());
+
+            Map<Symbol, Object> messageAnnotationsMap = messageAnnotations.getValue();
+
+            assertTrue(messageAnnotationsMap.containsKey(AmqpMessageSupport.JMS_MSG_TYPE));
+            if (toType != AmqpDestinationHelper.UNKNOWN_TYPE) {
+                assertTrue(messageAnnotationsMap.containsKey(AmqpDestinationHelper.JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL));
+                assertEquals(toType, messageAnnotationsMap.get(AmqpDestinationHelper.JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL));
+            } else {
+                assertFalse(messageAnnotationsMap.containsKey(AmqpDestinationHelper.JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL));
+            }
+            if (replyToType != AmqpDestinationHelper.UNKNOWN_TYPE) {
+                assertTrue(messageAnnotationsMap.containsKey(AmqpDestinationHelper.JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL));
+                assertEquals(replyToType, messageAnnotationsMap.get(AmqpDestinationHelper.JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL));
+            } else {
+                assertFalse(messageAnnotationsMap.containsKey(AmqpDestinationHelper.JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL));
+            }
+        }
+    }
+
+    private JmsDestination createDestinationFromTypeId(byte destinationType) {
+        final JmsDestination destination;
+        switch (destinationType) {
+            case AmqpDestinationHelper.QUEUE_TYPE:
+                destination = new JmsQueue("test");
+                break;
+            case AmqpDestinationHelper.TOPIC_TYPE:
+                destination = new JmsTopic("test");
+                break;
+            case AmqpDestinationHelper.TEMP_QUEUE_TYPE:
+                destination = new JmsTemporaryQueue("test");
+                break;
+            case AmqpDestinationHelper.TEMP_TOPIC_TYPE:
+                destination = new JmsTemporaryTopic("test");
+                break;
+            case AmqpDestinationHelper.UNKNOWN_TYPE:
+                destination = null;
+                break;
+            default:
+                throw new IllegalArgumentException("Unknown JMS Destination Type");
+        }
+
+        return destination;
+    }
+
+    private AmqpJmsMessageFacade createMessageFacadeFromTypeId(byte msgType) {
+        final AmqpJmsMessageFacade message;
+        switch (msgType) {
+            case AmqpMessageSupport.JMS_MESSAGE:
+                message = new AmqpJmsMessageFacade();
+                break;
+            case AmqpMessageSupport.JMS_BYTES_MESSAGE:
+                message = new AmqpJmsBytesMessageFacade();
+                break;
+            case AmqpMessageSupport.JMS_MAP_MESSAGE:
+                message = new AmqpJmsMapMessageFacade();
+                break;
+            case AmqpMessageSupport.JMS_OBJECT_MESSAGE:
+                message = new AmqpJmsObjectMessageFacade();
+                break;
+            case AmqpMessageSupport.JMS_STREAM_MESSAGE:
+                message = new AmqpJmsStreamMessageFacade();
+                break;
+            case AmqpMessageSupport.JMS_TEXT_MESSAGE:
+                message = new AmqpJmsTextMessageFacade();
+                break;
+            default:
+                throw new IllegalArgumentException("Unknown JMS Message Type");
+        }
+
+        message.initialize(mockConnection);
+
+        return message;
+    }
 }

http://git-wip-us.apache.org/repos/asf/qpid-jms/blob/0089ed18/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpDestinationHelperTest.java
----------------------------------------------------------------------
diff --git a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpDestinationHelperTest.java b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpDestinationHelperTest.java
index 893a637..8d63ab2 100644
--- a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpDestinationHelperTest.java
+++ b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpDestinationHelperTest.java
@@ -735,7 +735,9 @@ public class AmqpDestinationHelperTest {
         AmqpDestinationHelper.setToAddressFromDestination(message, destination);
 
         Mockito.verify(message).setToAddress(testAddress);
-        Mockito.verify(message).setMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL, QUEUE_TYPE);
+        Mockito.verify(message).removeMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL);
+
+        assertNull(message.getMessageAnnotations());
     }
 
     @Test
@@ -749,7 +751,9 @@ public class AmqpDestinationHelperTest {
         AmqpDestinationHelper.setToAddressFromDestination(message, destination);
 
         Mockito.verify(message).setToAddress(testAddress);
-        Mockito.verify(message).setMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL, TOPIC_TYPE);
+        Mockito.verify(message).removeMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL);
+
+        assertNull(message.getMessageAnnotations());
     }
 
     @Test
@@ -761,7 +765,9 @@ public class AmqpDestinationHelperTest {
         AmqpDestinationHelper.setToAddressFromDestination(message, destination);
 
         Mockito.verify(message).setToAddress(testAddress);
-        Mockito.verify(message).setMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL, TEMP_QUEUE_TYPE);
+        Mockito.verify(message).removeMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL);
+
+        assertNull(message.getMessageAnnotations());
     }
 
     @Test
@@ -773,7 +779,9 @@ public class AmqpDestinationHelperTest {
         AmqpDestinationHelper.setToAddressFromDestination(message, destination);
 
         Mockito.verify(message).setToAddress(testAddress);
-        Mockito.verify(message).setMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL, TEMP_TOPIC_TYPE);
+        Mockito.verify(message).removeMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL);
+
+        assertNull(message.getMessageAnnotations());
     }
 
     @Test
@@ -787,6 +795,8 @@ public class AmqpDestinationHelperTest {
 
         Mockito.verify(message).setToAddress(testAddress);
         Mockito.verify(message).removeMessageAnnotation(JMS_DEST_TYPE_MSG_ANNOTATION_SYMBOL);
+
+        assertNull(message.getMessageAnnotations());
     }
 
     //========================================================================//
@@ -823,7 +833,9 @@ public class AmqpDestinationHelperTest {
         AmqpDestinationHelper.setReplyToAddressFromDestination(message, destination);
 
         Mockito.verify(message).setReplyToAddress(testAddress);
-        Mockito.verify(message).setMessageAnnotation(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL, QUEUE_TYPE);
+        Mockito.verify(message).removeMessageAnnotation(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL);
+
+        assertNull(message.getMessageAnnotations());
     }
 
     @Test
@@ -851,7 +863,9 @@ public class AmqpDestinationHelperTest {
         AmqpDestinationHelper.setReplyToAddressFromDestination(message, destination);
 
         Mockito.verify(message).setReplyToAddress(testAddress);
-        Mockito.verify(message).setMessageAnnotation(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL, TOPIC_TYPE);
+        Mockito.verify(message).removeMessageAnnotation(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL);
+
+        assertNull(message.getMessageAnnotations());
     }
 
     @Test
@@ -863,7 +877,9 @@ public class AmqpDestinationHelperTest {
         AmqpDestinationHelper.setReplyToAddressFromDestination(message, destination);
 
         Mockito.verify(message).setReplyToAddress(testAddress);
-        Mockito.verify(message).setMessageAnnotation(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL, TEMP_QUEUE_TYPE);
+        Mockito.verify(message).removeMessageAnnotation(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL);
+
+        assertNull(message.getMessageAnnotations());
     }
 
     @Test
@@ -875,7 +891,9 @@ public class AmqpDestinationHelperTest {
         AmqpDestinationHelper.setReplyToAddressFromDestination(message, destination);
 
         Mockito.verify(message).setReplyToAddress(testAddress);
-        Mockito.verify(message).setMessageAnnotation(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL, TEMP_TOPIC_TYPE);
+        Mockito.verify(message).removeMessageAnnotation(JMS_REPLY_TO_TYPE_MSG_ANNOTATION_SYMBOL);
+
+        assertNull(message.getMessageAnnotations());
     }
 
     @Test

http://git-wip-us.apache.org/repos/asf/qpid-jms/blob/0089ed18/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsBytesMessageFacadeTest.java
----------------------------------------------------------------------
diff --git a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsBytesMessageFacadeTest.java b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsBytesMessageFacadeTest.java
index 6d2e53b..e982bed 100644
--- a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsBytesMessageFacadeTest.java
+++ b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsBytesMessageFacadeTest.java
@@ -17,10 +17,10 @@
 package org.apache.qpid.jms.provider.amqp.message;
 
 import static org.apache.qpid.jms.provider.amqp.message.AmqpMessageSupport.JMS_BYTES_MESSAGE;
-import static org.apache.qpid.jms.provider.amqp.message.AmqpMessageSupport.JMS_MSG_TYPE;
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertFalse;
 import static org.junit.Assert.assertNotNull;
+import static org.junit.Assert.assertNull;
 import static org.junit.Assert.assertSame;
 import static org.junit.Assert.assertTrue;
 import static org.junit.Assert.fail;
@@ -31,10 +31,8 @@ import java.io.OutputStream;
 import java.lang.reflect.Field;
 import java.util.ArrayList;
 import java.util.Arrays;
-import java.util.Map;
 
 import org.apache.qpid.proton.amqp.Binary;
-import org.apache.qpid.proton.amqp.Symbol;
 import org.apache.qpid.proton.amqp.messaging.AmqpSequence;
 import org.apache.qpid.proton.amqp.messaging.AmqpValue;
 import org.apache.qpid.proton.amqp.messaging.Data;
@@ -59,17 +57,13 @@ public class AmqpJmsBytesMessageFacadeTest extends AmqpJmsMessageTypesTestCase {
     // ---------- Test initial state of newly created message -----------------//
 
     @Test
-    public void testNewMessageContainsMessageTypeAnnotation() throws Exception {
+    public void testNewMessageDoesNotContainMessageTypeAnnotation() throws Exception {
         AmqpJmsBytesMessageFacade amqpBytesMessageFacade = createNewBytesMessageFacade();
 
         MessageAnnotations annotations = amqpBytesMessageFacade.getMessageAnnotations();
-        Map<Symbol, Object> annotationsMap = annotations.getValue();
 
-        assertNotNull("MessageAnnotations section was not present", annotations);
-        assertNotNull("MessageAnnotations section value was not present", annotationsMap);
+        assertNull("MessageAnnotations section was present", annotations);
 
-        assertTrue("expected message type annotation to be present", annotationsMap.containsKey(JMS_MSG_TYPE));
-        assertEquals("unexpected value for message type annotation value", JMS_BYTES_MESSAGE, annotationsMap.get(JMS_MSG_TYPE));
         assertEquals(JMS_BYTES_MESSAGE, amqpBytesMessageFacade.getJmsMsgType());
     }
 

http://git-wip-us.apache.org/repos/asf/qpid-jms/blob/0089ed18/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMapMessageFacadeTest.java
----------------------------------------------------------------------
diff --git a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMapMessageFacadeTest.java b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMapMessageFacadeTest.java
index 7037ab5..da9130c 100644
--- a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMapMessageFacadeTest.java
+++ b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMapMessageFacadeTest.java
@@ -16,6 +16,7 @@
  */
 package org.apache.qpid.jms.provider.amqp.message;
 
+import static org.apache.qpid.jms.provider.amqp.message.AmqpMessageSupport.JMS_MAP_MESSAGE;
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertFalse;
 import static org.junit.Assert.assertNotNull;
@@ -29,7 +30,6 @@ import java.util.HashMap;
 import java.util.Map;
 
 import org.apache.qpid.proton.amqp.Binary;
-import org.apache.qpid.proton.amqp.Symbol;
 import org.apache.qpid.proton.amqp.messaging.AmqpSequence;
 import org.apache.qpid.proton.amqp.messaging.AmqpValue;
 import org.apache.qpid.proton.amqp.messaging.MessageAnnotations;
@@ -44,17 +44,14 @@ public class AmqpJmsMapMessageFacadeTest extends AmqpJmsMessageTypesTestCase {
     //---------- Test initial state of newly created message -----------------//
 
     @Test
-    public void testNewMessageToSendContainsMessageTypeAnnotation() throws Exception {
+    public void testNewMessageToSendDoesnNotContainMessageTypeAnnotation() throws Exception {
         AmqpJmsMapMessageFacade amqpMapMessageFacade = createNewMapMessageFacade();
 
         MessageAnnotations annotations = amqpMapMessageFacade.getMessageAnnotations();
-        Map<Symbol, Object> annotationsMap = annotations.getValue();
 
-        assertNotNull("MessageAnnotations section was not present", annotations);
-        assertNotNull("MessageAnnotations section value was not present", annotationsMap);
+        assertNull("MessageAnnotations section was present", annotations);
 
-        assertTrue("expected message type annotation to be present", annotationsMap.containsKey(AmqpMessageSupport.JMS_MSG_TYPE));
-        assertEquals("unexpected value for message type annotation value", AmqpMessageSupport.JMS_MAP_MESSAGE, annotationsMap.get(AmqpMessageSupport.JMS_MSG_TYPE));
+        assertEquals(JMS_MAP_MESSAGE, amqpMapMessageFacade.getJmsMsgType());
     }
 
     @Test

http://git-wip-us.apache.org/repos/asf/qpid-jms/blob/0089ed18/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMessageFacadeTest.java
----------------------------------------------------------------------
diff --git a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMessageFacadeTest.java b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMessageFacadeTest.java
index 24a42a4..0cb5937 100644
--- a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMessageFacadeTest.java
+++ b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsMessageFacadeTest.java
@@ -20,6 +20,7 @@
  */
 package org.apache.qpid.jms.provider.amqp.message;
 
+import static org.apache.qpid.jms.provider.amqp.message.AmqpMessageSupport.JMS_MESSAGE;
 import static org.junit.Assert.assertArrayEquals;
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertFalse;
@@ -1661,17 +1662,15 @@ public class AmqpJmsMessageFacadeTest extends AmqpJmsMessageTypesTestCase  {
     public void testNewMessageHasUnderlyingMessageAnnotationsSectionWithTypeAnnotation() {
         AmqpJmsMessageFacade amqpMessageFacade = createNewMessageFacade();;
 
-        assertNotNull(amqpMessageFacade.getMessageAnnotations());
-        Symbol annotationKey = AmqpMessageSupport.JMS_MSG_TYPE;
-        assertEquals(AmqpMessageSupport.JMS_MESSAGE, amqpMessageFacade.getMessageAnnotations().getValue().get(annotationKey));
+        assertNull(amqpMessageFacade.getMessageAnnotations());
+        assertEquals(JMS_MESSAGE, amqpMessageFacade.getJmsMsgType());
     }
 
     @Test
     public void testNewMessageDoesNotHaveUnderlyingMessageAnnotationsSectionWithDeliveryTime() {
         AmqpJmsMessageFacade amqpMessageFacade = createNewMessageFacade();;
 
-        assertNotNull(amqpMessageFacade.getMessageAnnotations());
-        assertNull(amqpMessageFacade.getMessageAnnotations().getValue().get(AmqpMessageSupport.JMS_DELIVERY_TIME));
+        assertNull(amqpMessageFacade.getMessageAnnotations());
     }
 
     @Test

http://git-wip-us.apache.org/repos/asf/qpid-jms/blob/0089ed18/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsObjectMessageFacadeTest.java
----------------------------------------------------------------------
diff --git a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsObjectMessageFacadeTest.java b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsObjectMessageFacadeTest.java
index caa16fc..5a3b9af 100644
--- a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsObjectMessageFacadeTest.java
+++ b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsObjectMessageFacadeTest.java
@@ -16,7 +16,6 @@
  */
 package org.apache.qpid.jms.provider.amqp.message;
 
-import static org.apache.qpid.jms.provider.amqp.message.AmqpMessageSupport.JMS_MSG_TYPE;
 import static org.apache.qpid.jms.provider.amqp.message.AmqpMessageSupport.JMS_OBJECT_MESSAGE;
 import static org.junit.Assert.assertArrayEquals;
 import static org.junit.Assert.assertEquals;
@@ -36,7 +35,6 @@ import java.util.HashMap;
 import java.util.Map;
 
 import org.apache.qpid.proton.amqp.Binary;
-import org.apache.qpid.proton.amqp.Symbol;
 import org.apache.qpid.proton.amqp.messaging.AmqpValue;
 import org.apache.qpid.proton.amqp.messaging.Data;
 import org.apache.qpid.proton.amqp.messaging.MessageAnnotations;
@@ -52,17 +50,12 @@ public class AmqpJmsObjectMessageFacadeTest extends AmqpJmsMessageTypesTestCase
     // ---------- Test initial state of newly created message -----------------//
 
     @Test
-    public void testNewMessageToSendContainsMessageTypeAnnotation() throws Exception {
+    public void testNewMessageToSendDoesNotContainMessageTypeAnnotation() throws Exception {
         AmqpJmsObjectMessageFacade amqpObjectMessageFacade = createNewObjectMessageFacade(false);
 
         MessageAnnotations annotations = amqpObjectMessageFacade.getMessageAnnotations();
-        Map<Symbol, Object> annotationsMap = annotations.getValue();
 
-        assertNotNull("MessageAnnotations section was not present", annotations);
-        assertNotNull("MessageAnnotations section value was not present", annotationsMap);
-
-        assertTrue("expected message type annotation to be present", annotationsMap.containsKey(JMS_MSG_TYPE));
-        assertEquals("unexpected value for message type annotation value", JMS_OBJECT_MESSAGE, annotationsMap.get(JMS_MSG_TYPE));
+        assertNull("MessageAnnotations section was present", annotations);
         assertEquals(JMS_OBJECT_MESSAGE, amqpObjectMessageFacade.getJmsMsgType());
     }
 

http://git-wip-us.apache.org/repos/asf/qpid-jms/blob/0089ed18/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsStreamMessageFacadeTest.java
----------------------------------------------------------------------
diff --git a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsStreamMessageFacadeTest.java b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsStreamMessageFacadeTest.java
index eb8869e..a360e00 100644
--- a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsStreamMessageFacadeTest.java
+++ b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsStreamMessageFacadeTest.java
@@ -20,20 +20,20 @@
  */
 package org.apache.qpid.jms.provider.amqp.message;
 
+import static org.apache.qpid.jms.provider.amqp.message.AmqpMessageSupport.JMS_STREAM_MESSAGE;
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertFalse;
 import static org.junit.Assert.assertNotNull;
+import static org.junit.Assert.assertNull;
 import static org.junit.Assert.assertTrue;
 import static org.junit.Assert.fail;
 
 import java.util.ArrayList;
 import java.util.List;
-import java.util.Map;
 
 import javax.jms.MessageEOFException;
 
 import org.apache.qpid.proton.amqp.Binary;
-import org.apache.qpid.proton.amqp.Symbol;
 import org.apache.qpid.proton.amqp.messaging.AmqpSequence;
 import org.apache.qpid.proton.amqp.messaging.AmqpValue;
 import org.apache.qpid.proton.amqp.messaging.Data;
@@ -51,17 +51,13 @@ public class AmqpJmsStreamMessageFacadeTest extends AmqpJmsMessageTypesTestCase
     }
 
     @Test
-    public void testNewMessageToSendContainsMessageTypeAnnotation() throws Exception {
+    public void testNewMessageToSendDoesnNotContainMessageTypeAnnotation() throws Exception {
         AmqpJmsStreamMessageFacade amqpStreamMessageFacade = createNewStreamMessageFacade();
 
         MessageAnnotations annotations = amqpStreamMessageFacade.getMessageAnnotations();
-        Map<Symbol, Object> annotationsMap = annotations.getValue();
 
-        assertNotNull("MessageAnnotations section was not present", annotations);
-        assertNotNull("MessageAnnotations section value was not present", annotationsMap);
-
-        assertTrue("expected message type annotation to be present", annotationsMap.containsKey(AmqpMessageSupport.JMS_MSG_TYPE));
-        assertEquals("unexpected value for message type annotation value", AmqpMessageSupport.JMS_STREAM_MESSAGE, annotationsMap.get(AmqpMessageSupport.JMS_MSG_TYPE));
+        assertNull("MessageAnnotations section was not present", annotations);
+        assertEquals(JMS_STREAM_MESSAGE, amqpStreamMessageFacade.getJmsMsgType());
     }
 
     @Test

http://git-wip-us.apache.org/repos/asf/qpid-jms/blob/0089ed18/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsTextMessageFacadeTest.java
----------------------------------------------------------------------
diff --git a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsTextMessageFacadeTest.java b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsTextMessageFacadeTest.java
index 3011605..b9f94f6 100644
--- a/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsTextMessageFacadeTest.java
+++ b/qpid-jms-client/src/test/java/org/apache/qpid/jms/provider/amqp/message/AmqpJmsTextMessageFacadeTest.java
@@ -16,7 +16,6 @@
  */
 package org.apache.qpid.jms.provider.amqp.message;
 
-import static org.apache.qpid.jms.provider.amqp.message.AmqpMessageSupport.JMS_MSG_TYPE;
 import static org.apache.qpid.jms.provider.amqp.message.AmqpMessageSupport.JMS_TEXT_MESSAGE;
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertFalse;
@@ -27,13 +26,11 @@ import static org.junit.Assert.fail;
 
 import java.nio.charset.Charset;
 import java.util.ArrayList;
-import java.util.Map;
 
 import javax.jms.JMSException;
 
 import org.apache.qpid.jms.test.testpeer.describedtypes.sections.DataDescribedType;
 import org.apache.qpid.proton.amqp.Binary;
-import org.apache.qpid.proton.amqp.Symbol;
 import org.apache.qpid.proton.amqp.messaging.AmqpSequence;
 import org.apache.qpid.proton.amqp.messaging.AmqpValue;
 import org.apache.qpid.proton.amqp.messaging.Data;
@@ -48,17 +45,10 @@ public class AmqpJmsTextMessageFacadeTest extends AmqpJmsMessageTypesTestCase {
     //---------- Test initial state of newly created message -----------------//
 
     @Test
-    public void testNewMessageToSendContainsMessageTypeAnnotation() throws Exception {
+    public void testNewMessageToSendDoesNotContainMessageTypeAnnotation() throws Exception {
         AmqpJmsTextMessageFacade amqpTextMessageFacade = createNewTextMessageFacade();
 
-        assertNotNull("MessageAnnotations section was not present", amqpTextMessageFacade.getMessageAnnotations());
-
-        Map<Symbol, Object> annotationsMap = amqpTextMessageFacade.getMessageAnnotations().getValue();
-
-        assertNotNull("MessageAnnotations section value was not present", annotationsMap);
-
-        assertTrue("expected message type annotation to be present", annotationsMap.containsKey(JMS_MSG_TYPE));
-        assertEquals("unexpected value for message type annotation value", JMS_TEXT_MESSAGE, annotationsMap.get(JMS_MSG_TYPE));
+        assertNull("MessageAnnotations section was not present", amqpTextMessageFacade.getMessageAnnotations());
         assertEquals(JMS_TEXT_MESSAGE, amqpTextMessageFacade.getJmsMsgType());
     }
 


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@qpid.apache.org
For additional commands, e-mail: commits-help@qpid.apache.org