25
1
mirror of https://github.com/processone/ejabberd.git synced 2024-10-19 15:32:08 +02:00

Add mod_matrix_gw

This commit is contained in:
Alexey Shchepin 2024-02-14 06:57:59 +03:00
parent 67a6776fba
commit f44e23b8cc
6 changed files with 4301 additions and 0 deletions

28
include/mod_matrix_gw.hrl Normal file
View File

@ -0,0 +1,28 @@
%%%----------------------------------------------------------------------
%%%
%%% ejabberd, Copyright (C) 2002-2024 ProcessOne
%%%
%%% This program is free software; you can redistribute it and/or
%%% modify it under the terms of the GNU General Public License as
%%% published by the Free Software Foundation; either version 2 of the
%%% License, or (at your option) any later version.
%%%
%%% This program is distributed in the hope that it will be useful,
%%% but WITHOUT ANY WARRANTY; without even the implied warranty of
%%% MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
%%% General Public License for more details.
%%%
%%% You should have received a copy of the GNU General Public License along
%%% with this program; if not, write to the Free Software Foundation, Inc.,
%%% 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301 USA.
%%%
%%%----------------------------------------------------------------------
-record(room_version,
{id :: binary(),
%% use the same field names as in Synapse
knock_restricted_join_rule :: boolean(),
enforce_int_power_levels :: boolean(),
implicit_room_creator :: boolean(),
updated_redaction_rules :: boolean()
}).

913
src/mod_matrix_gw.erl Normal file
View File

@ -0,0 +1,913 @@
%%%----------------------------------------------------------------------
%%% File : mod_matrix_gw.erl
%%% Author : Alexey Shchepin <alexey@process-one.net>
%%% Purpose : Matrix gateway
%%% Created : 23 Apr 2022 by Alexey Shchepin <alexey@process-one.net>
%%%
%%%
%%% ejabberd, Copyright (C) 2002-2022 ProcessOne
%%%
%%% This program is free software; you can redistribute it and/or
%%% modify it under the terms of the GNU General Public License as
%%% published by the Free Software Foundation; either version 2 of the
%%% License, or (at your option) any later version.
%%%
%%% This program is distributed in the hope that it will be useful,
%%% but WITHOUT ANY WARRANTY; without even the implied warranty of
%%% MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
%%% General Public License for more details.
%%%
%%% You should have received a copy of the GNU General Public License along
%%% with this program; if not, write to the Free Software Foundation, Inc.,
%%% 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301 USA.
%%%
%%%----------------------------------------------------------------------
-module(mod_matrix_gw).
-if(?OTP_RELEASE >= 24).
-author('alexey@process-one.net').
-ifndef(GEN_SERVER).
-define(GEN_SERVER, gen_server).
-endif.
-behaviour(?GEN_SERVER).
-behaviour(gen_mod).
-export([start/2, stop/1, reload/3, process/2,
start_link/1,
procname/1,
init/1, handle_call/3, handle_cast/2,
handle_info/2, terminate/2, code_change/3,
depends/2, mod_opt_type/1, mod_options/1, mod_doc/0]).
-export([parse_auth/1, encode_canonical_json/1,
get_id_domain_exn/1,
base64_decode/1, base64_encode/1,
prune_event/2, get_event_id/2, content_hash/1,
sign_event/3, sign_pruned_event/2, sign_json/2,
send_request/8, s2s_out_bounce_packet/2, user_receive_packet/1,
route/1]).
-include_lib("xmpp/include/xmpp.hrl").
-include("logger.hrl").
-include("ejabberd_http.hrl").
-include("translate.hrl").
-include("ejabberd_web_admin.hrl").
-include("mod_matrix_gw.hrl").
-define(MAX_REQUEST_SIZE, 1000000).
process([<<"key">>, <<"v2">>, <<"server">> | _],
#request{method = 'GET', host = _Host} = _Request) ->
Host = ejabberd_config:get_myname(),
KeyName = mod_matrix_gw_opt:key_name(Host),
KeyID = <<"ed25519:", KeyName/binary>>,
ServerName = mod_matrix_gw_opt:matrix_domain(Host),
TS = erlang:system_time(millisecond) + timer:hours(24 * 7),
{PubKey, _PrivKey} = mod_matrix_gw_opt:key(Host),
JSON = #{<<"old_verify_keys">> => #{},
<<"server_name">> => ServerName,
<<"valid_until_ts">> => TS,
<<"verify_keys">> => #{
KeyID => #{
<<"key">> => base64_encode(PubKey)
}
}},
SJSON = sign_json(Host, JSON),
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(SJSON)};
process([<<"federation">>, <<"v1">>, <<"version">>],
#request{method = 'GET', host = _Host} = _Request) ->
JSON = #{<<"server">> => #{<<"name">> => <<"ejabberd/mod_matrix_gw">>,
<<"version">> => <<"0.1">>}},
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(JSON)};
process([<<"federation">>, <<"v1">>, <<"query">>, <<"profile">>],
#request{method = 'GET', host = _Host} = Request) ->
case proplists:get_value(<<"user_id">>, Request#request.q) of
UserID when is_binary(UserID) ->
Field =
case proplists:get_value(<<"field">>, Request#request.q) of
<<"displayname">> -> displayname;
<<"avatar_url">> -> avatar_url;
undefined -> all;
_ -> error
end,
case Field of
error ->
{400, [], <<"400 Bad Request: bad 'field' parameter">>};
_ ->
case preprocess_federation_request(Request) of
{ok, _JSON, _Origin} ->
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}], <<"{}">>};
{result, HTTPResult} ->
HTTPResult
end
end;
undefined ->
{400, [], <<"400 Bad Request: missing 'user_id' parameter">>}
end;
process([<<"federation">>, <<"v1">>, <<"user">>, <<"devices">>, UserID],
#request{method = 'GET', host = _Host} = Request) ->
case preprocess_federation_request(Request) of
{ok, _JSON, _Origin} ->
Res = #{<<"devices">> =>
[#{<<"device_id">> => <<"ejabberd/mod_matrix_gw">>,
<<"keys">> => []}],
<<"stream_id">> => 1,
<<"user_id">> => UserID},
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}], jiffy:encode(Res)};
{result, HTTPResult} ->
HTTPResult
end;
process([<<"federation">>, <<"v1">>, <<"user">>, <<"keys">>, <<"query">>],
#request{method = 'POST', host = _Host} = Request) ->
case preprocess_federation_request(Request, false) of
{ok, #{<<"device_keys">> := DeviceKeys}, _Origin} ->
DeviceKeys2 = maps:map(fun(_Key, _) -> #{} end, DeviceKeys),
Res = #{<<"device_keys">> => DeviceKeys2},
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)};
{ok, _JSON, _Origin} ->
{400, [], <<"400 Bad Request: invalid format">>};
{result, HTTPResult} ->
HTTPResult
end;
process([<<"federation">>, <<"v2">>, <<"invite">>, RoomID, EventID],
#request{method = 'PUT', host = _Host} = Request) ->
case preprocess_federation_request(Request) of
{ok, #{<<"event">> := #{%<<"origin">> := Origin,
<<"room_id">> := RoomID,
<<"sender">> := Sender,
<<"state_key">> := UserID} = Event,
<<"room_version">> := RoomVer},
Origin} ->
case mod_matrix_gw_room:binary_to_room_version(RoomVer) of
#room_version{} = RoomVersion ->
%% TODO: check type and userid
Host = ejabberd_config:get_myname(),
PrunedEvent = prune_event(Event, RoomVersion),
?DEBUG("invite ~p~n", [{RoomID, EventID, Event, RoomVer, catch mod_matrix_gw_s2s:check_signature(Host, PrunedEvent), get_pruned_event_id(PrunedEvent)}]),
case mod_matrix_gw_s2s:check_signature(Host, PrunedEvent) of
true ->
case get_pruned_event_id(PrunedEvent) of
EventID ->
SEvent = sign_pruned_event(Host, PrunedEvent),
?DEBUG("sign event ~p~n", [SEvent]),
ResJSON = #{<<"event">> => SEvent},
mod_matrix_gw_room:join(Host, Origin, RoomID, Sender, UserID),
?DEBUG("res ~s~n", [jiffy:encode(ResJSON)]),
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}], jiffy:encode(ResJSON)};
_ ->
{400, [], <<"400 Bad Request: bad event id">>}
end;
false ->
{400, [], <<"400 Bad Request: signature check failed">>}
end;
false ->
{400, [], <<"400 Bad Request: unsupported room version">>}
end;
{ok, _JSON, _Origin} ->
{400, [], <<"400 Bad Request: invalid format">>};
{result, HTTPResult} ->
HTTPResult
end;
process([<<"federation">>, <<"v1">>, <<"send">>, _TxnID],
#request{method = 'PUT', host = _Host} = Request) ->
case preprocess_federation_request(Request, false) of
{ok, #{<<"origin">> := Origin,
<<"pdus">> := PDUs} = JSON,
Origin} ->
?DEBUG("send request ~p~n", [JSON]),
Host = ejabberd_config:get_myname(),
Res = lists:map(
fun(PDU) ->
case mod_matrix_gw_room:process_pdu(Host, Origin, PDU) of
{ok, EventID} -> {EventID, #{}};
{error, Error} ->
{get_event_id(PDU, mod_matrix_gw_room:binary_to_room_version(<<"9">>)),
#{<<"error">> => Error}}
end
end, PDUs),
?DEBUG("send res ~p~n", [Res]),
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(maps:from_list(Res))};
{ok, _JSON, _Origin} ->
{400, [], <<"400 Bad Request: invalid format">>};
{result, HTTPResult} ->
HTTPResult
end;
process([<<"federation">>, <<"v1">>, <<"get_missing_events">>, RoomID],
#request{method = 'POST', host = _Host} = Request) ->
case preprocess_federation_request(Request, false) of
{ok, #{<<"earliest_events">> := EarliestEvents,
<<"latest_events">> := LatestEvents} = JSON,
Origin} ->
?DEBUG("get_missing_events request ~p~n", [JSON]),
Limit = maps:get(<<"limit">>, JSON, 10),
MinDepth = maps:get(<<"min_depth">>, JSON, 0),
Host = ejabberd_config:get_myname(),
PDUs = mod_matrix_gw_room:get_missing_events(
Host, Origin, RoomID, EarliestEvents, LatestEvents, Limit, MinDepth),
?DEBUG("get_missing_events res ~p~n", [PDUs]),
Res = #{<<"events">> => PDUs},
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)};
{ok, _JSON, _Origin} ->
{400, [], <<"400 Bad Request: invalid format">>};
{result, HTTPResult} ->
HTTPResult
end;
process([<<"federation">>, <<"v1">>, <<"backfill">>, RoomID],
#request{method = 'GET', host = _Host} = Request) ->
case catch binary_to_integer(proplists:get_value(<<"limit">>, Request#request.q)) of
Limit when is_integer(Limit) ->
case preprocess_federation_request(Request, false) of
{ok, _JSON, Origin} ->
LatestEvents = proplists:get_all_values(<<"v">>, Request#request.q),
?DEBUG("backfill request ~p~n", [{Limit, LatestEvents}]),
Host = ejabberd_config:get_myname(),
PDUs1 = mod_matrix_gw_room:get_missing_events(
Host, Origin, RoomID, [], LatestEvents, Limit, 0),
PDUs2 = lists:flatmap(
fun(EventID) ->
case mod_matrix_gw_room:get_event(Host, RoomID, EventID) of
{ok, PDU} ->
[PDU];
_ ->
[]
end
end, LatestEvents),
PDUs = PDUs2 ++ PDUs1,
?DEBUG("backfill res ~p~n", [PDUs]),
MatrixServer = mod_matrix_gw_opt:matrix_domain(Host),
Res = #{<<"origin">> => MatrixServer,
<<"origin_server_ts">> => erlang:system_time(millisecond),
<<"pdus">> => PDUs},
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)};
{result, HTTPResult} ->
HTTPResult
end;
_ ->
{400, [], <<"400 Bad Request: bad 'limit' parameter">>}
end;
process([<<"federation">>, <<"v1">>, <<"state_ids">>, RoomID],
#request{method = 'GET', host = _Host} = Request) ->
case proplists:get_value(<<"event_id">>, Request#request.q) of
EventID when is_binary(EventID) ->
case preprocess_federation_request(Request) of
{ok, _JSON, Origin} ->
Host = ejabberd_config:get_myname(),
case mod_matrix_gw_room:get_state_ids(Host, Origin, RoomID, EventID) of
{ok, AuthChain, PDUs} ->
Res = #{<<"auth_chain_ids">> => AuthChain,
<<"pdu_ids">> => PDUs},
?DEBUG("get_state_ids res ~p~n", [Res]),
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)};
{error, room_not_found} ->
{400, [], <<"400 Bad Request: room not found">>};
{error, not_allowed} ->
{403, [], <<"403 Forbidden: origin not in room">>};
{error, event_not_found} ->
{400, [], <<"400 Bad Request: 'event_id' not found">>}
end;
{result, HTTPResult} ->
HTTPResult
end;
undefined ->
{400, [], <<"400 Bad Request: missing 'event_id' parameter">>}
end;
process([<<"federation">>, <<"v1">>, <<"event">>, EventID],
#request{method = 'GET', host = _Host} = Request) ->
case preprocess_federation_request(Request) of
{ok, _JSON, _Origin} ->
Host = ejabberd_config:get_myname(),
%% TODO: very inefficient, replace with an SQL call
PDU =
lists:foldl(
fun(RoomID, undefined) ->
case mod_matrix_gw_room:get_event(Host, RoomID, EventID) of
{ok, PDU} ->
PDU;
_ ->
undefined
end;
(_, Acc) ->
Acc
end, undefined, mod_matrix_gw_room:get_rooms_list()),
?DEBUG("get_event res ~p~n", [PDU]),
case PDU of
undefined ->
{400, [], <<"400 Bad Request: event not found">>};
_ ->
MatrixServer = mod_matrix_gw_opt:matrix_domain(Host),
Res = #{<<"origin">> => MatrixServer,
<<"origin_server_ts">> => erlang:system_time(millisecond),
<<"pdus">> => [PDU]},
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)}
end;
{result, HTTPResult} ->
HTTPResult
end;
process([<<"federation">>, <<"v1">>, <<"make_join">>, RoomID, UserID],
#request{method = 'GET', host = _Host, q = Params} = Request) ->
case preprocess_federation_request(Request) of
{ok, _JSON, Origin} ->
Host = ejabberd_config:get_myname(),
case get_id_domain_exn(UserID) of
Origin ->
case mod_matrix_gw_room:make_join(Host, RoomID, UserID, Params) of
{error, room_not_found} ->
Res = #{<<"errcode">> => <<"M_NOT_FOUND">>,
<<"error">> => <<"Unknown room">>},
{404, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)};
{error, not_invited} ->
Res = #{<<"errcode">> => <<"M_FORBIDDEN">>,
<<"error">> => <<"You are not invited to this room">>},
{403, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)};
{error, {incompatible_version, Ver}} ->
Res = #{<<"errcode">> => <<"M_INCOMPATIBLE_ROOM_VERSION">>,
<<"error">> => <<"Your homeserver does not support the features required to join this room">>,
<<"room_version">> => Ver},
{400, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)};
{ok, Res} ->
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)}
end;
_ ->
Res = #{<<"errcode">> => <<"M_FORBIDDEN">>,
<<"error">> => <<"User not from origin">>},
{403, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)}
end;
{result, HTTPResult} ->
HTTPResult
end;
process([<<"federation">>, <<"v2">>, <<"send_join">>, RoomID, EventID],
#request{method = 'PUT', host = _Host} = Request) ->
case preprocess_federation_request(Request) of
{ok, #{<<"content">> := #{<<"membership">> := <<"join">>},
%<<"origin">> := Origin,
<<"room_id">> := RoomID,
<<"sender">> := Sender,
<<"state_key">> := Sender,
<<"type">> := <<"m.room.member">>} = JSON, Origin} ->
Host = ejabberd_config:get_myname(),
case get_id_domain_exn(Sender) of
Origin ->
case mod_matrix_gw_room:send_join(Host, Origin, RoomID, EventID, JSON) of
{error, room_not_found} ->
Res = #{<<"errcode">> => <<"M_NOT_FOUND">>,
<<"error">> => <<"Unknown room">>},
{404, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)};
{error, not_invited} ->
Res = #{<<"errcode">> => <<"M_FORBIDDEN">>,
<<"error">> => <<"You are not invited to this room">>},
{403, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)};
{error, Error} when is_binary(Error) ->
Res = #{<<"errcode">> => <<"M_BAD_REQUEST">>,
<<"error">> => Error},
{403, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)};
{ok, Res} ->
?DEBUG("send_join res: ~p~n", [Res]),
{200, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)}
end;
_ ->
Res = #{<<"errcode">> => <<"M_FORBIDDEN">>,
<<"error">> => <<"User not from origin">>},
{403, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)}
end;
{ok, _JSON, _Origin} ->
Res = #{<<"errcode">> => <<"M_BAD_REQUEST">>,
<<"error">> => <<"Invalid event format">>},
{400, [{<<"Content-Type">>, <<"application/json;charset=UTF-8">>}],
jiffy:encode(Res)};
{result, HTTPResult} ->
HTTPResult
end;
process(_Path, _Request) ->
?DEBUG("matrix 404: ~p~n~p~n", [_Path, _Request]),
ejabberd_web:error(not_found).
preprocess_federation_request(Request) ->
preprocess_federation_request(Request, true).
preprocess_federation_request(Request, DoSignCheck) ->
?DEBUG("matrix federation: ~p~n", [Request]),
case proplists:get_value('Authorization', Request#request.headers) of
Auth when is_binary(Auth) ->
case parse_auth(Auth) of
#{<<"origin">> := MatrixServer,
<<"key">> := _,
<<"sig">> := _} = AuthParams ->
?DEBUG("auth ~p~n", [AuthParams]),
if
Request#request.length =< ?MAX_REQUEST_SIZE ->
Request2 = recv_data(Request),
JSON =
if
Request#request.length > 0 ->
try
jiffy:decode(Request2#request.data,
[return_maps])
catch
_:_ ->
error
end;
true ->
none
end,
?DEBUG("json ~p~n", [JSON]),
case JSON of
error ->
{result, {400, [], <<"400 Bad Request: invalid JSON">>}};
JSON when not DoSignCheck ->
{ok, JSON, MatrixServer};
JSON ->
Host = ejabberd_config:get_myname(),
case mod_matrix_gw_s2s:check_auth(
Host, MatrixServer,
AuthParams, JSON,
Request2) of
true ->
?DEBUG("auth ok~n", []),
{ok, JSON, MatrixServer};
false ->
?DEBUG("auth failed~n", []),
{result, {401, [], <<"401 Unauthorized">>}}
end
end;
true ->
{result, {400, [], <<"400 Bad Request: size limit">>}}
end;
_ ->
{result, {400, [], <<"400 Bad Request: bad 'Authorization' header">>}}
end;
undefined ->
{result, {400, [], <<"400 Bad Request: no 'Authorization' header">>}}
end.
recv_data(#request{length = Len, data = Trail,
sockmod = SockMod, socket = Socket} = Request) ->
NewLen = Len - byte_size(Trail),
if NewLen > 0 ->
case SockMod:recv(Socket, NewLen, 60000) of
{ok, Data} ->
Request#request{data = <<Trail/binary, Data/binary>>};
{error, _} -> Request
end;
true ->
Request
end.
-record(state,
{host :: binary(),
server_host :: binary()}).
-type state() :: #state{}.
start(Host, _Opts) ->
case mod_matrix_gw_sup:start(Host) of
{ok, _} ->
{ok, [{hook, s2s_out_bounce_packet, s2s_out_bounce_packet, 50},
{hook, user_receive_packet, user_receive_packet, 50}]};
Err ->
Err
end.
stop(Host) ->
Proc = mod_matrix_gw_sup:procname(Host),
supervisor:terminate_child(ejabberd_gen_mod_sup, Proc),
supervisor:delete_child(ejabberd_gen_mod_sup, Proc).
reload(_Host, _NewOpts, _OldOpts) ->
ok.
start_link(Host) ->
Proc = procname(Host),
?GEN_SERVER:start_link({local, Proc}, ?MODULE, [Host],
ejabberd_config:fsm_limit_opts([])).
-spec init(list()) -> {ok, state()}.
init([Host]) ->
process_flag(trap_exit, true),
mod_matrix_gw_s2s:create_db(),
mod_matrix_gw_room:create_db(),
Opts = gen_mod:get_module_opts(Host, ?MODULE),
MyHost = gen_mod:get_opt(host, Opts),
register_routes(Host, [MyHost]),
{ok, #state{server_host = Host, host = MyHost}}.
-spec handle_call(term(), {pid(), term()}, state()) ->
{reply, ok | {ok, pid()} | {error, any()}, state()} |
{stop, normal, ok, state()}.
handle_call(stop, _From, State) ->
{stop, normal, ok, State}.
-spec handle_cast(term(), state()) -> {noreply, state()}.
handle_cast(Msg, State) ->
?WARNING_MSG("Unexpected cast: ~p", [Msg]),
{noreply, State}.
-spec handle_info(term(), state()) -> {noreply, state()}.
handle_info(Info, State) ->
?WARNING_MSG("Unexpected info: ~p", [Info]),
{noreply, State}.
-spec terminate(term(), state()) -> any().
terminate(_Reason, #state{host = Host}) ->
unregister_routes([Host]).
-spec code_change(term(), state(), term()) -> {ok, state()}.
code_change(_OldVsn, State, _Extra) -> {ok, State}.
-spec register_routes(binary(), [binary()]) -> ok.
register_routes(ServerHost, Hosts) ->
lists:foreach(
fun(Host) ->
ejabberd_router:register_route(
Host, ServerHost, {apply, ?MODULE, route})
end, Hosts).
unregister_routes(Hosts) ->
lists:foreach(
fun(Host) ->
ejabberd_router:unregister_route(Host)
end, Hosts).
procname(Host) ->
binary_to_atom(
<<(atom_to_binary(?MODULE, latin1))/binary, "_", Host/binary>>, utf8).
parse_auth(<<"X-Matrix ", S/binary>>) ->
parse_auth1(S, <<>>, []);
parse_auth(_) ->
error.
parse_auth1(<<$=, Cs/binary>>, S, Ts) ->
parse_auth2(Cs, S, <<>>, Ts);
parse_auth1(<<$,, Cs/binary>>, <<>>, Ts) -> parse_auth1(Cs, [], Ts);
parse_auth1(<<$\s, Cs/binary>>, <<>>, Ts) -> parse_auth1(Cs, [], Ts);
parse_auth1(<<C, Cs/binary>>, S, Ts) -> parse_auth1(Cs, <<S/binary, C>>, Ts);
parse_auth1(<<>>, <<>>, T) -> maps:from_list(T);
parse_auth1(<<>>, _S, _T) -> error.
parse_auth2(<<$", Cs/binary>>, Key, Val, Ts) ->
parse_auth3(Cs, Key, Val, Ts);
parse_auth2(<<C, Cs/binary>>, Key, Val, Ts) ->
parse_auth4(Cs, Key, <<Val/binary, C>>, Ts);
parse_auth2(<<>>, _, _, _) -> error.
parse_auth3(<<$", Cs/binary>>, Key, Val, Ts) ->
parse_auth4(Cs, Key, Val, Ts);
parse_auth3(<<$\\, C, Cs/binary>>, Key, Val, Ts) ->
parse_auth3(Cs, Key, <<Val/binary, C>>, Ts);
parse_auth3(<<C, Cs/binary>>, Key, Val, Ts) ->
parse_auth3(Cs, Key, <<Val/binary, C>>, Ts);
parse_auth3(<<>>, _, _, _) -> error.
parse_auth4(<<$,, Cs/binary>>, Key, Val, Ts) ->
parse_auth1(Cs, <<>>, [{Key, Val} | Ts]);
parse_auth4(<<$\s, Cs/binary>>, Key, Val, Ts) ->
parse_auth4(Cs, Key, Val, Ts);
parse_auth4(<<C, Cs/binary>>, Key, Val, Ts) ->
parse_auth4(Cs, Key, <<Val/binary, C>>, Ts);
parse_auth4(<<>>, Key, Val, Ts) ->
parse_auth1(<<>>, <<>>, [{Key, Val} | Ts]).
prune_event(#{<<"type">> := Type, <<"content">> := Content} = Event,
RoomVersion) ->
Event2 =
case RoomVersion#room_version.updated_redaction_rules of
false ->
maps:with(
[<<"event_id">>, <<"type">>, <<"room_id">>, <<"sender">>,
<<"state_key">>, <<"content">>, <<"hashes">>,
<<"signatures">>, <<"depth">>, <<"prev_events">>,
<<"prev_state">>, <<"auth_events">>, <<"origin">>,
<<"origin_server_ts">>, <<"membership">>], Event);
true ->
maps:with(
[<<"event_id">>, <<"type">>, <<"room_id">>, <<"sender">>,
<<"state_key">>, <<"content">>, <<"hashes">>,
<<"signatures">>, <<"depth">>, <<"prev_events">>,
<<"auth_events">>, <<"origin_server_ts">>], Event)
end,
Content2 =
case Type of
<<"m.room.member">> ->
C3 = maps:with([<<"membership">>,
<<"join_authorised_via_users_server">>],
Content),
case RoomVersion#room_version.updated_redaction_rules of
false ->
C3;
true ->
case Content of
#{<<"third_party_invite">> :=
#{<<"signed">> := InvSign}} ->
C3#{<<"third_party_invite">> =>
#{<<"signed">> => InvSign}};
_ ->
C3
end
end;
<<"m.room.create">> ->
case RoomVersion#room_version.updated_redaction_rules of
false ->
maps:with([<<"creator">>], Content);
true ->
Content
end;
<<"m.room.join_rules">> ->
maps:with([<<"join_rule">>, <<"allow">>], Content);
<<"m.room.power_levels">> ->
case RoomVersion#room_version.updated_redaction_rules of
false ->
maps:with(
[<<"ban">>, <<"events">>, <<"events_default">>,
<<"kick">>, <<"redact">>, <<"state_default">>,
<<"users">>, <<"users_default">>], Content);
true ->
maps:with(
[<<"ban">>, <<"events">>, <<"events_default">>,
<<"invite">>,
<<"kick">>, <<"redact">>, <<"state_default">>,
<<"users">>, <<"users_default">>], Content)
end;
<<"m.room.history_visibility">> ->
maps:with([<<"history_visibility">>], Content);
<<"m.room.redaction">> ->
case RoomVersion#room_version.updated_redaction_rules of
false ->
#{};
true ->
maps:with([<<"redacts">>], Content)
end;
_ -> #{}
end,
Event2#{<<"content">> := Content2}.
reference_hash(PrunedEvent) ->
Event2 = maps:without([<<"signatures">>, <<"age_ts">>, <<"unsigned">>],
PrunedEvent),
S = encode_canonical_json(Event2),
crypto:hash(sha256, S).
content_hash(Event) ->
Event2 = maps:without([<<"signatures">>, <<"age_ts">>, <<"unsigned">>,
<<"hashes">>, <<"outlier">>, <<"destinations">>],
Event),
S = encode_canonical_json(Event2),
crypto:hash(sha256, S).
get_event_id(Event, RoomVersion) ->
PrunedEvent = prune_event(Event, RoomVersion),
get_pruned_event_id(PrunedEvent).
get_pruned_event_id(PrunedEvent) ->
B = base64url_encode(reference_hash(PrunedEvent)),
<<$$, B/binary>>.
encode_canonical_json(JSON) ->
JSON2 = sort_json(JSON),
jiffy:encode(JSON2).
sort_json(#{} = Map) ->
Map2 = maps:map(fun(_K, V) ->
sort_json(V)
end, Map),
{lists:sort(maps:to_list(Map2))};
sort_json(List) when is_list(List) ->
lists:map(fun sort_json/1, List);
sort_json(JSON) ->
JSON.
base64_decode(B) ->
Fixed =
case size(B) rem 4 of
0 -> B;
%1 -> <<B/binary, "===">>;
2 -> <<B/binary, "==">>;
3 -> <<B/binary, "=">>
end,
base64:decode(Fixed).
base64_encode(B) ->
D = base64:encode(B),
K = binary:longest_common_suffix([D, <<"==">>]),
binary:part(D, 0, size(D) - K).
base64url_encode(B) ->
D = base64_encode(B),
D1 = binary:replace(D, <<"+">>, <<"-">>, [global]),
binary:replace(D1, <<"/">>, <<"_">>, [global]).
sign_event(Host, Event, RoomVersion) ->
PrunedEvent = prune_event(Event, RoomVersion),
case sign_pruned_event(Host, PrunedEvent) of
#{<<"signatures">> := Signatures} ->
Event#{<<"signatures">> => Signatures}
end.
sign_pruned_event(Host, PrunedEvent) ->
Event2 = maps:without([<<"age_ts">>, <<"unsigned">>], PrunedEvent),
sign_json(Host, Event2).
sign_json(Host, JSON) ->
Signatures = maps:get(<<"signatures">>, JSON, #{}),
JSON2 = maps:without([<<"signatures">>, <<"unsigned">>], JSON),
Msg = encode_canonical_json(JSON2),
SignatureName = mod_matrix_gw_opt:matrix_domain(Host),
KeyName = mod_matrix_gw_opt:key_name(Host),
{PubKey, PrivKey} = mod_matrix_gw_opt:key(Host),
KeyID = <<"ed25519:", KeyName/binary>>,
Sig = public_key:sign(Msg, ignored, {ed_pri, ed25519, PubKey, PrivKey}),
Sig64 = base64_encode(Sig),
Signatures2 = Signatures#{SignatureName => #{KeyID => Sig64}},
JSON#{<<"signatures">> => Signatures2}.
send_request(Host, Method, MatrixServer, Path, Query, JSON,
HTTPOptions, Options) ->
URI1 = iolist_to_binary(
lists:map(fun(P) -> [$/, http_uri:encode(P)] end, Path)),
URI =
case Query of
[] -> URI1;
_ ->
URI2 = str:join(
lists:map(
fun({K, V}) ->
[http_uri:encode(K), $=, http_uri:encode(V)]
end, Query), $&),
<<URI1/binary, $?, URI2/binary>>
end,
% TODO
{MHost, MPort} = mod_matrix_gw_s2s:get_matrix_host_port(Host, MatrixServer),
%{MHost, MPort} = {MatrixServer, 8008},
URL = <<"https://", MHost/binary,
":", (integer_to_binary(MPort))/binary,
URI/binary>>,
SMethod =
case Method of
get -> <<"GET">>;
put -> <<"PUT">>;
post -> <<"POST">>
end,
Auth = make_auth_header(Host, MatrixServer, SMethod, URI, JSON),
Headers = [{"Authorization", binary_to_list(Auth)}],
Content =
case JSON of
none -> <<>>;
_ -> jiffy:encode(JSON)
end,
Request =
case Method of
get ->
{URL, Headers};
_ ->
{URL, Headers, "application/json;charset=UTF-8", Content}
end,
httpc:request(Method,
Request,
HTTPOptions,
Options).
make_auth_header(Host, MatrixServer, Method, URI, Content) ->
Origin = mod_matrix_gw_opt:matrix_domain(Host),
JSON = #{<<"method">> => Method,
<<"uri">> => URI,
<<"origin">> => Origin,
<<"destination">> => MatrixServer
},
JSON2 =
case Content of
none -> JSON;
_ ->
JSON#{<<"content">> => Content}
end,
JSON3 = sign_json(Host, JSON2),
#{<<"signatures">> := #{Origin := #{} = KeySig}} = JSON3,
{KeyID, Sig, _} = maps:next(maps:iterator(KeySig)),
<<"X-Matrix origin=", Origin/binary, ",key=\"", KeyID/binary,
"\",sig=\"", Sig/binary, "\",",
"destination=\"", MatrixServer/binary, "\"">>.
get_id_domain_exn(B) ->
case binary:split(B, <<":">>) of
[_, Tail] -> Tail;
_ -> error({invalid_id, B})
end.
s2s_out_bounce_packet(S2SState, Pkt) ->
#{server_host := Host} = S2SState,
case mod_matrix_gw_opt:matrix_id_as_jid(Host) of
false ->
S2SState;
true ->
To = xmpp:get_to(Pkt),
ServiceHost = mod_matrix_gw_opt:host(Host),
EscU = mod_matrix_gw_room:escape(To#jid.user),
EscS = mod_matrix_gw_room:escape(To#jid.lserver),
NewTo = jid:make(<<EscU/binary, $%, EscS/binary>>, ServiceHost),
ejabberd_router:route(xmpp:set_to(Pkt, NewTo)),
{stop, ignore}
end.
user_receive_packet({Pkt, C2SState} = Acc) ->
#{lserver := Host} = C2SState,
case mod_matrix_gw_opt:matrix_id_as_jid(Host) of
false ->
Acc;
true ->
ServiceHost = mod_matrix_gw_opt:host(Host),
From = xmpp:get_from(Pkt),
case From#jid.lserver of
ServiceHost ->
case binary:split(From#jid.user, <<"%">>) of
[EscU, EscS] ->
U = mod_matrix_gw_room:unescape(EscU),
S = mod_matrix_gw_room:unescape(EscS),
NewFrom = jid:make(U, S),
{xmpp:set_from(Pkt, NewFrom), C2SState};
_ ->
Acc
end;
_ ->
Acc
end
end.
route(Pkt) ->
mod_matrix_gw_room:route(Pkt).
depends(_Host, _Opts) ->
[].
mod_opt_type(host) ->
econf:host();
mod_opt_type(matrix_domain) ->
econf:binary();
mod_opt_type(key_name) ->
econf:binary();
mod_opt_type(key) ->
fun(Key) ->
Key1 = (yconf:binary())(Key),
Key2 = base64_decode(Key1),
crypto:generate_key(eddsa, ed25519, Key2)
end;
mod_opt_type(matrix_id_as_jid) ->
econf:bool();
mod_opt_type(persist) ->
econf:bool().
mod_options(Host) ->
[{matrix_domain, <<"@HOST@">>},
{host, <<"matrix.", Host/binary>>},
{key_name, <<"">>},
{key, {<<"">>, <<"">>}},
{matrix_id_as_jid, false},
{persist, false}].
mod_doc() ->
#{desc =>
[?T("TODO")],
example =>
[{?T("TODO"),
["listen:",
" -",
" port: 5280",
" module: ejabberd_http",
" request_handlers:",
" /bosh: mod_bosh",
" /websocket: ejabberd_http_ws",
" /conversejs: mod_conversejs",
"",
"modules:",
" mod_bosh: {}",
" mod_conversejs:",
" websocket_url: \"ws://@HOST@:5280/websocket\""]}
],
opts =>
[{matrix_domain,
#{value => ?T("Domain"),
desc =>
?T("TODO Specify a domain to act as the default for user JIDs. "
"The keyword '@HOST@' is replaced with the hostname. "
"The default value is '@HOST@'.")}}
]
}.
-endif.

48
src/mod_matrix_gw_opt.erl Normal file
View File

@ -0,0 +1,48 @@
%% Generated automatically
%% DO NOT EDIT: run `make options` instead
-module(mod_matrix_gw_opt).
-export([host/1]).
-export([key/1]).
-export([key_name/1]).
-export([matrix_domain/1]).
-export([matrix_id_as_jid/1]).
-export([persist/1]).
-spec host(gen_mod:opts() | global | binary()) -> binary().
host(Opts) when is_map(Opts) ->
gen_mod:get_opt(host, Opts);
host(Host) ->
gen_mod:get_module_opt(Host, mod_matrix_gw, host).
-spec key(gen_mod:opts() | global | binary()) -> any().
key(Opts) when is_map(Opts) ->
gen_mod:get_opt(key, Opts);
key(Host) ->
gen_mod:get_module_opt(Host, mod_matrix_gw, key).
-spec key_name(gen_mod:opts() | global | binary()) -> binary().
key_name(Opts) when is_map(Opts) ->
gen_mod:get_opt(key_name, Opts);
key_name(Host) ->
gen_mod:get_module_opt(Host, mod_matrix_gw, key_name).
-spec matrix_domain(gen_mod:opts() | global | binary()) -> binary().
matrix_domain(Opts) when is_map(Opts) ->
gen_mod:get_opt(matrix_domain, Opts);
matrix_domain(Host) ->
gen_mod:get_module_opt(Host, mod_matrix_gw, matrix_domain).
-spec matrix_id_as_jid(gen_mod:opts() | global | binary()) -> boolean().
matrix_id_as_jid(Opts) when is_map(Opts) ->
gen_mod:get_opt(matrix_id_as_jid, Opts);
matrix_id_as_jid(Host) ->
gen_mod:get_module_opt(Host, mod_matrix_gw, matrix_id_as_jid).
-spec persist(gen_mod:opts() | global | binary()) -> boolean().
persist(Opts) when is_map(Opts) ->
gen_mod:get_opt(persist, Opts);
persist(Host) ->
gen_mod:get_module_opt(Host, mod_matrix_gw, persist).

2652
src/mod_matrix_gw_room.erl Normal file

File diff suppressed because it is too large Load Diff

583
src/mod_matrix_gw_s2s.erl Normal file
View File

@ -0,0 +1,583 @@
%%%-------------------------------------------------------------------
%%% File : mod_matrix_gw_s2s.erl
%%% Author : Alexey Shchepin <alexey@process-one.net>
%%% Purpose : Matrix S2S
%%% Created : 1 May 2022 by Alexey Shchepin <alexey@process-one.net>
%%%
%%%
%%% ejabberd, Copyright (C) 2002-2022 ProcessOne
%%%
%%% This program is free software; you can redistribute it and/or
%%% modify it under the terms of the GNU General Public License as
%%% published by the Free Software Foundation; either version 2 of the
%%% License, or (at your option) any later version.
%%%
%%% This program is distributed in the hope that it will be useful,
%%% but WITHOUT ANY WARRANTY; without even the implied warranty of
%%% MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
%%% General Public License for more details.
%%%
%%% You should have received a copy of the GNU General Public License along
%%% with this program; if not, write to the Free Software Foundation, Inc.,
%%% 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301 USA.
%%%
%%%-------------------------------------------------------------------
-module(mod_matrix_gw_s2s).
-if(?OTP_RELEASE >= 24).
-behaviour(gen_statem).
%% API
-export([start_link/2, supervisor/1, create_db/0,
get_connection/2, check_auth/5, check_signature/2,
get_matrix_host_port/2]).
%% gen_statem callbacks
-export([init/1, terminate/3, code_change/4, callback_mode/0]).
-export([handle_event/4]).
-define(SERVER, ?MODULE).
-include("logger.hrl").
-include("ejabberd_http.hrl").
-include_lib("kernel/include/inet.hrl").
-record(matrix_s2s,
{to :: binary(),
pid :: pid()}).
-record(data,
{host :: binary(),
matrix_server :: binary(),
matrix_host_port :: {binary(), integer()} | undefined,
keys = #{},
key_queue = #{}}).
%%%===================================================================
%%% API
%%%===================================================================
%%--------------------------------------------------------------------
%% @doc
%% Creates a gen_statem process which calls Module:init/1 to
%% initialize. To ensure a synchronized start-up procedure, this
%% function does not return until Module:init/1 has returned.
%%
%% @end
%%--------------------------------------------------------------------
-spec start_link(binary(), binary()) ->
{ok, Pid :: pid()} |
ignore |
{error, Error :: term()}.
start_link(Host, MatrixServer) ->
gen_statem:start_link(?MODULE, [Host, MatrixServer],
ejabberd_config:fsm_limit_opts([])).
-spec supervisor(binary()) -> atom().
supervisor(Host) ->
gen_mod:get_module_proc(Host, mod_matrix_gw_s2s_sup).
create_db() ->
ejabberd_mnesia:create(
?MODULE, matrix_s2s,
[{ram_copies, [node()]},
{type, set},
{attributes, record_info(fields, matrix_s2s)}]),
ok.
get_connection(Host, MatrixServer) ->
case mnesia:dirty_read(matrix_s2s, MatrixServer) of
[] ->
case supervisor:start_child(supervisor(Host),
[Host, MatrixServer]) of
{ok, undefined} -> {error, ignored};
Res -> Res
end;
[#matrix_s2s{pid = Pid}] ->
{ok, Pid}
end.
get_key(Host, MatrixServer, KeyID) ->
case mod_matrix_gw_opt:matrix_domain(Host) of
MatrixServer ->
{PubKey, _PrivKey} = mod_matrix_gw_opt:key(Host),
TS = erlang:system_time(millisecond) + timer:hours(24 * 7),
{ok, PubKey, TS};
_ ->
case get_connection(Host, MatrixServer) of
{ok, S2SPid} ->
gen_statem:call(S2SPid, {get_key, KeyID});
Error -> Error
end
end.
get_matrix_host_port(Host, MatrixServer) ->
case mod_matrix_gw_opt:matrix_domain(Host) of
MatrixServer ->
error;
_ ->
case get_connection(Host, MatrixServer) of
{ok, S2SPid} ->
gen_statem:call(S2SPid, get_matrix_host_port);
Error -> Error
end
end.
%process_query(Host, MatrixServer, AuthParams, Query, JSON, Request) ->
% case get_connection(Host, MatrixServer) of
% {ok, S2SPid} ->
% #request{sockmod = SockMod, socket = Socket} = Request,
% SockMod:controlling_process(Socket, S2SPid),
% gen_statem:cast(S2SPid, {query, AuthParams, Query, JSON, Request}),
% ok;
% {error, _} = Error ->
% Error
% end.
check_auth(Host, MatrixServer, AuthParams, Content, Request) ->
case get_connection(Host, MatrixServer) of
{ok, S2SPid} ->
#{<<"key">> := KeyID} = AuthParams,
case catch gen_statem:call(S2SPid, {get_key, KeyID}) of
{ok, VerifyKey, _ValidUntil} ->
%% TODO: check ValidUntil
Destination = mod_matrix_gw_opt:matrix_domain(Host),
#{<<"sig">> := Sig} = AuthParams,
JSON = #{<<"method">> => atom_to_binary(Request#request.method, latin1),
<<"uri">> => Request#request.raw_path,
<<"origin">> => MatrixServer,
<<"destination">> => Destination,
<<"signatures">> => #{
MatrixServer => #{KeyID => Sig}
}
},
JSON2 =
case Content of
none -> JSON;
_ ->
JSON#{<<"content">> => Content}
end,
case check_signature(JSON2, MatrixServer, KeyID, VerifyKey) of
true ->
true;
false ->
?WARNING_MSG("Failed authentication: ~p", [JSON2]),
false
end;
_ ->
false
end;
{error, _} = _Error ->
false
end.
check_signature(Host, JSON) ->
case JSON of
#{<<"sender">> := Sender,
<<"signatures">> := Sigs} ->
MatrixServer = mod_matrix_gw:get_id_domain_exn(Sender),
case Sigs of
#{MatrixServer := #{} = KeySig} ->
case maps:next(maps:iterator(KeySig)) of
{KeyID, _Sig, _} ->
case catch get_key(Host, MatrixServer, KeyID) of
{ok, VerifyKey, _ValidUntil} ->
%% TODO: check ValidUntil
case check_signature(JSON, MatrixServer, KeyID, VerifyKey) of
true ->
true;
false ->
?WARNING_MSG("Failed authentication: ~p", [JSON]),
false
end;
_ ->
false
end;
_ ->
false
end;
_ ->
false
end;
_ ->
false
end.
%%%===================================================================
%%% gen_statem callbacks
%%%===================================================================
%%--------------------------------------------------------------------
%% @private
%% @doc
%% Whenever a gen_statem is started using gen_statem:start/[3,4] or
%% gen_statem:start_link/[3,4], this function is called by the new
%% process to initialize.
%% @end
%%--------------------------------------------------------------------
-spec init(Args :: term()) ->
{gen_statem:callback_mode(),
State :: term(), Data :: term()} |
{gen_statem:callback_mode(),
State :: term(), Data :: term(),
[gen_statem:action()] | gen_statem:action()} |
ignore |
{stop, Reason :: term()}.
init([Host, MatrixServer]) ->
mnesia:dirty_write(
#matrix_s2s{to = MatrixServer,
pid = self()}),
{ok, state_name,
#data{host = Host,
matrix_server = MatrixServer}}.
%%--------------------------------------------------------------------
%% @private
%% @doc
%% If the gen_statem runs with CallbackMode =:= handle_event_function
%% this function is called for every event a gen_statem receives.
%% @end
%%--------------------------------------------------------------------
-spec handle_event(
gen_statem:event_type(), Msg :: term(),
State :: term(), Data :: term()) ->
gen_statem:handle_event_result().
%handle_event({call, From}, _Msg, State, Data) ->
% {next_state, State, Data, [{reply, From, ok}]}.
handle_event({call, From}, get_matrix_host_port, _State, Data) ->
case Data#data.matrix_host_port of
undefined ->
Result = do_get_matrix_host_port(Data),
Data2 = Data#data{matrix_host_port = Result},
{keep_state, Data2, [{reply, From, Result}]};
Result ->
{keep_state_and_data, [{reply, From, Result}]}
end;
handle_event({call, From}, {get_key, KeyID}, State, Data) ->
case maps:find(KeyID, Data#data.keys) of
{ok, {ok, _, _} = Result} ->
{keep_state, Data, [{reply, From, Result}]};
{ok, error = Result} ->
{keep_state, Data, [{reply, From, Result}]};
{ok, pending} ->
KeyQueue = maps:update_with(
KeyID,
fun(Xs) ->
[From | Xs]
end,
[From],
Data#data.key_queue),
{next_state, State,
Data#data{key_queue = KeyQueue}, []};
error ->
{MHost, MPort} = do_get_matrix_host_port(Data),
URL = <<"https://", MHost/binary,
":", (integer_to_binary(MPort))/binary,
"/_matrix/key/v2/server/", KeyID/binary>>,
Self = self(),
httpc:request(get, {URL, []},
[{timeout, 5000}],
[{sync, false},
{receiver,
fun({_RequestId, Result}) ->
gen_statem:cast(
Self, {key_reply, KeyID, Result})
end}]),
Keys = (Data#data.keys)#{KeyID => pending},
KeyQueue = maps:update_with(
KeyID,
fun(Xs) ->
[From | Xs]
end,
[From],
Data#data.key_queue),
{next_state, State,
Data#data{keys = Keys,
key_queue = KeyQueue},
[]}
end;
handle_event(cast, {query, AuthParams, _Query, _JSON, _Request} = Msg,
State, Data) ->
#{<<"key">> := KeyID} = AuthParams,
case maps:find(KeyID, Data#data.keys) of
{ok, {ok, VerifyKey, _ValidUntil}} ->
Data2 = process_unverified_query(
KeyID, VerifyKey, Msg, Data),
{next_state, State, Data2, []};
{ok, error} ->
%TODO
{next_state, State, Data, []};
{ok, pending} ->
KeyQueue = maps:update_with(
KeyID,
fun(Xs) ->
[Msg | Xs]
end,
[Msg],
Data#data.key_queue),
{next_state, State,
Data#data{key_queue = KeyQueue}, []};
error ->
{MHost, MPort} = do_get_matrix_host_port(Data),
URL = <<"https://", MHost/binary,
":", (integer_to_binary(MPort))/binary,
"/_matrix/key/v2/server/", KeyID/binary>>,
Self = self(),
httpc:request(get, {URL, []},
[{timeout, 5000}],
[{sync, false},
{receiver,
fun({_RequestId, Result}) ->
gen_statem:cast(
Self, {key_reply, KeyID, Result})
end}]),
Keys = (Data#data.keys)#{KeyID => pending},
KeyQueue = maps:update_with(
KeyID,
fun(Xs) ->
[Msg | Xs]
end,
[Msg],
Data#data.key_queue),
{next_state, State,
Data#data{keys = Keys,
key_queue = KeyQueue},
[]}
end;
handle_event(cast, {key_reply, KeyID, HTTPResult}, State, Data) ->
case HTTPResult of
{{_, 200, _}, _, SJSON} ->
try
JSON = jiffy:decode(SJSON, [return_maps]),
?DEBUG("key ~p~n", [JSON]),
#{<<"verify_keys">> := VerifyKeys} = JSON,
#{KeyID := KeyData} = VerifyKeys,
#{<<"key">> := SKey} = KeyData,
VerifyKey = mod_matrix_gw:base64_decode(SKey),
?DEBUG("key ~p~n", [VerifyKey]),
?DEBUG("check ~p~n",
[catch check_signature(
JSON, Data#data.matrix_server,
KeyID, VerifyKey)]),
true = check_signature(
JSON, Data#data.matrix_server,
KeyID, VerifyKey),
#{<<"valid_until_ts">> := ValidUntil} = JSON,
ValidUntil2 =
min(ValidUntil,
erlang:system_time(millisecond) + timer:hours(24 * 7)),
Keys = (Data#data.keys)#{KeyID => {ok, VerifyKey, ValidUntil2}},
Froms = maps:get(KeyID, Data#data.key_queue, []),
KeyQueue = maps:remove(KeyID, Data#data.key_queue),
Data2 = Data#data{keys = Keys,
key_queue = KeyQueue},
Replies =
lists:map(
fun(From) ->
{reply, From, {ok, VerifyKey, ValidUntil2}}
end, Froms),
?DEBUG("KEYS ~p~n", [{Keys, Data2}]),
{next_state, State, Data2, Replies}
catch
_:_ ->
%% TODO
Keys2 = (Data#data.keys)#{KeyID => error},
{next_state, State, Data#data{keys = Keys2}, []}
end;
_ ->
%% TODO
Keys = (Data#data.keys)#{KeyID => error},
{next_state, State, Data#data{keys = Keys}, []}
end;
handle_event(cast, Msg, State, Data) ->
?WARNING_MSG("Unexpected cast: ~p", [Msg]),
{next_state, State, Data, []};
handle_event(info, Info, State, Data) ->
?WARNING_MSG("Unexpected info: ~p", [Info]),
{next_state, State, Data, []}.
%%--------------------------------------------------------------------
%% @private
%% @doc
%% This function is called by a gen_statem when it is about to
%% terminate. It should be the opposite of Module:init/1 and do any
%% necessary cleaning up. When it returns, the gen_statem terminates with
%% Reason. The return value is ignored.
%% @end
%%--------------------------------------------------------------------
-spec terminate(Reason :: term(), State :: term(), Data :: term()) ->
any().
terminate(_Reason, _State, Data) ->
mnesia:dirty_delete_object(
#matrix_s2s{to = Data#data.matrix_server,
pid = self()}),
%% TODO: wait for messages
ok.
%%--------------------------------------------------------------------
%% @private
%% @doc
%% Convert process state when code is changed
%% @end
%%--------------------------------------------------------------------
-spec code_change(
OldVsn :: term() | {down,term()},
State :: term(), Data :: term(), Extra :: term()) ->
{ok, NewState :: term(), NewData :: term()}.
code_change(_OldVsn, State, Data, _Extra) ->
{ok, State, Data}.
callback_mode() ->
handle_event_function.
%%%===================================================================
%%% Internal functions
%%%===================================================================
do_get_matrix_host_port(Data) ->
MatrixServer = Data#data.matrix_server,
case binary:split(MatrixServer, <<":">>) of
[Addr] ->
case inet:parse_address(binary_to_list(Addr)) of
{ok, _} ->
{Addr, 8448};
_ ->
URL = <<"https://", Addr/binary, "/.well-known/matrix/server">>,
HTTPRes =
httpc:request(get, {URL, []},
[{timeout, 5000}],
[{sync, true},
{body_format, binary}]),
?DEBUG("HTTPRes ~p~n", [HTTPRes]),
Res =
case HTTPRes of
{ok, {{_, 200, _}, _Headers, Body}} ->
try
case jiffy:decode(Body, [return_maps]) of
#{<<"m.server">> := Server} ->
case binary:split(Server, <<":">>) of
[ServerAddr] ->
{ServerAddr, 8448};
[ServerAddr, ServerPort] ->
{ServerAddr, binary_to_integer(ServerPort)}
end
end
catch
_:_ ->
error
end;
_ ->
error
end,
case Res of
error ->
SRVName =
"_matrix._tcp." ++ binary_to_list(MatrixServer),
case inet_res:getbyname(SRVName, srv, 5000) of
{ok, HostEntry} ->
case h_addr_list_to_host_ports(
HostEntry#hostent.h_addr_list) of
{ok, [{Host, Port} | _]} ->
{list_to_binary(Host), Port};
_ ->
{MatrixServer, 8448}
end;
{error, _} ->
{MatrixServer, 8448}
end;
_ ->
Res
end
end;
[Addr, SPort] ->
case catch binary_to_integer(SPort) of
Port when is_integer(Port) ->
{Addr, Port};
_ ->
error
end
end.
%% Copied from xmpp_stream_out.erl
-type host_port() :: {inet:hostname(), inet:port_number()}.
-type h_addr_list() :: [{integer(), integer(), inet:port_number(), string()}].
-spec h_addr_list_to_host_ports(h_addr_list()) -> {ok, [host_port(),...]} |
{error, nxdomain}.
h_addr_list_to_host_ports(AddrList) ->
PrioHostPorts = lists:flatmap(
fun({Priority, Weight, Port, Host}) ->
N = case Weight of
0 -> 0;
_ -> (Weight + 1) * p1_rand:uniform()
end,
[{Priority * 65536 - N, Host, Port}];
(_) ->
[]
end, AddrList),
HostPorts = [{Host, Port}
|| {_Priority, Host, Port} <- lists:usort(PrioHostPorts)],
case HostPorts of
[] -> {error, nxdomain};
_ -> {ok, HostPorts}
end.
check_signature(JSON, SignatureName, KeyID, VerifyKey) ->
try
#{<<"signatures">> := Signatures} = JSON,
#{SignatureName := SignatureData} = Signatures,
#{KeyID := SSignature} = SignatureData,
Signature = mod_matrix_gw:base64_decode(SSignature),
JSON2 = maps:without([<<"signatures">>, <<"unsigned">>], JSON),
Msg = mod_matrix_gw:encode_canonical_json(JSON2),
public_key:verify(Msg, ignored, Signature, {ed_pub, ed25519, VerifyKey})
catch
_:_ ->
false
end.
%process_unverified_queries(KeyID, Data) ->
% case maps:find(KeyID, Data#data.keys) of
% {ok, {ok, VerifyKey, _ValidUntil}} ->
% Queue = maps:get(KeyID, Data#data.key_queue, []),
% KeyQueue = maps:remove(KeyID, Data#data.key_queue),
% Data2 = Data#data{key_queue = KeyQueue},
% lists:foldl(
% fun(Query, DataAcc) ->
% process_unverified_query(KeyID, VerifyKey, Query, DataAcc)
% end, Data2, Queue);
% _ ->
% %% TODO
% Data
% end.
process_unverified_query(
KeyID, VerifyKey, {query, AuthParams, _Query, Content, Request} = _Msg, Data) ->
Destination = mod_matrix_gw_opt:matrix_domain(Data#data.host),
#{<<"sig">> := Sig} = AuthParams,
JSON = #{<<"method">> => atom_to_binary(Request#request.method, latin1),
<<"uri">> => Request#request.raw_path,
<<"origin">> => Data#data.matrix_server,
<<"destination">> => Destination,
<<"signatures">> => #{
Data#data.matrix_server => #{KeyID => Sig}
}
},
JSON2 =
case Content of
none -> JSON;
_ ->
JSON#{<<"content">> => Content}
end,
case check_signature(JSON2, Data#data.matrix_server, KeyID, VerifyKey) of
true ->
todo_remove_me;
%process_query(Msg, Data);
false ->
?WARNING_MSG("Failed authentication: ~p", [JSON]),
%% TODO
Data
end.
-endif.

77
src/mod_matrix_gw_sup.erl Normal file
View File

@ -0,0 +1,77 @@
%%%----------------------------------------------------------------------
%%% Created : 1 May 2022 by Alexey Shchepin <alexey@process-one.net>
%%%
%%%
%%% ejabberd, Copyright (C) 2002-2022 ProcessOne
%%%
%%% This program is free software; you can redistribute it and/or
%%% modify it under the terms of the GNU General Public License as
%%% published by the Free Software Foundation; either version 2 of the
%%% License, or (at your option) any later version.
%%%
%%% This program is distributed in the hope that it will be useful,
%%% but WITHOUT ANY WARRANTY; without even the implied warranty of
%%% MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
%%% General Public License for more details.
%%%
%%% You should have received a copy of the GNU General Public License along
%%% with this program; if not, write to the Free Software Foundation, Inc.,
%%% 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301 USA.
%%%
%%%----------------------------------------------------------------------
-module(mod_matrix_gw_sup).
-if(?OTP_RELEASE >= 24).
-behaviour(supervisor).
%% API
-export([start/1, start_link/1, procname/1]).
%% Supervisor callbacks
-export([init/1]).
%%%===================================================================
%%% API functions
%%%===================================================================
start(Host) ->
Spec = #{id => procname(Host),
start => {?MODULE, start_link, [Host]},
restart => permanent,
shutdown => infinity,
type => supervisor,
modules => [?MODULE]},
supervisor:start_child(ejabberd_gen_mod_sup, Spec).
start_link(Host) ->
Proc = procname(Host),
supervisor:start_link({local, Proc}, ?MODULE, [Host]).
-spec procname(binary()) -> atom().
procname(Host) ->
gen_mod:get_module_proc(Host, ?MODULE).
%%%===================================================================
%%% Supervisor callbacks
%%%===================================================================
init([Host]) ->
S2SName = mod_matrix_gw_s2s:supervisor(Host),
RoomName = mod_matrix_gw_room:supervisor(Host),
Specs =
[#{id => S2SName,
start => {ejabberd_tmp_sup, start_link, [S2SName, mod_matrix_gw_s2s]},
restart => permanent,
shutdown => infinity,
type => supervisor,
modules => [ejabberd_tmp_sup]},
#{id => RoomName,
start => {ejabberd_tmp_sup, start_link, [RoomName, mod_matrix_gw_room]},
restart => permanent,
shutdown => infinity,
type => supervisor,
modules => [ejabberd_tmp_sup]},
#{id => mod_matrix_gw:procname(Host),
start => {mod_matrix_gw, start_link, [Host]},
restart => permanent,
shutdown => timer:minutes(1),
type => worker,
modules => [mod_matrix_gw]}],
{ok, {{one_for_one, 10, 1}, Specs}}.
-endif.