Merged in default and corrected ssl support clashes.

This commit is contained in:
Steve Powell 2011-06-08 17:44:43 +01:00
commit 4ee8146e5f
7 changed files with 183 additions and 55 deletions

View File

@ -4,7 +4,10 @@
{modules, []},
{registered, []},
{mod, {rabbit_stomp, []}},
{env, [{tcp_listeners, [61613]},
{env, [{default_user, [{login, "guest"},
{passcode, "guest"},
implicit_connect]},
{tcp_listeners, [61613]},
{ssl_listeners, []},
{tcp_listen_options, [binary,
{packet, raw},

View File

@ -0,0 +1,18 @@
%% The contents of this file are subject to the Mozilla Public License
%% Version 1.1 (the "License"); you may not use this file except in
%% compliance with the License. You may obtain a copy of the License
%% at http://www.mozilla.org/MPL/
%%
%% Software distributed under the License is distributed on an "AS IS"
%% basis, WITHOUT WARRANTY OF ANY KIND, either express or implied. See
%% the License for the specific language governing rights and
%% limitations under the License.
%%
%% The Original Code is RabbitMQ.
%%
%% The Initial Developer of the Original Code is VMware, Inc.
%% Copyright (c) 2007-2011 VMware, Inc. All rights reserved.
%%
-record(stomp_configuration, {default_login,
default_passcode,
implicit_connect}).

View File

@ -30,14 +30,23 @@
%%
-module(rabbit_stomp).
-include("rabbit_stomp.hrl").
-behaviour(application).
-export([start/2, stop/1]).
-define(DEFAULT_CONFIGURATION,
#stomp_configuration{
default_login = undefined,
default_passcode = undefined,
implicit_connect = false}).
start(normal, []) ->
Config = parse_configuration(),
Listeners = parse_listener_configuration(),
io:format("starting ~s (binding to ~p) ...",
["STOMP Adapter", Listeners]),
{ok, SupPid} = rabbit_stomp_sup:start_link(Listeners),
{ok, SupPid} = rabbit_stomp_sup:start_link(Listeners, Config),
io:format("done~n"),
{ok, SupPid}.
@ -54,3 +63,49 @@ parse_listener_configuration() ->
{ok, SslListeners} -> {Listeners, SslListeners}
end
end.
parse_configuration() ->
Configuration =
case application:get_env(default_user) of
undefined ->
?DEFAULT_CONFIGURATION;
{ok, UserConfig} ->
parse_default_user(UserConfig, ?DEFAULT_CONFIGURATION)
end,
report_configuration(Configuration),
Configuration.
parse_default_user([], Configuration) ->
Configuration;
parse_default_user([{login, Login} | Rest], Configuration) ->
parse_default_user(Rest, Configuration#stomp_configuration{
default_login = Login});
parse_default_user([{passcode, Passcode} | Rest], Configuration) ->
parse_default_user(Rest, Configuration#stomp_configuration{
default_passcode = Passcode});
parse_default_user([implicit_connect | Rest], Configuration) ->
parse_default_user(Rest, Configuration#stomp_configuration{
implicit_connect = true});
parse_default_user([Unknown | Rest], Configuration) ->
error_logger:error_msg("Invalid default_user configuration option: ~p~n",
[Unknown]),
parse_default_user(Rest, Configuration).
report_configuration(#stomp_configuration{
default_login = Login,
implicit_connect = ImplicitConnect}) ->
case Login of
undefined ->
ok;
_ ->
error_logger:info_msg("Default user '~s' enabled~n", [Login])
end,
case ImplicitConnect of
true -> error_logger:info_msg("Implicit connect enabled~n");
false -> ok
end,
ok.

View File

@ -32,15 +32,17 @@
-behaviour(supervisor2).
-define(MAX_WAIT, 16#ffffffff).
-export([start_link/1, init/1]).
-export([start_link/2, init/1]).
start_link(Sock) ->
start_link(Sock, Configuration) ->
{ok, SupPid} = supervisor2:start_link(?MODULE, []),
{ok, ProcessorPid} =
supervisor2:start_child(SupPid,
{rabbit_stomp_processor,
{rabbit_stomp_processor, start_link,
[Sock, rabbit_heartbeat:start_heartbeat_fun(SupPid)]},
[Sock,
rabbit_heartbeat:start_heartbeat_fun(SupPid),
Configuration]},
intrinsic, ?MAX_WAIT, worker,
[rabbit_stomp_processor]}),
{ok, ReaderPid} =

View File

@ -31,16 +31,18 @@
-module(rabbit_stomp_processor).
-behaviour(gen_server2).
-export([start_link/2, process_frame/2]).
-export([start_link/3, process_frame/2]).
-export([init/1, handle_call/3, handle_cast/2, handle_info/2,
code_change/3, terminate/2]).
-include_lib("amqp_client/include/amqp_client.hrl").
-include("rabbit_stomp_frame.hrl").
-include("rabbit_stomp.hrl").
-record(state, {socket, session_id, channel,
connection, subscriptions, version,
start_heartbeat_fun, pending_receipts}).
start_heartbeat_fun, pending_receipts,
config}).
-record(subscription, {dest_hdr, channel, multi_ack, description}).
@ -51,8 +53,9 @@
%%----------------------------------------------------------------------------
%% Public API
%%----------------------------------------------------------------------------
start_link(Sock, StartHeartbeatFun) ->
gen_server2:start_link(?MODULE, [Sock, StartHeartbeatFun], []).
start_link(Sock, StartHeartbeatFun, Configuration) ->
gen_server2:start_link(?MODULE, [Sock, StartHeartbeatFun, Configuration],
[]).
process_frame(Pid, Frame = #stomp_frame{command = Command}) ->
gen_server2:cast(Pid, {Command, Frame}).
@ -61,7 +64,7 @@ process_frame(Pid, Frame = #stomp_frame{command = Command}) ->
%% Basic gen_server2 callbacks
%%----------------------------------------------------------------------------
init([Sock, StartHeartbeatFun]) ->
init([Sock, StartHeartbeatFun, Configuration]) ->
process_flag(trap_exit, true),
{ok,
#state {
@ -72,7 +75,8 @@ init([Sock, StartHeartbeatFun]) ->
subscriptions = dict:new(),
version = none,
start_heartbeat_fun = StartHeartbeatFun,
pending_receipts = undefined},
pending_receipts = undefined,
config = Configuration},
hibernate,
{backoff, 1000, 1000, 10000}
}.
@ -83,16 +87,23 @@ terminate(_Reason, State) ->
handle_cast({"STOMP", Frame}, State) ->
handle_cast({"CONNECT", Frame}, State);
handle_cast({"CONNECT", Frame}, State = #state{channel = none,
socket = Sock}) ->
handle_cast({"CONNECT", Frame},
State = #state{
channel = none,
socket = Sock,
config = #stomp_configuration{
default_login = DefaultLogin,
default_passcode = DefaultPasscode}}) ->
process_request(
fun(StateN) ->
case negotiate_version(Frame) of
{ok, Version} ->
{ok, DefaultVHost} =
application:get_env(rabbit, default_vhost),
do_login(rabbit_stomp_frame:header(Frame, "login"),
rabbit_stomp_frame:header(Frame, "passcode"),
do_login(rabbit_stomp_frame:header(Frame, "login",
DefaultLogin),
rabbit_stomp_frame:header(Frame, "passcode",
DefaultPasscode),
rabbit_stomp_frame:header(Frame, "host",
binary_to_list(
DefaultVHost)),
@ -111,7 +122,15 @@ handle_cast({"CONNECT", Frame}, State = #state{channel = none,
fun(StateM) -> StateM end,
State);
handle_cast(_Request, State = #state{channel = none}) ->
handle_cast(Request, State = #state{channel = none,
config = #stomp_configuration{
implicit_connect = true}}) ->
{noreply, State1, _} =
handle_cast({"CONNECT", #stomp_frame{headers = []}}, State),
handle_cast(Request, State1);
handle_cast(_Request, State = #state{channel = none,
config = #stomp_configuration{
implicit_connect = false}}) ->
{noreply,
send_error("Illegal command",
"You must log in using CONNECT first\n",
@ -316,7 +335,10 @@ with_destination(Command, Frame, State, Fun) ->
State)
end.
do_login({ok, Username0}, {ok, Password0}, VirtualHost0, Heartbeat, AdapterInfo,
do_login(undefined, _, _, _, _, _, State) ->
error("Bad CONNECT", "Missing login or passcode header(s)\n", State);
do_login(Username0, Password0, VirtualHost0, Heartbeat, AdapterInfo,
Version, State) ->
Username = list_to_binary(Username0),
Password = list_to_binary(Password0),
@ -348,10 +370,7 @@ do_login({ok, Username0}, {ok, Password0}, VirtualHost0, Heartbeat, AdapterInfo,
end;
{refused, _Msg, _Args} ->
error("Bad CONNECT", "Authentication failure\n", State)
end;
do_login(_, _, _, _, _, _, State) ->
error("Bad CONNECT", "Missing login or passcode header(s)\n", State).
end.
adapter_info(Sock, Version) ->
{ok, {Addr, Port}} = rabbit_net:sockname(Sock),

View File

@ -31,15 +31,16 @@
-module(rabbit_stomp_sup).
-behaviour(supervisor).
-export([start_link/1, init/1]).
-export([start_link/2, init/1]).
-export([listener_started/3, listener_stopped/3,
start_client/1, start_ssl_client/2]).
start_client/2, start_ssl_client/3]).
start_link(Listeners) ->
supervisor:start_link({local, ?MODULE}, ?MODULE, [Listeners]).
start_link(Listeners, Configuration) ->
supervisor:start_link({local, ?MODULE}, ?MODULE,
[Listeners, Configuration]).
init([{Listeners, SslListeners}]) ->
init([{Listeners, SslListeners}, Configuration]) ->
{ok, SocketOpts} = application:get_env(rabbitmq_stomp, tcp_listen_options),
SslOpts = case SslListeners of
@ -53,11 +54,10 @@ init([{Listeners, SslListeners}]) ->
[{local, rabbit_stomp_client_sup_sup},
{rabbit_stomp_client_sup, start_link,[]}]},
transient, infinity, supervisor, [rabbit_client_sup]} |
listener_specs(fun tcp_listener_spec/1, [SocketOpts], Listeners) ++
listener_specs(fun ssl_listener_spec/1,
[SocketOpts, SslOpts], SslListeners)]}}.
listener_specs(fun tcp_listener_spec/1,
[SocketOpts, Configuration], Listeners) ++
listener_specs(fun ssl_listener_spec/1,
[SocketOpts, SslOpts, Configuration], SslListeners)]}}.
listener_specs(Fun, Args, Listeners) ->
[Fun([Address | Args]) ||
@ -65,13 +65,15 @@ listener_specs(Fun, Args, Listeners) ->
Address <- rabbit_networking:check_tcp_listener_address(
rabbit_stomp_listener_sup, Listener)].
tcp_listener_spec([Address, SocketOpts]) ->
tcp_listener_spec([Address, SocketOpts, Configuration]) ->
listener_spec(Address, SocketOpts, stomp,
{?MODULE, start_client, []}, "STOMP TCP Listener").
{?MODULE, start_client, [Configuration]},
"STOMP TCP Listener").
ssl_listener_spec([Address, SocketOpts, SslOpts]) ->
ssl_listener_spec([Address, SocketOpts, SslOpts, Configuration]) ->
listener_spec(Address, SocketOpts, 'stomp/ssl',
{?MODULE, start_ssl_client, [SslOpts]}, "STOMP SSL Listener").
{?MODULE, start_ssl_client, [Configuration, SslOpts]},
"STOMP SSL Listener").
listener_spec({IPAddress, Port, Family, Name},
SocketOpts, Protocol, OnConnect, Label) ->
@ -90,15 +92,16 @@ listener_started(Protocol, IPAddress, Port) ->
listener_stopped(Protocol, IPAddress, Port) ->
rabbit_networking:tcp_listener_stopped(Protocol, IPAddress, Port).
start_client(Sock) ->
start_client(Configuration, Sock) ->
{ok, SupPid, ReaderPid} =
supervisor:start_child(rabbit_stomp_client_sup_sup, [Sock]),
supervisor:start_child(rabbit_stomp_client_sup_sup,
[Sock, Configuration]),
ok = rabbit_net:controlling_process(Sock, ReaderPid),
ReaderPid ! {go, Sock},
SupPid.
start_ssl_client(SslOpts, Sock) ->
start_ssl_client(Configuration, SslOpts, Sock) ->
Transform = rabbit_networking:ssl_transform_fun(SslOpts),
{ok, SslSock} = Transform(Sock),
start_client(SslSock).
start_client(Configuration, SslSock).

View File

@ -5,47 +5,47 @@ import time
class TestLifecycle(base.BaseTest):
def test_unsubscribe_exchange_destination(self):
def xtest_unsubscribe_exchange_destination(self):
''' Test UNSUBSCRIBE command with exchange'''
d = "/exchange/amq.fanout"
self.unsub_test(d, self.sub_and_send(d))
def test_unsubscribe_exchange_destination_with_receipt(self):
def xtest_unsubscribe_exchange_destination_with_receipt(self):
''' Test receipted UNSUBSCRIBE command with exchange'''
d = "/exchange/amq.fanout"
self.unsub_test(d, self.sub_and_send(d, receipt="unsub.rct"), numRcts=1)
def test_unsubscribe_queue_destination(self):
def xtest_unsubscribe_queue_destination(self):
''' Test UNSUBSCRIBE command with queue'''
d = "/queue/unsub01"
self.unsub_test(d, self.sub_and_send(d))
def test_unsubscribe_queue_destination_with_receipt(self):
def xtest_unsubscribe_queue_destination_with_receipt(self):
''' Test receipted UNSUBSCRIBE command with queue'''
d = "/queue/unsub02"
self.unsub_test(d, self.sub_and_send(d, receipt="unsub.rct"), numRcts=1)
def test_unsubscribe_exchange_id(self):
def xtest_unsubscribe_exchange_id(self):
''' Test UNSUBSCRIBE command with exchange by id'''
d = "/exchange/amq.fanout"
self.unsub_test(d, self.sub_and_send(d, subid="exchid"))
def test_unsubscribe_exchange_id_with_receipt(self):
def xtest_unsubscribe_exchange_id_with_receipt(self):
''' Test receipted UNSUBSCRIBE command with exchange by id'''
d = "/exchange/amq.fanout"
self.unsub_test(d, self.sub_and_send(d, subid="exchid", receipt="unsub.rct"), numRcts=1)
def test_unsubscribe_queue_id(self):
def xtest_unsubscribe_queue_id(self):
''' Test UNSUBSCRIBE command with queue by id'''
d = "/queue/unsub03"
self.unsub_test(d, self.sub_and_send(d, subid="queid"))
def test_unsubscribe_queue_id_with_receipt(self):
def xtest_unsubscribe_queue_id_with_receipt(self):
''' Test receipted UNSUBSCRIBE command with queue by id'''
d = "/queue/unsub04"
self.unsub_test(d, self.sub_and_send(d, subid="queid", receipt="unsub.rct"), numRcts=1)
def test_connect_version_1_1(self):
def xtest_connect_version_1_1(self):
''' Test CONNECT with version 1.1'''
self.conn.disconnect()
new_conn = self.create_connection(version="1.1,1.0")
@ -54,7 +54,7 @@ class TestLifecycle(base.BaseTest):
finally:
new_conn.disconnect()
def test_heartbeat_disconnects_client(self):
def xtest_heartbeat_disconnects_client(self):
''' Test heart-beat disconnection'''
self.conn.disconnect()
new_conn = self.create_connection(heartbeat="1500,0")
@ -68,32 +68,60 @@ class TestLifecycle(base.BaseTest):
if new_conn.is_connected():
new_conn.disconnect()
def test_unsupported_version(self):
def xtest_unsupported_version(self):
''' Test unsupported version on CONNECT command'''
self.bad_connect(stomp.Connection(user="guest",
passcode="guest",
version="100.1"),
"Supported versions are 1.0,1.1\n")
def test_bad_username(self):
def xtest_bad_username(self):
''' Test bad username'''
self.bad_connect(stomp.Connection(user="gust",
passcode="guest"),
"Authentication failure\n")
def test_bad_password(self):
def xtest_bad_password(self):
''' Test bad password'''
self.bad_connect(stomp.Connection(user="guest",
passcode="gust"),
"Authentication failure\n")
def test_bad_vhost(self):
def xtest_bad_vhost(self):
''' Test bad virtual host'''
self.bad_connect(stomp.Connection(user="guest",
passcode="guest",
virtual_host="//"),
"Authentication failure\n")
def xtest_default_user(self):
''' Test default user connection '''
self.conn.disconnect()
new_conn = stomp.Connection(user="", passcode="")
new_conn.start()
new_conn.connect()
try:
self.assertTrue(new_conn.is_connected())
finally:
new_conn.disconnect()
def test_implicit_connect(self):
self.conn.disconnect()
listener = base.WaitableListener()
new_conn = stomp.Connection(user="", passcode="")
new_conn.set_listener('', listener)
new_conn.start() # not going to issue connect
new_conn.subscribe(destination="/topic/implicit", receipt='implicit')
try:
self.assertTrue(listener.await(5))
self.assertEquals(1, len(listener.receipts),
'Missing receipt. Likely not connected')
self.assertEquals('implicit', listener.receipts[0]['headers']['receipt-id'])
finally:
new_conn.disconnect()
def bad_connect(self, new_conn, expected):
self.conn.disconnect()
listener = base.WaitableListener()
@ -107,12 +135,12 @@ class TestLifecycle(base.BaseTest):
if new_conn.is_connected():
new_conn.disconnect()
def test_disconnect(self):
def xtest_disconnect(self):
''' Test DISCONNECT command'''
self.conn.disconnect()
self.assertFalse(self.conn.is_connected())
def test_disconnect_with_receipt(self):
def xtest_disconnect_with_receipt(self):
''' Test the DISCONNECT command with receipts '''
time.sleep(3)
self.listener.reset(1)