3434% % * Keeping track of consumers
3535% % * Keeping track of unacknowledged deliveries to consumers
3636% % * Keeping track of publisher confirms
37- % % * Keeping track of mandatory message routing confirmations
38- % % and returns
3937% % * Transaction management
4038% % * Authorisation (enforcing permissions)
4139% % * Publishing trace events if tracing is enabled
143141 % % a list of tags for published messages that were
144142 % % rejected but are yet to be sent to the client
145143 rejected ,
146- % % a dtree used to track oustanding notifications
147- % % for messages published as mandatory
148- mandatory ,
149144 % % same as capabilities in the reader
150145 capabilities ,
151146 % % tracing exchange resource if tracing is enabled,
@@ -469,7 +464,6 @@ init([Channel, ReaderPid, WriterPid, ConnPid, ConnName, Protocol, User, VHost,
469464 unconfirmed = dtree :empty (),
470465 rejected = [],
471466 confirmed = [],
472- mandatory = dtree :empty (),
473467 capabilities = Capabilities ,
474468 trace_state = rabbit_trace :init (VHost ),
475469 consumer_prefetch = Prefetch ,
@@ -502,7 +496,6 @@ prioritise_cast(Msg, _Len, _State) ->
502496 case Msg of
503497 {confirm , _MsgSeqNos , _QPid } -> 5 ;
504498 {reject_publish , _MsgSeqNos , _QPid } -> 5 ;
505- {mandatory_received , _MsgSeqNo , _QPid } -> 5 ;
506499 _ -> 0
507500 end .
508501
@@ -637,10 +630,6 @@ handle_cast({send_drained, CTagCredit}, State = #ch{writer_pid = WriterPid}) ->
637630 || {ConsumerTag , CreditDrained } <- CTagCredit ],
638631 noreply (State );
639632
640- handle_cast ({mandatory_received , MsgSeqNo }, State = # ch {mandatory = Mand }) ->
641- % % NB: don't call noreply/1 since we don't want to send confirms.
642- noreply_coalesce (State # ch {mandatory = dtree :drop (MsgSeqNo , Mand )});
643-
644633handle_cast ({reject_publish , MsgSeqNo , _QPid }, State = # ch {unconfirmed = UC }) ->
645634 % % It does not matter which queue rejected the message,
646635 % % if any queue rejected it - it should not be confirmed.
@@ -1707,17 +1696,13 @@ track_delivering_queue(NoAck, QPid, QName,
17071696 false -> sets :add_element (QRef , DQ )
17081697 end }.
17091698
1710- handle_publishing_queue_down (QPid , Reason , State = # ch {unconfirmed = UC ,
1711- mandatory = Mand })
1699+ handle_publishing_queue_down (QPid , Reason , State = # ch {unconfirmed = UC })
17121700 when ? IS_CLASSIC (QPid ) ->
1713- {MMsgs , Mand1 } = dtree :take (QPid , Mand ),
1714- [basic_return (Msg , State , no_route ) || {_ , Msg } <- MMsgs ],
1715- State1 = State # ch {mandatory = Mand1 },
17161701 case rabbit_misc :is_abnormal_exit (Reason ) of
17171702 true -> {MXs , UC1 } = dtree :take_all (QPid , UC ),
1718- record_rejects (MXs , State1 # ch {unconfirmed = UC1 });
1703+ record_rejects (MXs , State # ch {unconfirmed = UC1 });
17191704 false -> {MXs , UC1 } = dtree :take (QPid , UC ),
1720- record_confirms (MXs , State1 # ch {unconfirmed = UC1 })
1705+ record_confirms (MXs , State # ch {unconfirmed = UC1 })
17211706
17221707 end ;
17231708handle_publishing_queue_down (QPid , _Reason , _State ) when ? IS_QUORUM (QPid ) ->
@@ -1972,7 +1957,7 @@ foreach_per_queue(F, UAL, Acc) ->
19721957
19731958consumer_queue_refs (Consumers ) ->
19741959 lists :usort ([qpid_to_ref (QPid ) || {_Key , {# amqqueue {pid = QPid }, _CParams }}
1975- <- maps :to_list (Consumers )]).
1960+ <- maps :to_list (Consumers )]).
19761961
19771962% % tell the limiter about the number of acks that have been received
19781963% % for messages delivered to subscribed consumers, but not acks for
@@ -2038,11 +2023,11 @@ deliver_to_queues({Delivery = #delivery{message = Message = #basic_message{
20382023 queue_monitors = QMons1 },
20392024 % % NB: the order here is important since basic.returns must be
20402025 % % sent before confirms.
2041- State2 = process_routing_mandatory (Mandatory , AllDeliveredQRefs , MsgSeqNo ,
2042- Message , State1 ),
2043- State3 = process_routing_confirm (Confirm , AllDeliveredQRefs , MsgSeqNo ,
2044- XName , State2 ),
2045- case rabbit_event :stats_level (State3 , # ch .stats_timer ) of
2026+ ok = process_routing_mandatory (Mandatory , AllDeliveredQRefs ,
2027+ Message , State1 ),
2028+ State2 = process_routing_confirm (Confirm , AllDeliveredQRefs , MsgSeqNo ,
2029+ XName , State1 ),
2030+ case rabbit_event :stats_level (State , # ch .stats_timer ) of
20462031 fine ->
20472032 ? INCR_STATS (exchange_stats , XName , 1 , publish ),
20482033 [? INCR_STATS (queue_exchange_stats , {QName , XName }, 1 , publish ) ||
@@ -2051,16 +2036,13 @@ deliver_to_queues({Delivery = #delivery{message = Message = #basic_message{
20512036 _ ->
20522037 ok
20532038 end ,
2054- State3 # ch {queue_states = QueueStates }.
2039+ State2 # ch {queue_states = QueueStates }.
20552040
2056- process_routing_mandatory (false , _ , _ , _ , State ) ->
2057- State ;
2058- process_routing_mandatory (true , [], _ , Msg , State ) ->
2041+ process_routing_mandatory (true , [], Msg , State ) ->
20592042 ok = basic_return (Msg , State , no_route ),
2060- State ;
2061- process_routing_mandatory (true , QRefs , MsgSeqNo , Msg , State ) ->
2062- State # ch {mandatory = dtree :insert (MsgSeqNo , QRefs , Msg ,
2063- State # ch .mandatory )}.
2043+ ok ;
2044+ process_routing_mandatory (_ , _ , _ , _ ) ->
2045+ ok .
20642046
20652047process_routing_confirm (false , _ , _ , _ , State ) ->
20662048 State ;
0 commit comments