aboutsummaryrefslogtreecommitdiffstats
diff options
context:
space:
mode:
authorLoïc Hoguin <[email protected]>2016-08-10 11:49:31 +0200
committerLoïc Hoguin <[email protected]>2016-08-10 11:49:31 +0200
commitae0dd616737d8e1116de4a04be0bc84188997eb0 (patch)
tree57c95ae977f6b6c49c0fe06e4bb68157815faa46
parent0ba3a9a22269d21b2962fec78c03e5671294d20d (diff)
downloadcowboy-ae0dd616737d8e1116de4a04be0bc84188997eb0.tar.gz
cowboy-ae0dd616737d8e1116de4a04be0bc84188997eb0.tar.bz2
cowboy-ae0dd616737d8e1116de4a04be0bc84188997eb0.zip
Add tests for responses and request body reading
This is a large commit. The cowboy_req interface has largely changed, and will change a little more. It's possible that some examples or tests have not been converted to the new interface yet. The documentation has not yet been updated. All of this will be fixed in smaller subsequent commits. Gotta start somewhere...
-rw-r--r--examples/rest_pastebin/src/toppage_handler.erl5
-rw-r--r--src/cowboy.erl8
-rw-r--r--src/cowboy_http.erl48
-rw-r--r--src/cowboy_http2.erl83
-rw-r--r--src/cowboy_req.erl701
-rw-r--r--src/cowboy_stream_h.erl32
-rw-r--r--test/handlers/echo_h.erl32
-rw-r--r--test/handlers/multipart_h.erl65
-rw-r--r--test/handlers/resp_h.erl158
-rw-r--r--test/http_SUITE_data/http_body_qs.erl2
-rw-r--r--test/http_SUITE_data/http_loop_stream_recv.erl2
-rw-r--r--test/req_SUITE.erl444
12 files changed, 967 insertions, 613 deletions
diff --git a/examples/rest_pastebin/src/toppage_handler.erl b/examples/rest_pastebin/src/toppage_handler.erl
index 324fa4a..86bafb7 100644
--- a/examples/rest_pastebin/src/toppage_handler.erl
+++ b/examples/rest_pastebin/src/toppage_handler.erl
@@ -16,7 +16,6 @@
-export([paste_text/2]).
init(Req, Opts) ->
- random:seed(os:timestamp()),
{cowboy_rest, Req, Opts}.
allowed_methods(Req, State) ->
@@ -87,13 +86,13 @@ valid_path(<<$/, _T/binary>>) -> false;
valid_path(<<_Char, T/binary>>) -> valid_path(T).
new_paste_id() ->
- Initial = random:uniform(62) - 1,
+ Initial = rand:uniform(62) - 1,
new_paste_id(<<Initial>>, 7).
new_paste_id(Bin, 0) ->
Chars = <<"abcdefghijklmnopqrstuvwxyzABCDEFGHIJKLMNOPQRSTUVWXYZ1234567890">>,
<< <<(binary_part(Chars, B, 1))/binary>> || <<B>> <= Bin >>;
new_paste_id(Bin, Rem) ->
- Next = random:uniform(62) - 1,
+ Next = rand:uniform(62) - 1,
new_paste_id(<<Bin/binary, Next>>, Rem - 1).
format_html(Paste, plain) ->
diff --git a/src/cowboy.erl b/src/cowboy.erl
index 3387224..8ef9f8a 100644
--- a/src/cowboy.erl
+++ b/src/cowboy.erl
@@ -27,19 +27,15 @@
| {atom(), cowboy_constraints:constraint() | [cowboy_constraints:constraint()], any()}].
-export_type([fields/0]).
--type http_headers() :: [{binary(), iodata()}].
+-type http_headers() :: #{binary() => iodata()}.
-export_type([http_headers/0]).
-type http_status() :: non_neg_integer() | binary().
-export_type([http_status/0]).
--type http_version() :: 'HTTP/1.1' | 'HTTP/1.0'.
+-type http_version() :: 'HTTP/2' | 'HTTP/1.1' | 'HTTP/1.0'.
-export_type([http_version/0]).
--type onresponse_fun() ::
- fun((http_status(), http_headers(), iodata(), Req) -> Req).
--export_type([onresponse_fun/0]).
-
-spec start_clear(ranch:ref(), non_neg_integer(), ranch_tcp:opts(),
cowboy_protocol:opts()) -> {ok, pid()} | {error, any()}.
start_clear(Ref, NbAcceptors, TransOpts0, ProtoOpts)
diff --git a/src/cowboy_http.erl b/src/cowboy_http.erl
index 0231def..5cb00ad 100644
--- a/src/cowboy_http.erl
+++ b/src/cowboy_http.erl
@@ -111,8 +111,7 @@
%% The connection will be closed after this stream.
last_streamid = undefined :: pos_integer(),
- %% Currently active HTTP/1.1 streams. Streams may be initiated either
- %% by the client or by the server through PUSH_PROMISE frames.
+ %% Currently active HTTP/1.1 streams.
streams = [] :: [stream()],
%% Children which are in the process of shutting down.
@@ -202,7 +201,7 @@ loop(State=#state{parent=Parent, socket=Socket, transport=Transport,
loop(State, Buffer);
%% Unknown messages.
Msg ->
- error_logger:error_msg("Received stray message ~p.", [Msg]),
+ error_logger:error_msg("Received stray message ~p.~n", [Msg]),
loop(State, Buffer)
%% @todo Configurable timeout. This should be a global inactivity timeout
%% that triggers when really nothing happens (ie something went really wrong).
@@ -502,6 +501,8 @@ parse_hd_value(<< $\r, $\n, Rest/bits >>, S, Headers0, Name, SoFar) ->
Value = clean_value_ws_end(SoFar, byte_size(SoFar) - 1),
Headers = case maps:get(Name, Headers0, undefined) of
undefined -> Headers0#{Name => Value};
+ %% The cookie header does not use proper HTTP header lists.
+ Value0 when Name =:= <<"cookie">> -> Headers0#{Name => << Value0/binary, "; ", Value/binary >>};
Value0 -> Headers0#{Name => << Value0/binary, ", ", Value/binary >>}
end,
parse_header(Rest, S, Headers);
@@ -741,7 +742,7 @@ down(State=#state{children=Children0}, Pid, Msg) ->
{value, {_, StreamID, _}, Children} ->
info(State#state{children=Children}, StreamID, Msg);
false ->
- error_logger:error_msg("Received EXIT signal ~p for unknown process ~p.", [Msg, Pid]),
+ error_logger:error_msg("Received EXIT signal ~p for unknown process ~p.~n", [Msg, Pid]),
State
end.
@@ -762,16 +763,10 @@ info(State=#state{handler=Handler, streams=Streams0}, StreamID, Msg) ->
% 'Exception occurred in StreamHandler:info/3 call.'})
end;
false ->
- error_logger:error_msg("Received message ~p for unknown stream ~p.", [Msg, StreamID]),
+ error_logger:error_msg("Received message ~p for unknown stream ~p.~n", [Msg, StreamID]),
State
end.
-%% @todo commands/3
-%% @todo stream_reset
-
-
-
-
%% Commands.
commands(State, _, []) ->
@@ -800,20 +795,19 @@ commands(State, StreamID, [{flow, _Length}|Tail]) ->
%% @todo Set the body reading length to min(Length, BodyLength)
commands(State, StreamID, Tail);
-%% @todo Probably a good idea to have an atomic response send (single send call for resp+body).
%% Send a full response.
%%
%% @todo Kill the stream if it sent a response when one has already been sent.
%% @todo Keep IsFin in the state.
%% @todo Same two things above apply to DATA, possibly promise too.
-commands(State0=#state{socket=Socket, transport=Transport, streams=Streams}, StreamID,
+commands(State0=#state{socket=Socket, transport=Transport, out_state=wait, streams=Streams}, StreamID,
[{response, StatusCode, Headers0, Body}|Tail]) ->
%% @todo I'm pretty sure the last stream in the list is the one we want
%% considering all others are queued.
#stream{version=Version} = lists:keyfind(StreamID, #stream.id, Streams),
{State, Headers} = connection(State0, Headers0, StreamID, Version),
%% @todo Ensure content-length is set.
- Response = cow_http:response(StatusCode, 'HTTP/1.1', maps:to_list(Headers)),
+ Response = cow_http:response(StatusCode, 'HTTP/1.1', headers_to_list(Headers)),
case Body of
{sendfile, O, B, P} ->
Transport:send(Socket, Response),
@@ -838,7 +832,7 @@ commands(State0=#state{socket=Socket, transport=Transport, streams=Streams}, Str
{State0#state{last_streamid=StreamID}, Headers0}
end,
{State, Headers} = connection(State1, Headers1, StreamID, Version),
- Transport:send(Socket, cow_http:response(StatusCode, 'HTTP/1.1', maps:to_list(Headers))),
+ Transport:send(Socket, cow_http:response(StatusCode, 'HTTP/1.1', headers_to_list(Headers))),
commands(State#state{out_state=chunked}, StreamID, Tail);
%% Send a response body chunk.
%%
@@ -885,7 +879,17 @@ commands(State0=#state{ref=Ref, parent=Parent, socket=Socket, transport=Transpor
commands(State, StreamID, [stop|Tail]) ->
%% @todo Do we want to run the commands after a stop?
% commands(stream_terminate(State, StreamID, stop), StreamID, Tail).
- maybe_terminate(State, StreamID, Tail, fin).
+ maybe_terminate(State, StreamID, Tail, fin);
+%% HTTP/1.1 does not support push; ignore.
+commands(State, StreamID, [{push, _, _, _, _, _, _, _}|Tail]) ->
+ commands(State, StreamID, Tail).
+
+%% The set-cookie header is special; we can only send one cookie per header.
+headers_to_list(Headers0=#{<<"set-cookie">> := SetCookies}) ->
+ Headers1 = maps:to_list(maps:remove(<<"set-cookie">>, Headers0)),
+ Headers1 ++ [{<<"set-cookie">>, Value} || Value <- SetCookies];
+headers_to_list(Headers) ->
+ maps:to_list(Headers).
flush() ->
receive _ -> flush() after 0 -> ok end.
@@ -1013,18 +1017,6 @@ error_terminate(StatusCode, State=#state{socket=Socket, transport=Transport}, Re
terminate(_State, _Reason) ->
exit(normal). %% @todo
-
-
-
-
-
-
-
-
-
-
-
-
%% System callbacks.
-spec system_continue(_, _, #state{}) -> ok.
diff --git a/src/cowboy_http2.erl b/src/cowboy_http2.erl
index ffcc17f..7bd48a7 100644
--- a/src/cowboy_http2.erl
+++ b/src/cowboy_http2.erl
@@ -376,7 +376,7 @@ commands(State, Stream, []) ->
%% @todo Same two things above apply to DATA, possibly promise too.
commands(State=#state{socket=Socket, transport=Transport, encode_state=EncodeState0},
Stream=#stream{id=StreamID, local=idle}, [{response, StatusCode, Headers0, Body}|Tail]) ->
- Headers = Headers0#{<<":status">> => integer_to_binary(StatusCode)},
+ Headers = Headers0#{<<":status">> => status(StatusCode)},
{HeaderBlock, EncodeState} = headers_encode(Headers, EncodeState0),
case Body of
<<>> ->
@@ -387,17 +387,18 @@ commands(State=#state{socket=Socket, transport=Transport, encode_state=EncodeSta
commands(State#state{encode_state=EncodeState}, Stream#stream{local=nofin},
[{sendfile, fin, O, B, P}|Tail]);
_ ->
- Transport:send(Socket, [
- cow_http2:headers(StreamID, nofin, HeaderBlock),
- cow_http2:data(StreamID, fin, Body)
- ]),
+ Transport:send(Socket, cow_http2:headers(StreamID, nofin, HeaderBlock)),
+ %% @todo 16384 is the default SETTINGS_MAX_FRAME_SIZE.
+ %% Use the length set by the server instead, if any.
+ %% @todo Would be better if we didn't have to convert to binary.
+ send_data(Socket, Transport, StreamID, fin, iolist_to_binary(Body), 16384),
commands(State#state{encode_state=EncodeState}, Stream#stream{local=fin}, Tail)
end;
%% @todo response when local!=idle
%% Send response headers and initiate chunked encoding.
commands(State=#state{socket=Socket, transport=Transport, encode_state=EncodeState0},
Stream=#stream{id=StreamID, local=idle}, [{headers, StatusCode, Headers0}|Tail]) ->
- Headers = Headers0#{<<":status">> => integer_to_binary(StatusCode)},
+ Headers = Headers0#{<<":status">> => status(StatusCode)},
{HeaderBlock, EncodeState} = headers_encode(Headers, EncodeState0),
Transport:send(Socket, cow_http2:headers(StreamID, nofin, HeaderBlock)),
commands(State#state{encode_state=EncodeState}, Stream#stream{local=nofin}, Tail);
@@ -441,11 +442,20 @@ commands(State=#state{socket=Socket, transport=Transport}, Stream=#stream{id=Str
%% end up with an infinite loop of promises.
commands(State0=#state{socket=Socket, transport=Transport, server_streamid=PromisedStreamID,
encode_state=EncodeState0}, Stream=#stream{id=StreamID},
- [{promise, Method, Scheme, Authority, Path, Headers0}|Tail]) ->
+ [{push, Method, Scheme, Host, Port, Path, Qs, Headers0}|Tail]) ->
+ Authority = case {Scheme, Port} of
+ {<<"http">>, 80} -> Host;
+ {<<"https">>, 443} -> Host;
+ _ -> [Host, $:, integer_to_binary(Port)]
+ end,
+ PathWithQs = case Qs of
+ <<>> -> Path;
+ _ -> [Path, $?, Qs]
+ end,
Headers = Headers0#{<<":method">> => Method,
<<":scheme">> => Scheme,
<<":authority">> => Authority,
- <<":path">> => Path},
+ <<":path">> => PathWithQs},
{HeaderBlock, EncodeState} = headers_encode(Headers, EncodeState0),
Transport:send(Socket, cow_http2:push_promise(StreamID, PromisedStreamID, HeaderBlock)),
%% @todo iolist_to_binary(HeaderBlock) isn't optimal. Need a shortcut.
@@ -484,6 +494,22 @@ after_commands(State=#state{streams=Streams0}, Stream=#stream{id=StreamID}) ->
Streams = lists:keystore(StreamID, #stream.id, Streams0, Stream),
State#state{streams=Streams}.
+status(Status) when is_integer(Status) ->
+ integer_to_binary(Status);
+status(<< H, T, U, _/bits >>) when H >= $1, H =< $9, T >= $0, T =< $9, U >= $0, U =< $9 ->
+ << H, T, U >>.
+
+%% This same function is found in gun_http2.
+send_data(Socket, Transport, StreamID, IsFin, Data, Length) ->
+ if
+ Length < byte_size(Data) ->
+ << Payload:Length/binary, Rest/bits >> = Data,
+ Transport:send(Socket, cow_http2:data(StreamID, nofin, Payload)),
+ send_data(Socket, Transport, StreamID, IsFin, Rest, Length);
+ true ->
+ Transport:send(Socket, cow_http2:data(StreamID, IsFin, Data))
+ end.
+
terminate(#state{socket=Socket, transport=Transport, handler=Handler,
streams=Streams, children=Children}, Reason) ->
%% @todo Send GOAWAY frame; need to keep track of last good stream id; how?
@@ -511,6 +537,23 @@ stream_init(State0=#state{ref=Ref, socket=Socket, transport=Transport, peer=Peer
<<":path">> := PathWithQs}, DecodeState} ->
State = State0#state{decode_state=DecodeState},
Headers = maps:without([<<":method">>, <<":scheme">>, <<":authority">>, <<":path">>], Headers0),
+ BodyLength = case Headers of
+ _ when IsFin =:= fin ->
+ 0;
+ #{<<"content-length">> := <<"0">>} ->
+ 0;
+ #{<<"content-length">> := BinLength} ->
+ Length = try
+ cow_http_hd:parse_content_length(BinLength)
+ catch _:_ ->
+ terminate(State0, {stream_error, StreamID, protocol_error,
+ ''}) %% @todo
+ %% @todo Err should terminate here...
+ end,
+ Length;
+ _ ->
+ undefined
+ end,
{Host, Port} = cow_http_hd:parse_host(Authority),
{Path, Qs} = cow_http:parse_fullpath(PathWithQs),
Req = #{
@@ -527,7 +570,8 @@ stream_init(State0=#state{ref=Ref, socket=Socket, transport=Transport, peer=Peer
version => 'HTTP/2',
headers => Headers,
- has_body => IsFin =:= nofin
+ has_body => IsFin =:= nofin,
+ body_length => BodyLength
%% @todo multipart? keep state separate
%% meta values (cowboy_websocket, cowboy_rest)
@@ -609,9 +653,26 @@ stream_terminate_children([Child|Tail], StreamID, Acc) ->
headers_decode(HeaderBlock, DecodeState0) ->
{Headers, DecodeState} = cow_hpack:decode(HeaderBlock, DecodeState0),
- {maps:from_list(Headers), DecodeState}.
+ {headers_to_map(Headers, #{}), DecodeState}.
-%% @todo We will need to special-case the set-cookie header here.
+%% This function is necessary to properly handle duplicate headers
+%% and the special-case cookie header.
+headers_to_map([], Acc) ->
+ Acc;
+headers_to_map([{Name, Value}|Tail], Acc0) ->
+ Acc = case Acc0 of
+ %% The cookie header does not use proper HTTP header lists.
+ #{Name := Value0} when Name =:= <<"cookie">> -> Acc0#{Name => << Value0/binary, "; ", Value/binary >>};
+ #{Name := Value0} -> Acc0#{Name => << Value0/binary, ", ", Value/binary >>};
+ _ -> Acc0#{Name => Value}
+ end,
+ headers_to_map(Tail, Acc).
+
+%% The set-cookie header is special; we can only send one cookie per header.
+headers_encode(Headers0=#{<<"set-cookie">> := SetCookies}, EncodeState) ->
+ Headers1 = maps:to_list(maps:remove(<<"set-cookie">>, Headers0)),
+ Headers = Headers1 ++ [{<<"set-cookie">>, Value} || Value <- SetCookies],
+ cow_hpack:encode(Headers, EncodeState);
headers_encode(Headers0, EncodeState) ->
Headers = maps:to_list(Headers0),
cow_hpack:encode(Headers, EncodeState).
diff --git a/src/cowboy_req.erl b/src/cowboy_req.erl
index 77d4a79..bbcd2b7 100644
--- a/src/cowboy_req.erl
+++ b/src/cowboy_req.erl
@@ -15,7 +15,7 @@
-module(cowboy_req).
-%% Request API.
+%% Request.
-export([method/1]).
-export([version/1]).
-export([peer/1]).
@@ -41,109 +41,83 @@
-export([parse_cookies/1]).
-export([match_cookies/2]).
-%% Request body API.
+%% Request body.
-export([has_body/1]).
-export([body_length/1]).
-export([read_body/1]).
-export([read_body/2]).
+-export([read_urlencoded_body/1]).
+-export([read_urlencoded_body/2]).
--export([body/1]).
--export([body/2]).
--export([body_qs/1]).
--export([body_qs/2]).
+%% Multipart.
+-export([part/1]). %% @todo read_part?
+-export([part/2]). %% @todo read_part?
+-export([part_body/1]). %% @todo read_part_body?
+-export([part_body/2]). %% @todo read_part_body?
-%% Multipart API.
--export([part/1]).
--export([part/2]).
--export([part_body/1]).
--export([part_body/2]).
-
-%% Response API.
+%% Response.
+-export([set_resp_cookie/3]).
-export([set_resp_cookie/4]).
-export([set_resp_header/3]).
--export([set_resp_body/2]).
--export([set_resp_body_fun/2]).
--export([set_resp_body_fun/3]).
-export([has_resp_header/2]).
--export([has_resp_body/1]).
+%% @todo resp_header
-export([delete_resp_header/2]).
+-export([set_resp_body/2]). %% @todo Use set_resp_body for iodata() | {sendfile ...}
+-export([has_resp_body/1]).
-export([reply/2]).
-export([reply/3]).
-export([reply/4]).
-
--export([send_body/3]).
-
--export([chunked_reply/2]).
--export([chunked_reply/3]).
--export([chunk/2]).
--export([continue/1]).
--export([maybe_reply/2]).
--export([ensure_response/2]).
+-export([stream_reply/2]).
+-export([stream_reply/3]).
+-export([stream_body/3]).
+-export([push/3]).
+-export([push/4]).
-type cookie_opts() :: cow_cookie:cookie_opts().
-export_type([cookie_opts/0]).
--type content_decode_fun() :: fun((binary()) -> binary()).
--type transfer_decode_fun() :: fun((binary(), any())
- -> cow_http_te:decode_ret()).
-
--type body_opts() :: [{continue, boolean()} %% doesn't apply
- | {length, non_neg_integer()}
- | {read_length, non_neg_integer()} %% to be added back later as optimization
- | {read_timeout, timeout()} %% same
- | {transfer_decode, transfer_decode_fun(), any()} %% doesn't apply
- | {content_decode, content_decode_fun()}]. %% does apply
+-type body_opts() :: #{
+ length => non_neg_integer(),
+ period => non_neg_integer(),
+ timeout => timeout()
+}.
-export_type([body_opts/0]).
--type resp_body_fun() :: fun((any(), module()) -> ok).
--type send_chunk_fun() :: fun((iodata()) -> ok).
--type resp_chunked_fun() :: fun((send_chunk_fun()) -> ok).
-
--record(http_req, {
- %% Transport.
- socket = undefined :: any(),
- transport = undefined :: undefined | module(),
- connection = keepalive :: keepalive | close,
-
- %% Request.
- pid = undefined :: pid(),
- method = <<"GET">> :: binary(),
- version = 'HTTP/1.1' :: cowboy:http_version(),
- peer = undefined :: undefined | {inet:ip_address(), inet:port_number()},
- host = undefined :: undefined | binary(),
- host_info = undefined :: undefined | cowboy_router:tokens(),
- port = undefined :: undefined | inet:port_number(),
- path = undefined :: binary(),
- path_info = undefined :: undefined | cowboy_router:tokens(),
- qs = undefined :: binary(),
- bindings = undefined :: undefined | cowboy_router:bindings(),
- headers = [] :: cowboy:http_headers(),
- meta = [] :: [{atom(), any()}],
-
- %% Request body.
- body_state = waiting :: waiting | done | {stream, non_neg_integer(),
- transfer_decode_fun(), any(), content_decode_fun()},
- buffer = <<>> :: binary(),
- multipart = undefined :: undefined | {binary(), binary()},
-
- %% Response.
- resp_compress = false :: boolean(),
- resp_state = waiting :: locked | waiting | waiting_stream
- | chunks | stream | done,
- resp_headers = [] :: cowboy:http_headers(),
- resp_body = <<>> :: iodata() | resp_body_fun()
- | {non_neg_integer(), resp_body_fun()}
- | {chunked, resp_chunked_fun()},
-
- %% Functions.
- onresponse = undefined :: undefined | already_called
- | cowboy:onresponse_fun()
-}).
-
--opaque req() :: #http_req{}.
+%% While sendfile allows a Len of 0 that means "everything past Offset",
+%% Cowboy expects the real length as it is used as metadata.
+%% @todo We should probably explicitly reject it.
+-type resp_body() :: iodata()
+ | {sendfile, non_neg_integer(), pos_integer(), file:name_all()}.
+-export_type([resp_body/0]).
+
+-type req() :: map(). %% @todo #{
+% ref := ranch:ref(),
+% pid := pid(),
+% streamid := cowboy_stream:streamid(),
+% peer := {inet:ip_address(), inet:port_number()},
+%
+% method := binary(), %% case sensitive
+% version := cowboy:http_version(),
+% scheme := binary(), %% <<"http">> or <<"https">>
+% host := binary(), %% lowercase; case insensitive
+% port := inet:port_number(),
+% path := binary(), %% case sensitive
+% qs := binary(), %% case sensitive
+% headers := cowboy:http_headers(),
+%
+% host_info => cowboy_router:tokens(),
+% path_info => cowboy_router:tokens(),
+% bindings => cowboy_router:bindings(),
+%
+% has_body := boolean(),
+% has_read_body => true,
+% body_length := undefined | non_neg_integer()
+%
+%% @todo resp_*
+%}.
-export_type([req/0]).
-%% Request API.
+%% Request.
-spec method(req()) -> binary().
method(#{method := Method}) ->
@@ -403,206 +377,66 @@ parse_cookies(Req) ->
match_cookies(Fields, Req) ->
filter(Fields, kvlist_to_map(Fields, parse_cookies(Req))).
-%% Request Body API.
+%% Request body.
-spec has_body(req()) -> boolean().
has_body(#{has_body := HasBody}) ->
HasBody.
-%% The length may not be known if Transfer-Encoding is not identity,
-%% and the body hasn't been read at the time of the call.
+%% The length may not be known if HTTP/1.1 with a transfer-encoding;
+%% or HTTP/2 with no content-length header. The length is always
+%% known once the body has been completely read.
-spec body_length(req()) -> undefined | non_neg_integer().
body_length(#{body_length := Length}) ->
Length.
--spec body(Req) -> {ok, binary(), Req} | {more, binary(), Req} when Req::req().
-body(Req) ->
- body(Req, []).
-
-spec read_body(Req) -> {ok, binary(), Req} | {more, binary(), Req} when Req::req().
read_body(Req) ->
- read_body(Req, []).
+ read_body(Req, #{}).
-spec read_body(Req, body_opts()) -> {ok, binary(), Req} | {more, binary(), Req} when Req::req().
+read_body(Req=#{has_body := false}, _) ->
+ {ok, <<>>, Req};
+read_body(Req=#{has_read_body := true}, _) ->
+ {ok, <<>>, Req};
read_body(Req=#{pid := Pid, streamid := StreamID}, Opts) ->
- %% @todo Opts should be a map
- Length = case lists:keyfind(length, 1, Opts) of
- false -> 8000000;
- {_, ChunkLen0} -> ChunkLen0
- end,
- ReadTimeout = case lists:keyfind(read_timeout, 1, Opts) of
- false -> 15000;
- {_, ReadTimeout0} -> ReadTimeout0
- end,
+ Length = maps:get(length, Opts, 8000000),
+ Period = maps:get(period, Opts, 15000),
+ Timeout = maps:get(timeout, Opts, Period + 1000),
Ref = make_ref(),
- Pid ! {{Pid, StreamID}, {read_body, Ref, Length}},
+ Pid ! {{Pid, StreamID}, {read_body, Ref, Length, Period}},
receive
{request_body, Ref, nofin, Body} ->
{more, Body, Req};
{request_body, Ref, {fin, BodyLength}, Body} ->
{ok, Body, set_body_length(Req, BodyLength)}
- after ReadTimeout ->
- exit(read_body_timeout)
+ after Timeout ->
+ exit(timeout)
end.
set_body_length(Req=#{headers := Headers}, BodyLength) ->
Req#{
headers => Headers#{<<"content-length">> => integer_to_binary(BodyLength)},
- body_length => BodyLength
+ body_length => BodyLength,
+ has_read_body => true
}.
--spec body(Req, body_opts()) -> {ok, binary(), Req} | {more, binary(), Req} when Req::req().
-body(Req=#http_req{body_state=waiting}, Opts) ->
- %% Send a 100 continue if needed (enabled by default).
- case lists:keyfind(continue, 1, Opts) of
- {_, false} ->
- ok;
- _ ->
- ExpectHeader = parse_header(<<"expect">>, Req),
- ok = case ExpectHeader of
- continue -> continue(Req);
- _ -> ok
- end
- end,
- %% Initialize body streaming state.
- CFun = case lists:keyfind(content_decode, 1, Opts) of
- false ->
- fun body_content_decode_identity/1;
- {_, CFun0} ->
- CFun0
- end,
- case lists:keyfind(transfer_decode, 1, Opts) of
- false ->
- case parse_header(<<"transfer-encoding">>, Req) of
- [<<"chunked">>] ->
- body(Req#http_req{body_state={stream, 0,
- fun cow_http_te:stream_chunked/2, {0, 0}, CFun}}, Opts);
- [<<"identity">>] ->
- case body_length(Req) of
- 0 ->
- {ok, <<>>, Req#http_req{body_state=done}};
- Len ->
- body(Req#http_req{body_state={stream, Len,
- fun cow_http_te:stream_identity/2, {0, Len},
- CFun}}, Opts)
- end
- end;
- {_, TFun, TState} ->
- body(Req#http_req{body_state={stream, 0,
- TFun, TState, CFun}}, Opts)
- end;
-body(Req=#http_req{body_state=done}, _) ->
- {ok, <<>>, Req};
-body(Req, Opts) ->
- ChunkLen = case lists:keyfind(length, 1, Opts) of
- false -> 8000000;
- {_, ChunkLen0} -> ChunkLen0
- end,
- ReadLen = case lists:keyfind(read_length, 1, Opts) of
- false -> 1000000;
- {_, ReadLen0} -> ReadLen0
- end,
- ReadTimeout = case lists:keyfind(read_timeout, 1, Opts) of
- false -> 15000;
- {_, ReadTimeout0} -> ReadTimeout0
- end,
- body_loop(Req, ReadTimeout, ReadLen, ChunkLen, <<>>).
-
-%% Default identity function for content decoding.
-%% @todo Move into cowlib when more content decode functions get implemented.
-body_content_decode_identity(Data) -> Data.
+-spec read_urlencoded_body(Req) -> {ok, [{binary(), binary() | true}], Req} when Req::req().
+read_urlencoded_body(Req) ->
+ read_urlencoded_body(Req, #{length => 64000, period => 5000}).
-body_loop(Req=#http_req{buffer=Buffer, body_state={stream, Length, _, _, _}},
- ReadTimeout, ReadLength, ChunkLength, Acc) ->
- {Tag, Res, Req2} = case Buffer of
- <<>> ->
- body_recv(Req, ReadTimeout, min(Length, ReadLength));
- _ ->
- body_decode(Req, ReadTimeout)
- end,
- case {Tag, Res} of
- {ok, Data} ->
- {ok, << Acc/binary, Data/binary >>, Req2};
- {more, Data} ->
- Acc2 = << Acc/binary, Data/binary >>,
- case byte_size(Acc2) >= ChunkLength of
- true -> {more, Acc2, Req2};
- false -> body_loop(Req2, ReadTimeout, ReadLength, ChunkLength, Acc2)
- end
- end.
+-spec read_urlencoded_body(Req, body_opts()) -> {ok, [{binary(), binary() | true}], Req} when Req::req().
+read_urlencoded_body(Req0, Opts) ->
+ {ok, Body, Req} = read_body(Req0, Opts),
+ {ok, cow_qs:parse_qs(Body), Req}.
-body_recv(Req=#http_req{transport=Transport, socket=Socket, buffer=Buffer},
- ReadTimeout, ReadLength) ->
- {ok, Data} = Transport:recv(Socket, ReadLength, ReadTimeout),
- body_decode(Req#http_req{buffer= << Buffer/binary, Data/binary >>}, ReadTimeout).
-
-%% Two decodings happen. First a decoding function is applied to the
-%% transferred data, and then another is applied to the actual content.
-%%
-%% Transfer encoding is generally used for chunked bodies. The decoding
-%% function uses a state to keep track of how much it has read, which is
-%% also initialized through this function.
-%%
-%% Content encoding is generally used for compression.
-%%
-%% @todo Handle chunked after-the-facts headers.
-%% @todo Depending on the length returned we might want to 0 or +5 it.
-body_decode(Req=#http_req{buffer=Data, body_state={stream, _,
- TDecode, TState, CDecode}}, ReadTimeout) ->
- case TDecode(Data, TState) of
- more ->
- body_recv(Req#http_req{body_state={stream, 0,
- TDecode, TState, CDecode}}, ReadTimeout, 0);
- {more, Data2, TState2} ->
- {more, CDecode(Data2), Req#http_req{body_state={stream, 0,
- TDecode, TState2, CDecode}, buffer= <<>>}};
- {more, Data2, Length, TState2} when is_integer(Length) ->
- {more, CDecode(Data2), Req#http_req{body_state={stream, Length,
- TDecode, TState2, CDecode}, buffer= <<>>}};
- {more, Data2, Rest, TState2} ->
- {more, CDecode(Data2), Req#http_req{body_state={stream, 0,
- TDecode, TState2, CDecode}, buffer=Rest}};
- {done, TotalLength, Rest} ->
- {ok, <<>>, body_decode_end(Req, TotalLength, Rest)};
- {done, Data2, TotalLength, Rest} ->
- {ok, CDecode(Data2), body_decode_end(Req, TotalLength, Rest)}
- end.
-
-body_decode_end(Req=#http_req{headers=Headers}, TotalLength, Rest) ->
- Headers2 = lists:keystore(<<"content-length">>, 1, Headers,
- {<<"content-length">>, integer_to_binary(TotalLength)}),
- %% At this point we just assume TEs were all decoded.
- Headers3 = lists:keydelete(<<"transfer-encoding">>, 1, Headers2),
- Req#http_req{buffer=Rest, body_state=done, headers=Headers3}.
-
--spec body_qs(Req) -> {ok, [{binary(), binary() | true}], Req}
- | {badlength, Req} when Req::req().
-body_qs(Req) ->
- body_qs(Req, [
- {length, 64000},
- {read_length, 64000},
- {read_timeout, 5000}]).
-
--spec body_qs(Req, body_opts()) -> {ok, [{binary(), binary() | true}], Req}
- | {badlength, Req} when Req::req().
-body_qs(Req, Opts) ->
- case read_body(Req, Opts) of
- {ok, Body, Req2} ->
- {ok, cow_qs:parse_qs(Body), Req2};
- {more, _, Req2} ->
- {badlength, Req2}
- end.
-
-%% Multipart API.
+%% Multipart.
-spec part(Req)
-> {ok, cow_multipart:headers(), Req} | {done, Req}
when Req::req().
part(Req) ->
- part(Req, [
- {length, 64000},
- {read_length, 64000},
- {read_timeout, 5000}]).
+ part(Req, #{length => 64000, period => 5000}).
-spec part(Req, body_opts())
-> {ok, cow_multipart:headers(), Req} | {done, Req}
@@ -635,7 +469,7 @@ part(Buffer, Opts, Req=#{multipart := {Boundary, _}}) ->
-> {ok, binary(), Req} | {more, binary(), Req}
when Req::req().
part_body(Req) ->
- part_body(Req, []).
+ part_body(Req, #{}).
-spec part_body(Req, body_opts())
-> {ok, binary(), Req} | {more, binary(), Req}
@@ -649,11 +483,8 @@ part_body(Req, Opts) ->
end.
part_body(Buffer, Opts, Req=#{multipart := {Boundary, _}}, Acc) ->
- ChunkLen = case lists:keyfind(length, 1, Opts) of
- false -> 8000000;
- {_, ChunkLen0} -> ChunkLen0
- end,
- case byte_size(Acc) > ChunkLen of
+ Length = maps:get(length, Opts, 8000000),
+ case byte_size(Acc) > Length of
true ->
{more, Acc, Req#{multipart => {Boundary, Buffer}}};
false ->
@@ -686,7 +517,12 @@ stream_multipart(Req=#{multipart := {_, <<>>}}, Opts) ->
stream_multipart(Req=#{multipart := {Boundary, Buffer}}, _) ->
{Buffer, Req#{multipart => {Boundary, <<>>}}}.
-%% Response API.
+%% Response.
+
+-spec set_resp_cookie(iodata(), iodata(), Req)
+ -> Req when Req::req().
+set_resp_cookie(Name, Value, Req) ->
+ set_resp_cookie(Name, Value, #{}, Req).
%% The cookie name cannot contain any of the following characters:
%% =,;\s\t\r\n\013\014
@@ -696,9 +532,11 @@ stream_multipart(Req=#{multipart := {Boundary, Buffer}}, _) ->
-spec set_resp_cookie(iodata(), iodata(), cookie_opts(), Req)
-> Req when Req::req().
set_resp_cookie(Name, Value, Opts, Req) ->
- Cookie = cow_cookie:setcookie(Name, Value, Opts),
- %% @todo Nah, keep separate.
- set_resp_header(<<"set-cookie">>, Cookie, Req).
+ Cookie = cow_cookie:setcookie(Name, Value, maps:to_list(Opts)),
+ RespCookies = maps:get(resp_cookies, Req, #{}),
+ Req#{resp_cookies => RespCookies#{Name => Cookie}}.
+
+%% @todo We could add has_resp_cookie and delete_resp_cookie now.
-spec set_resp_header(binary(), iodata(), Req)
-> Req when Req::req().
@@ -707,29 +545,9 @@ set_resp_header(Name, Value, Req=#{resp_headers := RespHeaders}) ->
set_resp_header(Name,Value, Req) ->
Req#{resp_headers => #{Name => Value}}.
-%% @todo {sendfile, Offset, Bytes, Path} tuple
--spec set_resp_body(iodata(), Req) -> Req when Req::req().
+-spec set_resp_body(resp_body(), Req) -> Req when Req::req().
set_resp_body(Body, Req) ->
Req#{resp_body => Body}.
-%set_resp_body(Body, Req) ->
-% Req#http_req{resp_body=Body}.
-
--spec set_resp_body_fun(resp_body_fun(), Req) -> Req when Req::req().
-set_resp_body_fun(StreamFun, Req) when is_function(StreamFun) ->
- Req#http_req{resp_body=StreamFun}.
-
-%% If the body function crashes while writing the response body or writes
-%% fewer bytes than declared the behaviour is undefined.
--spec set_resp_body_fun(non_neg_integer(), resp_body_fun(), Req)
- -> Req when Req::req();
- (chunked, resp_chunked_fun(), Req)
- -> Req when Req::req().
-set_resp_body_fun(StreamLen, StreamFun, Req)
- when is_integer(StreamLen), is_function(StreamFun) ->
- Req#http_req{resp_body={StreamLen, StreamFun}};
-set_resp_body_fun(chunked, StreamFun, Req)
- when is_function(StreamFun) ->
- Req#http_req{resp_body={chunked, StreamFun}}.
-spec has_resp_header(binary(), req()) -> boolean().
has_resp_header(Name, #{resp_headers := RespHeaders}) ->
@@ -738,22 +556,13 @@ has_resp_header(_, _) ->
false.
-spec has_resp_body(req()) -> boolean().
-has_resp_body(#{resp_body := {sendfile, Len, _}}) ->
- Len > 0;
+has_resp_body(#{resp_body := {sendfile, _, _, _}}) ->
+ true;
has_resp_body(#{resp_body := RespBody}) ->
iolist_size(RespBody) > 0;
has_resp_body(_) ->
false.
-%has_resp_body(#http_req{resp_body=RespBody}) when is_function(RespBody) ->
-% true;
-%has_resp_body(#http_req{resp_body={chunked, _}}) ->
-% true;
-%has_resp_body(#http_req{resp_body={Length, _}}) ->
-% Length > 0;
-%has_resp_body(#http_req{resp_body=RespBody}) ->
-% iolist_size(RespBody) > 0.
-
-spec delete_resp_header(binary(), Req)
-> Req when Req::req().
delete_resp_header(Name, Req=#{resp_headers := RespHeaders}) ->
@@ -770,287 +579,89 @@ reply(Status, Headers, Req=#{resp_body := Body}) ->
reply(Status, Headers, Req) ->
reply(Status, Headers, <<>>, Req).
--spec reply(cowboy:http_status(), cowboy:http_headers(),
- iodata() | resp_body_fun() | {non_neg_integer(), resp_body_fun()}
- | {chunked, resp_chunked_fun()}, Req)
+-spec reply(cowboy:http_status(), cowboy:http_headers(), resp_body(), Req)
-> Req when Req::req().
-reply(Status, Headers, Stream = {stream, undefined, _}, Req) ->
- do_stream_reply(Status, Headers, Stream, Req);
-reply(Status, Headers, Stream = {stream, Len, _}, Req) ->
- do_stream_reply(Status, Headers#{
- <<"content-length">> => integer_to_binary(Len)
- }, Stream, Req);
-reply(Status, Headers, SendFile = {sendfile, _, Len, _}, Req) ->
+reply(Status, Headers, SendFile = {sendfile, _, Len, _}, Req)
+ when is_integer(Status); is_binary(Status) ->
do_reply(Status, Headers#{
<<"content-length">> => integer_to_binary(Len)
}, SendFile, Req);
-reply(Status, Headers, Body, Req) ->
+reply(Status, Headers, Body, Req)
+ when is_integer(Status); is_binary(Status) ->
do_reply(Status, Headers#{
<<"content-length">> => integer_to_binary(iolist_size(Body))
}, Body, Req).
-do_stream_reply(Status, Headers, {stream, _, Fun}, Req=#{pid := Pid, streamid := StreamID}) ->
- Pid ! {{Pid, StreamID}, {headers, Status, response_headers(Headers, Req)}},
- Fun(),
- ok.
-
+%% Don't send any body for HEAD responses. While the protocol code is
+%% supposed to enforce this rule, we prefer to avoid copying too much
+%% data around if we can avoid it.
+do_reply(Status, Headers, _, Req=#{pid := Pid, streamid := StreamID, method := <<"HEAD">>}) ->
+ Pid ! {{Pid, StreamID}, {response, Status, response_headers(Headers, Req), <<>>}},
+ ok;
do_reply(Status, Headers, Body, Req=#{pid := Pid, streamid := StreamID}) ->
Pid ! {{Pid, StreamID}, {response, Status, response_headers(Headers, Req), Body}},
ok.
--spec send_body(iodata(), fin | nofin, req()) -> ok.
-send_body(Data, IsFin, #{pid := Pid, streamid := StreamID}) ->
- Pid ! {{Pid, StreamID}, {data, IsFin, Data}},
- ok.
+-spec stream_reply(cowboy:http_status(), Req) -> Req when Req::req().
+stream_reply(Status, Req) ->
+ stream_reply(Status, #{}, Req).
-response_headers(Headers, Req) ->
- RespHeaders = maps:get(resp_headers, Req, #{}),
- maps:merge(#{
- <<"date">> => cowboy_clock:rfc1123(),
- <<"server">> => <<"Cowboy">>
- }, maps:merge(RespHeaders, Headers)).
-
-%reply(Status, Headers, Body, Req=#http_req{
-% socket=Socket, transport=Transport,
-% version=Version, connection=Connection,
-% method=Method, resp_compress=Compress,
-% resp_state=RespState, resp_headers=RespHeaders})
-% when RespState =:= waiting; RespState =:= waiting_stream ->
-% Req3 = case Body of
-% BodyFun when is_function(BodyFun) ->
-% %% We stream the response body until we close the connection.
-% RespConn = close,
-% {RespType, Req2} = if
-% true ->
-% response(Status, Headers, RespHeaders, [
-% {<<"connection">>, <<"close">>},
-% {<<"date">>, cowboy_clock:rfc1123()},
-% {<<"server">>, <<"Cowboy">>},
-% {<<"transfer-encoding">>, <<"identity">>}
-% ], <<>>, Req)
-% end,
-% if RespType =/= hook, Method =/= <<"HEAD">> ->
-% BodyFun(Socket, Transport);
-% true -> ok
-% end,
-% Req2#http_req{connection=RespConn};
-% {chunked, BodyFun} ->
-% %% We stream the response body in chunks.
-% {RespType, Req2} = chunked_response(Status, Headers, Req),
-% if RespType =/= hook, Method =/= <<"HEAD">> ->
-% ChunkFun = fun(IoData) -> chunk(IoData, Req2) end,
-% BodyFun(ChunkFun),
-% %% Send the last chunk if chunked encoding was used.
-% if
-% Version =:= 'HTTP/1.0'; RespState =:= waiting_stream ->
-% Req2;
-% true ->
-% last_chunk(Req2)
-% end;
-% true -> Req2
-% end;
-% {ContentLength, BodyFun} ->
-% %% We stream the response body for ContentLength bytes.
-% RespConn = response_connection(Headers, Connection),
-% {RespType, Req2} = response(Status, Headers, RespHeaders, [
-% {<<"content-length">>, integer_to_list(ContentLength)},
-% {<<"date">>, cowboy_clock:rfc1123()},
-% {<<"server">>, <<"Cowboy">>}
-% |HTTP11Headers], stream, Req),
-% if RespType =/= hook, Method =/= <<"HEAD">> ->
-% BodyFun(Socket, Transport);
-% true -> ok
-% end,
-% Req2#http_req{connection=RespConn};
-% _ when Compress ->
-% RespConn = response_connection(Headers, Connection),
-% Req2 = reply_may_compress(Status, Headers, Body, Req,
-% RespHeaders, HTTP11Headers, Method),
-% Req2#http_req{connection=RespConn};
-% _ ->
-% RespConn = response_connection(Headers, Connection),
-% Req2 = reply_no_compress(Status, Headers, Body, Req,
-% RespHeaders, HTTP11Headers, Method, iolist_size(Body)),
-% Req2#http_req{connection=RespConn}
-% end,
-% Req3#http_req{resp_state=done, resp_headers=[], resp_body= <<>>}.
-
-%reply_may_compress(Status, Headers, Body, Req,
-% RespHeaders, HTTP11Headers, Method) ->
-% BodySize = iolist_size(Body),
-% try parse_header(<<"accept-encoding">>, Req) of
-% Encodings ->
-% CanGzip = (BodySize > 300)
-% andalso (false =:= lists:keyfind(<<"content-encoding">>,
-% 1, Headers))
-% andalso (false =:= lists:keyfind(<<"content-encoding">>,
-% 1, RespHeaders))
-% andalso (false =:= lists:keyfind(<<"transfer-encoding">>,
-% 1, Headers))
-% andalso (false =:= lists:keyfind(<<"transfer-encoding">>,
-% 1, RespHeaders))
-% andalso (Encodings =/= undefined)
-% andalso (false =/= lists:keyfind(<<"gzip">>, 1, Encodings)),
-% case CanGzip of
-% true ->
-% GzBody = zlib:gzip(Body),
-% {_, Req2} = response(Status, Headers, RespHeaders, [
-% {<<"content-length">>, integer_to_list(byte_size(GzBody))},
-% {<<"content-encoding">>, <<"gzip">>},
-% |HTTP11Headers],
-% case Method of <<"HEAD">> -> <<>>; _ -> GzBody end,
-% Req),
-% Req2;
-% false ->
-% reply_no_compress(Status, Headers, Body, Req,
-% RespHeaders, HTTP11Headers, Method, BodySize)
-% end
-% catch _:_ ->
-% reply_no_compress(Status, Headers, Body, Req,
-% RespHeaders, HTTP11Headers, Method, BodySize)
-% end.
-%
-%reply_no_compress(Status, Headers, Body, Req,
-% RespHeaders, HTTP11Headers, Method, BodySize) ->
-% {_, Req2} = response(Status, Headers, RespHeaders, [
-% {<<"content-length">>, integer_to_list(BodySize)},
-% |HTTP11Headers],
-% case Method of <<"HEAD">> -> <<>>; _ -> Body end,
-% Req),
-% Req2.
-
--spec chunked_reply(cowboy:http_status(), Req) -> Req when Req::req().
-chunked_reply(Status, Req) ->
- chunked_reply(Status, #{}, Req).
-
--spec chunked_reply(cowboy:http_status(), cowboy:http_headers(), Req)
+-spec stream_reply(cowboy:http_status(), cowboy:http_headers(), Req)
-> Req when Req::req().
-chunked_reply(Status, Headers, Req=#{pid := Pid, streamid := StreamID}) ->
+stream_reply(Status, Headers=#{}, Req=#{pid := Pid, streamid := StreamID})
+ when is_integer(Status); is_binary(Status) ->
Pid ! {{Pid, StreamID}, {headers, Status, response_headers(Headers, Req)}},
- Req. %% @todo return ok
-% ok.
+ ok.
--spec chunk(iodata(), req()) -> ok.
-chunk(_Data, #{method := <<"HEAD">>}) ->
+-spec stream_body(iodata(), fin | nofin, req()) -> ok.
+%% Don't send any body for HEAD responses.
+stream_body(_, _, #{method := <<"HEAD">>}) ->
ok;
-chunk(Data, #{pid := Pid, streamid := StreamID}) ->
+%% Don't send a message if the data is empty, except for the
+%% very last message with IsFin=fin.
+stream_body(Data, IsFin=nofin, #{pid := Pid, streamid := StreamID}) ->
case iolist_size(Data) of
0 -> ok;
_ ->
- Pid ! {{Pid, StreamID}, {data, nofin, Data}},
+ Pid ! {{Pid, StreamID}, {data, IsFin, Data}},
ok
- end.
-
-%% If ever made public, need to send nothing if HEAD.
--spec last_chunk(Req) -> Req when Req::req().
-last_chunk(Req=#http_req{socket=Socket, transport=Transport}) ->
- _ = Transport:send(Socket, <<"0\r\n\r\n">>),
- Req#http_req{resp_state=done}.
-
--spec continue(req()) -> ok.
-continue(#http_req{socket=Socket, transport=Transport,
- version=Version}) ->
- HTTPVer = atom_to_binary(Version, latin1),
- ok = Transport:send(Socket,
- << HTTPVer/binary, " ", (status(100))/binary, "\r\n\r\n" >>).
-
-%% Meant to be used internally for sending errors after crashes.
--spec maybe_reply([{module(), atom(), arity() | [term()], _}], req()) -> ok.
-maybe_reply(Stacktrace, Req) ->
- receive
- {cowboy_req, resp_sent} -> ok
- after 0 ->
- _ = do_maybe_reply(Stacktrace, Req),
- ok
- end.
+ end;
+stream_body(Data, IsFin, #{pid := Pid, streamid := StreamID}) ->
+ Pid ! {{Pid, StreamID}, {data, IsFin, Data}},
+ ok.
-do_maybe_reply([{erlang, binary_to_integer, _, _}, {cow_http_hd, parse_content_length, _, _}|_], Req) ->
- cowboy_req:reply(400, Req);
-do_maybe_reply([{cow_http_hd, _, _, _}|_], Req) ->
- cowboy_req:reply(400, Req);
-do_maybe_reply(_, Req) ->
- cowboy_req:reply(500, Req).
+push(Path, Headers, Req) ->
+ push(Path, Headers, Req, #{}).
--spec ensure_response(req(), cowboy:http_status()) -> ok.
-%% The response has already been fully sent to the client.
-ensure_response(#http_req{resp_state=done}, _) ->
- ok;
-%% No response has been sent but everything apparently went fine.
-%% Reply with the status code found in the second argument.
-ensure_response(Req=#http_req{resp_state=RespState}, Status)
- when RespState =:= waiting; RespState =:= waiting_stream ->
- _ = reply(Status, [], [], Req),
- ok;
-%% Terminate the chunked body for HTTP/1.1 only.
-ensure_response(#http_req{method= <<"HEAD">>}, _) ->
- ok;
-ensure_response(Req=#http_req{resp_state=chunks}, _) ->
- _ = last_chunk(Req),
- ok;
-ensure_response(#http_req{}, _) ->
+%% @todo Optimization: don't send anything at all for HTTP/1.0 and HTTP/1.1.
+%% @todo Path, Headers, Opts, everything should be in proper binary,
+%% or normalized when creating the Req object.
+push(Path, Headers, #{pid := Pid, streamid := StreamID,
+ scheme := Scheme0, host := Host0, port := Port0}, Opts) ->
+ Method = maps:get(method, Opts, <<"GET">>),
+ Scheme = maps:get(scheme, Opts, Scheme0),
+ Host = maps:get(host, Opts, Host0),
+ Port = maps:get(port, Opts, Port0),
+ Qs = maps:get(qs, Opts, <<>>),
+ Pid ! {{Pid, StreamID}, {push, Method, Scheme, Host, Port, Path, Qs, Headers}},
ok.
%% Internal.
--spec status(cowboy:http_status()) -> binary().
-status(100) -> <<"100 Continue">>;
-status(101) -> <<"101 Switching Protocols">>;
-status(102) -> <<"102 Processing">>;
-status(200) -> <<"200 OK">>;
-status(201) -> <<"201 Created">>;
-status(202) -> <<"202 Accepted">>;
-status(203) -> <<"203 Non-Authoritative Information">>;
-status(204) -> <<"204 No Content">>;
-status(205) -> <<"205 Reset Content">>;
-status(206) -> <<"206 Partial Content">>;
-status(207) -> <<"207 Multi-Status">>;
-status(226) -> <<"226 IM Used">>;
-status(300) -> <<"300 Multiple Choices">>;
-status(301) -> <<"301 Moved Permanently">>;
-status(302) -> <<"302 Found">>;
-status(303) -> <<"303 See Other">>;
-status(304) -> <<"304 Not Modified">>;
-status(305) -> <<"305 Use Proxy">>;
-status(306) -> <<"306 Switch Proxy">>;
-status(307) -> <<"307 Temporary Redirect">>;
-status(400) -> <<"400 Bad Request">>;
-status(401) -> <<"401 Unauthorized">>;
-status(402) -> <<"402 Payment Required">>;
-status(403) -> <<"403 Forbidden">>;
-status(404) -> <<"404 Not Found">>;
-status(405) -> <<"405 Method Not Allowed">>;
-status(406) -> <<"406 Not Acceptable">>;
-status(407) -> <<"407 Proxy Authentication Required">>;
-status(408) -> <<"408 Request Timeout">>;
-status(409) -> <<"409 Conflict">>;
-status(410) -> <<"410 Gone">>;
-status(411) -> <<"411 Length Required">>;
-status(412) -> <<"412 Precondition Failed">>;
-status(413) -> <<"413 Request Entity Too Large">>;
-status(414) -> <<"414 Request-URI Too Long">>;
-status(415) -> <<"415 Unsupported Media Type">>;
-status(416) -> <<"416 Requested Range Not Satisfiable">>;
-status(417) -> <<"417 Expectation Failed">>;
-status(418) -> <<"418 I'm a teapot">>;
-status(422) -> <<"422 Unprocessable Entity">>;
-status(423) -> <<"423 Locked">>;
-status(424) -> <<"424 Failed Dependency">>;
-status(425) -> <<"425 Unordered Collection">>;
-status(426) -> <<"426 Upgrade Required">>;
-status(428) -> <<"428 Precondition Required">>;
-status(429) -> <<"429 Too Many Requests">>;
-status(431) -> <<"431 Request Header Fields Too Large">>;
-status(500) -> <<"500 Internal Server Error">>;
-status(501) -> <<"501 Not Implemented">>;
-status(502) -> <<"502 Bad Gateway">>;
-status(503) -> <<"503 Service Unavailable">>;
-status(504) -> <<"504 Gateway Timeout">>;
-status(505) -> <<"505 HTTP Version Not Supported">>;
-status(506) -> <<"506 Variant Also Negotiates">>;
-status(507) -> <<"507 Insufficient Storage">>;
-status(510) -> <<"510 Not Extended">>;
-status(511) -> <<"511 Network Authentication Required">>;
-status(B) when is_binary(B) -> B.
+%% @todo What about set-cookie headers set through set_resp_header or reply?
+response_headers(Headers0, Req) ->
+ RespHeaders = maps:get(resp_headers, Req, #{}),
+ Headers = maps:merge(#{
+ <<"date">> => cowboy_clock:rfc1123(),
+ <<"server">> => <<"Cowboy">>
+ }, maps:merge(RespHeaders, Headers0)),
+ %% The set-cookie header is special; we can only send one cookie per header.
+ %% We send the list of values for many cookies in one key of the map,
+ %% and let the protocols deal with it directly.
+ case maps:get(resp_cookies, Req, undefined) of
+ undefined -> Headers;
+ RespCookies -> Headers#{<<"set-cookie">> => maps:values(RespCookies)}
+ end.
%% Create map, convert keys to atoms and group duplicate keys into lists.
%% Keys that are not found in the user provided list are entirely skipped.
diff --git a/src/cowboy_stream_h.erl b/src/cowboy_stream_h.erl
index b834c17..54dcc2d 100644
--- a/src/cowboy_stream_h.erl
+++ b/src/cowboy_stream_h.erl
@@ -29,7 +29,8 @@
ref = undefined :: ranch:ref(),
pid = undefined :: pid(),
read_body_ref = undefined :: reference(),
- read_body_length = 0 :: non_neg_integer(),
+ read_body_timer_ref = undefined :: reference(),
+ read_body_length = 0 :: non_neg_integer() | infinity,
read_body_is_fin = nofin :: nofin | fin,
read_body_buffer = <<>> :: binary()
}).
@@ -58,9 +59,11 @@ data(_StreamID, IsFin, Data, State=#state{read_body_ref=undefined, read_body_buf
{[], State#state{read_body_is_fin=IsFin, read_body_buffer= << Buffer/binary, Data/binary >>}};
data(_StreamID, nofin, Data, State=#state{read_body_length=Length, read_body_buffer=Buffer}) when byte_size(Data) + byte_size(Buffer) < Length ->
{[], State#state{read_body_buffer= << Buffer/binary, Data/binary >>}};
-data(_StreamID, IsFin, Data, State=#state{pid=Pid, read_body_ref=Ref, read_body_buffer=Buffer}) ->
+data(_StreamID, IsFin, Data, State=#state{pid=Pid, read_body_ref=Ref,
+ read_body_timer_ref=TRef, read_body_buffer=Buffer}) ->
+ ok = erlang:cancel_timer(TRef, [{async, true}, {info, false}]),
Pid ! {request_body, Ref, IsFin, << Buffer/binary, Data/binary >>},
- {[], State#state{read_body_ref=undefined, read_body_buffer= <<>>}}.
+ {[], State#state{read_body_ref=undefined, read_body_timer_ref=undefined, read_body_buffer= <<>>}}.
%% @todo proper specs
-spec info(_,_,_) -> _.
@@ -90,17 +93,26 @@ info(StreamID, Exit = {'EXIT', Pid, {Reason, Stacktrace}}, State=#state{ref=Ref,
{internal_error, Exit, 'Stream process crashed.'}
], State};
%% Request body, no body buffer but IsFin=fin.
-info(_StreamID, {read_body, Ref, _}, State=#state{pid=Pid, read_body_is_fin=fin, read_body_buffer= <<>>}) ->
- Pid ! {request_body, Ref, fin, <<>>},
- {[], State};
+%info(_StreamID, {read_body, Ref, _, _}, State=#state{pid=Pid, read_body_is_fin=fin, read_body_buffer= <<>>}) ->
+% Pid ! {request_body, Ref, fin, <<>>},
+% {[], State};
%% Request body, body buffered large enough or complete.
-info(_StreamID, {read_body, Ref, Length}, State=#state{pid=Pid, read_body_is_fin=IsFin, read_body_buffer=Data})
+info(_StreamID, {read_body, Ref, Length, _},
+ State=#state{pid=Pid, read_body_is_fin=IsFin, read_body_buffer=Data})
when element(1, IsFin) =:= fin; byte_size(Data) >= Length ->
Pid ! {request_body, Ref, IsFin, Data},
{[], State#state{read_body_buffer= <<>>}};
%% Request body, not enough to send yet.
-info(_StreamID, {read_body, Ref, Length}, State) ->
- {[{flow, Length}], State#state{read_body_ref=Ref, read_body_length=Length}};
+info(StreamID, {read_body, Ref, Length, Period}, State) ->
+ TRef = erlang:send_after(Period, self(), {{self(), StreamID}, {read_body_timeout, Ref}}),
+ {[{flow, Length}], State#state{read_body_ref=Ref, read_body_timer_ref=TRef, read_body_length=Length}};
+%% Request body reading timeout; send what we got.
+info(_StreamID, {read_body_timeout, Ref}, State=#state{pid=Pid, read_body_ref=Ref,
+ read_body_is_fin=IsFin, read_body_buffer=Buffer}) ->
+ Pid ! {request_body, Ref, IsFin, Buffer},
+ {[], State#state{read_body_ref=undefined, read_body_timer_ref=undefined, read_body_buffer= <<>>}};
+info(_StreamID, {read_body_timeout, _}, State) ->
+ {[], State};
%% Response.
info(_StreamID, Response = {response, _, _, _}, State) ->
{[Response], State};
@@ -108,6 +120,8 @@ info(_StreamID, Headers = {headers, _, _}, State) ->
{[Headers], State};
info(_StreamID, Data = {data, _, _}, State) ->
{[Data], State};
+info(_StreamID, Push = {push, _, _, _, _, _, _, _}, State) ->
+ {[Push], State};
info(_StreamID, SwitchProtocol = {switch_protocol, _, _, _}, State) ->
{[SwitchProtocol], State};
%% Stray message.
diff --git a/test/handlers/echo_h.erl b/test/handlers/echo_h.erl
index fd45c5f..98594dc 100644
--- a/test/handlers/echo_h.erl
+++ b/test/handlers/echo_h.erl
@@ -12,10 +12,32 @@ init(Req, Opts) ->
echo_arg(Arg, Req, Opts)
end.
-echo(<<"body">>, Req0, Opts) ->
- {ok, Body, Req} = cowboy_req:read_body(Req0),
+echo(<<"read_body">>, Req0, Opts) ->
+ case Opts of
+ #{crash := true} -> ct_helper:ignore(cowboy_req, read_body, 2);
+ _ -> ok
+ end,
+ {_, Body, Req} = case cowboy_req:path(Req0) of
+ <<"/full", _/bits>> -> read_body(Req0, <<>>);
+ <<"/opts", _/bits>> -> cowboy_req:read_body(Req0, Opts);
+ _ -> cowboy_req:read_body(Req0)
+ end,
cowboy_req:reply(200, #{}, Body, Req),
{ok, Req, Opts};
+echo(<<"read_urlencoded_body">>, Req0, Opts) ->
+ Path = cowboy_req:path(Req0),
+ case {Path, Opts} of
+ {<<"/opts", _/bits>>, #{crash := true}} -> ct_helper:ignore(cowboy_req, read_body, 2);
+ {_, #{crash := true}} -> ct_helper:ignore(cowboy_req, read_urlencoded_body, 2);
+ _ -> ok
+ end,
+ {ok, Body, Req} = case Path of
+ <<"/opts", _/bits>> -> cowboy_req:read_urlencoded_body(Req0, Opts);
+ <<"/crash", _/bits>> -> cowboy_req:read_urlencoded_body(Req0, Opts);
+ _ -> cowboy_req:read_urlencoded_body(Req0)
+ end,
+ cowboy_req:reply(200, #{}, value_to_iodata(Body), Req),
+ {ok, Req, Opts};
echo(<<"uri">>, Req, Opts) ->
Value = case cowboy_req:path_info(Req) of
[<<"origin">>] -> cowboy_req:uri(Req, #{host => undefined});
@@ -55,6 +77,12 @@ echo_arg(Arg0, Req, Opts) ->
cowboy_req:reply(200, #{}, value_to_iodata(Value), Req),
{ok, Req, Opts}.
+read_body(Req0, Acc) ->
+ case cowboy_req:read_body(Req0) of
+ {ok, Data, Req} -> {ok, << Acc/binary, Data/binary >>, Req};
+ {more, Data, Req} -> read_body(Req, << Acc/binary, Data/binary >>)
+ end.
+
value_to_iodata(V) when is_integer(V) -> integer_to_binary(V);
value_to_iodata(V) when is_atom(V) -> atom_to_binary(V, latin1);
value_to_iodata(V) when is_list(V); is_tuple(V); is_map(V) -> io_lib:format("~p", [V]);
diff --git a/test/handlers/multipart_h.erl b/test/handlers/multipart_h.erl
new file mode 100644
index 0000000..289d2ed
--- /dev/null
+++ b/test/handlers/multipart_h.erl
@@ -0,0 +1,65 @@
+%% This module reads a multipart body and echoes it back as an Erlang term.
+
+-module(multipart_h).
+
+-export([init/2]).
+
+init(Req0, State) ->
+ {Result, Req} = case cowboy_req:binding(key, Req0) of
+ undefined -> acc_multipart(Req0, []);
+ <<"skip_body">> -> skip_body_multipart(Req0, []);
+ <<"read_part2">> -> read_part2_multipart(Req0, []);
+ <<"read_part_body2">> -> read_part_body2_multipart(Req0, [])
+ end,
+ {ok, cowboy_req:reply(200, #{}, term_to_binary(Result), Req), State}.
+
+acc_multipart(Req0, Acc) ->
+ case cowboy_req:part(Req0) of
+ {ok, Headers, Req1} ->
+ {ok, Body, Req} = stream_body(Req1, <<>>),
+ acc_multipart(Req, [{Headers, Body}|Acc]);
+ {done, Req} ->
+ {lists:reverse(Acc), Req}
+ end.
+
+stream_body(Req0, Acc) ->
+ case cowboy_req:part_body(Req0) of
+ {more, Data, Req} ->
+ stream_body(Req, << Acc/binary, Data/binary >>);
+ {ok, Data, Req} ->
+ {ok, << Acc/binary, Data/binary >>, Req}
+ end.
+
+skip_body_multipart(Req0, Acc) ->
+ case cowboy_req:part(Req0) of
+ {ok, Headers, Req} ->
+ skip_body_multipart(Req, [Headers|Acc]);
+ {done, Req} ->
+ {lists:reverse(Acc), Req}
+ end.
+
+read_part2_multipart(Req0, Acc) ->
+ case cowboy_req:part(Req0, #{length => 1, period => 1}) of
+ {ok, Headers, Req1} ->
+ {ok, Body, Req} = stream_body(Req1, <<>>),
+ acc_multipart(Req, [{Headers, Body}|Acc]);
+ {done, Req} ->
+ {lists:reverse(Acc), Req}
+ end.
+
+read_part_body2_multipart(Req0, Acc) ->
+ case cowboy_req:part(Req0) of
+ {ok, Headers, Req1} ->
+ {ok, Body, Req} = stream_body2(Req1, <<>>),
+ acc_multipart(Req, [{Headers, Body}|Acc]);
+ {done, Req} ->
+ {lists:reverse(Acc), Req}
+ end.
+
+stream_body2(Req0, Acc) ->
+ case cowboy_req:part_body(Req0, #{length => 1, period => 1}) of
+ {more, Data, Req} ->
+ stream_body(Req, << Acc/binary, Data/binary >>);
+ {ok, Data, Req} ->
+ {ok, << Acc/binary, Data/binary >>, Req}
+ end.
diff --git a/test/handlers/resp_h.erl b/test/handlers/resp_h.erl
new file mode 100644
index 0000000..36e6f13
--- /dev/null
+++ b/test/handlers/resp_h.erl
@@ -0,0 +1,158 @@
+%% This module echoes back the value the test is interested in.
+
+-module(resp_h).
+
+-export([init/2]).
+
+init(Req, Opts) ->
+ do(cowboy_req:binding(key, Req), Req, Opts).
+
+do(<<"set_resp_cookie3">>, Req0, Opts) ->
+ Req = case cowboy_req:binding(arg, Req0) of
+ undefined ->
+ cowboy_req:set_resp_cookie(<<"mycookie">>, "myvalue", Req0);
+ <<"multiple">> ->
+ Req1 = cowboy_req:set_resp_cookie(<<"mycookie">>, "myvalue", Req0),
+ cowboy_req:set_resp_cookie(<<"yourcookie">>, <<"yourvalue">>, Req1);
+ <<"overwrite">> ->
+ Req1 = cowboy_req:set_resp_cookie(<<"mycookie">>, "myvalue", Req0),
+ cowboy_req:set_resp_cookie(<<"mycookie">>, <<"overwrite">>, Req1)
+ end,
+ cowboy_req:reply(200, #{}, "OK", Req),
+ {ok, Req, Opts};
+do(<<"set_resp_cookie4">>, Req0, Opts) ->
+ Req = cowboy_req:set_resp_cookie(<<"mycookie">>, "myvalue", #{path => cowboy_req:path(Req0)}, Req0),
+ cowboy_req:reply(200, #{}, "OK", Req),
+ {ok, Req, Opts};
+do(<<"set_resp_header">>, Req0, Opts) ->
+ Req = cowboy_req:set_resp_header(<<"content-type">>, <<"text/plain">>, Req0),
+ cowboy_req:reply(200, #{}, "OK", Req),
+ {ok, Req, Opts};
+do(<<"set_resp_body">>, Req0, Opts) ->
+ Arg = cowboy_req:binding(arg, Req0),
+ Req = case Arg of
+ <<"sendfile">> ->
+ AppFile = code:where_is_file("cowboy.app"),
+ cowboy_req:set_resp_body({sendfile, 0, filelib:file_size(AppFile), AppFile}, Req0);
+ _ ->
+ cowboy_req:set_resp_body(<<"OK">>, Req0)
+ end,
+ case Arg of
+ <<"override">> ->
+ cowboy_req:reply(200, #{}, <<"OVERRIDE">>, Req);
+ _ ->
+ cowboy_req:reply(200, Req)
+ end,
+ {ok, Req, Opts};
+do(<<"has_resp_header">>, Req0, Opts) ->
+ false = cowboy_req:has_resp_header(<<"content-type">>, Req0),
+ Req = cowboy_req:set_resp_header(<<"content-type">>, <<"text/plain">>, Req0),
+ true = cowboy_req:has_resp_header(<<"content-type">>, Req),
+ cowboy_req:reply(200, #{}, "OK", Req),
+ {ok, Req, Opts};
+do(<<"has_resp_body">>, Req0, Opts) ->
+ case cowboy_req:binding(arg, Req0) of
+ <<"sendfile">> ->
+ %% @todo Cases for sendfile. Note that sendfile 0 is unallowed.
+ false = cowboy_req:has_resp_body(Req0),
+ Req = cowboy_req:set_resp_body({sendfile, 0, 10, code:where_is_file("cowboy.app")}, Req0),
+ true = cowboy_req:has_resp_body(Req),
+ cowboy_req:reply(200, #{}, <<"OK">>, Req),
+ {ok, Req, Opts};
+ undefined ->
+ false = cowboy_req:has_resp_body(Req0),
+ Req = cowboy_req:set_resp_body(<<"OK">>, Req0),
+ true = cowboy_req:has_resp_body(Req),
+ cowboy_req:reply(200, #{}, Req),
+ {ok, Req, Opts}
+ end;
+do(<<"delete_resp_header">>, Req0, Opts) ->
+ false = cowboy_req:has_resp_header(<<"content-type">>, Req0),
+ Req1 = cowboy_req:set_resp_header(<<"content-type">>, <<"text/plain">>, Req0),
+ true = cowboy_req:has_resp_header(<<"content-type">>, Req1),
+ Req = cowboy_req:delete_resp_header(<<"content-type">>, Req1),
+ false = cowboy_req:has_resp_header(<<"content-type">>, Req),
+ cowboy_req:reply(200, #{}, "OK", Req),
+ {ok, Req, Opts};
+do(<<"reply2">>, Req, Opts) ->
+ case cowboy_req:binding(arg, Req) of
+ <<"binary">> ->
+ cowboy_req:reply(<<"200 GOOD">>, Req);
+ <<"error">> ->
+ ct_helper:ignore(cowboy_req, reply, 4),
+ cowboy_req:reply(ok, Req);
+ <<"twice">> ->
+ cowboy_req:reply(200, Req),
+ cowboy_req:reply(200, Req);
+ Status ->
+ cowboy_req:reply(binary_to_integer(Status), Req)
+ end,
+ {ok, Req, Opts};
+do(<<"reply3">>, Req, Opts) ->
+ case cowboy_req:binding(arg, Req) of
+ <<"error">> ->
+ ct_helper:ignore(cowboy_req, reply, 4),
+ cowboy_req:reply(200, ok, Req);
+ Status ->
+ cowboy_req:reply(binary_to_integer(Status),
+ #{<<"content-type">> => <<"text/plain">>}, Req)
+ end,
+ {ok, Req, Opts};
+do(<<"reply4">>, Req, Opts) ->
+ case cowboy_req:binding(arg, Req) of
+ <<"error">> ->
+ ct_helper:ignore(erlang, iolist_size, 1),
+ cowboy_req:reply(200, #{}, ok, Req);
+ Status ->
+ cowboy_req:reply(binary_to_integer(Status), #{}, <<"OK">>, Req)
+ end,
+ {ok, Req, Opts};
+do(<<"stream_reply2">>, Req, Opts) ->
+ case cowboy_req:binding(arg, Req) of
+ <<"binary">> ->
+ cowboy_req:stream_reply(<<"200 GOOD">>, Req);
+ <<"error">> ->
+ ct_helper:ignore(cowboy_req, stream_reply, 3),
+ cowboy_req:stream_reply(ok, Req);
+ Status ->
+ cowboy_req:stream_reply(binary_to_integer(Status), Req)
+ end,
+ stream_body(Req),
+ {ok, Req, Opts};
+do(<<"stream_reply3">>, Req, Opts) ->
+ case cowboy_req:binding(arg, Req) of
+ <<"error">> ->
+ ct_helper:ignore(cowboy_req, stream_reply, 3),
+ cowboy_req:stream_reply(200, ok, Req);
+ Status ->
+ cowboy_req:stream_reply(binary_to_integer(Status),
+ #{<<"content-type">> => <<"text/plain">>}, Req)
+ end,
+ stream_body(Req),
+ {ok, Req, Opts};
+do(<<"stream_body">>, Req, Opts) ->
+ %% Call stream_body without initiating streaming.
+ cowboy_req:stream_body(<<0:800000>>, fin, Req),
+ {ok, Req, Opts};
+do(<<"push">>, Req, Opts) ->
+ case cowboy_req:binding(arg, Req) of
+ <<"method">> ->
+ cowboy_req:push("/static/style.css", #{<<"accept">> => <<"text/css">>}, Req,
+ #{method => <<"HEAD">>});
+ <<"origin">> ->
+ cowboy_req:push("/static/style.css", #{<<"accept">> => <<"text/css">>}, Req,
+ #{scheme => <<"ftp">>, host => <<"127.0.0.1">>, port => 21});
+ <<"qs">> ->
+ cowboy_req:push("/static/style.css", #{<<"accept">> => <<"text/css">>}, Req,
+ #{qs => <<"server=cowboy&version=2.0">>});
+ _ ->
+ cowboy_req:push("/static/style.css", #{<<"accept">> => <<"text/css">>}, Req),
+ %% The text/plain mime is not defined by default, so a 406 will be returned.
+ cowboy_req:push("/static/plain.txt", #{<<"accept">> => <<"text/plain">>}, Req)
+ end,
+ cowboy_req:reply(200, Req),
+ {ok, Req, Opts}.
+
+stream_body(Req) ->
+ _ = [cowboy_req:stream_body(<<0:800000>>, nofin, Req) || _ <- lists:seq(1,9)],
+ cowboy_req:stream_body(<<0:800000>>, fin, Req).
diff --git a/test/http_SUITE_data/http_body_qs.erl b/test/http_SUITE_data/http_body_qs.erl
index e0673cf..09ca5e4 100644
--- a/test/http_SUITE_data/http_body_qs.erl
+++ b/test/http_SUITE_data/http_body_qs.erl
@@ -10,7 +10,7 @@ init(Req, Opts) ->
{ok, maybe_echo(Method, HasBody, Req), Opts}.
maybe_echo(<<"POST">>, true, Req) ->
- case cowboy_req:body_qs(Req) of
+ case cowboy_req:read_urlencoded_body(Req) of
{badlength, Req2} ->
echo(badlength, Req2);
{ok, PostVals, Req2} ->
diff --git a/test/http_SUITE_data/http_loop_stream_recv.erl b/test/http_SUITE_data/http_loop_stream_recv.erl
index c006b6d..18b3d29 100644
--- a/test/http_SUITE_data/http_loop_stream_recv.erl
+++ b/test/http_SUITE_data/http_loop_stream_recv.erl
@@ -15,7 +15,7 @@ info(stream, Req, undefined) ->
stream(Req, 1, <<>>).
stream(Req, ID, Acc) ->
- case cowboy_req:body(Req) of
+ case cowboy_req:read_body(Req) of
{ok, <<>>, Req2} ->
{stop, cowboy_req:reply(200, Req2), undefined};
{_, Data, Req2} ->
diff --git a/test/req_SUITE.erl b/test/req_SUITE.erl
index 648ebcd..b0aabad 100644
--- a/test/req_SUITE.erl
+++ b/test/req_SUITE.erl
@@ -34,6 +34,13 @@ groups() ->
%% @todo With compression enabled.
].
+init_per_suite(Config) ->
+ ct_helper:create_static_dir(config(priv_dir, Config) ++ "/static"),
+ Config.
+
+end_per_suite(Config) ->
+ ct_helper:delete_static_dir(config(priv_dir, Config) ++ "/static").
+
init_per_group(Name, Config) ->
cowboy_test:init_common_groups(Name, Config, ?MODULE).
@@ -42,10 +49,20 @@ end_per_group(Name, _) ->
%% Routes.
-init_dispatch(_) ->
+init_dispatch(Config) ->
cowboy_router:compile([{"[...]", [
- {"/no/:key", echo_h, []},
+ {"/static/[...]", cowboy_static, {dir, config(priv_dir, Config) ++ "/static"}},
+ %% @todo Seriously InitialState should be optional.
+ {"/resp/:key[/:arg]", resp_h, []},
+ {"/multipart[/:key]", multipart_h, []},
{"/args/:key/:arg[/:default]", echo_h, []},
+ {"/crash/:key/period", echo_h, #{length => infinity, period => 1000, crash => true}},
+ {"/no-opts/:key", echo_h, #{crash => true}},
+ {"/opts/:key/length", echo_h, #{length => 1000}},
+ {"/opts/:key/period", echo_h, #{length => infinity, period => 1000}},
+ {"/opts/:key/timeout", echo_h, #{timeout => 1000, crash => true}},
+ {"/full/:key", echo_h, []},
+ {"/no/:key", echo_h, []},
{"/:key/[...]", echo_h, []}
]}]).
@@ -55,15 +72,32 @@ do_body(Method, Path, Config) ->
do_body(Method, Path, [], Config).
do_body(Method, Path, Headers, Config) ->
+ do_body(Method, Path, Headers, <<>>, Config).
+
+do_body(Method, Path, Headers, Body, Config) ->
ConnPid = gun_open(Config),
- Ref = gun:request(ConnPid, Method, Path, Headers),
+ Ref = case Body of
+ <<>> -> gun:request(ConnPid, Method, Path, Headers);
+ _ -> gun:request(ConnPid, Method, Path, Headers, Body)
+ end,
{response, IsFin, 200, _} = gun:await(ConnPid, Ref),
- {ok, Body} = case IsFin of
+ {ok, RespBody} = case IsFin of
nofin -> gun:await_body(ConnPid, Ref);
fin -> {ok, <<>>}
end,
gun:close(ConnPid),
- Body.
+ RespBody.
+
+do_get(Path, Config) ->
+ ConnPid = gun_open(Config),
+ Ref = gun:get(ConnPid, Path, []),
+ {response, IsFin, Status, Headers} = gun:await(ConnPid, Ref),
+ {ok, RespBody} = case IsFin of
+ nofin -> gun:await_body(ConnPid, Ref);
+ fin -> {ok, <<>>}
+ end,
+ gun:close(ConnPid),
+ {Status, Headers, RespBody}.
do_get_body(Path, Config) ->
do_get_body(Path, [], Config).
@@ -71,7 +105,7 @@ do_get_body(Path, Config) ->
do_get_body(Path, Headers, Config) ->
do_body("GET", Path, Headers, Config).
-%% Tests.
+%% Tests: Request.
binding(Config) ->
doc("Value bound from request URI path with/without default."),
@@ -109,6 +143,7 @@ host_info(Config) ->
<<"[<<\"localhost\">>]">> = do_get_body("/host_info", Config),
ok.
+%% @todo Actually write the related unit tests.
match_cookies(Config) ->
doc("Matched request cookies."),
<<"#{}">> = do_get_body("/match/cookies", [{<<"cookie">>, "a=b; c=d"}], Config),
@@ -119,6 +154,7 @@ match_cookies(Config) ->
%% This function is tested more extensively through unit tests.
ok.
+%% @todo Actually write the related unit tests.
match_qs(Config) ->
doc("Matched request URI query string."),
<<"#{}">> = do_get_body("/match/qs?a=b&c=d", Config),
@@ -131,7 +167,7 @@ match_qs(Config) ->
method(Config) ->
doc("Request method."),
<<"GET">> = do_body("GET", "/method", Config),
- <<"HEAD">> = do_body("HEAD", "/method", Config),
+ <<>> = do_body("HEAD", "/method", Config),
<<"OPTIONS">> = do_body("OPTIONS", "/method", Config),
<<"PATCH">> = do_body("PATCH", "/method", Config),
<<"POST">> = do_body("POST", "/method", Config),
@@ -147,6 +183,9 @@ parse_cookies(Config) ->
= do_get_body("/parse_cookies", [{<<"cookie">>, "cake=strawberry"}], Config),
<<"[{<<\"cake\">>,<<\"strawberry\">>},{<<\"color\">>,<<\"blue\">>}]">>
= do_get_body("/parse_cookies", [{<<"cookie">>, "cake=strawberry; color=blue"}], Config),
+ <<"[{<<\"cake\">>,<<\"strawberry\">>},{<<\"color\">>,<<\"blue\">>}]">>
+ = do_get_body("/parse_cookies",
+ [{<<"cookie">>, "cake=strawberry"}, {<<"cookie">>, "color=blue"}], Config),
ok.
parse_header(Config) ->
@@ -252,3 +291,394 @@ version(Config) ->
<<"HTTP/1.1">> when Protocol =:= http -> ok;
<<"HTTP/2">> when Protocol =:= http2 -> ok
end.
+
+%% Tests: Request body.
+
+body_length(Config) ->
+ doc("Request body length."),
+ <<"0">> = do_get_body("/body_length", Config),
+ <<"12">> = do_body("POST", "/body_length", [], "hello world!", Config),
+ ok.
+
+has_body(Config) ->
+ doc("Has a request body?"),
+ <<"false">> = do_get_body("/has_body", Config),
+ <<"true">> = do_body("POST", "/has_body", [], "hello world!", Config),
+ ok.
+
+read_body(Config) ->
+ doc("Request body."),
+ <<>> = do_get_body("/read_body", Config),
+ <<"hello world!">> = do_body("POST", "/read_body", [], "hello world!", Config),
+ %% We expect to have read *at least* 1000 bytes.
+ <<0:8000, _/bits>> = do_body("POST", "/opts/read_body/length", [], <<0:8000000>>, Config),
+ %% We read any length for at most 1 second.
+ %%
+ %% The body is sent twice, first with nofin, then wait 2 seconds, then again with fin.
+ <<0:8000000>> = do_read_body_period("/opts/read_body/period", <<0:8000000>>, Config),
+ %% The timeout value is set too low on purpose to ensure a crash occurs.
+ ok = do_read_body_timeout("/opts/read_body/timeout", <<0:8000000>>, Config),
+ %% 10MB body larger than default length.
+ <<0:80000000>> = do_body("POST", "/full/read_body", [], <<0:80000000>>, Config),
+ ok.
+
+do_read_body_period(Path, Body, Config) ->
+ ConnPid = gun_open(Config),
+ Ref = gun:request(ConnPid, "POST", Path, [
+ {<<"content-length">>, integer_to_binary(byte_size(Body) * 2)}
+ ]),
+ gun:data(ConnPid, Ref, nofin, Body),
+ timer:sleep(2000),
+ gun:data(ConnPid, Ref, fin, Body),
+ {response, nofin, 200, _} = gun:await(ConnPid, Ref),
+ {ok, RespBody} = gun:await_body(ConnPid, Ref),
+ gun:close(ConnPid),
+ RespBody.
+
+%% We expect a crash.
+do_read_body_timeout(Path, Body, Config) ->
+ ConnPid = gun_open(Config),
+ Ref = gun:request(ConnPid, "POST", Path, [
+ {<<"content-length">>, integer_to_binary(byte_size(Body))}
+ ]),
+ {response, _, 500, _} = gun:await(ConnPid, Ref),
+ gun:close(ConnPid).
+
+%% @todo Do we really want a key/value list here instead of a map?
+read_urlencoded_body(Config) ->
+ doc("application/x-www-form-urlencoded request body."),
+ <<"[]">> = do_body("POST", "/read_urlencoded_body", [], <<>>, Config),
+ <<"[{<<\"abc\">>,true}]">> = do_body("POST", "/read_urlencoded_body", [], "abc", Config),
+ <<"[{<<\"a\">>,<<\"b\">>},{<<\"c\">>,<<\"d e\">>}]">>
+ = do_body("POST", "/read_urlencoded_body", [], "a=b&c=d+e", Config),
+ %% Send a 10MB body, larger than the default length, to ensure a crash occurs.
+ ok = do_read_urlencoded_body_too_large("/no-opts/read_urlencoded_body",
+ string:chars($a, 10000000), Config),
+ %% We read any length for at most 1 second.
+ %%
+ %% The body is sent twice, first with nofin, then wait 1.1 second, then again with fin.
+ %% We expect the handler to crash because read_urlencoded_body expects the full body.
+ ok = do_read_urlencoded_body_too_long("/crash/read_urlencoded_body/period", <<"abc">>, Config),
+ %% The timeout value is set too low on purpose to ensure a crash occurs.
+ ok = do_read_body_timeout("/opts/read_urlencoded_body/timeout", <<"abc">>, Config),
+ ok.
+
+%% We expect a crash.
+do_read_urlencoded_body_too_large(Path, Body, Config) ->
+ ConnPid = gun_open(Config),
+ Ref = gun:request(ConnPid, "POST", Path, [
+ {<<"content-length">>, integer_to_binary(iolist_size(Body))}
+ ]),
+ gun:data(ConnPid, Ref, fin, Body),
+ {response, _, 500, _} = gun:await(ConnPid, Ref),
+ gun:close(ConnPid).
+
+%% We expect a crash.
+do_read_urlencoded_body_too_long(Path, Body, Config) ->
+ ConnPid = gun_open(Config),
+ Ref = gun:request(ConnPid, "POST", Path, [
+ {<<"content-length">>, integer_to_binary(byte_size(Body) * 2)}
+ ]),
+ gun:data(ConnPid, Ref, nofin, Body),
+ timer:sleep(1100),
+ gun:data(ConnPid, Ref, fin, Body),
+ {response, _, 500, _} = gun:await(ConnPid, Ref),
+ gun:close(ConnPid).
+
+multipart(Config) ->
+ doc("Multipart request body."),
+ do_multipart("/multipart", Config).
+
+do_multipart(Path, Config) ->
+ LargeBody = iolist_to_binary(string:chars($a, 10000000)),
+ ReqBody = [
+ "--deadbeef\r\nContent-Type: text/plain\r\n\r\nCowboy is an HTTP server.\r\n"
+ "--deadbeef\r\nContent-Type: application/octet-stream\r\nX-Custom: value\r\n\r\n", LargeBody, "\r\n"
+ "--deadbeef--"
+ ],
+ RespBody = do_body("POST", Path, [
+ {<<"content-type">>, <<"multipart/mixed; boundary=deadbeef">>}
+ ], ReqBody, Config),
+ [
+ {[{<<"content-type">>, <<"text/plain">>}], <<"Cowboy is an HTTP server.">>},
+ {LargeHeaders, LargeBody}
+ ] = binary_to_term(RespBody),
+ %% @todo Multipart header order is currently undefined.
+ [
+ {<<"content-type">>, <<"application/octet-stream">>},
+ {<<"x-custom">>, <<"value">>}
+ ] = lists:sort(LargeHeaders),
+ ok.
+
+read_part_skip_body(Config) ->
+ doc("Multipart request body skipping part bodies."),
+ LargeBody = iolist_to_binary(string:chars($a, 10000000)),
+ ReqBody = [
+ "--deadbeef\r\nContent-Type: text/plain\r\n\r\nCowboy is an HTTP server.\r\n"
+ "--deadbeef\r\nContent-Type: application/octet-stream\r\nX-Custom: value\r\n\r\n", LargeBody, "\r\n"
+ "--deadbeef--"
+ ],
+ RespBody = do_body("POST", "/multipart/skip_body", [
+ {<<"content-type">>, <<"multipart/mixed; boundary=deadbeef">>}
+ ], ReqBody, Config),
+ [
+ [{<<"content-type">>, <<"text/plain">>}],
+ LargeHeaders
+ ] = binary_to_term(RespBody),
+ %% @todo Multipart header order is currently undefined.
+ [
+ {<<"content-type">>, <<"application/octet-stream">>},
+ {<<"x-custom">>, <<"value">>}
+ ] = lists:sort(LargeHeaders),
+ ok.
+
+%% @todo When reading a multipart body, length and period
+%% only apply to a single read_body call. We may want a
+%% separate option to know how many reads we want to do
+%% before we give up.
+
+read_part2(Config) ->
+ doc("Multipart request body using read_part/2."),
+ %% Override the length and period values only, making
+ %% the request process use more read_body calls.
+ %%
+ %% We do not try a custom timeout value since this would
+ %% be the same test as read_body/2.
+ do_multipart("/multipart/read_part2", Config).
+
+read_part_body2(Config) ->
+ doc("Multipart request body using read_part_body/2."),
+ %% Override the length and period values only, making
+ %% the request process use more read_body calls.
+ %%
+ %% We do not try a custom timeout value since this would
+ %% be the same test as read_body/2.
+ do_multipart("/multipart/read_part_body2", Config).
+
+%% Tests: Response.
+
+%% @todo We want to crash when calling set_resp_* or related
+%% functions after the reply has been sent.
+
+set_resp_cookie(Config) ->
+ doc("Response using set_resp_cookie."),
+ %% Single cookie, no options.
+ {200, Headers1, _} = do_get("/resp/set_resp_cookie3", Config),
+ {_, <<"mycookie=myvalue; Version=1">>}
+ = lists:keyfind(<<"set-cookie">>, 1, Headers1),
+ %% Single cookie, with options.
+ {200, Headers2, _} = do_get("/resp/set_resp_cookie4", Config),
+ {_, <<"mycookie=myvalue; Version=1; Path=/resp/set_resp_cookie4">>}
+ = lists:keyfind(<<"set-cookie">>, 1, Headers2),
+ %% Multiple cookies.
+ {200, Headers3, _} = do_get("/resp/set_resp_cookie3/multiple", Config),
+ [_, _] = [H || H={<<"set-cookie">>, _} <- Headers3],
+ %% Overwrite previously set cookie.
+ {200, Headers4, _} = do_get("/resp/set_resp_cookie3/overwrite", Config),
+ {_, <<"mycookie=overwrite; Version=1">>}
+ = lists:keyfind(<<"set-cookie">>, 1, Headers4),
+ ok.
+
+set_resp_header(Config) ->
+ doc("Response using set_resp_header."),
+ {200, Headers, <<"OK">>} = do_get("/resp/set_resp_header", Config),
+ true = lists:keymember(<<"content-type">>, 1, Headers),
+ ok.
+
+set_resp_body(Config) ->
+ doc("Response using set_resp_body."),
+ {200, _, <<"OK">>} = do_get("/resp/set_resp_body", Config),
+ {200, _, <<"OVERRIDE">>} = do_get("/resp/set_resp_body/override", Config),
+ {ok, AppFile} = file:read_file(code:where_is_file("cowboy.app")),
+ {200, _, AppFile} = do_get("/resp/set_resp_body/sendfile", Config),
+ ok.
+
+has_resp_header(Config) ->
+ doc("Has response header?"),
+ {200, Headers, <<"OK">>} = do_get("/resp/has_resp_header", Config),
+ true = lists:keymember(<<"content-type">>, 1, Headers),
+ ok.
+
+has_resp_body(Config) ->
+ doc("Has response body?"),
+ {200, _, <<"OK">>} = do_get("/resp/has_resp_body", Config),
+ {200, _, <<"OK">>} = do_get("/resp/has_resp_body/sendfile", Config),
+ ok.
+
+delete_resp_header(Config) ->
+ doc("Delete response header."),
+ {200, Headers, <<"OK">>} = do_get("/resp/delete_resp_header", Config),
+ false = lists:keymember(<<"content-type">>, 1, Headers),
+ ok.
+
+reply2(Config) ->
+ doc("Response with default headers and no body."),
+ {200, _, _} = do_get("/resp/reply2/200", Config),
+ {201, _, _} = do_get("/resp/reply2/201", Config),
+ {404, _, _} = do_get("/resp/reply2/404", Config),
+ {200, _, _} = do_get("/resp/reply2/binary", Config),
+ {500, _, _} = do_get("/resp/reply2/error", Config),
+ %% @todo We want to crash when reply or stream_reply is called twice.
+ %% How to test this properly? This isn't enough.
+ {200, _, _} = do_get("/resp/reply2/twice", Config),
+ ok.
+
+reply3(Config) ->
+ doc("Response with additional headers and no body."),
+ {200, Headers1, _} = do_get("/resp/reply3/200", Config),
+ true = lists:keymember(<<"content-type">>, 1, Headers1),
+ {201, Headers2, _} = do_get("/resp/reply3/201", Config),
+ true = lists:keymember(<<"content-type">>, 1, Headers2),
+ {404, Headers3, _} = do_get("/resp/reply3/404", Config),
+ true = lists:keymember(<<"content-type">>, 1, Headers3),
+ {500, _, _} = do_get("/resp/reply3/error", Config),
+ ok.
+
+reply4(Config) ->
+ doc("Response with additional headers and body."),
+ {200, _, <<"OK">>} = do_get("/resp/reply4/200", Config),
+ {201, _, <<"OK">>} = do_get("/resp/reply4/201", Config),
+ {404, _, <<"OK">>} = do_get("/resp/reply4/404", Config),
+ {500, _, _} = do_get("/resp/reply4/error", Config),
+ ok.
+
+%% @todo Crash when stream_reply is called twice.
+
+stream_reply2(Config) ->
+ doc("Response with default headers and streamed body."),
+ Body = <<0:8000000>>,
+ {200, _, Body} = do_get("/resp/stream_reply2/200", Config),
+ {201, _, Body} = do_get("/resp/stream_reply2/201", Config),
+ {404, _, Body} = do_get("/resp/stream_reply2/404", Config),
+ {200, _, Body} = do_get("/resp/stream_reply2/binary", Config),
+ {500, _, _} = do_get("/resp/stream_reply2/error", Config),
+ ok.
+
+stream_reply3(Config) ->
+ doc("Response with additional headers and streamed body."),
+ Body = <<0:8000000>>,
+ {200, Headers1, Body} = do_get("/resp/stream_reply3/200", Config),
+ true = lists:keymember(<<"content-type">>, 1, Headers1),
+ {201, Headers2, Body} = do_get("/resp/stream_reply3/201", Config),
+ true = lists:keymember(<<"content-type">>, 1, Headers2),
+ {404, Headers3, Body} = do_get("/resp/stream_reply3/404", Config),
+ true = lists:keymember(<<"content-type">>, 1, Headers3),
+ {500, _, _} = do_get("/resp/stream_reply3/error", Config),
+ ok.
+
+%% @todo Crash when calling stream_body after the fin flag has been set.
+%% @todo Crash when calling stream_body after calling reply.
+%% @todo Crash when calling stream_body before calling stream_reply.
+
+%% Tests: Push.
+
+%% @todo We want to crash when push is called after reply has been initiated.
+
+push(Config) ->
+ case config(protocol, Config) of
+ http -> do_push_http("/resp/push", Config);
+ http2 -> do_push_http2(Config)
+ end.
+
+push_method(Config) ->
+ case config(protocol, Config) of
+ http -> do_push_http("/resp/push/method", Config);
+ http2 -> do_push_http2_method(Config)
+ end.
+
+
+push_origin(Config) ->
+ case config(protocol, Config) of
+ http -> do_push_http("/resp/push/origin", Config);
+ http2 -> do_push_http2_origin(Config)
+ end.
+
+push_qs(Config) ->
+ case config(protocol, Config) of
+ http -> do_push_http("/resp/push/qs", Config);
+ http2 -> do_push_http2_qs(Config)
+ end.
+
+do_push_http(Path, Config) ->
+ doc("Ignore pushed responses when protocol is HTTP/1.1."),
+ ConnPid = gun_open(Config),
+ Ref = gun:get(ConnPid, Path, []),
+ {response, fin, 200, _} = gun:await(ConnPid, Ref),
+ ok.
+
+do_push_http2(Config) ->
+ doc("Pushed responses."),
+ ConnPid = gun_open(Config),
+ Ref = gun:get(ConnPid, "/resp/push", []),
+ %% We expect two pushed resources.
+ Origin = iolist_to_binary([
+ case config(type, Config) of
+ tcp -> "http";
+ ssl -> "https"
+ end,
+ "://localhost:",
+ integer_to_binary(config(port, Config))
+ ]),
+ OriginLen = byte_size(Origin),
+ {push, PushCSS, <<"GET">>, <<Origin:OriginLen/binary, "/static/style.css">>,
+ [{<<"accept">>,<<"text/css">>}]} = gun:await(ConnPid, Ref),
+ {push, PushTXT, <<"GET">>, <<Origin:OriginLen/binary, "/static/plain.txt">>,
+ [{<<"accept">>,<<"text/plain">>}]} = gun:await(ConnPid, Ref),
+ %% Pushed CSS.
+ {response, nofin, 200, HeadersCSS} = gun:await(ConnPid, PushCSS),
+ {_, <<"text/css">>} = lists:keyfind(<<"content-type">>, 1, HeadersCSS),
+ {ok, <<"body{color:red}\n">>} = gun:await_body(ConnPid, PushCSS),
+ %% Pushed TXT is 406 because the pushed accept header uses an undefined type.
+ {response, fin, 406, _} = gun:await(ConnPid, PushTXT),
+ %% Let's not forget about the response to the client's request.
+ {response, fin, 200, _} = gun:await(ConnPid, Ref),
+ gun:close(ConnPid).
+
+do_push_http2_method(Config) ->
+ doc("Pushed response with non-GET method."),
+ ConnPid = gun_open(Config),
+ Ref = gun:get(ConnPid, "/resp/push/method", []),
+ %% Pushed CSS.
+ {push, PushCSS, <<"HEAD">>, _, [{<<"accept">>,<<"text/css">>}]} = gun:await(ConnPid, Ref),
+ {response, fin, 200, HeadersCSS} = gun:await(ConnPid, PushCSS),
+ {_, <<"text/css">>} = lists:keyfind(<<"content-type">>, 1, HeadersCSS),
+ %% Let's not forget about the response to the client's request.
+ {response, fin, 200, _} = gun:await(ConnPid, Ref),
+ gun:close(ConnPid).
+
+do_push_http2_origin(Config) ->
+ doc("Pushed response with custom scheme/host/port."),
+ ConnPid = gun_open(Config),
+ Ref = gun:get(ConnPid, "/resp/push/origin", []),
+ %% Pushed CSS.
+ {push, PushCSS, <<"GET">>, <<"ftp://127.0.0.1:21/static/style.css">>,
+ [{<<"accept">>,<<"text/css">>}]} = gun:await(ConnPid, Ref),
+ {response, nofin, 200, HeadersCSS} = gun:await(ConnPid, PushCSS),
+ {_, <<"text/css">>} = lists:keyfind(<<"content-type">>, 1, HeadersCSS),
+ {ok, <<"body{color:red}\n">>} = gun:await_body(ConnPid, PushCSS),
+ %% Let's not forget about the response to the client's request.
+ {response, fin, 200, _} = gun:await(ConnPid, Ref),
+ gun:close(ConnPid).
+
+do_push_http2_qs(Config) ->
+ doc("Pushed response with query string."),
+ ConnPid = gun_open(Config),
+ Ref = gun:get(ConnPid, "/resp/push/qs", []),
+ %% Pushed CSS.
+ Origin = iolist_to_binary([
+ case config(type, Config) of
+ tcp -> "http";
+ ssl -> "https"
+ end,
+ "://localhost:",
+ integer_to_binary(config(port, Config))
+ ]),
+ OriginLen = byte_size(Origin),
+ {push, PushCSS, <<"GET">>, <<Origin:OriginLen/binary, "/static/style.css?server=cowboy&version=2.0">>,
+ [{<<"accept">>,<<"text/css">>}]} = gun:await(ConnPid, Ref),
+ {response, nofin, 200, HeadersCSS} = gun:await(ConnPid, PushCSS),
+ {_, <<"text/css">>} = lists:keyfind(<<"content-type">>, 1, HeadersCSS),
+ {ok, <<"body{color:red}\n">>} = gun:await_body(ConnPid, PushCSS),
+ %% Let's not forget about the response to the client's request.
+ {response, fin, 200, _} = gun:await(ConnPid, Ref),
+ gun:close(ConnPid).