Make stream protocol route command return several streams

We expect to have 1 stream for each routing key, but
as binding can return several queues for a given key we
let that possibility open in the stream protocol.
This commit is contained in:
Arnaud Cogoluègnes 2021-09-13 17:53:25 +02:00
parent 3d7afcdc18
commit 8f207e3c5f
No known key found for this signature in database
GPG Key ID: D5C8C4DFAD43AFA8
5 changed files with 62 additions and 58 deletions

View File

@ -587,7 +587,7 @@ RouteQuery => Key Version CorrelationId RoutingKey SuperStream
RoutingKey => string RoutingKey => string
SuperStream => string SuperStream => string
RouteResponse => Key Version CorrelationId Stream RouteResponse => Key Version CorrelationId [Stream]
Key => uint16 // 24 Key => uint16 // 24
Version => uint16 Version => uint16
CorrelationId => uint32 CorrelationId => uint32

View File

@ -118,10 +118,8 @@ kill_connection(ConnectionName) ->
{ConnectionPid, {ConnectionPid,
#{<<"connection_name">> := ConnectionNameBin}} -> #{<<"connection_name">> := ConnectionNameBin}} ->
exit(ConnectionPid, kill); exit(ConnectionPid, kill);
{ConnectionPid, _ClientProperties} -> {ConnectionPid, _ClientProperties} -> ok
ok after 1000 -> ok
after 1000 ->
ok
end end
end, end,
pg_local:get_members(rabbit_stream_connections)). pg_local:get_members(rabbit_stream_connections)).

View File

@ -75,7 +75,7 @@ topology(VirtualHost, Stream) ->
gen_server:call(?MODULE, {topology, VirtualHost, Stream}). gen_server:call(?MODULE, {topology, VirtualHost, Stream}).
-spec route(binary(), binary(), binary()) -> -spec route(binary(), binary(), binary()) ->
{ok, binary() | no_route} | {error, stream_not_found}. {ok, [binary()] | no_route} | {error, stream_not_found}.
route(RoutingKey, VirtualHost, SuperStream) -> route(RoutingKey, VirtualHost, SuperStream) ->
gen_server:call(?MODULE, gen_server:call(?MODULE,
{route, RoutingKey, VirtualHost, SuperStream}). {route, RoutingKey, VirtualHost, SuperStream}).
@ -368,10 +368,9 @@ handle_call({route, RoutingKey, VirtualHost, SuperStream}, _From,
case rabbit_exchange:route(Exchange, Delivery) of case rabbit_exchange:route(Exchange, Delivery) of
[] -> [] ->
{ok, no_route}; {ok, no_route};
[#resource{name = Stream}] -> Routes ->
{ok, Stream}; %% FIXME filter non-stream resources
[#resource{name = Stream} | _] -> {ok, [Stream || #resource{name = Stream} <- Routes]}
{ok, Stream}
end end
catch catch
exit:Error -> exit:Error ->

View File

@ -445,7 +445,8 @@ tuned(info, Msg, StateData) ->
end). end).
state_timeout(State, Transport, Socket) -> state_timeout(State, Transport, Socket) ->
rabbit_log_connection:warning("Closing connection because of timeout in state '~s' likely due to lack of client action.", rabbit_log_connection:warning("Closing connection because of timeout in state "
"'~s' likely due to lack of client action.",
[State]), [State]),
close_immediately(Transport, Socket), close_immediately(Transport, Socket),
stop. stop.
@ -524,7 +525,8 @@ transition_to_opened(Transport,
config = Configuration}}. config = Configuration}}.
invalid_transition(Transport, Socket, From, To) -> invalid_transition(Transport, Socket, From, To) ->
rabbit_log_connection:warning("Closing socket ~w. Invalid transition from ~s to ~s.", rabbit_log_connection:warning("Closing socket ~w. Invalid transition from ~s "
"to ~s.",
[Socket, From, To]), [Socket, From, To]),
close_immediately(Transport, Socket), close_immediately(Transport, Socket),
stop. stop.
@ -882,8 +884,7 @@ open(cast,
Ids -> Ids ->
Acc#{PublisherId => [PublishingId | Ids]} Acc#{PublisherId => [PublishingId | Ids]}
end; end;
false -> false -> Acc
Acc
end end
end, end,
#{}, CorrelationList), #{}, CorrelationList),
@ -963,7 +964,8 @@ open(cast,
{queue_event, #resource{name = StreamName}, {queue_event, #resource{name = StreamName},
{osiris_offset, _QueueResource, -1}}, {osiris_offset, _QueueResource, -1}},
_StatemData) -> _StatemData) ->
rabbit_log:debug("Stream protocol connection received osiris offset event for ~p with offset ~p", rabbit_log:debug("Stream protocol connection received osiris offset "
"event for ~p with offset ~p",
[StreamName, -1]), [StreamName, -1]),
keep_state_and_data; keep_state_and_data;
open(cast, open(cast,
@ -982,11 +984,14 @@ open(cast,
{Connection1, State1} = {Connection1, State1} =
case maps:get(StreamName, StreamSubscriptions, undefined) of case maps:get(StreamName, StreamSubscriptions, undefined) of
undefined -> undefined ->
rabbit_log:debug("Stream protocol connection: osiris offset event for ~p, but no subscription (leftover messages after unsubscribe?)", rabbit_log:debug("Stream protocol connection: osiris offset event "
"for ~p, but no subscription (leftover messages "
"after unsubscribe?)",
[StreamName]), [StreamName]),
{Connection, State}; {Connection, State};
[] -> [] ->
rabbit_log:debug("Stream protocol connection: osiris offset event for ~p, but no registered consumers!", rabbit_log:debug("Stream protocol connection: osiris offset event "
"for ~p, but no registered consumers!",
[StreamName]), [StreamName]),
{Connection#stream_connection{stream_subscriptions = {Connection#stream_connection{stream_subscriptions =
maps:remove(StreamName, maps:remove(StreamName,
@ -999,15 +1004,15 @@ open(cast,
#consumer{credit = Credit} = Consumer, #consumer{credit = Credit} = Consumer,
Consumer1 = Consumer1 =
case Credit of case Credit of
0 -> 0 -> Consumer;
Consumer;
_ -> _ ->
case send_chunks(Transport, case send_chunks(Transport,
Consumer, Consumer,
SendFileOct) SendFileOct)
of of
{error, closed} -> {error, closed} ->
rabbit_log_connection:info("Stream protocol connection has been closed by peer", rabbit_log_connection:info("Stream protocol connection has been closed by "
"peer",
[]), []),
throw({stop, normal}); throw({stop, normal});
{error, Reason} -> {error, Reason} ->
@ -1058,7 +1063,8 @@ close_sent(state_timeout, close,
#statem_data{transport = Transport, #statem_data{transport = Transport,
connection = #stream_connection{socket = Socket}, connection = #stream_connection{socket = Socket},
connection_state = State}) -> connection_state = State}) ->
rabbit_log_connection:warning("Closing connection because of timeout in state '~s' likely due to lack of client action.", rabbit_log_connection:warning("Closing connection because of timeout in state "
"'~s' likely due to lack of client action.",
[?FUNCTION_NAME]), [?FUNCTION_NAME]),
close(Transport, Socket, State), close(Transport, Socket, State),
stop; stop;
@ -1089,13 +1095,15 @@ close_sent(info, {tcp_closed, S}, _StatemData) ->
stop; stop;
close_sent(info, {tcp_error, S, Reason}, close_sent(info, {tcp_error, S, Reason},
#statem_data{transport = Transport, connection_state = State}) -> #statem_data{transport = Transport, connection_state = State}) ->
rabbit_log_connection:error("Stream protocol connection socket error: ~p [~w] [~w]", rabbit_log_connection:error("Stream protocol connection socket error: ~p [~w] "
"[~w]",
[Reason, S, self()]), [Reason, S, self()]),
close(Transport, S, State), close(Transport, S, State),
stop; stop;
close_sent(info, {resource_alarm, IsThereAlarm}, close_sent(info, {resource_alarm, IsThereAlarm},
StatemData = #statem_data{connection = Connection}) -> StatemData = #statem_data{connection = Connection}) ->
rabbit_log:warning("Stream protocol connection ignored a resource alarm ~p in state ~s", rabbit_log:warning("Stream protocol connection ignored a resource "
"alarm ~p in state ~s",
[IsThereAlarm, ?FUNCTION_NAME]), [IsThereAlarm, ?FUNCTION_NAME]),
{keep_state, {keep_state,
StatemData#statem_data{connection = StatemData#statem_data{connection =
@ -1828,7 +1836,8 @@ handle_frame_post_auth(Transport,
SendFileOct) SendFileOct)
of of
{error, closed} -> {error, closed} ->
rabbit_log_connection:info("Stream protocol connection has been closed by peer", rabbit_log_connection:info("Stream protocol connection has been closed by "
"peer",
[]), []),
throw({stop, normal}); throw({stop, normal});
{{segment, Segment1}, {credit, Credit1}} -> {{segment, Segment1}, {credit, Credit1}} ->
@ -1909,7 +1918,8 @@ handle_frame_post_auth(Transport,
SendFileOct) SendFileOct)
of of
{error, closed} -> {error, closed} ->
rabbit_log_connection:info("Stream protocol connection has been closed by peer", rabbit_log_connection:info("Stream protocol connection has been closed by "
"peer",
[]), []),
throw({stop, normal}); throw({stop, normal});
{{segment, Segment1}, {credit, Credit1}} -> {{segment, Segment1}, {credit, Credit1}} ->
@ -2061,7 +2071,8 @@ handle_frame_post_auth(Transport,
{ok, {ok,
#{leader_node := LeaderPid, #{leader_node := LeaderPid,
replica_nodes := ReturnedReplicas}} -> replica_nodes := ReturnedReplicas}} ->
rabbit_log:debug("Created stream cluster with leader on ~p and replicas on ~p", rabbit_log:debug("Created stream cluster with leader on ~p and "
"replicas on ~p",
[LeaderPid, ReturnedReplicas]), [LeaderPid, ReturnedReplicas]),
response_ok(Transport, response_ok(Transport,
Connection, Connection,
@ -2222,8 +2233,7 @@ handle_frame_post_auth(Transport,
NodesAcc) NodesAcc)
end, end,
Acc1, ReplicaNodes); Acc1, ReplicaNodes);
{error, _} -> {error, _} -> Acc
Acc
end end
end, end,
#{}, Streams), #{}, Streams),
@ -2235,16 +2245,13 @@ handle_frame_post_auth(Transport,
lists:foldr(fun(Node, Acc) -> lists:foldr(fun(Node, Acc) ->
PortFunction = PortFunction =
case TransportLayer of case TransportLayer of
tcp -> tcp -> port;
port; ssl -> tls_port
ssl ->
tls_port
end, end,
Host = rpc:call(Node, rabbit_stream, host, []), Host = rpc:call(Node, rabbit_stream, host, []),
Port = rpc:call(Node, rabbit_stream, PortFunction, []), Port = rpc:call(Node, rabbit_stream, PortFunction, []),
case {is_binary(Host), is_integer(Port)} of case {is_binary(Host), is_integer(Port)} of
{true, true} -> {true, true} -> Acc#{Node => {Host, Port}};
Acc#{Node => {Host, Port}};
_ -> _ ->
rabbit_log:warning("Error when retrieving broker metadata: ~p ~p", rabbit_log:warning("Error when retrieving broker metadata: ~p ~p",
[Host, Port]), [Host, Port]),
@ -2256,25 +2263,21 @@ handle_frame_post_auth(Transport,
Metadata = Metadata =
lists:foldl(fun(Stream, Acc) -> lists:foldl(fun(Stream, Acc) ->
case maps:get(Stream, Topology) of case maps:get(Stream, Topology) of
{error, Err} -> {error, Err} -> Acc#{Stream => Err};
Acc#{Stream => Err};
{ok, {ok,
#{leader_node := LeaderNode, #{leader_node := LeaderNode,
replica_nodes := Replicas}} -> replica_nodes := Replicas}} ->
LeaderInfo = LeaderInfo =
case NodeEndpoints of case NodeEndpoints of
#{LeaderNode := Info} -> #{LeaderNode := Info} -> Info;
Info; _ -> undefined
_ ->
undefined
end, end,
ReplicaInfos = ReplicaInfos =
lists:foldr(fun(Replica, A) -> lists:foldr(fun(Replica, A) ->
case NodeEndpoints of case NodeEndpoints of
#{Replica := I} -> #{Replica := I} ->
[I | A]; [I | A];
_ -> _ -> A
A
end end
end, end,
[], Replicas), [], Replicas),
@ -2301,16 +2304,21 @@ handle_frame_post_auth(Transport,
case rabbit_stream_manager:route(RoutingKey, VirtualHost, SuperStream) case rabbit_stream_manager:route(RoutingKey, VirtualHost, SuperStream)
of of
{ok, no_route} -> {ok, no_route} ->
{?RESPONSE_CODE_OK, <<(-1):16>>}; {?RESPONSE_CODE_OK, <<0:32>>};
{ok, Stream} -> {ok, Streams} ->
StreamCount = length(Streams),
Bin = lists:foldl(fun(Stream, Acc) ->
StreamSize = byte_size(Stream), StreamSize = byte_size(Stream),
{?RESPONSE_CODE_OK, <<Acc/binary, StreamSize:16,
<<StreamSize:16, Stream:StreamSize/binary>>}; Stream:StreamSize/binary>>
end,
<<StreamCount:32>>, Streams),
{?RESPONSE_CODE_OK, Bin};
{error, _} -> {error, _} ->
rabbit_global_counters:increase_protocol_counter(stream, rabbit_global_counters:increase_protocol_counter(stream,
?STREAM_DOES_NOT_EXIST, ?STREAM_DOES_NOT_EXIST,
1), 1),
{?RESPONSE_CODE_STREAM_DOES_NOT_EXIST, <<(-1):16>>} {?RESPONSE_CODE_STREAM_DOES_NOT_EXIST, <<0:32>>}
end, end,
Frame = Frame =
@ -2362,7 +2370,8 @@ handle_frame_post_auth(Transport,
State, State,
{request, CorrelationId, {request, CorrelationId,
{close, ClosingCode, ClosingReason}}) -> {close, ClosingCode, ClosingReason}}) ->
rabbit_log:debug("Stream protocol reader received close command ~p ~p", rabbit_log:debug("Stream protocol reader received close command "
"~p ~p",
[ClosingCode, ClosingReason]), [ClosingCode, ClosingReason]),
Frame = Frame =
rabbit_stream_core:frame({response, CorrelationId, rabbit_stream_core:frame({response, CorrelationId,
@ -2485,8 +2494,7 @@ clean_state_after_stream_deletion_or_failure(Stream,
PubId), PubId),
{maps:remove(PubId, Pubs), {maps:remove(PubId, Pubs),
maps:remove({Stream, Ref}, PubToIds)}; maps:remove({Stream, Ref}, PubToIds)};
_ -> _ -> {Pubs, PubToIds}
{Pubs, PubToIds}
end end
end, end,
{Publishers, PublisherToIds}, Publishers), {Publishers, PublisherToIds}, Publishers),
@ -2603,8 +2611,7 @@ demonitor_stream(Stream,
Stream -> Stream ->
demonitor(MonitorRef, [flush]), demonitor(MonitorRef, [flush]),
Acc; Acc;
_ -> _ -> maps:put(MonitorRef, Strm, Acc)
maps:put(MonitorRef, Strm, Acc)
end end
end, end,
#{}, Monitors0), #{}, Monitors0),
@ -2625,10 +2632,8 @@ stream_has_publishers(Stream,
#stream_connection{publishers = Publishers}) -> #stream_connection{publishers = Publishers}) ->
lists:any(fun(#publisher{stream = S}) -> lists:any(fun(#publisher{stream = S}) ->
case S of case S of
Stream -> Stream -> true;
true; _ -> false
_ ->
false
end end
end, end,
maps:values(Publishers)). maps:values(Publishers)).

View File

@ -40,7 +40,9 @@ groups() ->
init_per_suite(Config) -> init_per_suite(Config) ->
case rabbit_ct_helpers:is_mixed_versions() of case rabbit_ct_helpers:is_mixed_versions() of
true -> true ->
{skip, "mixed version clusters are not supported for this suite"}; {skip,
"mixed version clusters are not supported for "
"this suite"};
_ -> _ ->
Config1 = Config1 =
rabbit_ct_helpers:set_config(Config, rabbit_ct_helpers:set_config(Config,