Skip to content

Commit 02fc85c

Browse files
committed
rabbit_fifo: Activate next SAC deterministically
This change extends the changes in 2db4843 (making `rabbit_fifo` deterministic). Determining the next active consumer uses a `maps:iterator()` which has undefined ordering. Like calls to `fold/3` or `keys/1` we should ensure ordering of the iterator so that two replicas are guaranteed to find the same consumer.
1 parent ae3fbbc commit 02fc85c

File tree

2 files changed

+51
-41
lines changed

2 files changed

+51
-41
lines changed

deps/rabbit/src/rabbit_fifo.erl

Lines changed: 37 additions & 33 deletions
Original file line numberDiff line numberDiff line change
@@ -416,7 +416,8 @@ apply(#{index := Index,
416416
{State, {dequeue, empty}, Effects}
417417
end
418418
end;
419-
apply(#{index := _Idx} = Meta,
419+
apply(#{index := _Idx,
420+
machine_version := Vsn} = Meta,
420421
#checkout{spec = Spec,
421422
consumer_id = ConsumerId}, State0)
422423
when Spec == cancel orelse
@@ -425,19 +426,19 @@ apply(#{index := _Idx} = Meta,
425426
{ok, ConsumerKey} ->
426427
{State1, Effects1} = activate_next_consumer(
427428
cancel_consumer(Meta, ConsumerKey, State0, [],
428-
Spec)),
429+
Spec), Vsn),
429430
Reply = {ok, consumer_cancel_info(ConsumerKey, State1)},
430431
{State, _, Effects} = checkout(Meta, State0, State1, Effects1),
431432
{State, Reply, Effects};
432433
error ->
433434
{State0, {error, consumer_not_found}, []}
434435
end;
435-
apply(#{index := Idx} = Meta,
436+
apply(#{index := Idx,
437+
machine_version := Vsn} = Meta,
436438
#checkout{spec = Spec0,
437439
meta = ConsumerMeta,
438440
consumer_id = {_, Pid} = ConsumerId}, State0) ->
439-
%% might be better to check machine_version
440-
IsV4 = tuple_size(Spec0) == 2,
441+
IsV4 = Vsn >= 4,
441442
%% normalise spec format
442443
Spec = case Spec0 of
443444
{_, _} ->
@@ -461,7 +462,7 @@ apply(#{index := Idx} = Meta,
461462
end,
462463
{Consumer, State1} = update_consumer(Meta, ConsumerKey, ConsumerId,
463464
ConsumerMeta, Spec, Priority, State0),
464-
{State2, Effs} = activate_next_consumer(State1, []),
465+
{State2, Effs} = activate_next_consumer(State1, [], Vsn),
465466
#consumer{checked_out = Checked,
466467
credit = Credit,
467468
delivery_count = DeliveryCount,
@@ -472,7 +473,7 @@ apply(#{index := Idx} = Meta,
472473
credit => Credit,
473474
key => ConsumerKey,
474475
delivery_count => DeliveryCount,
475-
is_active => is_active(ConsumerKey, State2),
476+
is_active => is_active(ConsumerKey, State2, Vsn),
476477
num_checked_out => map_size(Checked)}},
477478
checkout(Meta, State0, State2, [{monitor, process, Pid} | Effs], Reply);
478479
apply(#{index := Index}, #purge{},
@@ -553,7 +554,7 @@ apply(#{machine_version := Vsn,
553554

554555
%% select a new consumer from the waiting queue and run a checkout
555556
State2 = State1#?STATE{waiting_consumers = WaitingConsumers},
556-
{State, Effects1} = activate_next_consumer(State2, Effects0),
557+
{State, Effects1} = activate_next_consumer(State2, Effects0, Vsn),
557558

558559
%% mark any enquers as suspected
559560
Enqs = maps:map(fun(P, E) when node(P) =:= Node ->
@@ -602,8 +603,9 @@ apply(#{machine_version := Vsn,
602603
Effects = [{monitor, node, Node} | Effects1],
603604
checkout(Meta, State0, State#?STATE{enqueuers = Enqs,
604605
last_active = Ts}, Effects);
605-
apply(Meta, {down, Pid, _Info}, State0) ->
606-
{State1, Effects1} = activate_next_consumer(handle_down(Meta, Pid, State0)),
606+
apply(#{machine_version := Vsn} = Meta, {down, Pid, _Info}, State0) ->
607+
{State1, Effects1} = activate_next_consumer(handle_down(Meta, Pid, State0),
608+
Vsn),
607609
checkout(Meta, State0, State1, Effects1);
608610
apply(#{machine_version := Vsn} = Meta,
609611
{nodeup, Node},
@@ -639,7 +641,7 @@ apply(#{machine_version := Vsn} = Meta,
639641
Waiting = update_waiting_consumer_status(Node, State1, up),
640642
State2 = State1#?STATE{enqueuers = Enqs1,
641643
waiting_consumers = Waiting},
642-
{State, Effects} = activate_next_consumer(State2, Effects1),
644+
{State, Effects} = activate_next_consumer(State2, Effects1, Vsn),
643645
checkout(Meta, State0, State, Effects);
644646
apply(_, {nodedown, _Node}, State) ->
645647
{State, ok};
@@ -872,7 +874,7 @@ overview(#?STATE{consumers = Cons,
872874
msg_ttl => Cfg#cfg.msg_ttl,
873875
delivery_limit => Cfg#cfg.delivery_limit
874876
},
875-
SacOverview = case active_consumer(Cons) of
877+
SacOverview = case active_consumer(Cons, version()) of
876878
{SacConsumerKey, SacCon} ->
877879
SacConsumerId = consumer_id(SacCon),
878880
NumWaiting = length(WaitingConsumers),
@@ -1305,7 +1307,7 @@ query_consumers(#?STATE{consumers = Consumers,
13051307

13061308
query_single_active_consumer(#?STATE{cfg = #cfg{consumer_strategy = single_active},
13071309
consumers = Consumers}) ->
1308-
case active_consumer(Consumers) of
1310+
case active_consumer(Consumers, version()) of
13091311
undefined ->
13101312
{error, no_value};
13111313
{_CKey, ?CONSUMER_TAG_PID(Tag, Pid)} ->
@@ -1476,15 +1478,15 @@ cancel_consumer0(Meta, ConsumerKey,
14761478
{S0, Effects0}
14771479
end.
14781480

1479-
activate_next_consumer({State, Effects}) ->
1480-
activate_next_consumer(State, Effects).
1481+
activate_next_consumer({State, Effects}, MacVsn) ->
1482+
activate_next_consumer(State, Effects, MacVsn).
14811483

14821484
activate_next_consumer(#?STATE{cfg = #cfg{consumer_strategy = competing}} = State,
1483-
Effects) ->
1485+
Effects, _MacVsn) ->
14841486
{State, Effects};
14851487
activate_next_consumer(#?STATE{consumers = Cons0,
14861488
waiting_consumers = Waiting0} = State0,
1487-
Effects0) ->
1489+
Effects0, MacVsn) ->
14881490
%% invariant, the waiting list always need to be sorted by consumers that are
14891491
%% up - then by priority
14901492
NextConsumer =
@@ -1495,7 +1497,7 @@ activate_next_consumer(#?STATE{consumers = Cons0,
14951497
undefined
14961498
end,
14971499

1498-
case {active_consumer(Cons0), NextConsumer} of
1500+
case {active_consumer(Cons0, MacVsn), NextConsumer} of
14991501
{undefined, {NextCKey, #consumer{cfg = NextCCfg} = NextC}} ->
15001502
Remaining = tl(Waiting0),
15011503
%% TODO: can this happen?
@@ -1564,17 +1566,19 @@ active_consumer({CKey, #consumer{status = Status} = Consumer, _I})
15641566
active_consumer({_CKey, #consumer{status = _}, I}) ->
15651567
active_consumer(maps:next(I));
15661568
active_consumer(none) ->
1567-
undefined;
1568-
active_consumer(M) when is_map(M) ->
1569-
I = maps:iterator(M),
1569+
undefined.
1570+
1571+
active_consumer(M, MacVsn) when is_map(M) ->
1572+
I = rabbit_fifo_maps:iterator(M, MacVsn),
15701573
active_consumer(maps:next(I)).
15711574

1572-
is_active(_ConsumerKey, #?STATE{cfg = #cfg{consumer_strategy = competing}}) ->
1575+
is_active(_ConsumerKey,
1576+
#?STATE{cfg = #cfg{consumer_strategy = competing}}, _MacVsn) ->
15731577
%% all competing consumers are potentially active
15741578
true;
15751579
is_active(ConsumerKey, #?STATE{cfg = #cfg{consumer_strategy = single_active},
1576-
consumers = Consumers}) ->
1577-
ConsumerKey == active_consumer(Consumers).
1580+
consumers = Consumers}, MacVsn) ->
1581+
ConsumerKey == active_consumer(Consumers, MacVsn).
15781582

15791583
maybe_return_all(#{system_time := Ts} = Meta, ConsumerKey,
15801584
#consumer{cfg = CCfg} = Consumer, S0,
@@ -1835,26 +1839,26 @@ increase_credit(#consumer{cfg = #consumer_cfg{credit_mode =
18351839
increase_credit(#consumer{credit = Current}, Credit) ->
18361840
Current + Credit.
18371841

1838-
complete_and_checkout(#{} = Meta, MsgIds, ConsumerKey,
1842+
complete_and_checkout(#{machine_version := Vsn} = Meta, MsgIds, ConsumerKey,
18391843
#consumer{} = Con0,
18401844
Effects0, State0) ->
18411845
State1 = complete(Meta, ConsumerKey, MsgIds, Con0, State0),
18421846
%% a completion could have removed the active/quiescing consumer
1843-
Effects1 = add_active_effect(Con0, State1, Effects0),
1844-
{State2, Effects2} = activate_next_consumer(State1, Effects1),
1847+
Effects1 = add_active_effect(Con0, State1, Effects0, Vsn),
1848+
{State2, Effects2} = activate_next_consumer(State1, Effects1, Vsn),
18451849
checkout(Meta, State0, State2, Effects2).
18461850

18471851
add_active_effect(#consumer{status = quiescing} = Consumer,
18481852
#?STATE{cfg = #cfg{consumer_strategy = single_active},
18491853
consumers = Consumers} = State,
1850-
Effects) ->
1851-
case active_consumer(Consumers) of
1854+
Effects, MacVsn) ->
1855+
case active_consumer(Consumers, MacVsn) of
18521856
undefined ->
18531857
consumer_update_active_effects(State, Consumer, false, waiting, Effects);
18541858
_ ->
18551859
Effects
18561860
end;
1857-
add_active_effect(_, _, Effects) ->
1861+
add_active_effect(_, _, Effects, _) ->
18581862
Effects.
18591863

18601864
cancel_consumer_effects(ConsumerId,
@@ -2340,16 +2344,16 @@ update_consumer(Meta, ConsumerKey, {Tag, Pid}, ConsumerMeta,
23402344
delivery_count = DeliveryCount}
23412345
end,
23422346
{Consumer, update_or_remove_con(Meta, ConsumerKey, Consumer, State0)};
2343-
update_consumer(Meta, ConsumerKey, {Tag, Pid}, ConsumerMeta,
2344-
{Life, Mode} = Spec, Priority,
2347+
update_consumer(#{machine_version := Vsn} = Meta, ConsumerKey, {Tag, Pid},
2348+
ConsumerMeta, {Life, Mode} = Spec, Priority,
23452349
#?STATE{cfg = #cfg{consumer_strategy = single_active},
23462350
consumers = Cons0,
23472351
waiting_consumers = Waiting0,
23482352
service_queue = _ServiceQueue0} = State) ->
23492353
%% if it is the current active consumer, just update
23502354
%% if it is a cancelled active consumer, add to waiting unless it is the only
23512355
%% one, then merge
2352-
case active_consumer(Cons0) of
2356+
case active_consumer(Cons0, Vsn) of
23532357
{ConsumerKey, #consumer{status = up} = Consumer0} ->
23542358
Consumer = merge_consumer(Meta, Consumer0, ConsumerMeta,
23552359
Spec, Priority),

deps/rabbit/src/rabbit_fifo_maps.erl

Lines changed: 14 additions & 8 deletions
Original file line numberDiff line numberDiff line change
@@ -8,7 +8,8 @@
88
-module(rabbit_fifo_maps).
99

1010
-export([keys/2,
11-
fold/4]).
11+
fold/4,
12+
iterator/2]).
1213

1314
-spec keys(Map, ra_machine:version()) -> Keys when
1415
Map :: #{Key => _},
@@ -29,13 +30,18 @@ keys(Map, Vsn) ->
2930
AccIn :: Init | AccOut,
3031
Map :: #{Key => Value}.
3132
fold(Fun, Init, Map, Vsn) ->
32-
Iterable = case is_deterministic(Vsn) of
33-
true ->
34-
maps:iterator(Map, ordered);
35-
false ->
36-
Map
37-
end,
38-
maps:fold(Fun, Init, Iterable).
33+
maps:fold(Fun, Init, iterator(Map, Vsn)).
34+
35+
-spec iterator(Map, ra_machine:version()) -> Iterator when
36+
Map :: #{Key => Value},
37+
Iterator :: maps:iterator(Key, Value).
38+
iterator(Map, Vsn) ->
39+
case is_deterministic(Vsn) of
40+
true ->
41+
maps:iterator(Map, ordered);
42+
false ->
43+
maps:iterator(Map)
44+
end.
3945

4046
is_deterministic(Vsn) when is_integer(Vsn) ->
4147
Vsn > 5.

0 commit comments

Comments
 (0)