[Why]
Up until now, a user had to run the following three commands to expand a
cluster:
1. stop_app
2. join_cluster
3. start_app
Stopping and starting the `rabbit` application and taking care of the
underlying Mnesia application could be handled by `join_cluster`
directly.
[How]
After the call to `can_join/1` and before proceeding with the actual
join, the code remembers the state of `rabbit`, the Feature flags
controler and Mnesia.
After the join, it restarts whatever needs to be restarted to. It does
so regardless of the success or failure of the join. One exception is
when the node switched from Mnesia to Khepri as part of that join. In
this case, Mnesia is left stopped.
[Why]
When a Khepri-based node joins a Mnesia-based cluster, it is reset and
switches back from Khepri to Mnesia. If there are Mnesia files left in
its data directory, Mnesia will restart with stale/incorrect data and
the operation will fail.
After a migration to Khepri, we need to make sure there is no stale
Mnesia files.
[How]
We use `rabbit_mnesia` to query the Mnesia files and delete them.
Currently these are not allowed for use with stream queues
which is a bit too strict. Some client impl will automatically
nack or reject messages that are pending when an application
requests to stop consuming. Treating all message outcomes the same
makes as much sense as not to.
Because both `add_member` and `grow` default to Membership status `promotable`,
new members will have to catch up before they are considered cluster members.
This can be overridden with either `voter` or (permanent `non_voter` statuses.
The latter one is useless without additional tooling so kept undocumented.
- non-voters do not affect quorum size for election purposes
- `observer_cli` reports their status with lowercase 'f'
- `rabbitmq-queues check_if_node_is_quorum_critical` takes voter status into
account
[Why]
Mnesia is a very powerful and convenient tool for Erlang applications:
it is a persistent disc-based database, it handles replication accross
multiple Erlang nodes and it is available out-of-the-box from the
Erlang/OTP distribution. RabbitMQ relies on Mnesia to manage all its
metadata:
* virtual hosts' properties
* intenal users
* queue, exchange and binding declarations (not queues data)
* runtime parameters and policies
* ...
Unfortunately Mnesia makes it difficult to handle network partition and,
as a consequence, the merge conflicts between Erlang nodes once the
network partition is resolved. RabbitMQ provides several partition
handling strategies but they are not bullet-proof. Users still hit
situations where it is a pain to repair a cluster following a network
partition.
[How]
@kjnilsson created Ra [1], a Raft consensus library that RabbitMQ
already uses successfully to implement quorum queues and streams for
instance. Those queues do not suffer from network partitions.
We created Khepri [2], a new persistent and replicated database engine
based on Ra and we want to use it in place of Mnesia in RabbitMQ to
solve the problems with network partitions.
This patch integrates Khepri as an experimental feature. When enabled,
RabbitMQ will store all its metadata in Khepri instead of Mnesia.
This change comes with behavior changes. While Khepri remains disabled,
you should see no changes to the behavior of RabbitMQ. If there are
changes, it is a bug. After Khepri is enabled, there are significant
changes of behavior that you should be aware of.
Because it is based on the Raft consensus algorithm, when there is a
network partition, only the cluster members that are in the partition
with at least `(Number of nodes in the cluster ÷ 2) + 1` number of nodes
can "make progress". In other words, only those nodes may write to the
Khepri database and read from the database and expect a consistent
result.
For instance in a cluster of 5 RabbitMQ nodes:
* If there are two partitions, one with 3 nodes, one with 2 nodes, only
the group of 3 nodes will be able to write to the database.
* If there are three partitions, two with 2 nodes, one with 1 node, none
of the group can write to the database.
Because the Khepri database will be used for all kind of metadata, it
means that RabbitMQ nodes that can't write to the database will be
unable to perform some operations. A list of operations and what to
expect is documented in the associated pull request and the RabbitMQ
website.
This requirement from Raft also affects the startup of RabbitMQ nodes in
a cluster. Indeed, at least a quorum number of nodes must be started at
once to allow nodes to become ready.
To enable Khepri, you need to enable the `khepri_db` feature flag:
rabbitmqctl enable_feature_flag khepri_db
When the `khepri_db` feature flag is enabled, the migration code
performs the following two tasks:
1. It synchronizes the Khepri cluster membership from the Mnesia
cluster. It uses `mnesia_to_khepri:sync_cluster_membership/1` from
the `khepri_mnesia_migration` application [3].
2. It copies data from relevant Mnesia tables to Khepri, doing some
conversion if necessary on the way. Again, it uses
`mnesia_to_khepri:copy_tables/4` from `khepri_mnesia_migration` to do
it.
This can be performed on a running standalone RabbitMQ node or cluster.
Data will be migrated from Mnesia to Khepri without any service
interruption. Note that during the migration, the performance may
decrease and the memory footprint may go up.
Because this feature flag is considered experimental, it is not enabled
by default even on a brand new RabbitMQ deployment.
More about the implementation details below:
In the past months, all accesses to Mnesia were isolated in a collection
of `rabbit_db*` modules. This is where the integration of Khepri mostly
takes place: we use a function called `rabbit_khepri:handle_fallback/1`
which selects the database and perform the query or the transaction.
Here is an example from `rabbit_db_vhost`:
* Up until RabbitMQ 3.12.x:
get(VHostName) when is_binary(VHostName) ->
get_in_mnesia(VHostName).
* Starting with RabbitMQ 3.13.0:
get(VHostName) when is_binary(VHostName) ->
rabbit_khepri:handle_fallback(
#{mnesia => fun() -> get_in_mnesia(VHostName) end,
khepri => fun() -> get_in_khepri(VHostName) end}).
This `rabbit_khepri:handle_fallback/1` function relies on two things:
1. the fact that the `khepri_db` feature flag is enabled, in which case
it always executes the Khepri-based variant.
4. the ability or not to read and write to Mnesia tables otherwise.
Before the feature flag is enabled, or during the migration, the
function will try to execute the Mnesia-based variant. If it succeeds,
then it returns the result. If it fails because one or more Mnesia
tables can't be used, it restarts from scratch: it means the feature
flag is being enabled and depending on the outcome, either the
Mnesia-based variant will succeed (the feature flag couldn't be enabled)
or the feature flag will be marked as enabled and it will call the
Khepri-based variant. The meat of this function really lives in the
`khepri_mnesia_migration` application [3] and
`rabbit_khepri:handle_fallback/1` is a wrapper on top of it that knows
about the feature flag.
However, some calls to the database do not depend on the existence of
Mnesia tables, such as functions where we need to learn about the
members of a cluster. For those, we can't rely on exceptions from
Mnesia. Therefore, we just look at the state of the feature flag to
determine which database to use. There are two situations though:
* Sometimes, we need the feature flag state query to block because the
function interested in it can't return a valid answer during the
migration. Here is an example:
case rabbit_khepri:is_enabled(RemoteNode) of
true -> can_join_using_khepri(RemoteNode);
false -> can_join_using_mnesia(RemoteNode)
end
* Sometimes, we need the feature flag state query to NOT block (for
instance because it would cause a deadlock). Here is an example:
case rabbit_khepri:get_feature_state() of
enabled -> members_using_khepri();
_ -> members_using_mnesia()
end
Direct accesses to Mnesia still exists. They are limited to code that is
specific to Mnesia such as classic queue mirroring or network partitions
handling strategies.
Now, to discover the Mnesia tables to migrate and how to migrate them,
we use an Erlang module attribute called
`rabbit_mnesia_tables_to_khepri_db` which indicates a list of Mnesia
tables and an associated converter module. Here is an example in the
`rabbitmq_recent_history_exchange` plugin:
-rabbit_mnesia_tables_to_khepri_db(
[{?RH_TABLE, rabbit_db_rh_exchange_m2k_converter}]).
The converter module — `rabbit_db_rh_exchange_m2k_converter` in this
example — is is fact a "sub" converter module called but
`rabbit_db_m2k_converter`. See the documentation of a `mnesia_to_khepri`
converter module to learn more about these modules.
[1] https://github.com/rabbitmq/ra
[2] https://github.com/rabbitmq/khepri
[3] https://github.com/rabbitmq/khepri_mnesia_migration
See #7206.
Co-authored-by: Jean-Sébastien Pédron <jean-sebastien@rabbitmq.com>
Co-authored-by: Diana Parra Corbacho <dparracorbac@vmware.com>
Co-authored-by: Michael Davis <mcarsondavis@gmail.com>
This includes a new ra:key_metrics/1 API that is more available
than parsing the output of sys:get_status/1.
the rabbit_quorum_queue:status/1 function has been ported to use
this API instead as well as now inludes a few new fields.
it was named by copying and pasting an adjacent
one that indeed had to do with queue type-specific
policies but "version-specific" policies is not
something RabbitMQ supports
References #9547#9541
Just valid policies are effectively applied on each queue type,
but they need to be added to 'unsupported-capabilities' to be
excluded from the queue info.
[Why]
The CLI is only compatible with the version of RabbitMQ it is shipped
with. It does not pretend to be backward- or forward-compatible with
other versions.
Unfortunately, `rabbit_control_helper` always use the CLI's module from
the first RabbitMQ node and is executed against any nodes in a testcase.
This may break for the reason described above.
[How]
There is no reason to fix `rabbit_control_helper`, we just need to
switch to the initial way of using the CLI,
`rabbit_ct_broker_helper:rabbitmqctl()`. This one was already fixed to
use the appropriate copy of the CLI.
This patch only fixes `clustering_management_SUITE` and
`rabbitmq_4_0_deprecations_SUITE`. The former because it broke because
of this, the latter as a liow hanging fruit.
Following up on failures detected by Java project test
suites after the merge of the message container PR.
These tests are ported to Erlang in the broker test suite.
Fixes#9371
Since each AMQP 1.0 connection opens several direct AMQP connections, we
must assign each direct connection a unique name to prevent multiple
entries in the `connection_created_stats` table.
Also, use `pg_local` to track AMQP 1.0 connections instead of walking
the supervisor trees.
Nuke authz_backends from connection created event 💥
Fix regex for connection name because UniqueId is part of it now (channel number)
* Translate AMQP 0.9.1 CC headers to AMQP 1.0 x-cc
Translate AMQP 0.9.1 CC headers to AMQP 1.0 x-cc message annotations.
We want CC headers to be kept an AMQP legacy feature and therefore
special case its conversion to AMQP 1.0.
* Translate x-cc from 1.0 message annotation to 091 CC header
for the case where you publish via 091 with CC to a stream and consume
via 091 from a stream in which case the 091 consuming client would like
to know the original CC headers.
* AMQP encoded bodies should be converted to amqp correctly
Fix for AMQP encoded amqpl payloads.
Also removing some headers added during amqpl->amqpl conversions that
duplicate information in the amqp header.
* we should not need to prepre for read toset annotations
* fix tagged_prop() type spec
* tagged_prop() -> tagged_value()
As the unit_access_control_SUITE topic test is the only testcase
that covers topic routing, it makes sense to extract it and run
it as a standalone test suite. It eases the development and testing
of topic routing features.
This PR implements an approach for a "protocol (data format) agnostic core" where the format of the message isn't converted at point of reception.
Currently all non AMQP 0.9.1 originating messages are converted into a AMQP 0.9.1 flavoured basic_message record before sent to a queue. If the messages are then consumed by the originating protocol they are converted back from AMQP 0.9.1. For some protocols such as MQTT 3.1 this isn't too expensive as MQTT is mostly a fairly easily mapped subset of AMQP 0.9.1 but for others such as AMQP 1.0 the conversions are awkward and in some cases lossy even if consuming from the originating protocol.
This PR instead wraps all incoming messages in their originating form into a generic, extensible message container type (mc). The container module exposes an API to get common message details such as size and various properties (ttl, priority etc) directly from the source data type. Each protocol needs to implement the mc behaviour such that when a message originating form one protocol is consumed by another protocol we convert it to the target protocol at that point.
The message container also contains annotations, dead letter records and other meta data we need to record during the lifetime of a message. The original protocol message is never modified unless it is consumed.
This includes conversion modules to and from amqp, amqpl (AMQP 0.9.1) and mqtt.
COMMIT HISTORY:
* Refactor away from using the delivery{} record
In many places including exchange types. This should make it
easier to move towards using a message container type instead of
basic_message.
Add mc module and move direct replies outside of exchange
Lots of changes incl classic queues
Implement stream support incl amqp conversions
simplify mc state record
move mc.erl
mc dlx stuff
recent history exchange
Make tracking work
But doesn't take a protocol agnostic approach as we just convert
everything into AMQP legacy and back. Might be good enough for now.
Tracing as a whole may want a bit of a re-vamp at some point.
tidy
make quorum queue peek work by legacy conversion
dead lettering fixes
dead lettering fixes
CMQ fixes
rabbit_trace type fixes
fixes
fix
Fix classic queue props
test assertion fix
feature flag and backwards compat
Enable message_container feature flag in some SUITEs
Dialyzer fixes
fixes
fix
test fixes
Various
Manually update a gazelle generated file
until a gazelle enhancement can be made
https://github.com/rabbitmq/rules_erlang/issues/185
Add message_containers_SUITE to bazel
and regen bazel files with gazelle from rules_erlang@main
Simplify essential proprty access
Such as durable, ttl and priority by extracting them into annotations
at message container init time.
Move type
to remove dependenc on amqp10 stuff in mc.erl
mostly because I don't know how to make bazel do the right thing
add more stuff
Refine routing header stuff
wip
Cosmetics
Do not use "maybe" as type name as "maybe" is a keyword since OTP 25
which makes Erlang LS complain.
* Dedup death queue names
* Fix function clause crashes
Fix failing tests in the MQTT shared_SUITE:
A classic queue message ID can be undefined as set in
fbe79ff47b/deps/rabbit/src/rabbit_classic_queue_index_v2.erl (L1048)
Fix failing tests in the MQTT shared_SUITE-mixed:
When feature flag message_containers is disabled, the
message is not an #mc{} record, but a #basic_message{} record.
* Fix is_utf8_no_null crash
Prior to this commit, the function crashed if invalid UTF-8 was
provided, e.g.:
```
1> rabbit_misc:is_valid_shortstr(<<"😇"/utf16>>).
** exception error: no function clause matching rabbit_misc:is_utf8_no_null(<<216,61,222,7>>) (rabbit_misc.erl, line 1481)
```
* Implement mqtt mc behaviour
For now via amqp translation.
This is still work in progress, but the following SUITEs pass:
```
make -C deps/rabbitmq_mqtt ct-shared t=[mqtt,v5,cluster_size_1] FULL=1
make -C deps/rabbitmq_mqtt ct-v5 t=[mqtt,cluster_size_1] FULL=1
```
* Shorten mc file names
Module name length matters because for each persistent message the #mc{}
record is persisted to disk.
```
1> iolist_size(term_to_iovec({mc, rabbit_mc_amqp_legacy})).
30
2> iolist_size(term_to_iovec({mc, mc_amqpl})).
17
```
This commit renames the mc modules:
```
ag -l rabbit_mc_amqp_legacy | xargs sed -i 's/rabbit_mc_amqp_legacy/mc_amqpl/g'
ag -l rabbit_mc_amqp | xargs sed -i 's/rabbit_mc_amqp/mc_amqp/g'
ag -l rabbit_mqtt_mc | xargs sed -i 's/rabbit_mqtt_mc/mc_mqtt/g'
```
* mc: make deaths an annotation + fixes
* Fix mc_mqtt protocol_state callback
* Fix test will_delay_node_restart
```
make -C deps/rabbitmq_mqtt ct-v5 t=[mqtt,cluster_size_3]:will_delay_node_restart FULL=1
```
* Bazel run gazelle
* mix format rabbitmqctl.ex
* Ensure ttl annotation is refelected in amqp legacy protocol state
* Fix id access in message store
* Fix rabbit_message_interceptor_SUITE
* dializer fixes
* Fix rabbit:rabbit_message_interceptor_SUITE-mixed
set_annotation/3 should not result in duplicate keys
* Fix MQTT shared_SUITE-mixed
Up to 3.12 non-MQTT publishes were always QoS 1 regardless of delivery_mode.
75a953ce28/deps/rabbitmq_mqtt/src/rabbit_mqtt_processor.erl (L2075-L2076)
From now on, non-MQTT publishes are QoS 1 if durable.
This makes more sense.
The MQTT plugin must send a #basic_message{} to an old node that does
not understand message containers.
* Field content of 'v1_0.data' can be binary
Fix
```
bazel test //deps/rabbitmq_mqtt:shared_SUITE-mixed \
--test_env FOCUS="-group [mqtt,v4,cluster_size_1] -case trace" \
-t- --test_sharding_strategy=disabled
```
* Remove route/2 and implement route/3 for all exchange types.
This removes the route/2 callback from rabbit_exchange_type and
makes route/3 mandatory instead. This is a breaking change and
will require all implementations of exchange types to update their
code, however this is necessary anyway for them to correctly handle
the mc type.
stream filtering fixes
* Translate directly from MQTT to AMQP 0.9.1
* handle undecoded properties in mc_compat
amqpl: put clause in right order
recover death deatails from amqp data
* Replace callback init_amqp with convert_from
* Fix return value of lists:keyfind/3
* Translate directly from AMQP 0.9.1 to MQTT
* Fix MQTT payload size
MQTT payload can be a list when converted from AMQP 0.9.1 for example
First conversions tests
Plus some other conversion related fixes.
bazel
bazel
translate amqp 1.0 null to undefined
mc: property/2 and correlation_id/message_id return type tagged values.
To ensure we can support a variety of types better.
The type type tags are AMQP 1.0 flavoured.
fix death recovery
mc_mqtt: impl new api
Add callbacks to allow protocols to compact data before storage
And make readable if needing to query things repeatedly.
bazel fix
* more decoding
* tracking mixed versions compat
* mc: flip default of `durable` annotation to save some data.
Assuming most messages are durable and that in memory messages suffer less
from persistence overhead it makes sense for a non existent `durable`
annotation to mean durable=true.
* mc conversion tests and tidy up
* mc make x_header unstrict again
* amqpl: death record fixes
* bazel
* amqp -> amqpl conversion test
* Fix crash in mc_amqp:size/1
Body can be a single amqp-value section (instead of
being a list) as shown by test
```
make -C deps/rabbitmq_amqp1_0/ ct-system t=java
```
on branch native-amqp.
* Fix crash in lists:flatten/1
Data can be a single amqp-value section (instead of
being a list) as shown by test
```
make -C deps/rabbitmq_amqp1_0 ct-system t=dotnet:roundtrip_to_amqp_091
```
on branch native-amqp.
* Fix crash in rabbit_writer
Running test
```
make -C deps/rabbitmq_amqp1_0 ct-system t=dotnet:roundtrip_to_amqp_091
```
on branch native-amqp resulted in the following crash:
```
crasher:
initial call: rabbit_writer:enter_mainloop/2
pid: <0.711.0>
registered_name: []
exception error: bad argument
in function size/1
called as size([<<0>>,<<"Sw">>,[<<160,2>>,<<"hi">>]])
*** argument 1: not tuple or binary
in call from rabbit_binary_generator:build_content_frames/7 (rabbit_binary_generator.erl, line 89)
in call from rabbit_binary_generator:build_simple_content_frames/4 (rabbit_binary_generator.erl, line 61)
in call from rabbit_writer:assemble_frames/5 (rabbit_writer.erl, line 334)
in call from rabbit_writer:internal_send_command_async/3 (rabbit_writer.erl, line 365)
in call from rabbit_writer:handle_message/2 (rabbit_writer.erl, line 265)
in call from rabbit_writer:handle_message/3 (rabbit_writer.erl, line 232)
in call from rabbit_writer:mainloop1/2 (rabbit_writer.erl, line 223)
```
because #content.payload_fragments_rev is currently supposed to
be a flat list of binaries instead of being an iolist.
This commit fixes this crash inefficiently by calling
iolist_to_binary/1. A better solution would be to allow AMQP legacy's #content.payload_fragments_rev
to be an iolist.
* Add accidentally deleted line back
* mc: optimise mc_amqp internal format
By removint the outer records for message and delivery annotations
as well as application properties and footers.
* mc: optimis mc_amqp map_add by using upsert
* mc: refactoring and bug fixes
* mc_SUITE routingheader assertions
* mc remove serialize/1 callback as only used by amqp
* mc_amqp: avoid returning a nested list from protocol_state
* test and bug fix
* move infer_type to mc_util
* mc fixes and additiona assertions
* Support headers exchange routing for MQTT messages
When a headers exchange is bound to the MQTT topic exchange, routing
will be performend based on both MQTT topic (by the topic exchange) and
MQTT User Property (by the headers exchange).
This combines the best worlds of both MQTT 5.0 and AMQP 0.9.1 and
enables powerful routing topologies.
When the User Property contains the same name multiple times, only the
last name (and value) will be considered by the headers exchange.
* Fix crash when sending from stream to amqpl
When publishing a message via the stream protocol and consuming it via
AMQP 0.9.1, the following crash occurred prior to this commit:
```
crasher:
initial call: rabbit_channel:init/1
pid: <0.818.0>
registered_name: []
exception exit: {{badmatch,undefined},
[{rabbit_channel,handle_deliver0,4,
[{file,"rabbit_channel.erl"},
{line,2728}]},
{lists,foldl,3,[{file,"lists.erl"},{line,1594}]},
{rabbit_channel,handle_cast,2,
[{file,"rabbit_channel.erl"},
{line,728}]},
{gen_server2,handle_msg,2,
[{file,"gen_server2.erl"},{line,1056}]},
{proc_lib,wake_up,3,
[{file,"proc_lib.erl"},{line,251}]}]}
```
This commit first gives `mc:init/3` the chance to set exchange and
routing_keys annotations.
If not set, `rabbit_stream_queue` will set these annotations assuming
the message was originally published via the stream protocol.
* Support consistent hash exchange routing for MQTT 5.0
When a consistent hash exchange is bound to the MQTT topic exchange,
MQTT 5.0 messages can be routed to queues consistently based on the
Correlation-Data in the PUBLISH packet.
* Convert MQTT 5.0 User Property
* to AMQP 0.9.1 headers
* from AMQP 0.9.1 headers
* to AMQP 1.0 application properties and message annotations
* from AMQP 1.0 application properties and message annotations
* Make use of Annotations in mc_mqtt:protocol_state/2
mc_mqtt:protocol_state/2 includes Annotations as parameter.
It's cleaner to make use of these Annotations when computing the
protocol state instead of relying on the caller (rabbitmq_mqtt_processor)
to compute the protocol state.
* Enforce AMQP 0.9.1 field name length limit
The AMQP 0.9.1 spec prohibits field names longer than 128 characters.
Therefore, when converting AMQP 1.0 message annotations, application
properties or MQTT 5.0 User Property to AMQP 0.9.1 headers, drop any
names longer than 128 characters.
* Fix type specs
Apply feedback from Michael Davis
Co-authored-by: Michael Davis <mcarsondavis@gmail.com>
* Add mc_mqtt unit test suite
Implement mc_mqtt:x_header/2
* Translate indicator that payload is UTF-8 encoded
when converting between MQTT 5.0 and AMQP 1.0
* Translate single amqp-value section from AMQP 1.0 to MQTT
Convert to a text representation, if possible, and indicate to MQTT
client that the payload is UTF-8 encoded. This way, the MQTT client will
be able to parse the payload.
If conversion to text representation is not possible, encode the payload
using the AMQP 1.0 type system and indiate the encoding via Content-Type
message/vnd.rabbitmq.amqp.
This Content-Type is not registered.
Type "message" makes sense since it's a message.
Vendor tree "vnd.rabbitmq.amqp" makes sense since merely subtype "amqp" is not
registered.
* Fix payload conversion
* Translate Response Topic between MQTT and AMQP
Translate MQTT 5.0 Response Topic to AMQP 1.0 reply-to address and vice
versa.
The Response Topic must be a UTF-8 encoded string.
This commit re-uses the already defined RabbitMQ target addresses:
```
"/topic/" RK Publish to amq.topic with routing key RK
"/exchange/" X "/" RK Publish to exchange X with routing key RK
```
By default, the MQTT topic exchange is configure dto be amq.topic using
the 1st target address.
When an operator modifies the mqtt.exchange, the 2nd target address is
used.
* Apply PR feedback
and fix formatting
Co-authored-by: Michael Davis <mcarsondavis@gmail.com>
* tidy up
* Add MQTT message_containers test
* consistent hash exchange: avoid amqp legacy conversion
When hashing on a header value.
* Avoid converting to amqp legacy when using exchange federation
* Fix test flake
* test and dialyzer fixes
* dialyzer fix
* Add MQTT protocol interoperability tests
Test receiving from and sending to MQTT 5.0 and
* AMQP 0.9.1
* AMQP 1.0
* STOMP
* Streams
* Regenerate portions of deps/rabbit/app.bzl with gazelle
I'm not exactly sure how this happened, but gazell seems to have been
run with an older version of the rules_erlang gazelle extension at
some point. This caused generation of a structure that is no longer
used. This commit updates the structure to the current pattern.
* mc: refactoring
* mc_amqpl: handle delivery annotations
Just in case they are included.
Also use iolist_to_iovec to create flat list of binaries when
converting from amqp with amqp encoded payload.
---------
Co-authored-by: David Ansari <david.ansari@gmx.de>
Co-authored-by: Michael Davis <mcarsondavis@gmail.com>
Co-authored-by: Rin Kuryloski <kuryloskip@vmware.com>
[Why]
The `enable` callback is executed on each node of the cluster. It could
succeed on some of them and fail on other nodes. If it succeeds
everywhere, the controller could still fail to mark the feature flag as
enabled on some of the nodes.
When this happens, we correctly mark the feature flag back as disabled
everywhere. However, the controller never gave a chance to the feature
flag callbacks to roll back anything.
[How]
Now, the controller always runs the `post_enable` callback (if any)
after it ran the `enable` callback. It adds the following field to the
passed map of arguments to indicate if the feature flag was enabled or
not:
#{enabled => boolean()}
While here, fix two things:
1. One call to `restore_feature_flag_state()` was passed an older
"version" of the inventory, instead of the latest modified one.
2. One log message had no domain set.
[Why]
The feature flags controller ensures all nodes in a cluster are running
before a feature flag can be enabled. It continues to do so whenever it
wants to record a state change because it requires that all nodes get
the new state otherwise the task in aborted.
However, it's difficult to verify that through out the entire process if
the feature flag has an `enable` callback. But again, if we loose a node
during the execution of the callback or between its execution and the
time we mark the feature flag as enabled on all nodes, that's ok because
the feature flag will be marked as disabled everywhere: the remaining
running nodes will go back from `state_changing` to `false` and the
stopped nodes will keep their initial state of `false`.
Nonetheless, we can increase the chance of letting an `enable` operation
to finish if the controller waits for anything in-flight before is
actually exits.
[How]
The `terminate/3` function now tries to register globally, like if the
controller wanted to lock the cluster and run a task. If it succeeds to
register, it means nothing is running in parallel and it can exit. If it
fails, it waits for the globally-registered controller to finish and
tries to register again.
We expose a new `wait_for_task_and_stop/0` function to explicitly stop
the feature flags controller and call it from the `rabbit` application
pre-stop phase. The reason is that when the supervisor asks the
controller to stop as part of the regular shutdown of a supervision
tree, it has a timeout and could kill the controller if an in-flight
operation takes too much time. To avoid this kill, we prefer to use
`wait_for_task_and_stop/0` which has no timeout.
When subscribing using a consumer tag that is already in the quorum
queues state (but perhaps with a cancelled status) and that has
pending messages the next_msg_id which is used to initialise the
queue type consumer state did not take the in flight message ids into
account. This resulted in some messages occasionally not being delivered
to the clint and thus would appear stuck as awaiting acknowledgement
for the consumer.
When a new checkout operation detects there are in-flight messages
we set the last_msg_id to `undefined` and just accept the next message
that arrives, irrespective of their message id. This isn't 100% fool proof
as there may be cases where messages are lost between queue and channel
where we'd miss to trigger the fallback query for missing messages.
It is however much better than what we have atm.
NB: really the ideal solution would be to make checkout operations
async so that any inflight messages are delivered before the checkout
result. That is a much bigger change for another day.
[Why]
They are deprecated. Currently, we simply got a warning in the logs but
in a few minor versions, the testcase will start to fail because it
may not be able to declare a queue.
[Why]
We were running the check to make sure the exchange was declared, but we
didn't verify the result of that check. The testcase would still fail
later but if we verify its existence early, the testcase can fail early
too.
since MQTT 5.0 supports negative acknowledgements thanks to reason codes
in the PUBACK packet.
We could either choose reason code 128 or 131. The description code for
131 applies for rejected messages, hence this commit uses 131:
> The PUBLISH is valid but the receiver is not willing to accept it.
[Why]
The testcase used to set the `cluster_formation` proplist twice. It is
very ambiguous what we should do: is only one of them relevant or should
they be merged?
[How]
We merge both proplists into a single one.
[Why]
The previous detection was based on a reuse of the channel to get the
error from an exit exception. The problem is that it is very dependent
on the timing: if the channel process exits before it is reused, the
test fails for two possible reasons:
1. The channel and connection processes exit before they are reused and
the channel manager opens a new pair. The problem is that the declare
suceeds but the test expected a failure.
2. The channel and connection processes exit during the reuse and
`rabbit_ct_client_helpers:open_channel` in
`retry_if_coordinator_unavailable()` waits a response from the
channel manager forever (this is probably a weakness of the channel
manager in rabbitmq_ct_client_helpers). This indefinite wait causes
the testcase to timeout.
[How]
A simpler solution is to monitor the exit reason of the channel process
that triggers the error on the server side.
[Why]
We don't record the state of deprecated features because it is
controlled from configuration and they can be disabled (the deprecated
feature can be turned back on) if the deprecated feature allows it.
However, some feature flags may depend on deprecated features. If those
feature flags are enabled, we need to enable the deprecated features
(turn off the deprecated features) they depend on regardless of the
configuration.
[How]
During the (re)initialization of the registry, we go through all enabled
feature flags and deprecated features' `depends_on` declarations and
consider all their dependencies to be implicitly enabled.
[Why]
A database reset removes the enabled feature flags file on disc. A reset
of the registry ensures that the next time the registry is reloaded, it
is also initialized from scratch.
[How]
We call `rabbit_feature_flags:reset_registry/0` after both a regular
reset and a forced reset.
The `reset_registry/0` is also exposed by the `rabbit_feature_flags`
module now. The actual implementation in `rabbit_ff_registry_factory`
should only be called by the Feature flags subsystem itself.
[Why]
Testcases fail with various system errors in CI, like the inability to
spawn system processes or to open a TCP port.
[How]
We check if the `$RABBITMQ_RUN` environment variable is set. It is only
set by Bazel and not make(1). Based on that, we compute the test group
options to include `parallel` or not.
[Why]
The CLI may be used against a remote node running a different version.
We took that into account in several uses of the `rabbit_db*` modules on
remote nodes, but not everywhere. Likewise in the
`clustering_management_SUITE` testsuite.
[How]
This patch falls back to previous `rabbit_mnesia`-based calls if the
initial calls throws an `undef` exception.
If the target for at least once dead lettering included the source queue
the dead letter outbound queue in the quorum queue would never be cleared.
This changes the queue -> dead letter worker message format to better distinguish
between those and queue events for "normal" queue type interactions.
[Why]
We want the code to depend less on Mnesia (and not at all in the
future). We also want to make room to introduce the use of Khepri.
[How]
For now, we simply store each list in a variable. This give them a name
to better understand what each one is.
`rabbit_mnsia:cluster_nodes(all)` is also replaced by
`rabbit_db_cluster:members()`. The other two calls to `rabbit_mnesia`
are left alone as they are quite specific to Mnesia.
[Why]
Peer discovery is not Mnesia-specific and will be used once we introduce
Khepri.
[How]
The whole peer discovery driving code is moved from `rabbit_mnesia` to
`rabbit_peer_discovery`. When `rabbit_mnesia` calls that code, it simply
passes a callback for the Mnesia-specific cluster expansion code.
[Why]
Now that feature flags compatibility is tested first, before
Mnesia-specific checks, when a peer is not started yet, the feature
flags check lasts the entire timeout, so one minute. This retry
mechanism was added to feature flags in #8411.
Thus, instead of 20 seconds, the testcase takes 10 minutes now (10
retries of one minute each).
[Why]
Transient queues are queues that are removed upon node restart. An
application developer can't rely on such an arbitrary event to reason
about a queue's lifetime.
The only exception are exclusive transient queues which have a lifetime
linked to that of a client connection.
[How]
Non-exclusive transient queues are marked as deprecated in the code
using the Deprecated features subsystem (based on feature flags). See
pull request #7390 for a description of that subsystem.
To test RabbitMQ behavior as if the feature was removed, the following
configuration setting can be used:
deprecated_features.permit.transient_nonexcl_queues = false
Non-exclusive transient queues can be turned off anytime, there are no
conditions to do that.
Once non-exclusive transient queues are turned off, declaring a new
queue with those arguments will be rejected with a protocol error.
Note that given the marketing calendar, the deprecated feature will go
directly from "permitted by default" to "removed" in RabbitMQ 4.0. It
won't go through the gradual deprecation process.
[Why]
Classic queue mirroring will be removed in RabbitMQ 4.0. Quorum queues
provide a better safer alternative. Non-replicated classic queues remain
supported.
[How]
Classic queue mirroring is marked as deprecated in the code using the
Deprecated features subsystem (based on feature flags). See #7390 for a
description of that subsystem.
To test RabbitMQ behavior as if the feature was removed, the following
configuration setting can be used:
deprecated_features.permit.classic_queue_mirroring = false
To turn off classic queue mirroring, there must be no classic mirrored
queues declared and no HA policy defined. A node with classic mirrored
queues will refuse to start if classic queue mirroring is turned off.
Once classic queue mirroring is turned off, users will not be able to
declare HA policies. Trying to do that from the CLI or the management
API will be rejected with a warning in the logs. This impacts clustering
too: a node with classic queue mirroring turned off will only cluster
with another node which has no HA policy or has classic queue mirroring
turned off.
Note that given the marketing calendar, the deprecated feature will go
directly from "permitted by default" to "removed" in RabbitMQ 4.0. It
won't go through the gradual deprecation process.
V2: Renamed the deprecated feature from `classic_mirrored_queues` to
`classic_queue_mirroring` to better reflect the intention. Otherwise
it could be unclear is only the mirroring property is
deprecated/removed or classic queues entirely.
[Why]
RAM nodes provide no safety at all and they lost interest with recent
fast storage solutions.
[How]
RAM nodes are marked as deprecated in the code using the Deprecated
features subsystem (based on feature flags). See pull request #7390 for
a description of that subsystem.
To test RabbitMQ behavior as if the feature was removed, the following
configuration setting can be used:
deprecated_features.permit.ram_node_type = false
RAM nodes can be turned off anytime, there are no conditions to do that.
Once RAM nodes are turned off, an existing node previously created as a
RAM node will change itself to a disc node during boot. If a new node is
added to the cluster using peer discovery or the CLI, it will be as a
disc node and a warning will be logged if the requested node type is
RAM. The `change_cluster_node_type` CLI command will reject a change to
a RAM node with an error.
Note that given the marketing calendar, the deprecated feature will go
directly from "permitted by default" to "removed" in RabbitMQ 4.0. It
won't go through the gradual deprecation process.
[Why]
Global QoS, where a single shared prefetch is used for an entire
channel, is not recommended practice. Per-consumer QoS (non-global)
should be set instead.
[How]
The global QoS setting is marked as deprecated in the code using the
Deprecated features subsystem (based on feature flags). See #7390 for a
description of that subsystem.
To test RabbitMQ behavior as if the feature was removed, the following
configuration setting can be used:
deprecated_features.permit.global_qos = false
Global QoS can be turned off anytime, there are no conditions to do
that.
Once global QoS is turned off, the prefetch setting will always be
considered as non-global (i.e. per-consumer). A warning message will be
logged if the default prefetch setting enables global QoS or anytime a
client requests a global QoS on the channel.
Note that given the marketing calendar, the deprecated feature will go
directly from "permitted by default" to "removed" in RabbitMQ 4.0. It
won't go through the gradual deprecation process.
Hashing the #resource{} record is expensive.
Routing to 40k queues via the topic exchanges takes:
~150ms prior to this commit
~100ms after this commit
As rabbit_exchange already deduplicates destination queues and binding
keys, there's no need to use maps in rabbit_db_topic_exchange or
rabbit_exchange_type_topic.
For MQTT 5.0 destination queues, the topic exchange does not only have
to return the destination queue names, but also the matched binding
keys.
This is needed to implement MQTT 5.0 subscription options No Local,
Retain As Published and Subscription Identifiers.
Prior to this commit, as the trie was walked down, we remembered the
edges being walked and assembled the final binding key with
list_to_binary/1.
list_to_binary/1 is very expensive with long lists (long topic names),
even in OTP 26.
The CPU flame graph showed ~3% of CPU usage was spent only in
list_to_binary/1.
Unfortunately and unnecessarily, the current topic exchange
implementation stores topic levels as lists.
It would be better to store topic levels as binaries:
split_topic_key/1 should ideally use binary:split/3 similar as follows:
```
1> P = binary:compile_pattern(<<".">>).
{bm,#Ref<0.1273071188.1488322568.63736>}
2> Bin = <<"aaa.bbb..ccc">>.
<<"aaa.bbb..ccc">>
3> binary:split(Bin, P, [global]).
[<<"aaa">>,<<"bbb">>,<<>>,<<"ccc">>]
```
The compiled pattern could be placed into persistent term.
This commit decided to avoid migrating Mnesia tables to use binaries
instead of lists. Mnesia migrations are non-trivial, especially with the
current feature flag subsystem.
Furthermore the Mnesia topic tables are already getting migrated to
their Khepri counterparts in 3.13.
Adding additional migration only for Mnesia does not make sense.
So, instead of assembling the binding key as we walk down the trie and
then calling list_to_binary/1 in the leaf, it
would be better to just fetch the binding key from the database in the leaf.
As we reach the leaf of the trie, we know both source and destination.
Unfortunately, we cannot fetch the binding key efficiently with the
current rabbit_route (sorted by source exchange) and
rabbit_reverse_route (sorted by destination) tables as the key is in
the middle between source and destination.
If there are a huge number of bindings for a given sourc exchange (very
realistic in MQTT use cases) or a large number of bindings for a given
destination (also realistic), it would require scanning these large
number of bindings.
Therefore this commit takes the simplest possible solution:
The solution leverages the fact that binding arguments are already part of
table rabbit_topic_trie_binding.
So, if we simply include the binding key into the binding arguments, we
can fetch and return it efficiently in the topic exchange
implementation.
The following patch omitting fetching the empty list binding argument
(the default) makes routing slower because function
`analyze_pattern.constprop.0` requires significantly more (~2.5%) CPU time
```
@@ -273,7 +273,11 @@ trie_bindings(X, Node) ->
node_id = Node,
destination = '$1',
arguments = '$2'}},
- mnesia:select(?MNESIA_BINDING_TABLE, [{MatchHead, [], [{{'$1', '$2'}}]}]).
+ mnesia:select(
+ ?MNESIA_BINDING_TABLE,
+ [{MatchHead, [{'andalso', {'is_list', '$2'}, {'=/=', '$2', []}}], [{{'$1', '$2'}}]},
+ {MatchHead, [], ['$1']}
+ ]).
```
Hence, this commit always fetches the binding arguments.
All MQTT 5.0 destination queues will create a binding that
contains the binding key in the binding arguments.
Not only does this solution avoid expensive list_to_binay/1 calls, but
it also means that Erlang app rabbit (specifically the topic exchange
implementation) does not need to be aware of MQTT anymore:
It just returns the binding key when the binding args tell to do so.
In future, once the Khepri migration completed, we should be able to
relatively simply remove the binding key from the binding arguments
again to free up some storage space.
Note that one of the advantages of a trie data structue is its space
efficiency that you don't have to store the same prefixes multiple
times.
However, for RabbitMQ the binding key is already stored at least N times
in various routing tables, so storing it a few times more via the
binding arguments should be acceptable.
The speed improvements are favoured over a few more MBs ETS usage.
Support subscription options "No Local" and "Retain As Published"
as well as Subscription Identifiers.
All three MQTT 5.0 features can be set on a per subscription basis.
Due to wildcards in topic filters, multiple subscriptions
can match a given topic. Therefore, to implement Retain As Published and
Subscription Identifiers, the destination MQTT connection process needs
to know what subscription(s) caused it to receive the message.
There are a few ways how this could be implemented:
1. The destination MQTT connection process is aware of all its
subscriptions. Whenever, it receives a message, it can match the
message's routing key / topic against all its known topic filters.
However, to iteratively match the routing key against all topic
filters for every received message can become very expensive in the
worst case when the MQTT client creates many subscriptions containing
wildcards. This could be the case for an MQTT client that acts as a
bridge or proxy or dispatcher: It could subscribe via a wildcard for
each of its own clients.
2. Instead of interatively matching the topic of the received message
against all topic filters that contain wildcards, a better approach
would be for every MQTT subscriber connection process to maintain a
local trie datastructure (similar to how topic exchanges are
implemented) and perform matching therefore more efficiently.
However, this does not sound optimal either because routing is
effectively performed twice: in the topic exchange and again against
a much smaller trie in each destination connection process.
3. Given that the topic exchange already perform routing, a much more
sensible way would be to send the matched binding key(s) to the
destination MQTT connection process. A subscription (topic filter)
maps to a binding key in AMQP 0.9.1 routing. Therefore, for the first
time in RabbitMQ, the routing function should not only output a list
of unique destination queues, but also the binding keys (subscriptions)
that caused the message to be routed to the destination queue.
This commit therefore implements the 3rd approach.
The downside of the 3rd approach is that it requires API changes to the
routing function and topic exchange.
Specifically, this commit adds a new function rabbit_exchange:route/3
that accepts a list of routing options. If that list contains version 2,
the caller of the routing function knows how to handle the return value
that could also contain binding keys.
This commits allows an MQTT connection process, the channel process, and
at-most-once dead lettering to handle binding keys. Binding keys are
included as AMQP 0.9.1 headers into the basic message.
Therefore, whenever a message is sent from an MQTT client or AMQP 0.9.1
client or AMQP 1.0 client or STOMP client, the MQTT receiver will know
the subscription identifier that caused the message to be received.
Note that due to the low number of allowed wildcard characters (# and
+), the cardinality of matched binding keys shouldn't be high even if
the topic contains for example 3 levels and the message is sent to for
example 5 million destination queues. In other words, sending multiple
distinct basic messages to the destination shouldn't hurt the delegate
optimisation too much. The delegate optimisation implemented for classic
queues and rabbit_mqtt_qos0_queue(s) still takes place for all basic
messages that contain the same set of matched binding keys.
The topic exchange returns all matched binding keys by remembering the
edges walked down to the leaves. As an optimisation, only for MQTT
queues are binding keys being returned. This does add a small dependency
from app rabbit to app rabbitmq_mqtt which is not optimal. However, this
dependency should be simple to remove when omitting this optimisation.
Another important feature of this commit is persisting subscription
options and subscription identifiers because they are part of the
MQTT 5.0 session state.
In MQTT v3 and v4, the only subscription information that were part of
the session state was the topic filter and the QoS level.
Both information were implicitly stored in the form of bindings:
The topic filter as the binding key and the QoS level as the destination
queue name of the binding.
For MQTT v5 we need to persist more subscription information.
From a domain perspective, it makes sense to store subscription options
as part of subscriptions, i.e. bindings, even though they are currently
not used in routing.
Therefore, this commits stores subscription options as binding arguments.
Storing subscription options as binding arguments comes in turn with
new challenges: How to handle mixed version clusters and upgrading an
MQTT session from v3 or v4 to v5?
Imagine an MQTT client connects via v5 with Session Expiry Interval > 0
to a new node in a mixed version cluster, creates a subscription,
disconnects, and subsequently connects via v3 to an old node. The
client should continue to receive messages.
To simplify such edge cases, this commit introduces a new feature flag
called mqtt_v5. If mqtt_v5 is disabled, clients cannot connect to
RabbitMQ via MQTT 5.0.
This still doesn't entirely solve the problem of MQTT session upgrades
(v4 to v5 client) or session downgrades (v5 to v4 client).
Ideally, once mqtt_v5 is enabled, all MQTT bindings contain non-empty binding
arguments. However, this will require a feature flag migration function
to modify all MQTT bindings. To be more precise, all MQTT bindings need
to be deleted and added because the binding argument is part of the
Mnesia table key.
Since feature flag migration functions are non-trivial to implement in
RabbitMQ (they can run on every node multiple times and concurrently),
this commit takes a simpler approach:
All v3 / v4 sessions keep the empty binding argument [].
All v5 sessions use the new binding argument [#mqtt_subscription_opts{}].
This requires only handling a session upgrade / downgrade by
creating a binding (with the new binding arg) and deleting the old
binding (with the old binding arg) when processing the CONNECT packet.
Note that such session upgrades or downgrades should be rather rare in
practice. Therefore these binding transactions shouldn't hurt peformance.
The No Local option is implemented within the MQTT publishing connection
process: The message is not sent to the MQTT destination if the
destination queue name matches the current MQTT client ID and the
message was routed due to a subscription that has the No Local flag set.
This avoids unnecessary traffic on the MQTT queue.
The alternative would have been that the "receiving side" (same process)
filters the message out - which would have been more consistent in how
Retain As Published and Subscription Identifiers are implemented, but
would have caused unnecessary load on the MQTT queue.
"Allow the Client and Server to independently specify the maximum
packet size they support. It is an error for the session partner
to send a larger packet."
This commit implements the part where the Client specifies the maximum
packet size.
As per protocol spec, instead of sending, the server drops the MQTT packet
if it's too large.
A debug message is logged for "infrequent" packet types.
For PUBLISH packets, the messages is rejected to the queue such that it
will be dead lettered, if dead lettering is configured.
At the very least, Prometheus metrics for dead lettered messages will
be increased, even if dead lettering is not configured.
* CQ: Don't use FHC for writes in shared store
* CQ: Send confirms when flushing to disk in shared store
Before they would only be sent periodically or when
rolling over to a new file.
* CQ: Fast-confirm when flushing data to disk
We know the messages are on disk or were acked so there is no
need to do sets intersections/subtracts in this scenario.
* Fix a Dialyzer warning
* Faster confirms for unwritten messages
Instead of having the message store send a message to the queue
with the confirms for messages ignored due to the flying
optimisation, we have the queue handle the confirms directly
when removing the messages.
This avoids sending potentially 1 Erlang message per 1 AMQP
message to the queue.
* Refactor rabbit_msg_file:pread into rabbit_msg_store
Also make use of the opened file for multi-reads instead
of opening/reading/closing each time.
* CQ: Make sure we keep the updated CState when using read_many
* CQ shared store: Run compaction on older file candidates
The way I initially did this the maybe_gc would be triggered
based on candidates from 15s ago, but run against candidates
from just now. This is sub-optimal because when messages are
consumed rapidly, just-now candidates are likely to be in a
file about to be deleted, and we don't want to run compaction
on those.
Instead, when sending the maybe_gc we also send the candidates
we had at the time. Then 15s later we check if the file still
exists. If it's gone, great! No compaction to do.
* CQ: Add a few todos for later
Since CMQs are on their way out, we are only willing
to spend so much time on it.
The test covers a scenario where four nodes are stopped, then
one force booted and then immediately removed from the cluster.
In other words, a scenario that's quite unrealistic.
This introduces a way to declare deprecated features in the code, not
only in our communication. The new module allows to disallow the use of
a deprecated feature and/or warn the user when he relies on such a
feature.
[Why]
Currently, we only tell people about deprecated features through blog
posts and the mailing-list. This might be insufficiant for our users
that a feature they use will be removed in a future version:
* They may not read our blog or mailing-list
* They may not understand that they use such a deprecated feature
* They might wait for the big removal before they plan testing
* They might not take it seriously enough
The idea behind this patch is to increase the chance that users notice
that they are using something which is about to be dropped from
RabbitMQ. Anopther benefit is that they should be able to test how
RabbitMQ will behave in the future before the actual removal. This
should allow them to test and plan changes.
[How]
When a feature is deprecated in other large projects (such as FreeBSD
where I took the idea from), it goes through a lifecycle:
1. The feature is still available, but users get a warning somehow when
they use it. They can disable it to test.
2. The feature is still available, but disabled out-of-the-box. Users
can re-enable it (and get a warning).
3. The feature is disconnected from the build. Therefore, the code
behind it is still there, but users have to recompile the thing to be
able to use it.
4. The feature is removed from the source code. Users have to adapt or
they can't upgrade anymore.
The solution in this patch offers the same lifecycle. A deprecated
feature will be in one of these deprecation phases:
1. `permitted_by_default`: The feature is available. Users get a warning
if they use it. They can disable it from the configuration.
2. `denied_by_default`: The feature is available but disabled by
default. Users get an error if they use it and RabbitMQ behaves like
the feature is removed. They can re-enable is from the configuration
and get a warning.
3. `disconnected`: The feature is present in the source code, but is
disabled and can't be re-enabled without recompiling RabbitMQ. Users
get the same behavior as if the code was removed.
4. `removed`: The feature's code is gone.
The whole thing is based on the feature flags subsystem, but it has the
following differences with other feature flags:
* The semantic is reversed: the feature flag behind a deprecated feature
is disabled when the deprecated feature is permitted, or enabled when
the deprecated feature is denied.
* The feature flag behind a deprecated feature is enabled out-of-the-box
(meaning the deprecated feature is denied):
* if the deprecation phase is `permitted_by_default` and the
configuration denies the deprecated feature
* if the deprecation phase is `denied_by_default` and the
configuration doesn't permit the deprecated feature
* if the deprecation phase is `disconnected` or `removed`
* Feature flags behind deprecated feature don't appear in feature flags
listings.
Otherwise, deprecated features' feature flags are managed like other
feature flags, in particular inside clusters.
To declare a deprecated feature:
-rabbit_deprecated_feature(
{my_deprecated_feature,
#{deprecation_phase => permitted_by_default,
msgs => #{when_permitted => "This feature will be removed in RabbitMQ X.0"},
}}).
Then, to check the state of a deprecated feature in the code:
case rabbit_deprecated_features:is_permitted(my_deprecated_feature) of
true ->
%% The deprecated feature is still permitted.
ok;
false ->
%% The deprecated feature is gone or should be considered
%% unavailable.
error
end.
Warnings and errors are logged automatically. A message is generated
automatically, but it is possible to define a message in the deprecated
feature flag declaration like in the example above.
Here is an example of a logged warning that was generated automatically:
Feature `my_deprecated_feature` is deprecated.
By default, this feature can still be used for now.
Its use will not be permitted by default in a future minor RabbitMQ version and the feature will be removed from a future major RabbitMQ version; actual versions to be determined.
To continue using this feature when it is not permitted by default, set the following parameter in your configuration:
"deprecated_features.permit.my_deprecated_feature = true"
To test RabbitMQ as if the feature was removed, set this in your configuration:
"deprecated_features.permit.my_deprecated_feature = false"
To override the default state of `permitted_by_default` and
`denied_by_default` deprecation phases, users can set the following
configuration:
# In rabbitmq.conf:
deprecated_features.permit.my_deprecated_feature = true # or false
The actual behavior protected by a deprecated feature check is out of
scope for this subsystem. It is the repsonsibility of each deprecated
feature code to determine what to do when the deprecated feature is
denied.
V1: Deprecated feature states are initially computed during the
initialization of the registry, based on their deprecation phase and
possibly the configuration. They don't go through the `enable/1`
code at all.
V2: Manage deprecated feature states as any other non-required
feature flags. This allows to execute an `is_feature_used()`
callback to determine if a deprecated feature can be denied. This
also allows to prevent the RabbitMQ node from starting if it
continues to use a deprecated feature.
V3: Manage deprecated feature states from the registry initialization
again. This is required because we need to know very early if some
of them are denied, so that an upgrade to a version of RabbitMQ
where a deprecated feature is disconnected or removed can be
performed.
To still prevent the start of a RabbitMQ node when a denied
deprecated feature is actively used, we run the `is_feature_used()`
callback of all denied deprecated features as part of the
`sync_cluster()` task. This task is executed as part of a feature
flag refresh executed when RabbitMQ starts or when plugins are
enabled. So even though a deprecated feature is marked as denied in
the registry early in the boot process, we will still abort the
start of a RabbitMQ node if the feature is used.
V4: Support context-dependent warnings. It is now possible to set a
specific message when deprecated feature is permitted, when it is
denied and when it is removed. Generic per-context messages are
still generated.
V5: Improve default warning messages, thanks to @pstack2021.
V6: Rename the configuration variable from `permit_deprecated_features.*`
to `deprecated_features.permit.*`. As @michaelklishin said, we tend
to use shorter top-level names.
[Why]
There could be a transient network issue. Let's give a few more chances
to perform the requested RPC call.
[How]
We retry until the given timeout is reached, if any.
To honor that timeout, we measure the time taken by the RPC call itself.
We also sleep between retries. Before each retry, the timeout is reduced
by the total of the time taken by the RPC call and the sleep.
References #8346.
V2: Treat `infinity` timeout differently. In this case, we never retry
following a `noconnection` error. The reason is that this timeout is
used specifically for callbacks executed remotely. We don't know how
long they take (for instance if there is a lot of data to migrate).
We don't want an infinite retry loop either, so in this case, we
don't retry.
[Why]
During peer discovery, when the feature flags state is synchronized on a
starting node that joins a cluster thanks to peer discovery, the list of
nodes returned by `rabbit_nodes:list_running()` is incorrect because
Mnesia is not initiliazed yet.
Because of that, the synchronization works on the wrong inventory of
feature flags. In the end, the states of feature flags is incorrect
across the cluster.
[How]
`rabbit_mnesia` passes a list of nodes to
`rabbit_feature_flags:sync_feature_flags_with_cluster/2`. We can use
this list, as we did in feature flags V1. This makes sure that the
synchronization works with a valid list of cluster members, in case the
cluster state is not ready yet.
V2: Filter the given list of nodes to only keep those where `rabbit` is
running. This avoids trying to collect inventory from nodes which
are stopped.
Instead of doing a complicated +1/-1 we do an update_counter
of an integer value using 2^n values. We always know exactly
in which state we are when looking at the ets table. We also
can avoid some ets operations as a result although the
performance improvements are minimal.
In mixed version cluster tests where the new node
uses CQv2, when mirror synchronisation happens,
v2 (source) overloads v1 (destination) leading to
a memory spike and a crash (in a memory-constrained
CI environment). Given that in 3.12 we switch to
a lazy-like mode for all classic queues, I think
we can make use a lazy queue in the test.
[Why]
The background reason for this fix is about the same as the one
explained in the previous version of this fix; see commit
e0a2f10272.
This time, the order of events that led to a similar deadlock is the
following:
0. No `rabbit_ff_registry` is loaded yet.
1. Process A, B and C call `rabbit_ff_registry:something()` indirectly
which triggers two initializations in parallel.
* Process A did it from an explicit call to
`rabbit_ff_registry_factory:initialize_factory()` during RabbitMQ
boot.
* Process B and C indirectly called it because they checked if a
feature flag was enabled.
2. Process B acquires the lock first and finishes the initialization. A
new registry is loaded and the old `rabbit_ff_registry` module copy
is marked as "old". At this point, process A and C still reference
that old copy because `rabbit_ff_registry:something()` is up above in
its call stack.
3. Process A acquires the lock, prepares the new registry and tries to
soft-purge the old `rabbit_ff_registry` copy before loading the new
one.
This is where the deadlock happens: process A requests the Code server
to purge the old copy, but the Code server waits for process C to stop
using it.
The difference between the steps described in the first bug fix
attempt's commit and these ones is that the process which lingers on the
deleted `rabbit_ff_registry` (process C above) isn't the one who
acquired the lock; process A has it.
That's why the first bug fix isn't effective in this case: it relied on
the fact that the process which lingers on the deleted
`rabbit_ff_registry` is the process which attempts to purge the module.
[How]
In this commit, we go with a more drastic change. This time, we put a
wrapper in front of `rabbit_ff_registry` called
`rabbit_ff_registry_wrapper`. This wrapper is responsible for doing the
automatic initialization if the loaded registry is the stub module. The
`rabbit_ff_registry` stub now always returns `init_required` instead of
performing the initialization and calling itself recursively.
This way, processes linger on `rabbit_ff_registry_wrapper`, not on
`rabbit_ff_registry`. Thanks to this, the Code server can proceed with
the purge.
See #8112.
as it nicer categorises if there will be a future
"message_interceptors.outgoing.*" key.
We leave the advanced config file key because simple single value
settings should not require using the advanced config file.
As reported in https://groups.google.com/g/rabbitmq-users/c/x8ACs4dBlkI/
plugins that implement rabbit_channel_interceptor break with
Native MQTT in 3.12 because Native MQTT does not use rabbit_channel anymore.
Specifically, these plugins don't work anymore in 3.12 when sending a message
from an MQTT publisher to an AMQP 0.9.1 consumer.
Two of these plugins are
https://github.com/rabbitmq/rabbitmq-message-timestamp
and
https://github.com/rabbitmq/rabbitmq-routing-node-stamp
This commit moves both plugins into rabbitmq-server.
Therefore, these plugins are deprecated starting in 3.12.
Instead of using these plugins, the user gets the same behaviour by
configuring rabbitmq.conf as follows:
```
incoming_message_interceptors.set_header_timestamp.overwrite = false
incoming_message_interceptors.set_header_routing_node.overwrite = false
```
While both plugins were incompatible to be used together, this commit
allows setting both headers.
We name the top level configuration key `incoming_message_interceptors`
because only incoming messages are intercepted.
Currently, only `set_header_timestamp` and `set_header_routing_node` are
supported. (We might support more in the future.)
Both can set `overwrite` to `false` or `true`.
The meaning of `overwrite` is the same as documented in
https://github.com/rabbitmq/rabbitmq-message-timestamp#always-overwrite-timestamps
i.e. whether headers should be overwritten if they are already present
in the message.
Both `set_header_timestamp` and `set_header_routing_node` behave exactly
to plugins `rabbitmq-message-timestamp` and `rabbitmq-routing-node-stamp`,
respectively.
Upon node boot, the configuration is put into persistent_term to not
cause any performance penalty in the default case where these settings
are disabled.
The channel and MQTT connection process will intercept incoming messages
and - if configured - add the desired AMQP 0.9.1 headers.
For now, this allows using Native MQTT in 3.12 with the old plugins
behaviour.
In the future, once "message containers" are implemented,
we can think about more generic message interceptors where plugins can be
written to modify arbitrary headers or message contents for various protocols.
Likewise, in the future, once MQTT 5.0 is implemented, we can think
about an MQTT connection interceptor which could function similar to a
`rabbit_channel_interceptor` allowing to modify any MQTT packet.
[Why]
The Feature flags registry is implemented as a module called
`rabbit_ff_registry` recompiled and reloaded at runtime.
There is a copy on disk which is a stub responsible for triggering the
first initialization of the real registry and please Dialyzer. Once the
initialization is done, this stub calls `rabbit_ff_registry` again to
get an actual return value. This is kind of recursive: the on-disk
`rabbit_ff_registry` copy calls the `rabbit_ff_registry` copy generated
at runtime.
Early during RabbitMQ startup, there could be multiple processes
indirectly calling `rabbit_ff_registry` and possibly triggering that
first initialization concurrently. Unfortunately, there is a slight
chance of race condition and deadlock:
0. No `rabbit_ff_registry` is loaded yet.
1. Both process A and B call `rabbit_ff_registry:something()` indirectly
which triggers two initializations in parallel.
2. Process A acquires the lock first and finishes the initialization. A
new registry is loaded and the old `rabbit_ff_registry` module copy
is marked as "old". At this point, process B still references that
old copy because `rabbit_ff_registry:something()` is up above in its
call stack.
3. Process B acquires the lock, prepares the new registry and tries to
soft-purge the old `rabbit_ff_registry` copy before loading the new
one.
This is where the deadlock happens: process B requests the Code server
to purge the old copy, but the Code server waits for process B to stop
using it.
[How]
With this commit, process B calls `erlang:check_process_code/2` before
asking for a soft purge. If it is using an old copy, it skips the purge
because it will deadlock anyway.
Bazel build files are now maintained primarily with `bazel run
gazelle`. This will analyze and merge changes into the build files as
necessitated by certain code changes (e.g. the introduction of new
modules).
In some cases there hints to gazelle in the build files, such as `#
gazelle:erlang...` or `# keep` comments. xref checks on plugins that
depend on the cli are a good example.
OTP-26 changed the default version for binary_to_term from 1 to 2.
There's one place where we explicitly ask for version 1 anyway
(in the STOMP plugin) and seems like we need to keep it like this.
Previously osiris did not support uncorrelated writes which means
we could not use a "stateless" queue type delivery and these were
silently dropped.
This had the impact that at-most-once dead letter was not possible
where the dead letter target is a stream.
This change bumps the osiris version that has the required API
to allow for uncorrelated writes (osiris:write/2).
Currently there is no feature flag to control this as osiris writer
processes just logs and drops any messages they don't understand.
Returns reaching a Ra member that used to be leader but now has stepped
down would cause that follower to crash and restart.
This commit avoids this scenario as well as giving the return commands
a good chance of being resent to the new leader in a timeley manner.
(see the Ra release for this).
vhost_precondition_failed => vhost_limit_exceeded
vhost_limit_exceeded is the error type used by
definition import when a per-vhost is exceeded.
It feels appropriate for this case, too.
The x-delivery-count header only needs to be added when a message is
redelivered. Adding it on the first delivery attempt is unnecessary,
not recorded in the quorum queue documentation and causes additional work
deserialising the binary basic properties data to add this header.
This could be notable for messages with substantial property data incl.
heavy use of headers for example.
This is useful for understanding if a deleted queue was matching any
policies given the more selective policies introduced in #7601.
Does not apply to bulk deletion of transient queues on node down.
Rather than relying on queue name conventions, allow applying policies
based on the queue type. For example, this allows multiple policies that
apply to all queue names (".*") that specify different parameters for
different queue types.
This puts a limit to the amount of message data that is added
to the process heap at the same time to around 128KB.
Large prefetch values combined with large messages could cause
excessive garbage collection work.
Also similify the intermediate delivery message format to avoid
allocations that aren't necessary.
This new module sits on top of `rabbit_mnesia` and provide an API with
all cluster-related functions.
`rabbit_mnesia` should be called directly inside Mnesia-specific code
only, `rabbit_mnesia_rename` or classic mirrored queues for instance.
Otherwise, `rabbit_db_cluster` must be used.
Several modules, in particular in `rabbitmq_cli`, continue to call
`rabbit_mnesia` as a fallback option if the `rabbit_db_cluster` module
unavailable. This will be the case when the CLI will interact with an
older RabbitMQ version.
This will help with the introduction of a new database backend.
[Why]
If a plugin was already enabled when RabbitMQ starts, its required
feature flags were correctly handled and thus enabled. However, this was
not the case for a plugin enabled at runtime.
Here is an example with the `drop_unroutable_metric` from the
rabbitmq_management_agent plugin:
Feature flags: `drop_unroutable_metric`: required feature flag not
enabled! It must be enabled before upgrading RabbitMQ.
Supporting required feature flags in plugin is trickier than in the
core broker. Indeed, with the broker, we know when this is the first
time the broker is started. Therefore we are sure that a required
feature flag can be enabled directly, there is no existing data/context
that could conflict with the code behind the required feature flag.
For plugins, this is different: a plugin can be enabled/disabled at
runtime and between broker restarts (and thus upgrades). So, when a
plugin is enabled and it has a required feature flag, we have no way to
make sure that there is no existing and conflicting data/context.
[How]
In this patch, if the required feature flag is provided by a plugin
(i.e. not `rabbit`), we always mark it as enabled.
The plugin is responsible for handling any existing data/context and
perform any cleanup/conversion.
Reported by: @ansd
So far, we had the following functions to list nodes in a RabbitMQ
cluster:
* `rabbit_mnesia:cluster_nodes/1` to get members of the Mnesia cluster;
the argument was used to select members (all members or only those
running Mnesia and participating in the cluster)
* `rabbit_nodes:all/0` to get all members of the Mnesia cluster
* `rabbit_nodes:all_running/0` to get all members who currently run
Mnesia
Basically:
* `rabbit_nodes:all/0` calls `rabbit_mnesia:cluster_nodes(all)`
* `rabbit_nodes:all_running/0` calls `rabbit_mnesia:cluster_nodes(running)`
We also have:
* `rabbit_node_monitor:alive_nodes/1` which filters the given list of
nodes to only select those currently running Mnesia
* `rabbit_node_monitor:alive_rabbit_nodes/1` which filters the given
list of nodes to only select those currently running RabbitMQ
Most of the code uses `rabbit_mnesia:cluster_nodes/1` or the
`rabbit_nodes:all*/0` functions. `rabbit_mnesia:cluster_nodes(running)`
or `rabbit_nodes:all_running/0` is often used as a close approximation
of "all cluster members running RabbitMQ". This list might be incorrect
in times where a node is joining the clustered or is being worked on
(i.e. Mnesia is running but not RabbitMQ).
With Khepri, there won't be the same possible approximation because we
will try to keep Khepri/Ra running even if RabbitMQ is stopped to
expand/shrink the cluster.
So in order to clarify what we want when we query a list of nodes, this
patch introduces the following functions:
* `rabbit_nodes:list_members/0` to get all cluster members, regardless
of their state
* `rabbit_nodes:list_reachable/0` to get all cluster members we can
reach using Erlang distribution, regardless of the state of RabbitMQ
* `rabbit_nodes:list_running/0` to get all cluster members who run
RabbitMQ, regardless of the maintenance state
* `rabbit_nodes:list_serving/0` to get all cluster members who run
RabbitMQ and are accepting clients
In addition to the list functions, there are the corresponding
`rabbit_nodes:is_*(Node)` checks and `rabbit_nodes:filter_*(Nodes)`
filtering functions.
The code is modified to use these new functions. One possible
significant change is that the new list functions will perform RPC calls
to query the nodes' state, unlike `rabbit_mnesia:cluster_nodes(running)`.
RabbitMQ 3.12 requires feature flag `feature_flags_v2` which got
introduced in 3.11.0 (see
https://github.com/rabbitmq/rabbitmq-server/pull/6810).
Therefore, we can mark all feature flags that got introduced in 3.11.0
or before 3.11.0 as required because users will have to upgrade to
3.11.x first, before upgrading to 3.12.x
The advantage of marking these feature flags as required is that we can
start deleting any compatibliy code for these feature flags, similarly
as done in https://github.com/rabbitmq/rabbitmq-server/issues/5215
This list shows when a given feature flag was first introduced:
```
classic_mirrored_queue_version 3.11.0
stream_single_active_consumer 3.11.0
direct_exchange_routing_v2 3.11.0
listener_records_in_ets 3.11.0
tracking_records_in_ets 3.11.0
empty_basic_get_metric 3.8.10
drop_unroutable_metric 3.8.10
```
In this commit, we also force all required feature flags in Erlang
application `rabbit` to be enabled in mixed version cluster testing
and delete any tests that were about a feature flag starting as disabled.
Furthermore, this commit already deletes the callback (migration) functions
given they do not run anymore in 3.12.x.
All other clean up (i.e. branching depending on whether a feature flag
is enabled) will be done in separate commits.
* Mark AMQP 1.0 properties chunk as binary
It is marked as an UTF8 string, which is not, so
strict AMQP 1.0 codecs can fail.
* Re-use AMQP 1.0 binary chunks if available
Instead of converting from AMQP 091 back to AMQP 1.0.
This is for AMQP 1.0 properties, application properties,
and message annotations.
* Test AMQP 1.0 binary chunk reuse
* Support AMQP 1.0 multi-value body better
In the rabbit_msg_record module, mostly. Before this commit,
only one Data section was supported. Now multiple Data sections,
multiple Sequence sections, and an AMQP value section are supported.
* Add test for non-single-data-section AMQP 1.0 message
* Squash some Dialyzer warnings
* Silent dialyzer for a function for now
* Fix type declaration, use type, not atom
* Address review comments
as it was unnecessary to introduce it in the first place.
Remove the queue name from all queue type clients and pass the queue
name to the queue type callbacks that need it.
We have to leave feature flag classic_queue_type_delivery_support
required because we removed the monitor registry
1fd4a6d353/deps/rabbit/src/rabbit_queue_type.erl (L322-L325)
Implements review from Karl:
"rather than changing the message format we could amend the queue type
callbacks involved with the stateful operation to also take the queue
name record as an argument. This way we don't need to maintain the extra
queue name (which uses memory for known but obscurely technical reasons
with how maps work) in the queue type state (as it is used in the queue
type state map as the key)"
We want the build to fail if there are any dialyzer warnings in
rabbitmq_mqtt or rabbitmq_web_mqtt. Otherwise we rely on people manually
executing and checking the results of dialyzer.
Also, we want any test to fail that is flaky.
Flaky tests can indicate subtle errors in either test or program execution.
Instead of marking them as flaky, we should understand and - if possible -
fix the underlying root cause.
Fix OTP 25.0 dialyzer warning
Type gen_server:format_status() is known in OTP 25.2, but not in 25.0
Prior to this commit, when connecting or disconnecting many thousands of
MQTT subscribers, RabbitMQ printed many times:
```
[warning] <0.241.0> Mnesia('rabbit@mqtt-rabbit-1-server-0.mqtt-rabbit-1-nodes.default'): ** WARNING ** Mnesia is overloaded: {dump_log,write_threshold}
```
Each MQTT subscription causes queues and bindings to be written into Mnesia.
In order to allow for higher Mnesia load, the user can configure
```
[
{mnesia,[
{dump_log_write_threshold, 10000}
]}
].
```
in advanced.config
or set this value via
```
RABBITMQ_SERVER_ADDITIONAL_ERL_ARGS="-mnesia dump_log_write_threshold 10000"
```
The Mnesia default for dump_log_write_threshold is 1,000.
The Mnesia default for dump_log_time_threshold is 180,000 ms.
It is reasonable to increase the default for dump_log_write_threshold from
1,000 to 5,000 and in return decrease the default dump_log_time_threshold
from 3 minutes to 1.5 minutes.
This way, users can achieve higher MQTT scalability by default.
This setting cannot be changed at Mnesia runtime, it needs to be set
before Mnesia gets started.
Since the rabbitmq_mqtt plugin can be enabled dynamically after Mnesia
started, this setting must therefore apply globally to RabbitMQ.
Users can continue to set their own defaults via advanced.config or
RABBITMQ_SERVER_ADDITIONAL_ERL_ARGS. They continue to be respected
as shown by the new test suite included in this commit.
Tests sporadically fail with:
```
=== Ended at 2022-11-17 20:27:09
=== Location: [{rabbit_fifo_dlx_integration_SUITE,assert_active_dlx_workers,938},
{test_server,ts_tc,1782},
{test_server,run_test_case_eval1,1291},
{test_server,run_test_case_eval,1223}]
=== === Reason: {assertMatch,
[{module,rabbit_fifo_dlx_integration_SUITE},
{line,938},
{expression,
"rabbit_ct_broker_helpers : rpc ( Config , Server , supervisor , count_children , [ rabbit_fifo_dlx_sup ] , 1000 )"},
{pattern,"[ _ , { active , N } , _ , _ ]"},
{value,
[{specs,1},
{active,2},
{supervisors,0},
{workers,2}]}]}
in function rabbit_fifo_dlx_integration_SUITE:assert_active_dlx_workers/3 (rabbit_fifo_dlx_integration_SUITE.erl, line 938)
in call from test_server:ts_tc/3 (test_server.erl, line 1782)
in call from test_server:run_test_case_eval1/6 (test_server.erl, line 1291)
in call from test_server:run_test_case_eval/9 (test_server.erl, line 1223)
```
This commits attempts to remove that failure by using
supervisor:which_children/1 because the docs for
supervisor:count_children/1 say:
"active - The count of all actively running child processes managed by this supervisor.
For a simple_one_for_one supervisors, no check is done to ensure that each child process
is still alive, although the result provided here is likely to be very accurate unless
the supervisor is heavily overloaded."
Instead of performing credit_flow within quorum queue and stream queue
clients, return new {block | unblock, QueueName} actions.
The queue client process can then decide what to do.
For example, the channel continues to use credit_flow such that the
channel gets blocked sending any more credits to rabbit_reader.
However, the MQTT connection process does not use credit_flow. It
instead blocks its reader directly.
Prior to this commit, 1 MQTT publisher publishing to 1 Million target
classic queues requires around 680 MB of process memory.
After this commit, it requires around 290 MB of process memory.
This commit requires feature flag classic_queue_type_delivery_support
and introduces a new one called no_queue_name_in_classic_queue_client.
Instead of storing the binary queue name 4 times, this commit now stores
it only 1 time.
The monitor_registry is removed since only classic queue clients monitor
their classic queue server processes.
The classic queue client does not store the queue name anymore. Instead
the queue name is included in messages handled by the classic queue
client.
Storing the queue name in the record ctx was unnecessary.
More potential future memory optimisations:
* When routing to destination queues, looking up the queue record,
delivering to queue: Use streaming / batching instead of fetching all
at once
* Only fetch ETS columns that are necessary instead of whole queue
records
* Do not hold the same vhost binary in memory many times. Instead,
maintain a mapping.
* Remove unnecessary tuple fields.
"Each Client connecting to the Server has a unique ClientId"
"If the ClientId represents a Client already connected to
the Server then the Server MUST disconnect the existing
Client [MQTT-3.1.4-2]."
Instead of tracking client IDs via Raft, we use local ETS tables in this
commit.
Previous tracking of client IDs via Raft:
(+) consistency (does the right thing)
(-) state of Ra process becomes large > 1GB with many (> 1 Million) MQTT clients
(-) Ra process becomes a bottleneck when many MQTT clients (e.g. 300k)
disconnect at the same time because monitor (DOWN) Ra commands get
written resulting in Ra machine timeout.
(-) if we need consistency, we ideally want a single source of truth,
e.g. only Mnesia, or only Khepri (but not Mnesia + MQTT ra process)
While above downsides could be fixed (e.g. avoiding DOWN commands by
instead doing periodic cleanups of client ID entries using session interval
in MQTT 5 or using subscription_ttl parameter in current RabbitMQ MQTT config),
in this case we do not necessarily need the consistency guarantees Raft provides.
In this commit, we try to comply with [MQTT-3.1.4-2] on a best-effort
basis: If there are no network failures and no messages get lost,
existing clients with duplicate client IDs get disconnected.
In the presence of network failures / lost messages, two clients with
the same client ID can end up publishing or receiving from the same
queue. Arguably, that's acceptable and less worse than the scaling
issues we experience when we want stronger consistency.
Note that it is also the responsibility of the client to not connect
twice with the same client ID.
This commit also ensures that the client ID is a binary to save memory.
A new feature flag is introduced, which when enabled, deletes the Ra
cluster named 'mqtt_node'.
Independent of that feature flag, client IDs are tracked locally in ETS
tables.
If that feature flag is disabled, client IDs are additionally tracked in
Ra.
The feature flag is required such that clients can continue to connect
to all nodes except for the node being udpated in a rolling update.
This commit also fixes a bug where previously all MQTT connections were
cluster-wide closed when one RabbitMQ node was put into maintenance
mode.
These functions sit on top of their equivalent in `rabbit_mnesia`. In
the future, they will take care of picking the right database layer,
whatever it is.
The start of `mnesia_sync` is now part of this initialization instead of
a separate boot step in `rabbit` because it is specific to our use of
Mnesia.
In addition, `rabbit_db` provides `is_virgin_node/1` to query the state
of a remote node. This is used by `rabbit_ff_controller` in the feature
flags subsystem.
At this point, the underlying equivalent functions in `rabbit_mnesia`
become private to this module (and other modules implementing the
interaction with Mnesia). Other parts of RabbitMQ, including plugins,
should now use `rabbit_db`, not `rabbit_mnesia`.
With `rpc:call/5`, the `throw(reason)` in a migration function would be
detected as an error by the feature flags subsystem, but the return
value of `sync_cluster/0` would be `reason` instead of `{error, reason}`
(which was expected by the caller).
This should make sure that the call to
`rabbit_feature_flags:sync_feature_flags_with_cluster/2` in
`rabbit_mnesia` gets the proper return value and aborts the node
startup.
When a node joins a cluster, it will synchronize its feature flags
states with the cluster. As part of that, it will run the migration
functions of the feature flags which must be enabled.
The migration function will be executed on the joining node only. It was
already executed on each member and supposedly succeeded at the time the
feature flag was enabled initially.
On the joining node, if the migration function fails, we used to mark
the feature flag state as disabled. This is a bug because existing
cluster members see the state going from enabled to disabled.
Instead of marking it as disabled everywhere, we should restore the
state as it was before we tried to enable it on the joining node:
* enabled for the cluster members
* disabled for the joining node
This fixes a bug discovered as part of the investigation on an issue in
the migration function of the `direct_exchange_routing_v2` feature flag
(#6847).
The testcase was moved to another testsuite but called functions from
its initial testsuite. That code code relies on the name of the
testsuite so it broke.
At the same time, make it depend on the testsuite's feature flag, not
stream queues. The stream queues feature flag will become required at
some point and break the testcase.
All existing feature flags either use v2 callbacks or don't have any
associated migration code. Future flags will have to use v2 callbacks if
they need to.
v1 migration functions were deprecated with the introduction of the
feature flags controller in RabbitMQ 3.11.0. Removing support for v1
migration functions simplifies the code.
This means users of RabbitMQ 3.10.x or older will have to upgrade to
RabbitMQ 3.11.x and enable `feature_flags_v2` before they can upgrade to
a more recent release.
If a virgin node starts as clustered (thanks to peer discovery), we need
to mark feature flags already enabled remotely as enabled locally too.
We can't do a regular cluster sync because remote nodes may have
required feature flags which are disabled locally on the virgin node.
Therefore, those nodes don't have the migration code for these feature
flags anymore and the feature flags' state can't be changed.
By doing this special sync, we allow a clustered virgin node to join a
cluster with remote nodes having required feature flags.
Store directory names as binary instead of string.
This commit saves >1GB of memory per 100,000 classic queues v2.
With longish node names, the memory savings are even much higher.
This commit is especially a prerequisite for scalalbe MQTT where every
subscribing MQTT connection creates its own classic queue.
So, with 3 million MQTT subscribers, this commit saves >30 GB of memory.
This commits stores file names as binaries and converts back to
file:filename() when passed to file API functions.
This is to reduce risk of breaking behaviour for path names containing
unicode chars on certain platforms.
Alternatives to the implementation in this commit:
1. Store common directory list prefix only once (e.g. put it into
persistent_term) and store per queue directory names in ETS.
2. Use file:filename_all() instead of file:filename() and pass binaries
to the file module functions. However this might be brittle on some
platforms since these binaries are interpreted as "raw filenames".
Using raw filenames requires more changes to classic queues which we
want to avoid to reduce risk.
The downside of the implemenation in this commit is that the binary gets
converted to a list sometimes.
This happens whenever a file is flushed or a new file gets created for
example.
Following perf tests did not show any regression in performance:
```
java -jar target/perf-test.jar -s 10 -x 1 -y 0 -u q -f persistent -z 30
java -jar target/perf-test.jar -s 10000 -x 1 -y 0 -u q -f persistent -z 30
java -jar target/perf-test.jar -s 10 -x 100 -qp q%d -qpf 1 -qpt 100 -y 0 -f persistent -z 60 -c 1000
```
Furthermore `rabbit_file` did not show up in the CPU flame graphs
either.
One testsuite was using strings to check the non-existence of a user and
a virtual host. Given these names are expected to be binaries, for sure
strings won't match.
as `raft.adaptive_failure_detector.poll_interval`.
On systems under peak load, inter-node communication link congestion
can result in false positives and trigger QQ leader re-elections that
are unnecessary and could make the situation worse.
Using a higher poll interval would at least reduce the probability of
false positives.
Per discussion with @kjnilsson @mkuratczyk.
This is a follow-up commit to the parent commit. To quote part of the
parent commit's message:
> Historically, this was the Mnesia directory. But semantically, this
> should be the reverse: RabbitMQ owns the data directory and Mnesia is
> configured to put its files there too.
Now all subsystems call `rabbit:data_dir/0`. They are not tied to Mnesia
anymore.
Allow the restart_stream command / API to provide a preferrred leader
hint. If this leader replies to the coordinator stop phase as one of the
first n/2+1 members and has a sufficientl up-to-date stream tail it will be
selected as the leader, else it will fall back and use the modulus logic to
select the next leader.
sufficiently up to
Refactor stream coordinator code not to rely on the node field in the
members record as this information is already present in the members map
as the key. This way we could repurpose this field.
Implement "tie-break" selection when more than one replica has the
potential to become the next writer. For now only a simple modulo based
selection is made.
This may improve writer distribution in cases where a rabbit node goes down
and there are many streams.
which tests that messages get delivered to target quorum queue
eventually when it gets rejected initially due to target
quorum queue not being available.
Previously it was not possible to see code coverage for the majority of
test cases: integration tests that create RabbitMQ nodes.
It was only possible to see code coverage for unit tests.
This commit allows to see code coverage for tests that create RabbitMQ
nodes.
The only thing you need to do is setting the `COVER` variable, for example
```
make -C deps/rabbitmq_mqtt ct COVER=1
```
will show you coverage across all tests in the MQTT plugin.
Whenever a RabbitMQ node is started `ct_cover:add_nodes/1` is called.
Contrary to the documentation which states
> To have effect, this function is to be called from init_per_suite/1 (see common_test) before any tests are performed.
I found that it also works in init_per_group/1 or even within the test cases themselves.
Whenever a RabbitMQ node is stopped or killed `ct_cover:remove_nodes/1`
is called to transfer results from the RabbitMQ node to the CT node.
Since the erlang.mk file writes a file called `test/ct.cover.spec`
including the line:
```
{export,".../rabbitmq-server/deps/rabbitmq_mqtt/cover/ct.coverdata"}.
```
results across all test suites will be accumulated in that file.
The accumulated result can be seen through the link `Coverage log` on the test suite result pages.
Previously publisher counts would be decremented only if "publishing_mode"
was set to false resulting in ever decreasing global counters.
No consumer counts were decremented in terminate previously resulting
in ever growing consumer counts.
Add assertion to ensure global counters are decremented
Else it is possible that any mnesia update actions that are in
progress when a stream coordinator leader change occurs get stuck
and never restart.
Exclude test from mixed versions
The assertion fixes a bug and requires the new code to be running.
Limits are defined in the instance config:
default_limits.vhosts.1.pattern = ^device
default_limits.vhosts.1.max_connections = 10
default_limits.vhosts.1.max_queues = 10
default_limits.vhosts.2.pattern = ^system
default_limits.vhosts.2.max_connections = 100
default_limits.vhosts.3.pattern = .*
default_limits.vhosts.3.max_connections = 20
default_limits.vhosts.3.max_queues = 20
Where pattern is a regular expression used to match limits to a newly
created vhost, and the limits are non-negative integers. First matching
set of limits is applied, only once, during vhost creation.
Some systems may incur a substantial latency penalty when attempting
reconnections to down nodes so to avoid this some stat related functions
that gather info from all QQ member nodes no only try those nodes
that are connected. This should help keeping things like the mgmt API
functions and ctl commands a bit more responsive.
Previously if a command was issued from a rabbit node where there is
not yet a local stream coordinator member it would try to create
a new stream coordinator cluster, which would fail even if there
is a functional cluster spanning the other rabbit nodes.
This was due to use of global:whereis_name to discover stream coordinator
members on other nodes, however, Ra members never register globally
they only do local name registrations.
Replacing with a simple erpc that does erlang:whereis/1
Pairing with @acogoluegnes
This category should be unused with the decommissioning of the old
upgrade subsystem (in favor of the feature flags subsystem). It means:
1. The upgrade log file will not be created by default anymore.
2. The `$RABBITMQ_UPGRADE_LOG` environment variable is now unsupported.
The configuration variables remain to avoid breaking an existing and
working configuration.
Primarily to avoid the transient process that is spawned every
Ra "tick" to query the quorum queue process for information that could
be passed in when the process is spawned.
Refactored to make use of the overview map instead of a custom
tuple.
Also use ets:lookup_element where applicable
When enabling a plugin on the fly, the dir of the plugin and its
dependencies are added to the load path (and their modules are
loaded). There are some libraries which require their dependencies to be
loaded before they are (e.g lz4 requires host_triple to load the NIF
module).
During `rabbit_plugins:prepare_plugins/1`, `dependencies/3` gathers the
wanted apps in the right order (every app comes after its deps) however
`validate_plugins/1` used to reverse their order, so later code loading
in `prepare_plugin/2` happened in the wrong order.
This is now fixed, so `validate_plugins/1` preserves the order of apps.
As since QQ v2 we don't ever keep any messages in memory and we need
to read them from the log. The only way to do this is by using an
aux command.
Execute get_checked_out query on local members if possible
This reduces the change of crashing a QQ member during a live upgrade
where the follower does not have the appropriate code in handle_aux
By returning the next msg id for a merged consumer the rabbit_fifo_client
can set it's next expected msg_id accordingly and avoid triggering
a fetch of "missing" messages from the queue.
Since ram_pending_acks is now a map the test must support both
map and gb_trees to continue working. Also updated the state to
reflect a field name change.
For the following flags I see an improvement of
30k/s to 34k/s on my machine:
-x 1 -y 1 -A 1000 -q 1000 -c 1000 -s 1000 -f persistent
-u cqv2 --queue-args=x-queue-version=2
Prior to this commit test credit() was flaky.
It used to fail with:
```
rabbit_fifo_int_SUITE:credit failed on line 437
Reason: {badmatch,{[{{resource,"/",queue,<<"credit">>},credit,1,fals...}
```
In that case returned events were:
```
Events=[{ra_event,{credit,ct_rabbit@nuc},{applied,[{3,ok}]}},
{ra_event,{credit,ct_rabbit@nuc},
{machine,{delivery,<<"tag">>,[{1,{0,m2}}]}}}]
```
instead of:
```
Events=[{ra_event,
{credit,ct_rabbit@nuc},
{machine,{delivery,<<"tag">>,[{1,{0,m2}}]}}},
{ra_event,
{credit,ct_rabbit@nuc},
{applied,
[{3,ok},
{4,
{multi,
[{send_credit_reply,0},
{send_drained,{<<"tag">>,3}}]}}]}}]
```
The fix is we wait for both ra 'applied' events.
Stop sending connection_stats from protocol readers to rabbit_event.
Stop sending queue_stats from queues to rabbit_event.
Sending these stats every 5 seconds to the event manager process is
superfluous because noone handles these events.
They seem to be a relict from before rabbit_core_metrics ETS tables got
introduced in 2016.
Delete test head_message_timestamp_statistics because it tests that
head_message_timestamp is set correctly in queue_stats events
although queue_stats events are used nowhere.
The functionality of head_message_timestamp itself is still tested in
deps/rabbit/test/priority_queue_SUITE.erl and
deps/rabbit/test/temp/head_message_timestamp_tests.py
* Crash when a sub-command times out
* Use atom `NaN` when free space can not be determined
Fixes#5721
Use port to run /bin/sh on `unix` systems to then run `df` command
Update disk monitor tests to not use mocks because we no longer use rabbit_misc:os_cmd/1
In rabbit_fifo Ra machine v3 instead of using credit_mode
simple_prefetch, use credit_mode {simple_prefetch, MaxCredits}.
The goal is to rely less on consumer metadata which is supposed to just be a
map of informational metadata.
We know that the prefetch is part of consumer metadata up until now.
However, the prefetch might not be part anymore of consumer metadata in
a future Ra version.
This commit therefore ensures that:
1. in the conversion from v2 to v3, {simple_prefetch, MaxCredits} is
set as credit_mode if the consumer uses simple_prefetch, and
2. whenever a new credit_mode is set (in merge_consumer() or
update_consumer()), ensure that the credit_mode is set correctly if
the machine runs in v3
Prior to this commit, when a consumer NACKed a message with requeue=true
(resulting in a Ra #return{} command) and that message got
dropped or dead-lettered (for example because the delivery-limit
was exceeded), that consumer got too many credits granted,
which could lead to exceeding (and therefore violating) the
configured Prefetch value.
The consumer got credit granted twice:
1. for completing the message, and
2. for returning the message.
The same issue occurs in the scenario where a consumer (or its node)
is DOWN. In that case the consumer got credit granted twice:
1. for completing the message (if delivery limit is exceeded), and
2. for returning the message (so that other live consumers can consume)
This bug has existed since 3.8. It got reported in
https://groups.google.com/g/rabbitmq-users/c/iMcX0oXzURQ
From now on, credit for a consumer is increased in only 3 places:
a message is completed, or
a message is returned, or
a message is requeued using the new #requeue{} Ra command (when
delivery-limit is not configured)
Furthermore, this commit also fixes a somewhat related bug:
When a consumer with checked out messages gets cancelled, and
a new consumer subscribes with the same consumer ID and on the same channel
but with a lower Prefetch value, that new consumer's Prefetch value was
not always respected. This commit fixes this issue by ensuring that
the credit in simple_prefetch mode does not exceed the consumer
Prefetch value.
This commit requires a new Ra machine version v3 which is done in-place
in rabbit_fifo due to the small number of changes compared to v2.
When a non-mirrored durable classic queue is hosted on a node
that goes down, prior to #4563 not only was the behaviour
that the queue gets deleted from the rabbit_queue table,
but also that its corresponding bindings get deleted.
The purpose of this test was to make sure that bindings
get also properly deleted from the new rabbit_index_route
table.
Given that the behaviour now changed #4563 we can either
delete this test or - as done in this commit - adapt this test.
When a full recovery was done it was possible to lose messages
for v1 queues when the queues only had a journal file and no
segment files.
In practice it should be a rare event because it requires the
queue (or maybe the node) to crash first and then the vhost or
the node to be restarted gracefully.
Quorum queues were introduced in RabbitMQ 3.8.0. This was first time we
added a breaking change protected behind a feature flag. This allowed a
RabbitMQ cluster to be upgraded one node at a time, without having to
stop the entire cluster.
The breaking change was a new field in the `#amqqueue{}` record. This
broke the API and the ABI because records are a compile-time thing in
Erlang.
The compatibility code is in the wild for long enough that we want to
support the new `#amqqueue{}` record only from now on. The
`quorum_queue` feature flag was marked as required in a previous commit
(see #5202). This allows us to remove code in this patch.
References #5215.
Thoas is more efficient both in terms of encoding
time and peak memory footprint.
In the process we have discovered an issue:
https://github.com/lpil/thoas/issues/15
Pair: @pjk25
In the initial Feature flags subsystem implementation, we used a
migration function taking three arguments:
* the name of the feature flag
* its properties
* a command (`enable` or `is_enabled`)
However we wanted to implement a command (`post_enable`) and pass more
variables to the migration function. With the rework in #3940, the
migration function was therefore changed to take a single argument. That
argument was a record containing the command and much more information.
The content of the record could be different, depending on the command.
This solved the extensibility and the flexilibity of how we could call
the migration function. Unfortunately, this didn't improve its return
value as we wanted to return different things depending on the command.
In this patch, we change this completely by using a map of callbacks,
one per command, instead of that single migration function.
So before, where we had:
#{migration_fun => {Module, Function}}
The new property is now:
#{callbacks => #{enable => {Module, Function},
post_enable => {Module, Function}}}
All callbacks are still optional. You don't have to implement a fallback
empty function clause to skip commands you don't want to use oryou don't
support, as you would have to with the migration function.
Callbacks may be called with a callback-specific map of argument and
they should return the expected callback-specific return values.
Everything is defined with specs.
If a feature flag uses this new callbacks map, it will automatically
depend on `feature_flags_v2`, like the previous arity-1 migration
function. A feature flag can't define both the old migration function
and the new callbacks map.
Note that this arity-1 migration function never made it to a release of
RabbitMQ, so its support is entirely dropped with no backward
compatibility support. Likewise for the `#ffcommand{}` record.
The `feature_flags_v2` test group is only relevant if all nodes support
the `feature_flags_v2` feature flags. When doing mixed-version testing,
we must ensure that both umbrellas support that feature flags. If they
are not, we can skip the entire test group.
While here, add a dot at the end of a comment title.
We had this setup step to work around the by-design circular dependency
in the CLI (the broker depends on the CLI to start and the CLI depends
on the broker to work).
Unfortunately, it pollutes the code path and breaks the testsuite when
doing mixed-version testing: the copied `rabbit` taken from the main
umbrella ends up in the code path of the secondary umbrella, overriding
the `rabbit` there.
This patch removes this workaround to see what breaks, but it seems to
work fine so far. Let's see how it goes!
Up to RabbitMQ 3.10.x, this function took an application attribute. In
`master`, it takes the feature flags map only.
The testsuite now verifies if the remote node supports
`feature_flags_v2` to determine if it should call the function with the
feature flags map directly or convert it to an application attribute
first.
This fixes some failure of the testsuite in mixed-version cluster
testing.
Set the policy _before_ creating the stream as there is a current
limitation which means streams won't be updated immediately when
only changing the segment size.
Prior to this commit, when bindings were deleted while enabling feature
flag direct_exchange_routing_v2 - specifically AFTER
rabbit_binding:populate_index_route_table/0 ran and BEFORE the feature
flags controller changed the feature state from 'state_changing' to 'enabled' -
some bindings were incorrectly present in table rabbit_index_route.
This commit fixes this issue.
If the state is 'state_changing', bindings must be deleted when the
table already got created.
(Somewhat unexpectedly) checking for table existence within a Mnesia
transaction can return 'true' although the subsequent Mnesia table operation
will fail with {no_exists, rabbit_index_route}.
Therefore, a lock on the schema table must be set when checking for
table existence.
(Mnesia itself creates a write lock on the schema table when creating a
table, see
09c601fa21/lib/mnesia/src/mnesia_schema.erl (L1525) )
An alternative fix would have been to catch {no_exists,
rabbit_index_route} outside the Mnesia transaction, i.e. in all the callers of
the rabbit_binding:remove... functions and then retry the
rabbit_binding:remove... function.
This allows operators to override the default queue type on a per-vhost
basis by setting the default_queue_type metadata key on the vhost record.
When a queue is declared without specifiying a queue type (x-queue-type)
and there is a default set for the vhost we will augment the declare arguments
with the default. This allows future declares with and without the x-queue-type
argument to succeed.
Also only change the default _if_ the queue is durable and not
exclusive.
Changes in this commit:
1.
Use feature_flags_v2 API for feature flag direct_exchange_routing_v2.
Both feature flags feature_flags_v2 and direct_exchange_routing_v2
will be introduced in 3.11.
Therefore, direct_exchange_routing_v2 can depend on feature_flags_v2.
2.
rabbit_binding:populate_index_route_table/0 should be run only during the
feature flag migration. Thereafter, upon node start-ups, binding recovery takes
care of populating the table.
3.
Merge direct_exchange_routing_v2_post_3_11_SUITE into
direct_exchange_routing_v2_SUITE. We don't need two separate test
suites with almost identical setup and teardown.
`quorum_queue` is now required and can't be used in tests.
Also, part of the `enable_feature_flag_when_ff_file_is_unwritable`
testcase was commentted out because it relied on the `is_enabled` thing
which was dropped in `rabbit_ff_controller`. This should be introduced
at some point with a more robust design.
Our test framework can inject feature flags, but we would need a special
handling for required injected feature flags which would not test the
regular code path.
Therefore we rely on the `quorum_queue` feature flag, now required, to
test that it is correctly enabled on boot and when clustering.
This gen_statem-based process is responsible for handling concurrency
when feature flags are enabled and synchronized when a cluster is
expanded.
This clarifies and stabilizes the behavior of the feature flag subsystem
w.r.t. situations where e.g. a feature flag migration function takes
time to update data and a new node joins a cluster and synchronizes its
feature flag states with the cluster. There was a chance that the
feature flag was marked as enabled on the joining node, even though the
migration function didn't take care of that node.
With this new feature flags controller, enabling or synchronizing
feature flags blocks and delays any concurrent operations which try to
modify feature flags states too.
This change also clarifies where and when the migration function is
called: it is called at least once on each node who knows the feature
flag and when the state goes from "disabled" to "enabled" on that node.
Note that even if the feature flag is being enabled on a subset of the
nodes (because other nodes already have it enabled), it is marked as
"state_changing" everywhere during the migration. This is to prevent
that a node where it is enabled assumes it is enabled on all nodes who
know the feature flag.
There is a new feature as well: just after a feature flag is enabled,
the migration function is called a second time for any post-enable
actions. The feature flag is marked as enabled between these "enable"
and "post-enable" steps. The success or failure of this "post-enable"
run does not affect the state of the feature flag (i.e. it is ignored).
A new migration function API is introduced to allow more advanced
things. The new API is:
my_migration_function(
#ffcommand{name = ...,
props = ...,
command = enable | post_enable,
extra = #{...}})
The record is defined in `include/feature_flags.hrl`. Here is the
meaning of each field:
* `name` and `props` are the equivalent of the `FeatureName` and
`FeatureProps` arguments of the previous migration function API.
* `command` is basically the same as the previous `Arg` arguments.
* `extra` is map containing context-specific information. For instance, it
contains the list of nodes where the feature flag state changes.
This whole new behavior is behind a new feature flag called
`feature_flags_v2`. If a feature flag uses the new migration function
API, `feature_flags_v2` will be automatically enabled.
If many feature flags are enabled at once (like when a fresh RabbitMQ
node is started for the first time), `feature_flags_v2` will be enabled
first if it is in the list.
The format string started with "~s" but there was no corresponding
argument. I just removed the "~s".
While here, the log message now contains the stacktrace too.
Also rework elixir dependency handling, so we no longer rely on mix to
fetch the rabbitmq_cli deps
Also:
- Specify ra version with a commit rather than a branch
- Fixup compilation options for erlang 23
- Add missing ra reference in MODULE.bazel
- Add missing flag in oci.yaml
- Reduce bazel rbe jobs to try to save memory
- Use bazel built erlang for erlang git master tests
- Use the same cache for all the workflows but windows
- Avoid using `mix local.hex --force` in elixir rules
- Fetching seems blocked in CI, and this should reduce hex api usage in
all builds, which is always nice
- Remove xref and dialyze tags since rules_erlang 3 includes them in
the defaults
Prior to this commit, for at-most-once and at-least-once dead lettering
a quorum queue crashed if:
1. no delivery-limit set (causing Ra #requeue{} instead of #enqueue{} command
when message gets requeued)
2. message got rejected with requeue = true
3. requeued message got dead lettered
Fixes#4940
When applications accidentally set an unreasonable high value for
the message TTL expiration field, e.g. 6779303336614035452,
before this commit quorum queue and classic queue processes crashed:
```
2022-05-17 13:35:26.488670+00:00 [notice] <0.1000.0> queue 'test' in vhost '/': candidate -> leader in term: 2 machine version: 2
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> crasher:
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> initial call: ra_server_proc:init/1
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> pid: <0.1000.0>
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> registered_name: '%2F_test'
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> exception error: bad argument
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> in function erlang:start_timer/4
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> called as erlang:start_timer(6779303336614035351,<0.1000.0>,
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> {timeout,expire_msgs},
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> [])
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> *** argument 1: exceeds the maximum supported time value
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> in call from gen_statem:loop_timeouts_start/16 (gen_statem.erl, line 2108)
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> ancestors: [<0.999.0>,ra_server_sup_sup,<0.250.0>,ra_systems_sup,ra_sup,
2022-05-17 13:35:26.489492+00:00 [error] <0.1000.0> <0.186.0>]
```
In this commit, we disallow expiry fields higher than 100 years.
This causes the channel to be closed which is better than crashing the
queue process.
This new validation applies to message TTLs and queue expiry.
From the docs of erlang:start_timer:
"The absolute point in time, the timer is set to expire on, must be in the interval
[erlang:convert_time_unit(erlang:system_info(start_time), native, millisecond),
erlang:convert_time_unit(erlang:system_info(end_time), native, millisecond)].
If a relative time is specified, the Time value is not allowed to be negative.
end_time:
The last Erlang monotonic time in native time unit that can be represented
internally in the current Erlang runtime system instance.
The time between the start time and the end time is at least a quarter of a millennium."
This commit increases consumption throughput from a stream via AMQP 0.9.1
for 1 consumer by 83k msg/s or 55%,
for 4 consumers by 140k msg/s or 44%.
This commit tries to follow https://www.erlang.org/doc/efficiency_guide/binaryhandling.html
by reusing match contexts instead of creating new sub-binaries.
The CPU and mmap() memory flame graphs show that
when producing and consuming from a stream via AMQP 0.9.1
module amqp10_binary_parser requires
before this commit: 10.1% CPU time and 8.0% of mmap system calls
after this commit: 2.6% CPU time 2.5% of mmap system calls
Performance tests
Start rabbitmq-server without any plugins enabled and with 4 schedulers:
```
make run-broker PLUGINS="" RABBITMQ_SERVER_ADDITIONAL_ERL_ARGS="+JPperf true +S 4"
```
Test 1
Perf test client:
```
-x 1 -y 2 -qa x-queue-type=stream -ad false -f persistent -u s1 --qos 10000 --multi-ack-every 1000 -z 30
```
master branch:
sending rate avg msg/s 143k - 146k
receiving rate avg msg/s 188k - 194k
PR:
sending rate avg 133k - 138k
receiving rate avg 266k - 276k
This shows that with AMQP 0.9.1 and a stream, prior to this commit the broker could not
deliver messages to consumers as fast as they were published.
After this commit, it can.
Test 2
First, produce a few millions messages:
```
-x 1 -y 0 -qa x-queue-type=stream -ad false -f persistent -u s2
```
Then, consume them:
```
-x 0 -y 1 -qa x-queue-type=stream -ad false -f persistent -u s2 --qos 10000 --multi-ack-every 1000 -ca x-stream-offset=first -z 30
```
receving rate avg msg/s
master branch:
147k - 156k
PR:
230k - 237k
Improvement: 83k / 55%
Test 3
-x 0 -y 4 -qa x-queue-type=stream -ad false -f persistent -u s2 --qos 10000 --multi-ack-every 1000 -ca x-stream-offset=first -z 30
receving rate avg msg/s
master branch:
313k - 319k
PR:
450k - 461k
Improvement: 140k / 44%
A queue (Q1) can have an extra_bcc queue (Q2).
Whenever a message is routed to Q1, it must also be routed to Q2.
Commit fc2d37ed1c
puts the logic to determine extra_bcc queues into
rabbit_exchange:route/2.
That is functionally correct because it ensures that messages being dead
lettered to target queues will also route to the target queues'
extra_bcc queues.
For every message being routed, that commit uses ets:lookup/2
just to check for an extra_bcc queue.
(Technically, that commit is not a regression because it does not slow
down the common case where a message is routed to a single target queue
because before that commit rabbit_channel:deliver_to_queues/3
used two ets:lookup/2 calls.)
However we can do better by avoiding the ets:lookup/2 for the common
case where there is no extra_bcc queue set.
One option is to use ets:lookup_element/3 to only fetch the queue
'options' field.
A better option (implemented in this commit) is determining whether to
send to an extra_bcc queue in the rabbit_channel and in the at-most
and at-least once dead lettering modules where the queue records
are already looked up.
This commit speeds up sending throughput by a few thousand messages per
second.
Throw a PRECONDITION_FAILED error when a queue is created with an
invalid argument.
There can be RabbitMQ plugins and extensions out there defining arbitrary
queue arguments. Therefore we only check that a queue is not declared
with arguments that we know are supported only by OTHER queue types.
The benefit of this change is that queues are not mistakenly declared
with the wrong arguments. For example, a stream should not be
declared with `x-dead-letter-exchange` because this argument is not
supported in streams. It is only supported by classic and quorum queues.
Instead of silently allowing this argument and users wondering why the stream
does not dead letter any messages, it's better to fail early with an
meaningful error message.
We still allow any other arbitrary queue arguments.
(Therefore and unfortunately, when `x-dead-letter-exchange` is misspelled,
it will still be accepted as an argument).
For argument equivalence, we only validate the arguments that are
supported for a queue type. There is no benefit in validating any other
arguments that are not supported anyways.
because deleting a quorum queue in a mixed version cluster doesn't
delete the Ra server on time in the RabbitMQ node with the lower
version.
Therefore, this mixed versions test is skipped for the same reason that
test delete_declare is skipped.
Reimporting a queue (i.e. same vhost and same name) with different properties
or queue arguments should be a no-op because it might be dangerous to
blindly override queue properties and arguments of existing queues.
(cherry picked from commit 3ae2befb30)
Reimporting a queue (i.e. same vhost and same name) with different properties
or queue arguments should be a no-op because it might be dangerous to
blindly override queue properties and arguments of existing queues.
Deprecate queue-leader-locator values 'random' and 'least-leaders'.
Both become value 'balanced'.
From now on only queue-leader-locator values 'client-local', and
'balanced' should be set.
'balanced' will place the leader on the node with the least leaders if
there are few queues and will select a random leader if there are many
queues.
This avoid expensive least leaders calculation if there are many queues.
This change also allows us to change the implementation of 'balanced' in
the future. For example 'balanced' could place a leader on a node
depending on resource usage or available node resources.
There is no need to expose implementation details like 'random' or
'least-leaders' as configuration to users.
RA applies machine upgrades from any version to any version,
e.g. 0 to 2. This commit "fills in the gaps" in the stream coordinator,
to make sure all 1-to-1 upgrades are applied, e.g. 0 to 1 and 1 to 2
in the previous example.
Fixes#4510
The amqqueue record does not exist when `rabbit_amqqueue:is_policy_applicable/2`
is called. It is available in `rabbit_policy`, so let's send it through.
When declaring a quorum queue or a stream, select its replicas in the
following order:
1. local RabbitMQ node (to have data locality for declaring client)
2. running RabbitMQ nodes
3. RabbitMQ nodes with least quorum queue or stream replicas (to have a "balanced" RabbitMQ cluster).
From now on, quorum queues and streams behave the same way for replica
selection strategy and leader locator strategy.
Prior to this commit:
1. When a new quorum queue was created, the local node + random nodes
were selected as replicas.
2. Always local node became leader.
For example, when an AMQP client connects to a single RabbitMQ node and
creates N quorum queues, all N leaders will be on that node and replicas
are not evenly distributed across the RabbitMQ cluster.
If N is small and the RabbitMQ cluster has many nodes, some nodes might
not host any quorum queue replicas at all.
After this commit:
1. When a new quorum queue is created, the local node + RabbitMQ nodes
with least quorum queue replicas are selected.
This will nicely distribute the quorum queue replicas across the
RabbitMQ cluster.
2. Support (x-)queue-leader-locator argument / policy with
* client-local (stays the default)
* random
* least-leaders
The same settings are already available for streams.
This very small patch requires extended explanations. The patch
swaps two lines in a rabbit_variable_queue setup: one which sets
the memory hint to 0 which results in reduce_memory_usage to
always flush to disk and fsync; and another which publishes a
lot of messages to the queue that will after that point be
manipulated further to get the queue in the exact right state
for the relevant tests.
The problem with calling reduce_memory_usage after every single
message has been published is not writing to disk (v2 tests do
not suffer from performance issues in that regard) but rather
that rabbit_queue_index will always flush its journal (containing
the one message), which results in opening the segment file,
appending to it, and closing it. The file handling is done
by file_handle_cache which, in this case, will always fsync
the data before closing the file. And that's this one fsync
per message that makes the relevant tests very slow.
By swapping the lines, meaning we publish all messages first
and then set the memory hint to 0, we end up with a single
reduce_memory_usage call that results in an fsync, at the
end. (There may be other fsyncs as part of normal operations.)
And still get the same result because all messages will have
been flushed to disk, only this time in far fewer operations.
This doesn't seem to have been causing problems on CI which
already runs the tests very fast but should help macOS and
possibly other development environments.
Unlikely to pass as it randomly stops a node, which
can be the stream coordinator leader. The 2 remaining nodes
then cannot elect a new leader because they don't have
the same version.
References #4133
These 2 tests can fail in mixed-version, 2-node cluster testing. If
the stream coordinator leader ends up on the lower version, it
does not contain the fixes and the tests fail.
With this commit, the 2 tests are skipped under the appropriate
conditions.
References #4133