@@ -192,7 +192,6 @@ init(#{name := Name,
192
192
update_config (Conf , State ) ->
193
193
DLH = maps :get (dead_letter_handler , Conf , undefined ),
194
194
BLH = maps :get (become_leader_handler , Conf , undefined ),
195
- RCI = maps :get (release_cursor_interval , Conf , ? RELEASE_CURSOR_EVERY ),
196
195
Overflow = maps :get (overflow_strategy , Conf , drop_head ),
197
196
MaxLength = maps :get (max_length , Conf , undefined ),
198
197
MaxBytes = maps :get (max_bytes , Conf , undefined ),
@@ -206,11 +205,9 @@ update_config(Conf, State) ->
206
205
competing
207
206
end ,
208
207
Cfg = State #? STATE .cfg ,
209
- RCISpec = {RCI , RCI },
210
208
211
209
LastActive = maps :get (created , Conf , undefined ),
212
- State #? STATE {cfg = Cfg # cfg {release_cursor_interval = RCISpec ,
213
- dead_letter_handler = DLH ,
210
+ State #? STATE {cfg = Cfg # cfg {dead_letter_handler = DLH ,
214
211
become_leader_handler = BLH ,
215
212
overflow_strategy = Overflow ,
216
213
max_length = MaxLength ,
@@ -485,7 +482,7 @@ apply(#{index := Index}, #purge{},
485
482
returns = lqueue :new (),
486
483
msg_bytes_enqueue = 0
487
484
},
488
- Effects0 = [garbage_collection ],
485
+ Effects0 = [{ aux , force_checkpoint }, garbage_collection ],
489
486
Reply = {purge , NumReady },
490
487
{State , _ , Effects } = evaluate_limit (Index , false , State0 ,
491
488
State1 , Effects0 ),
@@ -580,9 +577,8 @@ apply(#{system_time := Ts} = Meta,
580
577
Effects = [{monitor , node , Node } | Effects1 ],
581
578
checkout (Meta , State0 , State #? STATE {enqueuers = Enqs ,
582
579
last_active = Ts }, Effects );
583
- apply (#{index := _Idx } = Meta , {down , Pid , _Info }, State0 ) ->
584
- {State1 , Effects1 } = activate_next_consumer (
585
- handle_down (Meta , Pid , State0 )),
580
+ apply (Meta , {down , Pid , _Info }, State0 ) ->
581
+ {State1 , Effects1 } = activate_next_consumer (handle_down (Meta , Pid , State0 )),
586
582
checkout (Meta , State0 , State1 , Effects1 );
587
583
apply (Meta , {nodeup , Node }, #? STATE {consumers = Cons0 ,
588
584
enqueuers = Enqs0 ,
@@ -670,7 +666,8 @@ convert_v3_to_v4(#{} = _Meta, StateV3) ->
670
666
end , Returns0 )),
671
667
672
668
Messages = rabbit_fifo_q :from_lqueue (Messages0 ),
673
- #? STATE {cfg = rabbit_fifo_v3 :get_field (cfg , StateV3 ),
669
+ Cfg = rabbit_fifo_v3 :get_field (cfg , StateV3 ),
670
+ #? STATE {cfg = Cfg # cfg {unused_1 = ? NIL },
674
671
messages = Messages ,
675
672
messages_total = rabbit_fifo_v3 :get_field (messages_total , StateV3 ),
676
673
returns = Returns ,
@@ -813,8 +810,7 @@ state_enter0(_, _, Effects) ->
813
810
Effects .
814
811
815
812
-spec tick (non_neg_integer (), state ()) -> ra_machine :effects ().
816
- tick (Ts , #? STATE {cfg = # cfg {name = _Name ,
817
- resource = QName }} = State ) ->
813
+ tick (Ts , #? STATE {cfg = # cfg {resource = QName }} = State ) ->
818
814
case is_expired (Ts , State ) of
819
815
true ->
820
816
[{mod_call , rabbit_quorum_queue , spawn_deleter , [QName ]}];
@@ -835,7 +831,6 @@ overview(#?STATE{consumers = Cons,
835
831
waiting_consumers = WaitingConsumers } = State ) ->
836
832
Conf = #{name => Cfg # cfg .name ,
837
833
resource => Cfg # cfg .resource ,
838
- release_cursor_interval => Cfg # cfg .release_cursor_interval ,
839
834
dead_lettering_enabled => undefined =/= Cfg # cfg .dead_letter_handler ,
840
835
max_length => Cfg # cfg .max_length ,
841
836
max_bytes => Cfg # cfg .max_bytes ,
@@ -908,9 +903,10 @@ which_module(4) -> ?MODULE.
908
903
909
904
-record (checkpoint , {index :: ra :index (),
910
905
timestamp :: milliseconds (),
911
- enqueue_count :: non_neg_integer (),
912
906
smallest_index :: undefined | ra :index (),
913
- messages_total :: non_neg_integer ()}).
907
+ messages_total :: non_neg_integer (),
908
+ indexes = ? CHECK_MIN_INDEXES :: non_neg_integer (),
909
+ unused_1 = ? NIL }).
914
910
-record (aux_gc , {last_raft_idx = 0 :: ra :index ()}).
915
911
-record (aux , {name :: atom (),
916
912
capacity :: term (),
@@ -921,7 +917,6 @@ which_module(4) -> ?MODULE.
921
917
gc = # aux_gc {} :: # aux_gc {},
922
918
tick_pid :: undefined | pid (),
923
919
cache = #{} :: map (),
924
- % % TODO: we need a state conversion for this
925
920
last_checkpoint :: # checkpoint {}}).
926
921
927
922
init_aux (Name ) when is_atom (Name ) ->
@@ -934,8 +929,8 @@ init_aux(Name) when is_atom(Name) ->
934
929
capacity = {inactive , Now , 1 , 1.0 },
935
930
last_checkpoint = # checkpoint {index = 0 ,
936
931
timestamp = erlang :system_time (millisecond ),
937
- enqueue_count = 0 ,
938
- messages_total = 0 }}.
932
+ messages_total = 0 ,
933
+ unused_1 = ? NIL }}.
939
934
940
935
handle_aux (RaftState , Tag , Cmd , # aux {name = Name ,
941
936
capacity = Cap ,
@@ -950,6 +945,35 @@ handle_aux(RaftState, Tag, Cmd, AuxV2, RaAux)
950
945
Name = element (2 , AuxV2 ),
951
946
AuxV3 = init_aux (Name ),
952
947
handle_aux (RaftState , Tag , Cmd , AuxV3 , RaAux );
948
+ handle_aux (leader , cast , eval ,
949
+ #? AUX {last_decorators_state = LastDec ,
950
+ last_checkpoint = Check0 } = Aux0 ,
951
+ RaAux ) ->
952
+ #? STATE {cfg = # cfg {resource = QName }} = MacState =
953
+ ra_aux :machine_state (RaAux ),
954
+
955
+ Ts = erlang :system_time (millisecond ),
956
+ {Check , Effects0 } = do_checkpoints (Ts , Check0 , RaAux , false ),
957
+
958
+ % % this is called after each batch of commands have been applied
959
+ % % set timer for message expire
960
+ % % should really be the last applied index ts but this will have to do
961
+ Effects1 = timer_effect (Ts , MacState , Effects0 ),
962
+ case query_notify_decorators_info (MacState ) of
963
+ LastDec ->
964
+ {no_reply , Aux0 #? AUX {last_checkpoint = Check }, RaAux , Effects1 };
965
+ {MaxActivePriority , IsEmpty } = NewLast ->
966
+ Effects = [notify_decorators_effect (QName , MaxActivePriority , IsEmpty )
967
+ | Effects1 ],
968
+ {no_reply , Aux0 #? AUX {last_checkpoint = Check ,
969
+ last_decorators_state = NewLast }, RaAux , Effects }
970
+ end ;
971
+ handle_aux (_RaftState , cast , eval ,
972
+ #? AUX {last_checkpoint = Check0 } = Aux0 ,
973
+ RaAux ) ->
974
+ Ts = erlang :system_time (millisecond ),
975
+ {Check , Effects } = do_checkpoints (Ts , Check0 , RaAux , false ),
976
+ {no_reply , Aux0 #? AUX {last_checkpoint = Check }, RaAux , Effects };
953
977
handle_aux (_RaftState , cast , {# return {msg_ids = MsgIds ,
954
978
consumer_key = Key } = Ret , Corr , Pid },
955
979
Aux0 , RaAux0 ) ->
@@ -959,18 +983,18 @@ handle_aux(_RaftState, cast, {#return{msg_ids = MsgIds,
959
983
case find_consumer (Key , Consumers ) of
960
984
{ConsumerKey , # consumer {checked_out = Checked }} ->
961
985
{RaAux , ToReturn } =
962
- maps :fold (
963
- fun (MsgId , ? MSG (Idx , Header ), {RA0 , Acc }) ->
964
- % % it is possible this is not found if the consumer
965
- % % crashed and the message got removed
966
- case ra_aux :log_fetch (Idx , RA0 ) of
967
- {{_Term , _Meta , Cmd }, RA } ->
968
- Msg = get_msg (Cmd ),
969
- {RA , [{MsgId , Idx , Header , Msg } | Acc ]};
970
- {undefined , RA } ->
971
- {RA , Acc }
972
- end
973
- end , {RaAux0 , []}, maps :with (MsgIds , Checked )),
986
+ maps :fold (
987
+ fun (MsgId , ? MSG (Idx , Header ), {RA0 , Acc }) ->
988
+ % % it is possible this is not found if the consumer
989
+ % % crashed and the message got removed
990
+ case ra_aux :log_fetch (Idx , RA0 ) of
991
+ {{_Term , _Meta , Cmd }, RA } ->
992
+ Msg = get_msg (Cmd ),
993
+ {RA , [{MsgId , Idx , Header , Msg } | Acc ]};
994
+ {undefined , RA } ->
995
+ {RA , Acc }
996
+ end
997
+ end , {RaAux0 , []}, maps :with (MsgIds , Checked )),
974
998
975
999
Appends = make_requeue (ConsumerKey , {notify , Corr , Pid },
976
1000
lists :sort (ToReturn ), []),
@@ -1020,35 +1044,6 @@ handle_aux(_, _, {get_checked_out, ConsumerKey, MsgIds}, Aux0, RaAux0) ->
1020
1044
_ ->
1021
1045
{reply , {error , consumer_not_found }, Aux0 , RaAux0 }
1022
1046
end ;
1023
- handle_aux (leader , cast , eval ,
1024
- #? AUX {last_decorators_state = LastDec ,
1025
- last_checkpoint = Check0 } = Aux0 ,
1026
- RaAux ) ->
1027
- #? STATE {cfg = # cfg {resource = QName }} = MacState =
1028
- ra_aux :machine_state (RaAux ),
1029
-
1030
- Ts = erlang :system_time (millisecond ),
1031
- {Check , Effects0 } = do_checkpoints (Ts , Check0 , RaAux ),
1032
-
1033
- % % this is called after each batch of commands have been applied
1034
- % % set timer for message expire
1035
- % % should really be the last applied index ts but this will have to do
1036
- Effects1 = timer_effect (Ts , MacState , Effects0 ),
1037
- case query_notify_decorators_info (MacState ) of
1038
- LastDec ->
1039
- {no_reply , Aux0 #? AUX {last_checkpoint = Check }, RaAux , Effects1 };
1040
- {MaxActivePriority , IsEmpty } = NewLast ->
1041
- Effects = [notify_decorators_effect (QName , MaxActivePriority , IsEmpty )
1042
- | Effects1 ],
1043
- {no_reply , Aux0 #? AUX {last_checkpoint = Check ,
1044
- last_decorators_state = NewLast }, RaAux , Effects }
1045
- end ;
1046
- handle_aux (_RaftState , cast , eval ,
1047
- #? AUX {last_checkpoint = Check0 } = Aux0 ,
1048
- RaAux ) ->
1049
- Ts = erlang :system_time (millisecond ),
1050
- {Check , Effects } = do_checkpoints (Ts , Check0 , RaAux ),
1051
- {no_reply , Aux0 #? AUX {last_checkpoint = Check }, RaAux , Effects };
1052
1047
handle_aux (_RaState , cast , Cmd , #? AUX {capacity = Use0 } = Aux0 , RaAux )
1053
1048
when Cmd == active orelse Cmd == inactive ->
1054
1049
{no_reply , Aux0 #? AUX {capacity = update_use (Use0 , Cmd )}, RaAux };
@@ -1107,6 +1102,11 @@ handle_aux(_RaState, {call, _From}, {peek, Pos}, Aux0,
1107
1102
end ;
1108
1103
handle_aux (_ , _ , garbage_collection , Aux , RaAux ) ->
1109
1104
{no_reply , force_eval_gc (RaAux , Aux ), RaAux };
1105
+ handle_aux (_RaState , _ , force_checkpoint ,
1106
+ #? AUX {last_checkpoint = Check0 } = Aux , RaAux ) ->
1107
+ Ts = erlang :system_time (millisecond ),
1108
+ {Check , Effects } = do_checkpoints (Ts , Check0 , RaAux , true ),
1109
+ {no_reply , Aux #? AUX {last_checkpoint = Check }, RaAux , Effects };
1110
1110
handle_aux (RaState , _ , {dlx , _ } = Cmd , Aux0 , RaAux ) ->
1111
1111
#? STATE {dlx = DlxState ,
1112
1112
cfg = # cfg {dead_letter_handler = DLH ,
@@ -2639,8 +2639,8 @@ suspected_pids_for(Node, #?STATE{consumers = Cons0,
2639
2639
end , Enqs , WaitingConsumers0 ).
2640
2640
2641
2641
is_expired (Ts , #? STATE {cfg = # cfg {expires = Expires },
2642
- last_active = LastActive ,
2643
- consumers = Consumers })
2642
+ last_active = LastActive ,
2643
+ consumers = Consumers })
2644
2644
when is_number (LastActive ) andalso is_number (Expires ) ->
2645
2645
% % TODO: should it be active consumers?
2646
2646
Active = maps :filter (fun (_ , # consumer {status = suspected_down }) ->
@@ -2845,53 +2845,53 @@ priority_tag(Msg) ->
2845
2845
lo
2846
2846
end .
2847
2847
2848
- -define (CHECK_ENQ_MIN_INTERVAL_MS , 500 ).
2849
- -define (CHECK_ENQ_MIN_INDEXES , 4096 ).
2850
- -define (CHECK_MIN_INTERVAL_MS , 5000 ).
2851
- -define (CHECK_MIN_INDEXES , 65456 ).
2852
2848
2853
2849
do_checkpoints (Ts ,
2854
2850
# checkpoint {index = ChIdx ,
2855
2851
timestamp = ChTime ,
2856
- enqueue_count = ChEnqCnt ,
2857
2852
smallest_index = LastSmallest ,
2858
- messages_total = LastMsgsTot } = Check0 , RaAux ) ->
2853
+ indexes = MinIndexes } = Check0 , RaAux , Force ) ->
2859
2854
LastAppliedIdx = ra_aux :last_applied (RaAux ),
2860
- #? STATE {enqueue_count = EnqCnt } = MacState = ra_aux :machine_state (RaAux ),
2855
+ IndexesSince = LastAppliedIdx - ChIdx ,
2856
+ #? STATE {} = MacState = ra_aux :machine_state (RaAux ),
2857
+ TimeSince = Ts - ChTime ,
2858
+ NewSmallest = case smallest_raft_index (MacState ) of
2859
+ undefined ->
2860
+ LastAppliedIdx ;
2861
+ Smallest ->
2862
+ Smallest
2863
+ end ,
2861
2864
MsgsTot = messages_total (MacState ),
2862
- Mult = case MsgsTot > 200_000 of
2863
- true ->
2864
- min (4 , MsgsTot div 100_000 );
2865
- false ->
2866
- 1
2867
- end ,
2868
- Since = Ts - ChTime ,
2869
- NewSmallest = case smallest_raft_index (MacState ) of
2870
- undefined ->
2871
- LastAppliedIdx ;
2872
- Smallest ->
2873
- Smallest
2874
- end ,
2875
- {Check , Effects } = case (EnqCnt - ChEnqCnt > ? CHECK_ENQ_MIN_INDEXES andalso
2876
- Since > (? CHECK_ENQ_MIN_INTERVAL_MS * Mult )) orelse
2877
- (LastAppliedIdx - ChIdx > ? CHECK_MIN_INDEXES andalso
2878
- Since > (? CHECK_MIN_INTERVAL_MS * Mult )) orelse
2879
- (LastMsgsTot > 0 andalso MsgsTot == 0 ) of
2880
- true ->
2881
- % % take a checkpoint;
2882
- {# checkpoint {index = LastAppliedIdx ,
2883
- timestamp = Ts ,
2884
- enqueue_count = EnqCnt ,
2885
- smallest_index = NewSmallest ,
2886
- messages_total = MsgsTot },
2887
- [{checkpoint , LastAppliedIdx , MacState } |
2888
- release_cursor (LastSmallest , NewSmallest )]};
2889
- false ->
2890
- {Check0 # checkpoint {smallest_index = NewSmallest },
2891
- release_cursor (LastSmallest , NewSmallest )}
2892
- end ,
2893
-
2894
- {Check , Effects }.
2865
+ {CheckMinInterval , CheckMinIndexes , CheckMaxIndexes } =
2866
+ persistent_term :get (quorum_queue_checkpoint_config ,
2867
+ {? CHECK_MIN_INTERVAL_MS , ? CHECK_MIN_INDEXES ,
2868
+ ? CHECK_MAX_INDEXES }),
2869
+ EnoughTimeHasPassed = TimeSince > CheckMinInterval ,
2870
+
2871
+ % % enough time has passed and enough indexes have been committed
2872
+ case (IndexesSince > MinIndexes andalso
2873
+ EnoughTimeHasPassed ) orelse
2874
+ % % the queue is empty and some commands have been
2875
+ % % applied since the last checkpoint
2876
+ (MsgsTot == 0 andalso
2877
+ IndexesSince > CheckMinIndexes andalso
2878
+ EnoughTimeHasPassed ) orelse
2879
+ Force of
2880
+ true ->
2881
+ % % take fewer checkpoints the more messages there are on queue
2882
+ NextIndexes = min (max (MsgsTot , CheckMinIndexes ), CheckMaxIndexes ),
2883
+ % % take a checkpoint;
2884
+ {# checkpoint {index = LastAppliedIdx ,
2885
+ timestamp = Ts ,
2886
+ smallest_index = NewSmallest ,
2887
+ messages_total = MsgsTot ,
2888
+ indexes = NextIndexes },
2889
+ [{checkpoint , LastAppliedIdx , MacState } |
2890
+ release_cursor (LastSmallest , NewSmallest )]};
2891
+ false ->
2892
+ {Check0 # checkpoint {smallest_index = NewSmallest },
2893
+ release_cursor (LastSmallest , NewSmallest )}
2894
+ end .
2895
2895
2896
2896
release_cursor (LastSmallest , Smallest )
2897
2897
when is_integer (LastSmallest ) andalso
0 commit comments