Implement FrameBuffer3Proxy
This emulates behaviour from frame buffer 2, but does not handle stats.
In contrast to frame buffer 2, all work happens on the same task queue.
FrameBuffer3Proxy encapsulates FrameBuffer3 and scheduler behind
a field trial WebRTC-FrameBuffer3.
This separates frame scheduling behaviour into a few components,
VideoReceiveStreamTimeoutTracker
* Handles the stream timeouts.
FrameDecodeScheduler
* Manages the scheduling and cancelling of frames being sent to the
decoder.
FrameDecodeTiming
* Handles the timing and ordering of frames to be decoded.
Other changes
* Adds CurrentSize() method to FrameBuffer3
* Move timing to a separate library
* Does a thread check for Receive statistics as this is now
on the worker thread.
* Adds `FlushImmediate` method to RunLoop so that
video_receive_stream2_unittest can pass when scheduling is happening
on the worker thread.
Change-Id: Ia8d2e5650d1708cdc1be3631a5214134583a0721
Bug: webrtc:13343
Tested: Ran webrtc_perf_tests, video_engine_tests, rtc_unittests forcing frame buffer3
Reviewed-on: https://webrtc-review.googlesource.com/c/src/+/241603
Reviewed-by: Ilya Nikolaevskiy <ilnik@webrtc.org>
Reviewed-by: Markus Handell <handellm@webrtc.org>
Reviewed-by: Mirko Bonadei <mbonadei@webrtc.org>
Reviewed-by: Philip Eliasson <philipel@webrtc.org>
Reviewed-by: Tomas Gunnarsson <tommi@webrtc.org>
Commit-Queue: Evan Shrubsole <eshr@webrtc.org>
Cr-Commit-Position: refs/heads/main@{#35847}
diff --git a/video/BUILD.gn b/video/BUILD.gn
index 92531fe..b89e5e4 100644
--- a/video/BUILD.gn
+++ b/video/BUILD.gn
@@ -55,7 +55,9 @@
deps = [
":frame_buffer_proxy",
":frame_cadence_adapter",
+ ":frame_decode_scheduler",
":frame_dumping_decoder",
+ ":video_receive_stream_timeout_tracker",
":video_stream_encoder_impl",
"../api:array_view",
"../api:fec_controller_api",
@@ -96,8 +98,11 @@
"../modules/rtp_rtcp:rtp_video_header",
"../modules/video_coding",
"../modules/video_coding:codec_globals_headers",
+ "../modules/video_coding:frame_buffer",
+ "../modules/video_coding:frame_helpers",
"../modules/video_coding:nack_requester",
"../modules/video_coding:packet_buffer",
+ "../modules/video_coding:timing",
"../modules/video_coding:video_codec_interface",
"../modules/video_coding:video_coding_utility",
"../modules/video_processing",
@@ -116,6 +121,7 @@
"../rtc_base/experiments:min_video_bitrate_experiment",
"../rtc_base/experiments:quality_scaling_experiment",
"../rtc_base/experiments:rate_control_settings",
+ "../rtc_base/experiments:rtt_mult_experiment",
"../rtc_base/synchronization:mutex",
"../rtc_base/system:no_unique_address",
"../rtc_base/system:thread_registry",
@@ -131,6 +137,8 @@
absl_deps = [
"//third_party/abseil-cpp/absl/algorithm:container",
"//third_party/abseil-cpp/absl/base:core_headers",
+ "//third_party/abseil-cpp/absl/container:inlined_vector",
+ "//third_party/abseil-cpp/absl/functional:bind_front",
"//third_party/abseil-cpp/absl/memory",
"//third_party/abseil-cpp/absl/strings",
"//third_party/abseil-cpp/absl/types:optional",
@@ -228,6 +236,7 @@
"../api/video:video_stream_decoder",
"../api/video_codecs:video_codecs_api",
"../modules/video_coding",
+ "../modules/video_coding:timing",
"../rtc_base:rtc_base_approved",
"../rtc_base:rtc_task_queue",
"../rtc_base/synchronization:mutex",
@@ -292,16 +301,74 @@
"frame_buffer_proxy.h",
]
deps = [
+ ":frame_decode_scheduler",
+ ":frame_decode_timing",
+ ":video_receive_stream_timeout_tracker",
+ "../api:sequence_checker",
"../api/task_queue",
"../api/video:encoded_frame",
"../modules/video_coding",
+ "../modules/video_coding:frame_buffer",
+ "../modules/video_coding:frame_helpers",
+ "../modules/video_coding:timing",
"../modules/video_coding:video_codec_interface",
+ "../rtc_base:logging",
+ "../rtc_base:macromagic",
"../rtc_base:rtc_task_queue",
"../system_wrappers",
+ "../system_wrappers:field_trial",
+ ]
+ absl_deps = [
+ "//third_party/abseil-cpp/absl/functional:bind_front",
+ "//third_party/abseil-cpp/absl/types:optional",
+ ]
+}
+
+rtc_library("frame_decode_scheduler") {
+ sources = [
+ "frame_decode_scheduler.cc",
+ "frame_decode_scheduler.h",
+ ]
+ deps = [
+ ":frame_decode_timing",
+ "../api:sequence_checker",
+ "../api/task_queue",
+ "../api/units:timestamp",
+ "../rtc_base/task_utils:pending_task_safety_flag",
+ "../rtc_base/task_utils:to_queued_task",
+ "../system_wrappers",
]
absl_deps = [ "//third_party/abseil-cpp/absl/types:optional" ]
}
+rtc_library("frame_decode_timing") {
+ sources = [
+ "frame_decode_timing.cc",
+ "frame_decode_timing.h",
+ ]
+ deps = [
+ "../api/task_queue",
+ "../modules/video_coding:timing",
+ "../rtc_base:logging",
+ "../rtc_base/task_utils:pending_task_safety_flag",
+ "../system_wrappers",
+ ]
+ absl_deps = [ "//third_party/abseil-cpp/absl/types:optional" ]
+}
+
+rtc_library("video_receive_stream_timeout_tracker") {
+ sources = [
+ "video_receive_stream_timeout_tracker.cc",
+ "video_receive_stream_timeout_tracker.h",
+ ]
+ deps = [
+ "../api/task_queue",
+ "../api/units:time_delta",
+ "../rtc_base/task_utils:repeating_task",
+ "../system_wrappers",
+ ]
+}
+
rtc_library("video_stream_encoder_impl") {
visibility = [ "*" ]
@@ -659,6 +726,8 @@
"end_to_end_tests/transport_feedback_tests.cc",
"frame_buffer_proxy_unittest.cc",
"frame_cadence_adapter_unittest.cc",
+ "frame_decode_scheduler_unittest.cc",
+ "frame_decode_timing_unittest.cc",
"frame_encode_metadata_writer_unittest.cc",
"picture_id_tests.cc",
"quality_limitation_reason_tracker_unittest.cc",
@@ -673,6 +742,7 @@
"stats_counter_unittest.cc",
"stream_synchronization_unittest.cc",
"video_receive_stream2_unittest.cc",
+ "video_receive_stream_timeout_tracker_unittest.cc",
"video_send_stream_impl_unittest.cc",
"video_send_stream_tests.cc",
"video_source_sink_controller_unittest.cc",
@@ -682,8 +752,11 @@
deps = [
":frame_buffer_proxy",
":frame_cadence_adapter",
+ ":frame_decode_scheduler",
+ ":frame_decode_timing",
":video",
":video_mocks",
+ ":video_receive_stream_timeout_tracker",
":video_stream_decoder_impl",
":video_stream_encoder_impl",
"../api:create_frame_generator",
@@ -709,6 +782,7 @@
"../api/task_queue:default_task_queue_factory",
"../api/test/video:function_video_factory",
"../api/units:data_rate",
+ "../api/units:frequency",
"../api/units:time_delta",
"../api/units:timestamp",
"../api/video:builtin_video_bitrate_allocator_factory",
@@ -749,6 +823,7 @@
"../modules/video_coding:codec_globals_headers",
"../modules/video_coding:encoded_frame",
"../modules/video_coding:packet_buffer",
+ "../modules/video_coding:timing",
"../modules/video_coding:video_codec_interface",
"../modules/video_coding:video_coding_utility",
"../modules/video_coding:webrtc_h264",
@@ -767,6 +842,7 @@
"../rtc_base:rtc_task_queue",
"../rtc_base:task_queue_for_test",
"../rtc_base:threading",
+ "../rtc_base/containers:flat_map",
"../rtc_base/experiments:alr_experiment",
"../rtc_base/experiments:encoder_info_settings",
"../rtc_base/synchronization:mutex",
@@ -796,6 +872,7 @@
"//third_party/abseil-cpp/absl/algorithm:container",
"//third_party/abseil-cpp/absl/memory",
"//third_party/abseil-cpp/absl/types:optional",
+ "//third_party/abseil-cpp/absl/types:variant",
]
if (!build_with_mozilla) {
deps += [ "../media:rtc_media_base" ]
diff --git a/video/frame_buffer_proxy.cc b/video/frame_buffer_proxy.cc
index 48658f0..3182bf9 100644
--- a/video/frame_buffer_proxy.cc
+++ b/video/frame_buffer_proxy.cc
@@ -10,10 +10,21 @@
#include "video/frame_buffer_proxy.h"
+#include <algorithm>
#include <memory>
#include <utility>
+#include "absl/functional/bind_front.h"
+#include "api/sequence_checker.h"
#include "modules/video_coding/frame_buffer2.h"
+#include "modules/video_coding/frame_buffer3.h"
+#include "modules/video_coding/frame_helpers.h"
+#include "rtc_base/logging.h"
+#include "rtc_base/thread_annotations.h"
+#include "system_wrappers/include/field_trial.h"
+#include "video/frame_decode_scheduler.h"
+#include "video/frame_decode_timing.h"
+#include "video/video_receive_stream_timeout_tracker.h"
namespace webrtc {
@@ -114,7 +125,381 @@
PendingTaskSafetyFlag::CreateDetached();
};
-// TODO(bugs.webrtc.org/13343): Create FrameBuffer3Proxy when complete.
+// Max number of frames the buffer will hold.
+static constexpr size_t kMaxFramesBuffered = 800;
+// Max number of decoded frame info that will be saved.
+static constexpr int kMaxFramesHistory = 1 << 13;
+
+// Default value for the maximum decode queue size that is used when the
+// low-latency renderer is used.
+static constexpr size_t kZeroPlayoutDelayDefaultMaxDecodeQueueSize = 8;
+
+// Encapsulates use of the new frame buffer for use in VideoReceiveStream. This
+// behaves the same as the FrameBuffer2Proxy but uses frame_buffer3 instead.
+// Responsiblities from frame_buffer2, like stats, jitter and frame timing
+// accounting are moved into this pro
+class FrameBuffer3Proxy : public FrameBufferProxy {
+ public:
+ FrameBuffer3Proxy(Clock* clock,
+ TaskQueueBase* worker_queue,
+ VCMTiming* timing,
+ VCMReceiveStatisticsCallback* stats_proxy,
+ rtc::TaskQueue* decode_queue,
+ FrameSchedulingReceiver* receiver,
+ TimeDelta max_wait_for_keyframe,
+ TimeDelta max_wait_for_frame)
+ : max_wait_for_keyframe_(max_wait_for_keyframe),
+ max_wait_for_frame_(max_wait_for_frame),
+ clock_(clock),
+ worker_queue_(worker_queue),
+ decode_queue_(decode_queue),
+ stats_proxy_(stats_proxy),
+ receiver_(receiver),
+ timing_(timing),
+ jitter_estimator_(clock_),
+ inter_frame_delay_(clock_->TimeInMilliseconds()),
+ buffer_(std::make_unique<FrameBuffer>(kMaxFramesBuffered,
+ kMaxFramesHistory)),
+ frame_decode_scheduler_(
+ clock_,
+ worker_queue,
+ absl::bind_front(&FrameBuffer3Proxy::OnFrameReadyForExtraction,
+ this)),
+ decode_timing_(clock_, timing_),
+ timeout_tracker_(clock_,
+ worker_queue_,
+ VideoReceiveStreamTimeoutTracker::Timeouts{
+ .max_wait_for_keyframe = max_wait_for_keyframe,
+ .max_wait_for_frame = max_wait_for_frame},
+ absl::bind_front(&FrameBuffer3Proxy::OnTimeout, this)),
+ zero_playout_delay_max_decode_queue_size_(
+ "max_decode_queue_size",
+ kZeroPlayoutDelayDefaultMaxDecodeQueueSize) {
+ RTC_DCHECK(decode_queue_);
+ RTC_DCHECK(stats_proxy_);
+ RTC_DCHECK(receiver_);
+ RTC_DCHECK(timing_);
+ RTC_DCHECK(worker_queue_);
+ RTC_DCHECK(clock_);
+ RTC_LOG(LS_WARNING) << "Using FrameBuffer3";
+
+ ParseFieldTrial({&zero_playout_delay_max_decode_queue_size_},
+ field_trial::FindFullName("WebRTC-ZeroPlayoutDelay"));
+ }
+
+ // FrameBufferProxy implementation.
+ void StopOnWorker() override {
+ RTC_DCHECK_RUN_ON(&worker_sequence_checker_);
+ timeout_tracker_.Stop();
+ frame_decode_scheduler_.CancelOutstanding();
+ decoder_ready_for_new_frame_ = false;
+ decode_queue_->PostTask([this] {
+ RTC_DCHECK_RUN_ON(decode_queue_);
+ decode_safety_->SetNotAlive();
+ });
+ }
+
+ void SetProtectionMode(VCMVideoProtection protection_mode) override {
+ RTC_DCHECK_RUN_ON(&worker_sequence_checker_);
+ protection_mode_ = kProtectionNackFEC;
+ }
+
+ void Clear() override {
+ RTC_DCHECK_RUN_ON(&worker_sequence_checker_);
+ stats_proxy_->OnDroppedFrames(buffer_->CurrentSize());
+ buffer_ =
+ std::make_unique<FrameBuffer>(kMaxFramesBuffered, kMaxFramesHistory);
+ frame_decode_scheduler_.CancelOutstanding();
+ }
+
+ absl::optional<int64_t> InsertFrame(
+ std::unique_ptr<EncodedFrame> frame) override {
+ RTC_DCHECK_RUN_ON(&worker_sequence_checker_);
+ if (frame->is_last_spatial_layer)
+ stats_proxy_->OnCompleteFrame(frame->is_keyframe(), frame->size(),
+ frame->contentType());
+ if (!frame->delayed_by_retransmission())
+ timing_->IncomingTimestamp(frame->Timestamp(), frame->ReceivedTime());
+
+ buffer_->InsertFrame(std::move(frame));
+ MaybeScheduleFrameForRelease();
+
+ return buffer_->LastContinuousFrameId();
+ }
+
+ void UpdateRtt(int64_t max_rtt_ms) override {
+ RTC_DCHECK_RUN_ON(&worker_sequence_checker_);
+ jitter_estimator_.UpdateRtt(max_rtt_ms);
+ }
+
+ void StartNextDecode(bool keyframe_required) override {
+ if (!worker_queue_->IsCurrent()) {
+ worker_queue_->PostTask(ToQueuedTask(
+ worker_safety_,
+ [this, keyframe_required] { StartNextDecode(keyframe_required); }));
+ return;
+ }
+
+ RTC_DCHECK_RUN_ON(&worker_sequence_checker_);
+ if (!timeout_tracker_.Running())
+ timeout_tracker_.Start(keyframe_required);
+ keyframe_required_ = keyframe_required;
+ if (keyframe_required_) {
+ timeout_tracker_.SetWaitingForKeyframe();
+ }
+ decoder_ready_for_new_frame_ = true;
+ MaybeScheduleFrameForRelease();
+ }
+
+ int Size() override {
+ RTC_DCHECK_RUN_ON(&worker_sequence_checker_);
+ return buffer_->CurrentSize();
+ }
+
+ void OnFrameReady(
+ absl::InlinedVector<std::unique_ptr<EncodedFrame>, 4> frames,
+ Timestamp render_time) {
+ RTC_DCHECK_RUN_ON(&worker_sequence_checker_);
+ RTC_DCHECK(!frames.empty());
+
+ timeout_tracker_.OnEncodedFrameReleased();
+
+ int64_t now_ms = clock_->TimeInMilliseconds();
+ bool superframe_delayed_by_retransmission = false;
+ size_t superframe_size = 0;
+ const EncodedFrame& first_frame = *frames.front();
+ int64_t receive_time_ms = first_frame.ReceivedTime();
+
+ if (first_frame.is_keyframe())
+ keyframe_required_ = false;
+
+ // Gracefully handle bad RTP timestamps and render time issues.
+ if (FrameHasBadRenderTiming(render_time.ms(), now_ms,
+ timing_->TargetVideoDelay())) {
+ jitter_estimator_.Reset();
+ timing_->Reset();
+ render_time = Timestamp::Millis(
+ timing_->RenderTimeMs(first_frame.Timestamp(), now_ms));
+ }
+
+ for (std::unique_ptr<EncodedFrame>& frame : frames) {
+ frame->SetRenderTime(render_time.ms());
+
+ superframe_delayed_by_retransmission |=
+ frame->delayed_by_retransmission();
+ receive_time_ms = std::max(receive_time_ms, frame->ReceivedTime());
+ superframe_size += frame->size();
+ }
+
+ if (!superframe_delayed_by_retransmission) {
+ int64_t frame_delay;
+
+ if (inter_frame_delay_.CalculateDelay(first_frame.Timestamp(),
+ &frame_delay, receive_time_ms)) {
+ jitter_estimator_.UpdateEstimate(frame_delay, superframe_size);
+ }
+
+ float rtt_mult = protection_mode_ == kProtectionNackFEC ? 0.0 : 1.0;
+ absl::optional<float> rtt_mult_add_cap_ms = absl::nullopt;
+ if (rtt_mult_settings_.has_value()) {
+ rtt_mult = rtt_mult_settings_->rtt_mult_setting;
+ rtt_mult_add_cap_ms = rtt_mult_settings_->rtt_mult_add_cap_ms;
+ }
+ timing_->SetJitterDelay(
+ jitter_estimator_.GetJitterEstimate(rtt_mult, rtt_mult_add_cap_ms));
+ timing_->UpdateCurrentDelay(render_time.ms(), now_ms);
+ } else if (RttMultExperiment::RttMultEnabled()) {
+ jitter_estimator_.FrameNacked();
+ }
+
+ // Update stats.
+ UpdateDroppedFrames();
+ UpdateJitterDelay();
+ UpdateTimingFrameInfo();
+
+ std::unique_ptr<EncodedFrame> frame =
+ CombineAndDeleteFrames(std::move(frames));
+
+ decoder_ready_for_new_frame_ = false;
+ // VideoReceiveStream2 wants frames on the decoder thread.
+ decode_queue_->PostTask(ToQueuedTask(
+ decode_safety_, [this, frame = std::move(frame)]() mutable {
+ receiver_->OnEncodedFrame(std::move(frame));
+ }));
+ }
+
+ void OnTimeout() {
+ RTC_DCHECK_RUN_ON(&worker_sequence_checker_);
+ // If the stream is paused then ignore the timeout.
+ if (!decoder_ready_for_new_frame_) {
+ timeout_tracker_.Stop();
+ return;
+ }
+ receiver_->OnDecodableFrameTimeout(MaxWait());
+ // Stop sending timeouts until receive starts waiting for a new frame.
+ timeout_tracker_.Stop();
+ decoder_ready_for_new_frame_ = false;
+ }
+
+ private:
+ void OnFrameReadyForExtraction(uint32_t rtp_timestamp,
+ Timestamp render_time) {
+ RTC_DCHECK_RUN_ON(&worker_sequence_checker_);
+ RTC_DCHECK(buffer_->NextDecodableTemporalUnitRtpTimestamp() ==
+ rtp_timestamp)
+ << "Frame buffer's next decodable frame was not the one sent for "
+ "extraction rtp="
+ << rtp_timestamp << " next="
+ << buffer_->NextDecodableTemporalUnitRtpTimestamp().value_or(-1);
+ auto frames = buffer_->ExtractNextDecodableTemporalUnit();
+ OnFrameReady(std::move(frames), render_time);
+ }
+
+ TimeDelta MaxWait() const RTC_RUN_ON(&worker_sequence_checker_) {
+ return keyframe_required_ ? max_wait_for_keyframe_ : max_wait_for_frame_;
+ }
+
+ void UpdateDroppedFrames() RTC_RUN_ON(&worker_sequence_checker_) {
+ const int dropped_frames = buffer_->GetTotalNumberOfDroppedFrames() -
+ frames_dropped_before_last_new_frame_;
+ if (dropped_frames > 0)
+ stats_proxy_->OnDroppedFrames(dropped_frames);
+ frames_dropped_before_last_new_frame_ =
+ buffer_->GetTotalNumberOfDroppedFrames();
+ }
+
+ void UpdateJitterDelay() {
+ int max_decode_ms;
+ int current_delay_ms;
+ int target_delay_ms;
+ int jitter_buffer_ms;
+ int min_playout_delay_ms;
+ int render_delay_ms;
+ if (timing_->GetTimings(&max_decode_ms, ¤t_delay_ms, &target_delay_ms,
+ &jitter_buffer_ms, &min_playout_delay_ms,
+ &render_delay_ms)) {
+ stats_proxy_->OnFrameBufferTimingsUpdated(
+ max_decode_ms, current_delay_ms, target_delay_ms, jitter_buffer_ms,
+ min_playout_delay_ms, render_delay_ms);
+ }
+ }
+
+ void UpdateTimingFrameInfo() {
+ absl::optional<TimingFrameInfo> info = timing_->GetTimingFrameInfo();
+ if (info)
+ stats_proxy_->OnTimingFrameInfoUpdated(*info);
+ }
+
+ bool IsTooManyFramesQueued() const RTC_RUN_ON(&worker_sequence_checker_) {
+ return buffer_->CurrentSize() > zero_playout_delay_max_decode_queue_size_;
+ }
+
+ void ForceKeyFrameReleaseImmediately() RTC_RUN_ON(&worker_sequence_checker_) {
+ RTC_DCHECK(keyframe_required_);
+ // Iterate through the frame buffer until there is a complete keyframe and
+ // release this right away.
+ while (buffer_->NextDecodableTemporalUnitRtpTimestamp()) {
+ auto next_frame = buffer_->ExtractNextDecodableTemporalUnit();
+ if (next_frame.empty()) {
+ RTC_DCHECK_NOTREACHED()
+ << "Frame buffer should always return at least 1 frame.";
+ continue;
+ }
+ // Found keyframe - decode right away.
+ if (next_frame.front()->is_keyframe()) {
+ auto render_time = Timestamp::Millis(timing_->RenderTimeMs(
+ next_frame.front()->Timestamp(), clock_->TimeInMilliseconds()));
+ OnFrameReady(std::move(next_frame), render_time);
+ return;
+ }
+ }
+ }
+
+ void MaybeScheduleFrameForRelease() RTC_RUN_ON(&worker_sequence_checker_) {
+ if (!decoder_ready_for_new_frame_ ||
+ !buffer_->NextDecodableTemporalUnitRtpTimestamp())
+ return;
+
+ if (keyframe_required_) {
+ return ForceKeyFrameReleaseImmediately();
+ }
+
+ // TODO(https://bugs.webrtc.org/13343): Make [next,last] decodable returned
+ // as an optional pair and remove this check.
+ RTC_CHECK(buffer_->LastDecodableTemporalUnitRtpTimestamp());
+ auto last_rtp = *buffer_->LastDecodableTemporalUnitRtpTimestamp();
+
+ // If already scheduled then abort.
+ if (frame_decode_scheduler_.scheduled_rtp() ==
+ buffer_->NextDecodableTemporalUnitRtpTimestamp())
+ return;
+
+ absl::optional<FrameDecodeTiming::FrameSchedule> schedule;
+ while (buffer_->NextDecodableTemporalUnitRtpTimestamp()) {
+ auto next_rtp = *buffer_->NextDecodableTemporalUnitRtpTimestamp();
+ schedule = decode_timing_.OnFrameBufferUpdated(next_rtp, last_rtp,
+ IsTooManyFramesQueued());
+ if (schedule) {
+ // Don't schedule if already waiting for the same frame.
+ if (frame_decode_scheduler_.scheduled_rtp() != next_rtp) {
+ frame_decode_scheduler_.CancelOutstanding();
+ frame_decode_scheduler_.ScheduleFrame(next_rtp, *schedule);
+ }
+ return;
+ }
+ // If no schedule for current rtp, drop and try again.
+ buffer_->DropNextDecodableTemporalUnit();
+ }
+ }
+
+ RTC_NO_UNIQUE_ADDRESS SequenceChecker worker_sequence_checker_;
+ const TimeDelta max_wait_for_keyframe_;
+ const TimeDelta max_wait_for_frame_;
+ const absl::optional<RttMultExperiment::Settings> rtt_mult_settings_ =
+ RttMultExperiment::GetRttMultValue();
+ Clock* const clock_;
+ TaskQueueBase* const worker_queue_;
+ rtc::TaskQueue* const decode_queue_;
+ VCMReceiveStatisticsCallback* const stats_proxy_;
+ FrameSchedulingReceiver* const receiver_;
+ VCMTiming* const timing_;
+
+ VCMJitterEstimator jitter_estimator_
+ RTC_GUARDED_BY(&worker_sequence_checker_);
+ VCMInterFrameDelay inter_frame_delay_
+ RTC_GUARDED_BY(&worker_sequence_checker_);
+ bool keyframe_required_ RTC_GUARDED_BY(&worker_sequence_checker_) = false;
+ std::unique_ptr<FrameBuffer> buffer_
+ RTC_GUARDED_BY(&worker_sequence_checker_);
+ FrameDecodeScheduler frame_decode_scheduler_
+ RTC_GUARDED_BY(&worker_sequence_checker_);
+ FrameDecodeTiming decode_timing_ RTC_GUARDED_BY(&worker_sequence_checker_);
+ VideoReceiveStreamTimeoutTracker timeout_tracker_
+ RTC_GUARDED_BY(&worker_sequence_checker_);
+ int frames_dropped_before_last_new_frame_
+ RTC_GUARDED_BY(&worker_sequence_checker_) = 0;
+ VCMVideoProtection protection_mode_
+ RTC_GUARDED_BY(&worker_sequence_checker_) = kProtectionNack;
+
+ // This flag guards frames from queuing in front of the decoder. Without this
+ // guard, encoded frames will not wait for the decoder to finish decoding a
+ // frame and just queue up, meaning frames will not be dropped or
+ // fast-forwarded when the decoder is slow or hangs.
+ bool decoder_ready_for_new_frame_ RTC_GUARDED_BY(&worker_sequence_checker_) =
+ false;
+
+ // Maximum number of frames in the decode queue to allow pacing. If the
+ // queue grows beyond the max limit, pacing will be disabled and frames will
+ // be pushed to the decoder as soon as possible. This only has an effect
+ // when the low-latency rendering path is active, which is indicated by
+ // the frame's render time == 0.
+ FieldTrialParameter<unsigned> zero_playout_delay_max_decode_queue_size_;
+
+ rtc::scoped_refptr<PendingTaskSafetyFlag> decode_safety_ =
+ PendingTaskSafetyFlag::CreateDetached();
+ ScopedTaskSafety worker_safety_;
+};
+
std::unique_ptr<FrameBufferProxy> FrameBufferProxy::CreateFromFieldTrial(
Clock* clock,
TaskQueueBase* worker_queue,
@@ -124,6 +509,10 @@
FrameSchedulingReceiver* receiver,
TimeDelta max_wait_for_keyframe,
TimeDelta max_wait_for_frame) {
+ if (field_trial::IsEnabled("WebRTC-FrameBuffer3"))
+ return std::make_unique<FrameBuffer3Proxy>(
+ clock, worker_queue, timing, stats_proxy, decode_queue, receiver,
+ max_wait_for_keyframe, max_wait_for_frame);
return std::make_unique<FrameBuffer2Proxy>(
clock, timing, stats_proxy, decode_queue, receiver, max_wait_for_keyframe,
max_wait_for_frame);
diff --git a/video/frame_buffer_proxy_unittest.cc b/video/frame_buffer_proxy_unittest.cc
index c8aa114..408fac9 100644
--- a/video/frame_buffer_proxy_unittest.cc
+++ b/video/frame_buffer_proxy_unittest.cc
@@ -12,21 +12,28 @@
#include <stdint.h>
+#include <limits>
#include <memory>
#include <string>
#include <utility>
#include <vector>
+#include "absl/types/optional.h"
+#include "absl/types/variant.h"
+#include "api/units/frequency.h"
#include "api/units/time_delta.h"
#include "api/units/timestamp.h"
#include "api/video/video_content_type.h"
+#include "rtc_base/checks.h"
#include "rtc_base/event.h"
+#include "system_wrappers/include/field_trial.h"
#include "test/field_trial.h"
#include "test/gmock.h"
#include "test/gtest.h"
#include "test/run_loop.h"
#include "test/time_controller/simulated_time_controller.h"
+using ::testing::_;
using ::testing::AllOf;
using ::testing::Contains;
using ::testing::Each;
@@ -38,6 +45,7 @@
using ::testing::Optional;
using ::testing::Pointee;
using ::testing::SizeIs;
+using ::testing::VariantWith;
namespace webrtc {
@@ -62,15 +70,18 @@
class FakeEncodedFrame : public EncodedFrame {
public:
// Always 10ms delay and on time.
- int64_t ReceivedTime() const override {
- if (Timestamp() == 0)
- return kClockStart.ms();
- return TimeDelta::Seconds(Timestamp() / 90000.0).ms() + kClockStart.ms();
- }
+ int64_t ReceivedTime() const override { return received_time_; }
int64_t RenderTime() const override { return _renderTimeMs; }
+
+ void SetReceivedTime(int64_t received_time) {
+ received_time_ = received_time;
+ }
+
+ private:
+ int64_t received_time_;
};
-MATCHER_P(FrameWithId, id, "") {
+MATCHER_P(WithId, id, "") {
return Matches(Eq(id))(arg.Id());
}
@@ -78,6 +89,14 @@
return Matches(Eq(id))(arg.size());
}
+auto TimedOut() {
+ return Optional(VariantWith<TimeDelta>(_));
+}
+
+auto Frame(testing::Matcher<EncodedFrame> m) {
+ return Optional(VariantWith<std::unique_ptr<EncodedFrame>>(Pointee(m)));
+}
+
class Builder {
public:
Builder& Time(uint32_t rtp_timestamp) {
@@ -104,6 +123,10 @@
spatial_layer_ = spatial_layer;
return *this;
}
+ Builder& ReceivedTime(Timestamp receive_time) {
+ received_time_ = receive_time;
+ return *this;
+ }
std::unique_ptr<FakeEncodedFrame> Build() {
RTC_CHECK_LE(references_.size(), EncodedFrame::kMaxFrameReferences);
@@ -126,6 +149,15 @@
if (spatial_layer_) {
frame->SetSpatialIndex(spatial_layer_);
}
+ if (received_time_) {
+ frame->SetReceivedTime(received_time_->ms());
+ } else {
+ if (*rtp_timestamp_ == 0)
+ frame->SetReceivedTime(kClockStart.ms());
+ frame->SetReceivedTime(
+ TimeDelta::Seconds(*rtp_timestamp_ / 90000.0).ms() +
+ kClockStart.ms());
+ }
return frame;
}
@@ -135,6 +167,7 @@
absl::optional<int64_t> frame_id_;
absl::optional<VideoPlayoutDelay> playout_delay_;
absl::optional<int> spatial_layer_;
+ absl::optional<Timestamp> received_time_;
bool last_spatial_layer_ = false;
std::vector<int64_t> references_;
};
@@ -200,31 +233,40 @@
}
void OnEncodedFrame(std::unique_ptr<EncodedFrame> frame) override {
- last_frame_ = std::move(frame);
- run_loop_.Quit();
+ RTC_DCHECK(frame);
+ SetWaitResult(std::move(frame));
}
void OnDecodableFrameTimeout(TimeDelta wait_time) override {
- timeouts_++;
- run_loop_.Quit();
+ SetWaitResult(wait_time);
}
- bool WaitForFrameOrTimeout(TimeDelta wait) {
- if (NewFrameOrTimeout()) {
- return true;
+ using WaitResult =
+ absl::variant<std::unique_ptr<EncodedFrame>, TimeDelta /*wait_time*/>;
+
+ absl::optional<WaitResult> WaitForFrameOrTimeout(TimeDelta wait) {
+ if (wait_result_) {
+ return std::move(wait_result_);
}
run_loop_.PostTask([&] { time_controller_.AdvanceTime(wait); });
run_loop_.PostTask([&] {
- // If run loop posted to a task queue, flush that.
+ if (wait_result_)
+ return;
+
+ // If run loop posted to a task queue, flush that if there is no result.
time_controller_.AdvanceTime(TimeDelta::Zero());
+ if (wait_result_)
+ return;
run_loop_.PostTask([&] {
time_controller_.AdvanceTime(TimeDelta::Zero());
- run_loop_.Quit();
+ // Quit if there is no result set.
+ if (!wait_result_)
+ run_loop_.Quit();
});
});
run_loop_.Run();
- return NewFrameOrTimeout();
+ return std::move(wait_result_);
}
void StartNextDecode() {
@@ -239,13 +281,8 @@
time_controller_.AdvanceTime(TimeDelta::Zero());
}
- void ResetLastResult() {
- last_frame_.reset();
- last_timeouts_ = timeouts_;
- }
+ void ResetLastResult() { wait_result_.reset(); }
- int timeouts() const { return timeouts_; }
- EncodedFrame* last_frame() const { return last_frame_.get(); }
int dropped_frames() const { return dropped_frames_; }
protected:
@@ -259,31 +296,31 @@
std::unique_ptr<FrameBufferProxy> proxy_;
private:
- bool NewFrameOrTimeout() const {
- return last_frame_ || timeouts_ != last_timeouts_;
+ void SetWaitResult(WaitResult result) {
+ RTC_DCHECK(!wait_result_);
+ if (absl::holds_alternative<std::unique_ptr<EncodedFrame>>(result)) {
+ RTC_DCHECK(absl::get<std::unique_ptr<EncodedFrame>>(result));
+ }
+ wait_result_.emplace(std::move(result));
+ run_loop_.Quit();
}
- int timeouts_ = 0;
- int last_timeouts_ = 0;
- std::unique_ptr<EncodedFrame> last_frame_;
uint32_t dropped_frames_ = 0;
+ absl::optional<WaitResult> wait_result_;
};
TEST_P(FrameBufferProxyTest, InitialTimeoutAfterKeyframeTimeoutPeriod) {
StartNextDecodeForceKeyframe();
// No frame insterted. Timeout expected.
- EXPECT_TRUE(WaitForFrameOrTimeout(kMaxWaitForKeyframe));
- EXPECT_EQ(timeouts(), 1);
+ EXPECT_THAT(WaitForFrameOrTimeout(kMaxWaitForKeyframe), TimedOut());
// No new timeout set since receiver has not started new decode.
ResetLastResult();
- EXPECT_FALSE(WaitForFrameOrTimeout(kMaxWaitForKeyframe));
- EXPECT_EQ(timeouts(), 1);
+ EXPECT_THAT(WaitForFrameOrTimeout(kMaxWaitForKeyframe), Eq(absl::nullopt));
// Now that receiver has asked for new frame, a new timeout can occur.
StartNextDecodeForceKeyframe();
- EXPECT_TRUE(WaitForFrameOrTimeout(kMaxWaitForKeyframe));
- EXPECT_EQ(timeouts(), 2);
+ EXPECT_THAT(WaitForFrameOrTimeout(kMaxWaitForKeyframe), TimedOut());
}
TEST_P(FrameBufferProxyTest, KeyFramesAreScheduled) {
@@ -293,10 +330,7 @@
auto frame = Builder().Id(0).Time(0).AsLast().Build();
proxy_->InsertFrame(std::move(frame));
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Zero()));
-
- ASSERT_THAT(last_frame(), Pointee(FrameWithId(0)));
- EXPECT_EQ(timeouts(), 0);
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()), Frame(WithId(0)));
}
TEST_P(FrameBufferProxyTest, DeltaFrameTimeoutAfterKeyframeExtracted) {
@@ -305,8 +339,7 @@
time_controller_.AdvanceTime(TimeDelta::Millis(50));
auto frame = Builder().Id(0).Time(0).AsLast().Build();
proxy_->InsertFrame(std::move(frame));
- EXPECT_TRUE(WaitForFrameOrTimeout(kMaxWaitForKeyframe));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
+ EXPECT_THAT(WaitForFrameOrTimeout(kMaxWaitForKeyframe), Frame(WithId(0)));
StartNextDecode();
time_controller_.AdvanceTime(TimeDelta::Millis(50));
@@ -314,28 +347,22 @@
// Timeouts should now happen at the normal frequency.
const int expected_timeouts = 5;
for (int i = 0; i < expected_timeouts; ++i) {
- EXPECT_TRUE(WaitForFrameOrTimeout(kMaxWaitForFrame));
+ EXPECT_THAT(WaitForFrameOrTimeout(kMaxWaitForFrame), TimedOut());
StartNextDecode();
}
-
- EXPECT_EQ(timeouts(), expected_timeouts);
}
TEST_P(FrameBufferProxyTest, DependantFramesAreScheduled) {
StartNextDecodeForceKeyframe();
proxy_->InsertFrame(Builder().Id(0).Time(0).AsLast().Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Zero()));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()), Frame(WithId(0)));
StartNextDecode();
time_controller_.AdvanceTime(kFps30Delay);
proxy_->InsertFrame(
Builder().Id(1).Time(kFps30Rtp).AsLast().Refs({0}).Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay));
-
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(1)));
- EXPECT_EQ(timeouts(), 0);
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(1)));
}
TEST_P(FrameBufferProxyTest, SpatialLayersAreScheduled) {
@@ -343,9 +370,8 @@
proxy_->InsertFrame(Builder().Id(0).SpatialLayer(0).Time(0).Build());
proxy_->InsertFrame(Builder().Id(1).SpatialLayer(1).Time(0).Build());
proxy_->InsertFrame(Builder().Id(2).SpatialLayer(2).Time(0).AsLast().Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Zero()));
- EXPECT_THAT(last_frame(),
- Pointee(AllOf(FrameWithId(0), FrameWithSize(3 * kFrameSize))));
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()),
+ Frame(AllOf(WithId(0), FrameWithSize(3 * kFrameSize))));
proxy_->InsertFrame(Builder().Id(3).Time(kFps30Rtp).SpatialLayer(0).Build());
proxy_->InsertFrame(Builder().Id(4).Time(kFps30Rtp).SpatialLayer(1).Build());
@@ -353,18 +379,15 @@
Builder().Id(5).Time(kFps30Rtp).SpatialLayer(2).AsLast().Build());
StartNextDecode();
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay * 10));
- EXPECT_THAT(last_frame(),
- Pointee(AllOf(FrameWithId(3), FrameWithSize(3 * kFrameSize))));
- EXPECT_EQ(timeouts(), 0);
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay * 10),
+ Frame(AllOf(WithId(3), FrameWithSize(3 * kFrameSize))));
}
TEST_P(FrameBufferProxyTest, OutstandingFrameTasksAreCancelledAfterDeletion) {
StartNextDecodeForceKeyframe();
proxy_->InsertFrame(Builder().Id(0).Time(0).AsLast().Build());
// Get keyframe. Delta frame should now be scheduled.
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Zero()));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()), Frame(WithId(0)));
StartNextDecode();
proxy_->InsertFrame(
@@ -372,8 +395,7 @@
proxy_->StopOnWorker();
// Wait for 2x max wait time. Since we stopped, this should cause no timeouts
// or frame-ready callbacks.
- EXPECT_FALSE(WaitForFrameOrTimeout(kMaxWaitForFrame * 2));
- EXPECT_EQ(timeouts(), 0);
+ EXPECT_THAT(WaitForFrameOrTimeout(kMaxWaitForFrame * 2), Eq(absl::nullopt));
}
TEST_P(FrameBufferProxyTest, FramesWaitForDecoderToComplete) {
@@ -381,8 +403,7 @@
// Start with a keyframe.
proxy_->InsertFrame(Builder().Id(0).Time(0).AsLast().Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Zero()));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()), Frame(WithId(0)));
ResetLastResult();
// Insert a delta frame.
@@ -391,11 +412,10 @@
// Advancing time should not result in a frame since the scheduler has not
// been signalled that we are ready.
- EXPECT_FALSE(WaitForFrameOrTimeout(kFps30Delay));
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Eq(absl::nullopt));
// Signal ready.
StartNextDecode();
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(1)));
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(1)));
}
TEST_P(FrameBufferProxyTest, LateFrameDropped) {
@@ -405,8 +425,7 @@
// F0 --> F2
proxy_->InsertFrame(Builder().Id(0).Time(0).AsLast().Build());
// Start with a keyframe.
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Zero()));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()), Frame(WithId(0)));
StartNextDecode();
@@ -414,16 +433,14 @@
time_controller_.AdvanceTime(kFps30Delay * 2);
proxy_->InsertFrame(
Builder().Id(2).Time(2 * kFps30Rtp).AsLast().Refs({0}).Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(2)));
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(2)));
StartNextDecode();
proxy_->InsertFrame(
Builder().Id(1).Time(1 * kFps30Rtp).AsLast().Refs({0}).Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(kMaxWaitForFrame));
// Confirm frame 1 is never scheduled by timing out.
- EXPECT_EQ(timeouts(), 1);
+ EXPECT_THAT(WaitForFrameOrTimeout(kMaxWaitForFrame), TimedOut());
}
TEST_P(FrameBufferProxyTest, FramesFastForwardOnSystemHalt) {
@@ -434,8 +451,7 @@
proxy_->InsertFrame(Builder().Id(0).Time(0).AsLast().Build());
// Start with a keyframe.
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Zero()));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()), Frame(WithId(0)));
time_controller_.AdvanceTime(kFps30Delay);
proxy_->InsertFrame(
@@ -447,8 +463,7 @@
// Halting time should result in F1 being skipped.
time_controller_.AdvanceTime(kFps30Delay * 2);
StartNextDecode();
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(2)));
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(2)));
EXPECT_EQ(dropped_frames(), 1);
}
@@ -456,8 +471,7 @@
StartNextDecodeForceKeyframe();
// Initial keyframe.
proxy_->InsertFrame(Builder().Id(0).Time(0).AsLast().Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Zero()));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()), Frame(WithId(0)));
StartNextDecodeForceKeyframe();
@@ -467,8 +481,7 @@
Builder().Id(1).Time(kFps30Rtp).AsLast().Refs({0}).Build());
proxy_->InsertFrame(Builder().Id(2).Time(kFps30Rtp * 2).AsLast().Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(kMaxWaitForFrame));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(2)));
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay * 3), Frame(WithId(2)));
}
TEST_P(FrameBufferProxyTest, SlowDecoderDropsTemporalLayers) {
@@ -480,9 +493,8 @@
// F0 --> F2 --> F4
proxy_->InsertFrame(Builder().Id(0).Time(0).AsLast().Build());
// Keyframe received.
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Zero()));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
// Don't start next decode until slow delay.
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()), Frame(WithId(0)));
time_controller_.AdvanceTime(kFps30Delay);
proxy_->InsertFrame(
@@ -494,9 +506,8 @@
// Simulate decode taking 3x FPS rate.
time_controller_.AdvanceTime(kFps30Delay * 1.5);
StartNextDecode();
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay * 2));
// F2 is the best frame since decoding was so slow that F1 is too old.
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(2)));
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay * 2), Frame(WithId(2)));
EXPECT_EQ(dropped_frames(), 1);
time_controller_.AdvanceTime(kFps30Delay / 2);
@@ -510,8 +521,7 @@
// F4 is the best frame since decoding was so slow that F1 is too old.
time_controller_.AdvanceTime(kFps30Delay);
StartNextDecode();
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(4)));
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(4)));
proxy_->InsertFrame(
Builder().Id(5).Time(5 * kFps30Rtp).Refs({3, 4}).AsLast().Build());
@@ -520,8 +530,7 @@
// F5 is not decodable since F4 was decoded, so a timeout is expected.
time_controller_.AdvanceTime(TimeDelta::Millis(10));
StartNextDecode();
- EXPECT_TRUE(WaitForFrameOrTimeout(kMaxWaitForFrame));
- EXPECT_EQ(timeouts(), 1);
+ EXPECT_THAT(WaitForFrameOrTimeout(kMaxWaitForFrame), TimedOut());
// TODO(bugs.webrtc.org/13343): This should be 2 dropped frames since frames 1
// and 3 were dropped. However, frame_buffer2 does not mark frame 3 as dropped
// which is a bug. Uncomment below when that is fixed for frame_buffer2 is
@@ -529,41 +538,23 @@
// EXPECT_EQ(dropped_frames(), 2);
}
-TEST_P(FrameBufferProxyTest, OldTimestampNotDecodable) {
- StartNextDecodeForceKeyframe();
-
- proxy_->InsertFrame(Builder().Id(0).Time(kFps30Rtp).AsLast().Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
-
- // Timestamp is before previous frame's.
- proxy_->InsertFrame(Builder().Id(1).Time(0).AsLast().Build());
- StartNextDecode();
- // F1 should be dropped since its timestamp went backwards.
- EXPECT_TRUE(WaitForFrameOrTimeout(kMaxWaitForFrame));
- EXPECT_EQ(timeouts(), 1);
-}
-
TEST_P(FrameBufferProxyTest, NewFrameInsertedWhileWaitingToReleaseFrame) {
StartNextDecodeForceKeyframe();
// Initial keyframe.
proxy_->InsertFrame(Builder().Id(0).Time(0).AsLast().Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Zero()));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()), Frame(WithId(0)));
time_controller_.AdvanceTime(kFps30Delay);
proxy_->InsertFrame(
Builder().Id(1).Time(kFps30Rtp).Refs({0}).AsLast().Build());
StartNextDecode();
- EXPECT_FALSE(WaitForFrameOrTimeout(TimeDelta::Zero()));
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()), Eq(absl::nullopt));
// Scheduler is waiting to deliver Frame 1 now. Insert Frame 2. Frame 1 should
// be delivered still.
proxy_->InsertFrame(
Builder().Id(2).Time(kFps30Rtp * 2).Refs({0}).AsLast().Build());
-
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(1)));
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(1)));
}
TEST_P(FrameBufferProxyTest, SameFrameNotScheduledTwice) {
@@ -580,16 +571,14 @@
// First keyframe.
proxy_->InsertFrame(Builder().Id(0).Time(0).AsLast().Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Millis(15)));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Millis(15)), Frame(WithId(0)));
StartNextDecode();
// Warmup VCMTiming for 30fps.
for (int i = 1; i <= 30; ++i) {
proxy_->InsertFrame(Builder().Id(i).Time(i * kFps30Rtp).AsLast().Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(i)));
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(i)));
StartNextDecode();
}
@@ -605,12 +594,10 @@
time_controller_.AdvanceTime(kFps30Delay / 2);
proxy_->InsertFrame(Builder().Id(31).Time(31 * kFps30Rtp).AsLast().Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(32)));
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(32)));
StartNextDecode();
- EXPECT_TRUE(WaitForFrameOrTimeout(kFps30Delay));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(33)));
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(33)));
EXPECT_EQ(dropped_frames(), 1);
}
@@ -624,15 +611,73 @@
clock_->TimeInMilliseconds());
StartNextDecodeForceKeyframe();
proxy_->InsertFrame(Builder().Id(0).Time(0).AsLast().Build());
- EXPECT_TRUE(WaitForFrameOrTimeout(TimeDelta::Zero()));
- EXPECT_THAT(last_frame(), Pointee(FrameWithId(0)));
+ EXPECT_THAT(WaitForFrameOrTimeout(TimeDelta::Zero()), Frame(WithId(0)));
// Flush stats posted on the decode queue.
time_controller_.AdvanceTime(TimeDelta::Zero());
}
+TEST_P(FrameBufferProxyTest, NextFrameWithOldTimestamp) {
+ // Test inserting 31 frames and pause the stream for a long time before
+ // frame 32.
+ StartNextDecodeForceKeyframe();
+ constexpr uint32_t kBaseRtp = std::numeric_limits<uint32_t>::max() / 2;
+
+ // First keyframe. The receive time must be explicitly set in this test since
+ // the RTP derived time used in all tests does not work when the long pause
+ // happens later in the test.
+ proxy_->InsertFrame(Builder()
+ .Id(0)
+ .Time(kBaseRtp)
+ .ReceivedTime(clock_->CurrentTime())
+ .AsLast()
+ .Build());
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(0)));
+
+ // 1 more frame to warmup VCMTiming for 30fps.
+ StartNextDecode();
+ proxy_->InsertFrame(Builder()
+ .Id(1)
+ .Time(kBaseRtp + kFps30Rtp)
+ .ReceivedTime(clock_->CurrentTime())
+ .AsLast()
+ .Build());
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(1)));
+
+ // Pause the stream for such a long time it incurs an RTP timestamp rollover
+ // by over half.
+ constexpr uint32_t kLastRtp = kBaseRtp + kFps30Rtp;
+ constexpr uint32_t kRolloverRtp =
+ kLastRtp + std::numeric_limits<uint32_t>::max() / 2 + 1;
+ constexpr Frequency kRtpHz = Frequency::KiloHertz(90);
+ // Pause for corresponding delay such that RTP timestamp would increase this
+ // much at 30fps.
+ constexpr TimeDelta kRolloverDelay =
+ (std::numeric_limits<uint32_t>::max() / 2 + 1) / kRtpHz;
+
+ // Avoid timeout being set while waiting for the frame and before the receiver
+ // is ready.
+ ResetLastResult();
+ EXPECT_THAT(WaitForFrameOrTimeout(kMaxWaitForFrame), Eq(absl::nullopt));
+ time_controller_.AdvanceTime(kRolloverDelay - kMaxWaitForFrame);
+ StartNextDecode();
+ proxy_->InsertFrame(Builder()
+ .Id(2)
+ .Time(kRolloverRtp)
+ .ReceivedTime(clock_->CurrentTime())
+ .AsLast()
+ .Build());
+ // FrameBuffer2 drops the frame, while FrameBuffer3 will continue the stream.
+ if (field_trial::IsEnabled("WebRTC-FrameBuffer3")) {
+ EXPECT_THAT(WaitForFrameOrTimeout(kFps30Delay), Frame(WithId(2)));
+ } else {
+ EXPECT_THAT(WaitForFrameOrTimeout(kMaxWaitForFrame), TimedOut());
+ }
+}
+
INSTANTIATE_TEST_SUITE_P(FrameBufferProxy,
FrameBufferProxyTest,
- ::testing::Values("WebRTC-FrameBuffer3/Disabled/"));
+ ::testing::Values("WebRTC-FrameBuffer3/Disabled/",
+ "WebRTC-FrameBuffer3/Enabled/"));
} // namespace webrtc
diff --git a/video/frame_decode_scheduler.cc b/video/frame_decode_scheduler.cc
new file mode 100644
index 0000000..5696e10
--- /dev/null
+++ b/video/frame_decode_scheduler.cc
@@ -0,0 +1,64 @@
+/*
+ * Copyright (c) 2022 The WebRTC project authors. All Rights Reserved.
+ *
+ * Use of this source code is governed by a BSD-style license
+ * that can be found in the LICENSE file in the root of the source
+ * tree. An additional intellectual property rights grant can be found
+ * in the file PATENTS. All contributing project authors may
+ * be found in the AUTHORS file in the root of the source tree.
+ */
+
+#include "video/frame_decode_scheduler.h"
+
+#include <algorithm>
+#include <utility>
+
+#include "api/sequence_checker.h"
+#include "rtc_base/task_utils/to_queued_task.h"
+
+namespace webrtc {
+
+FrameDecodeScheduler::FrameDecodeScheduler(
+ Clock* clock,
+ TaskQueueBase* const bookkeeping_queue,
+ FrameReleaseCallback callback)
+ : clock_(clock),
+ bookkeeping_queue_(bookkeeping_queue),
+ callback_(std::move(callback)) {
+ RTC_DCHECK(clock_);
+ RTC_DCHECK(bookkeeping_queue_);
+}
+
+FrameDecodeScheduler::~FrameDecodeScheduler() {
+ RTC_DCHECK(!scheduled_rtp_) << "Outstanding scheduled rtp=" << *scheduled_rtp_
+ << ". Call CancelOutstanding before destruction.";
+}
+
+void FrameDecodeScheduler::ScheduleFrame(
+ uint32_t rtp,
+ FrameDecodeTiming::FrameSchedule schedule) {
+ RTC_DCHECK(!scheduled_rtp_.has_value())
+ << "Can not schedule two frames for release at the same time.";
+ scheduled_rtp_ = rtp;
+
+ TimeDelta wait = std::max(TimeDelta::Zero(),
+ schedule.max_decode_time - clock_->CurrentTime());
+ bookkeeping_queue_->PostDelayedTask(
+ ToQueuedTask(task_safety_.flag(),
+ [this, rtp, schedule] {
+ RTC_DCHECK_RUN_ON(bookkeeping_queue_);
+ // If the next frame rtp has changed since this task was
+ // this scheduled release should be skipped.
+ if (scheduled_rtp_ != rtp)
+ return;
+ scheduled_rtp_ = absl::nullopt;
+ callback_(rtp, schedule.render_time);
+ }),
+ wait.ms());
+}
+
+void FrameDecodeScheduler::CancelOutstanding() {
+ scheduled_rtp_ = absl::nullopt;
+}
+
+} // namespace webrtc
diff --git a/video/frame_decode_scheduler.h b/video/frame_decode_scheduler.h
new file mode 100644
index 0000000..6e1c1bd
--- /dev/null
+++ b/video/frame_decode_scheduler.h
@@ -0,0 +1,56 @@
+/*
+ * Copyright (c) 2022 The WebRTC project authors. All Rights Reserved.
+ *
+ * Use of this source code is governed by a BSD-style license
+ * that can be found in the LICENSE file in the root of the source
+ * tree. An additional intellectual property rights grant can be found
+ * in the file PATENTS. All contributing project authors may
+ * be found in the AUTHORS file in the root of the source tree.
+ */
+
+#ifndef VIDEO_FRAME_DECODE_SCHEDULER_H_
+#define VIDEO_FRAME_DECODE_SCHEDULER_H_
+
+#include <stdint.h>
+
+#include <functional>
+
+#include "absl/types/optional.h"
+#include "api/task_queue/task_queue_base.h"
+#include "api/units/timestamp.h"
+#include "rtc_base/task_utils/pending_task_safety_flag.h"
+#include "system_wrappers/include/clock.h"
+#include "video/frame_decode_timing.h"
+
+namespace webrtc {
+
+class FrameDecodeScheduler {
+ public:
+ // Invoked when a frame with `rtp_timestamp` is ready for decoding.
+ using FrameReleaseCallback =
+ std::function<void(uint32_t rtp_timestamp, Timestamp render_time)>;
+
+ FrameDecodeScheduler(Clock* clock,
+ TaskQueueBase* const bookkeeping_queue,
+ FrameReleaseCallback callback);
+ ~FrameDecodeScheduler();
+ FrameDecodeScheduler(const FrameDecodeScheduler&) = delete;
+ FrameDecodeScheduler& operator=(const FrameDecodeScheduler&) = delete;
+
+ absl::optional<uint32_t> scheduled_rtp() const { return scheduled_rtp_; }
+
+ void ScheduleFrame(uint32_t rtp, FrameDecodeTiming::FrameSchedule schedule);
+ void CancelOutstanding();
+
+ private:
+ Clock* const clock_;
+ TaskQueueBase* const bookkeeping_queue_;
+ const FrameReleaseCallback callback_;
+
+ absl::optional<uint32_t> scheduled_rtp_;
+ ScopedTaskSafetyDetached task_safety_;
+};
+
+} // namespace webrtc
+
+#endif // VIDEO_FRAME_DECODE_SCHEDULER_H_
diff --git a/video/frame_decode_scheduler_unittest.cc b/video/frame_decode_scheduler_unittest.cc
new file mode 100644
index 0000000..e30863c
--- /dev/null
+++ b/video/frame_decode_scheduler_unittest.cc
@@ -0,0 +1,126 @@
+/*
+ * Copyright (c) 2022 The WebRTC project authors. All Rights Reserved.
+ *
+ * Use of this source code is governed by a BSD-style license
+ * that can be found in the LICENSE file in the root of the source
+ * tree. An additional intellectual property rights grant can be found
+ * in the file PATENTS. All contributing project authors may
+ * be found in the AUTHORS file in the root of the source tree.
+ */
+
+#include "video/frame_decode_scheduler.h"
+
+#include <stddef.h>
+
+#include <memory>
+#include <utility>
+
+#include "absl/types/optional.h"
+#include "api/units/time_delta.h"
+#include "api/units/timestamp.h"
+#include "rtc_base/task_queue.h"
+#include "test/gmock.h"
+#include "test/gtest.h"
+#include "test/time_controller/simulated_time_controller.h"
+
+namespace webrtc {
+
+using ::testing::Eq;
+using ::testing::Optional;
+
+class FrameDecodeSchedulerTest : public ::testing::Test {
+ public:
+ FrameDecodeSchedulerTest()
+ : time_controller_(Timestamp::Millis(2000)),
+ task_queue_(time_controller_.GetTaskQueueFactory()->CreateTaskQueue(
+ "scheduler",
+ TaskQueueFactory::Priority::NORMAL)),
+ scheduler_(std::make_unique<FrameDecodeScheduler>(
+ time_controller_.GetClock(),
+ task_queue_.Get(),
+ [this](uint32_t rtp, Timestamp render_time) {
+ OnFrame(rtp, render_time);
+ })) {}
+
+ ~FrameDecodeSchedulerTest() override {
+ if (scheduler_) {
+ OnQueue([&] {
+ scheduler_->CancelOutstanding();
+ scheduler_ = nullptr;
+ });
+ }
+ }
+
+ protected:
+ template <class Task>
+ void OnQueue(Task&& t) {
+ task_queue_.PostTask(std::forward<Task>(t));
+ time_controller_.AdvanceTime(TimeDelta::Zero());
+ }
+
+ GlobalSimulatedTimeController time_controller_;
+ rtc::TaskQueue task_queue_;
+ std::unique_ptr<FrameDecodeScheduler> scheduler_;
+ absl::optional<uint32_t> last_rtp_;
+ absl::optional<Timestamp> last_render_time_;
+
+ private:
+ void OnFrame(uint32_t timestamp, Timestamp render_time) {
+ last_rtp_ = timestamp;
+ last_render_time_ = render_time;
+ }
+};
+
+TEST_F(FrameDecodeSchedulerTest, FrameYieldedAfterSpecifiedPeriod) {
+ constexpr TimeDelta decode_delay = TimeDelta::Millis(5);
+ const Timestamp now = time_controller_.GetClock()->CurrentTime();
+ const uint32_t rtp = 90000;
+ const Timestamp render_time = now + TimeDelta::Millis(15);
+ OnQueue([&] {
+ scheduler_->ScheduleFrame(rtp, {.max_decode_time = now + decode_delay,
+ .render_time = render_time});
+ EXPECT_THAT(scheduler_->scheduled_rtp(), Optional(rtp));
+ });
+ EXPECT_THAT(last_rtp_, Eq(absl::nullopt));
+
+ time_controller_.AdvanceTime(decode_delay);
+ EXPECT_THAT(last_rtp_, Optional(rtp));
+ EXPECT_THAT(last_render_time_, Optional(render_time));
+}
+
+TEST_F(FrameDecodeSchedulerTest, NegativeDecodeDelayIsRoundedToZero) {
+ constexpr TimeDelta decode_delay = TimeDelta::Millis(-5);
+ const Timestamp now = time_controller_.GetClock()->CurrentTime();
+ const uint32_t rtp = 90000;
+ const Timestamp render_time = now + TimeDelta::Millis(15);
+ OnQueue([&] {
+ scheduler_->ScheduleFrame(rtp, {.max_decode_time = now + decode_delay,
+ .render_time = render_time});
+ EXPECT_THAT(scheduler_->scheduled_rtp(), Optional(rtp));
+ });
+ EXPECT_THAT(last_rtp_, Optional(rtp));
+ EXPECT_THAT(last_render_time_, Optional(render_time));
+}
+
+TEST_F(FrameDecodeSchedulerTest, CancelOutstanding) {
+ constexpr TimeDelta decode_delay = TimeDelta::Millis(50);
+ const Timestamp now = time_controller_.GetClock()->CurrentTime();
+ const uint32_t rtp = 90000;
+ OnQueue([&] {
+ scheduler_->ScheduleFrame(rtp,
+ {.max_decode_time = now + decode_delay,
+ .render_time = now + TimeDelta::Millis(75)});
+ EXPECT_THAT(scheduler_->scheduled_rtp(), Optional(rtp));
+ });
+ time_controller_.AdvanceTime(decode_delay / 2);
+ OnQueue([&] {
+ EXPECT_THAT(scheduler_->scheduled_rtp(), Optional(rtp));
+ scheduler_->CancelOutstanding();
+ EXPECT_THAT(scheduler_->scheduled_rtp(), Eq(absl::nullopt));
+ });
+ time_controller_.AdvanceTime(decode_delay / 2);
+ EXPECT_THAT(last_rtp_, Eq(absl::nullopt));
+ EXPECT_THAT(last_render_time_, Eq(absl::nullopt));
+}
+
+} // namespace webrtc
diff --git a/video/frame_decode_timing.cc b/video/frame_decode_timing.cc
new file mode 100644
index 0000000..7150bbc
--- /dev/null
+++ b/video/frame_decode_timing.cc
@@ -0,0 +1,58 @@
+/*
+ * Copyright (c) 2022 The WebRTC project authors. All Rights Reserved.
+ *
+ * Use of this source code is governed by a BSD-style license
+ * that can be found in the LICENSE file in the root of the source
+ * tree. An additional intellectual property rights grant can be found
+ * in the file PATENTS. All contributing project authors may
+ * be found in the AUTHORS file in the root of the source tree.
+ */
+
+#include "video/frame_decode_timing.h"
+
+#include "absl/types/optional.h"
+#include "rtc_base/logging.h"
+
+namespace webrtc {
+
+namespace {
+
+constexpr TimeDelta kMaxAllowedFrameDelay = TimeDelta::Millis(5);
+
+}
+
+FrameDecodeTiming::FrameDecodeTiming(Clock* clock,
+ webrtc::VCMTiming const* timing)
+ : clock_(clock), timing_(timing) {
+ RTC_DCHECK(clock_);
+ RTC_DCHECK(timing_);
+}
+
+absl::optional<FrameDecodeTiming::FrameSchedule>
+FrameDecodeTiming::OnFrameBufferUpdated(uint32_t next_temporal_unit_rtp,
+ uint32_t last_temporal_unit_rtp,
+ bool too_many_frames_queued) {
+ const Timestamp now = clock_->CurrentTime();
+ Timestamp render_time = Timestamp::Millis(
+ timing_->RenderTimeMs(next_temporal_unit_rtp, now.ms()));
+ TimeDelta max_wait = TimeDelta::Millis(timing_->MaxWaitingTime(
+ render_time.ms(), now.ms(), too_many_frames_queued));
+
+ // If the delay is not too far in the past, or this is the last decodable
+ // frame then it is the best frame to be decoded. Otherwise, fast-forward
+ // to the next frame in the buffer.
+ if (max_wait <= -kMaxAllowedFrameDelay &&
+ next_temporal_unit_rtp != last_temporal_unit_rtp) {
+ RTC_DLOG(LS_VERBOSE) << "Fast-forwarded frame " << next_temporal_unit_rtp
+ << " render time " << render_time.ms()
+ << " with delay " << max_wait.ms() << "ms";
+ return absl::nullopt;
+ }
+ RTC_DLOG(LS_VERBOSE) << "Selected frame with rtp " << next_temporal_unit_rtp
+ << " render time " << render_time.ms()
+ << " with a max wait of " << max_wait.ms() << "ms";
+ return FrameSchedule{.max_decode_time = now + max_wait,
+ .render_time = render_time};
+}
+
+} // namespace webrtc
diff --git a/video/frame_decode_timing.h b/video/frame_decode_timing.h
new file mode 100644
index 0000000..8c7353e
--- /dev/null
+++ b/video/frame_decode_timing.h
@@ -0,0 +1,49 @@
+/*
+ * Copyright (c) 2022 The WebRTC project authors. All Rights Reserved.
+ *
+ * Use of this source code is governed by a BSD-style license
+ * that can be found in the LICENSE file in the root of the source
+ * tree. An additional intellectual property rights grant can be found
+ * in the file PATENTS. All contributing project authors may
+ * be found in the AUTHORS file in the root of the source tree.
+ */
+
+#ifndef VIDEO_FRAME_DECODE_TIMING_H_
+#define VIDEO_FRAME_DECODE_TIMING_H_
+
+#include <stdint.h>
+
+#include <functional>
+
+#include "api/task_queue/task_queue_base.h"
+#include "modules/video_coding/timing.h"
+#include "rtc_base/task_utils/pending_task_safety_flag.h"
+#include "system_wrappers/include/clock.h"
+
+namespace webrtc {
+
+class FrameDecodeTiming {
+ public:
+ FrameDecodeTiming(Clock* clock, webrtc::VCMTiming const* timing);
+ ~FrameDecodeTiming() = default;
+ FrameDecodeTiming(const FrameDecodeTiming&) = delete;
+ FrameDecodeTiming& operator=(const FrameDecodeTiming&) = delete;
+
+ struct FrameSchedule {
+ Timestamp max_decode_time;
+ Timestamp render_time;
+ };
+
+ absl::optional<FrameSchedule> OnFrameBufferUpdated(
+ uint32_t next_temporal_unit_rtp,
+ uint32_t last_temporal_unit_rtp,
+ bool too_many_frames_queued);
+
+ private:
+ Clock* const clock_;
+ webrtc::VCMTiming const* const timing_;
+};
+
+} // namespace webrtc
+
+#endif // VIDEO_FRAME_DECODE_TIMING_H_
diff --git a/video/frame_decode_timing_unittest.cc b/video/frame_decode_timing_unittest.cc
new file mode 100644
index 0000000..ec77ed4
--- /dev/null
+++ b/video/frame_decode_timing_unittest.cc
@@ -0,0 +1,113 @@
+/*
+ * Copyright (c) 2022 The WebRTC project authors. All Rights Reserved.
+ *
+ * Use of this source code is governed by a BSD-style license
+ * that can be found in the LICENSE file in the root of the source
+ * tree. An additional intellectual property rights grant can be found
+ * in the file PATENTS. All contributing project authors may
+ * be found in the AUTHORS file in the root of the source tree.
+ */
+
+#include "video/frame_decode_timing.h"
+
+#include <stdint.h>
+
+#include "absl/types/optional.h"
+#include "api/units/time_delta.h"
+#include "modules/video_coding/timing.h"
+#include "rtc_base/containers/flat_map.h"
+#include "test/gmock.h"
+#include "test/gtest.h"
+
+namespace webrtc {
+
+using ::testing::AllOf;
+using ::testing::Eq;
+using ::testing::Field;
+using ::testing::Optional;
+
+namespace {
+
+class FakeVCMTiming : public webrtc::VCMTiming {
+ public:
+ explicit FakeVCMTiming(Clock* clock) : webrtc::VCMTiming(clock) {}
+
+ int64_t RenderTimeMs(uint32_t frame_timestamp,
+ int64_t now_ms) const override {
+ RTC_DCHECK(render_time_map_.contains(frame_timestamp));
+ auto it = render_time_map_.find(frame_timestamp);
+ return it->second.ms();
+ }
+
+ int64_t MaxWaitingTime(int64_t render_time_ms,
+ int64_t now_ms,
+ bool too_many_frames_queued) const override {
+ auto render_time = Timestamp::Millis(render_time_ms);
+ RTC_DCHECK(wait_time_map_.contains(render_time));
+ auto it = wait_time_map_.find(render_time);
+ return it->second.ms();
+ }
+
+ void SetTimes(uint32_t frame_timestamp,
+ Timestamp render_time,
+ TimeDelta max_decode_wait) {
+ render_time_map_.insert_or_assign(frame_timestamp, render_time);
+ wait_time_map_.insert_or_assign(render_time, max_decode_wait);
+ }
+
+ protected:
+ flat_map<uint32_t, Timestamp> render_time_map_;
+ flat_map<Timestamp, TimeDelta> wait_time_map_;
+};
+} // namespace
+
+class FrameDecodeTimingTest : public ::testing::Test {
+ public:
+ FrameDecodeTimingTest()
+ : clock_(Timestamp::Millis(1000)),
+ timing_(&clock_),
+ frame_decode_scheduler_(&clock_, &timing_) {}
+
+ protected:
+ SimulatedClock clock_;
+ FakeVCMTiming timing_;
+ FrameDecodeTiming frame_decode_scheduler_;
+};
+
+TEST_F(FrameDecodeTimingTest, ReturnsWaitTimesWhenValid) {
+ const TimeDelta decode_delay = TimeDelta::Millis(42);
+ const Timestamp render_time = clock_.CurrentTime() + TimeDelta::Millis(60);
+ timing_.SetTimes(90000, render_time, decode_delay);
+
+ EXPECT_THAT(
+ frame_decode_scheduler_.OnFrameBufferUpdated(90000, 180000, false),
+ Optional(AllOf(Field(&FrameDecodeTiming::FrameSchedule::max_decode_time,
+ Eq(clock_.CurrentTime() + decode_delay)),
+ Field(&FrameDecodeTiming::FrameSchedule::render_time,
+ Eq(render_time)))));
+}
+
+TEST_F(FrameDecodeTimingTest, FastForwardsFrameTooFarInThePast) {
+ const TimeDelta decode_delay = TimeDelta::Millis(-6);
+ const Timestamp render_time = clock_.CurrentTime();
+ timing_.SetTimes(90000, render_time, decode_delay);
+
+ EXPECT_THAT(
+ frame_decode_scheduler_.OnFrameBufferUpdated(90000, 180000, false),
+ Eq(absl::nullopt));
+}
+
+TEST_F(FrameDecodeTimingTest, NoFastForwardIfOnlyFrameToDecode) {
+ const TimeDelta decode_delay = TimeDelta::Millis(-6);
+ const Timestamp render_time = clock_.CurrentTime();
+ timing_.SetTimes(90000, render_time, decode_delay);
+
+ EXPECT_THAT(
+ frame_decode_scheduler_.OnFrameBufferUpdated(90000, 90000, false),
+ Optional(AllOf(Field(&FrameDecodeTiming::FrameSchedule::max_decode_time,
+ Eq(clock_.CurrentTime() + decode_delay)),
+ Field(&FrameDecodeTiming::FrameSchedule::render_time,
+ Eq(render_time)))));
+}
+
+} // namespace webrtc
diff --git a/video/receive_statistics_proxy2.cc b/video/receive_statistics_proxy2.cc
index 22da793..aabfa52 100644
--- a/video/receive_statistics_proxy2.cc
+++ b/video/receive_statistics_proxy2.cc
@@ -725,25 +725,33 @@
int jitter_buffer_ms,
int min_playout_delay_ms,
int render_delay_ms) {
- RTC_DCHECK_RUN_ON(&decode_queue_);
- worker_thread_->PostTask(ToQueuedTask(
- task_safety_,
- [max_decode_ms, current_delay_ms, target_delay_ms, jitter_buffer_ms,
- min_playout_delay_ms, render_delay_ms, this]() {
- RTC_DCHECK_RUN_ON(&main_thread_);
- stats_.max_decode_ms = max_decode_ms;
- stats_.current_delay_ms = current_delay_ms;
- stats_.target_delay_ms = target_delay_ms;
- stats_.jitter_buffer_ms = jitter_buffer_ms;
- stats_.min_playout_delay_ms = min_playout_delay_ms;
- stats_.render_delay_ms = render_delay_ms;
- jitter_buffer_delay_counter_.Add(jitter_buffer_ms);
- target_delay_counter_.Add(target_delay_ms);
- current_delay_counter_.Add(current_delay_ms);
- // Network delay (rtt/2) + target_delay_ms (jitter delay + decode time +
- // render delay).
- delay_counter_.Add(target_delay_ms + avg_rtt_ms_ / 2);
- }));
+ // Only called on main_thread_ with FrameBuffer3
+ if (!worker_thread_->IsCurrent()) {
+ RTC_DCHECK_RUN_ON(&decode_queue_);
+ worker_thread_->PostTask(ToQueuedTask(
+ task_safety_,
+ [max_decode_ms, current_delay_ms, target_delay_ms, jitter_buffer_ms,
+ min_playout_delay_ms, render_delay_ms, this]() {
+ OnFrameBufferTimingsUpdated(max_decode_ms, current_delay_ms,
+ target_delay_ms, jitter_buffer_ms,
+ min_playout_delay_ms, render_delay_ms);
+ }));
+ return;
+ }
+
+ RTC_DCHECK_RUN_ON(&main_thread_);
+ stats_.max_decode_ms = max_decode_ms;
+ stats_.current_delay_ms = current_delay_ms;
+ stats_.target_delay_ms = target_delay_ms;
+ stats_.jitter_buffer_ms = jitter_buffer_ms;
+ stats_.min_playout_delay_ms = min_playout_delay_ms;
+ stats_.render_delay_ms = render_delay_ms;
+ jitter_buffer_delay_counter_.Add(jitter_buffer_ms);
+ target_delay_counter_.Add(target_delay_ms);
+ current_delay_counter_.Add(current_delay_ms);
+ // Network delay (rtt/2) + target_delay_ms (jitter delay + decode time +
+ // render delay).
+ delay_counter_.Add(target_delay_ms + avg_rtt_ms_ / 2);
}
void ReceiveStatisticsProxy::OnUniqueFramesCounted(int num_unique_frames) {
@@ -753,25 +761,29 @@
void ReceiveStatisticsProxy::OnTimingFrameInfoUpdated(
const TimingFrameInfo& info) {
- RTC_DCHECK_RUN_ON(&decode_queue_);
- worker_thread_->PostTask(ToQueuedTask(task_safety_, [info, this]() {
- RTC_DCHECK_RUN_ON(&main_thread_);
- if (info.flags != VideoSendTiming::kInvalid) {
- int64_t now_ms = clock_->TimeInMilliseconds();
- timing_frame_info_counter_.Add(info, now_ms);
- }
+ // Only called on main_thread_ with FrameBuffer3
+ if (!worker_thread_->IsCurrent()) {
+ RTC_DCHECK_RUN_ON(&decode_queue_);
+ worker_thread_->PostTask(ToQueuedTask(
+ task_safety_, [info, this]() { OnTimingFrameInfoUpdated(info); }));
+ return;
+ }
+ RTC_DCHECK_RUN_ON(&main_thread_);
+ if (info.flags != VideoSendTiming::kInvalid) {
+ int64_t now_ms = clock_->TimeInMilliseconds();
+ timing_frame_info_counter_.Add(info, now_ms);
+ }
- // Measure initial decoding latency between the first frame arriving and
- // the first frame being decoded.
- if (!first_frame_received_time_ms_.has_value()) {
- first_frame_received_time_ms_ = info.receive_finish_ms;
- }
- if (stats_.first_frame_received_to_decoded_ms == -1 &&
- first_decoded_frame_time_ms_) {
- stats_.first_frame_received_to_decoded_ms =
- *first_decoded_frame_time_ms_ - *first_frame_received_time_ms_;
- }
- }));
+ // Measure initial decoding latency between the first frame arriving and
+ // the first frame being decoded.
+ if (!first_frame_received_time_ms_.has_value()) {
+ first_frame_received_time_ms_ = info.receive_finish_ms;
+ }
+ if (stats_.first_frame_received_to_decoded_ms == -1 &&
+ first_decoded_frame_time_ms_) {
+ stats_.first_frame_received_to_decoded_ms =
+ *first_decoded_frame_time_ms_ - *first_frame_received_time_ms_;
+ }
}
void ReceiveStatisticsProxy::RtcpPacketTypesCounterUpdated(
diff --git a/video/video_receive_stream2.cc b/video/video_receive_stream2.cc
index 395a15f..918413f 100644
--- a/video/video_receive_stream2.cc
+++ b/video/video_receive_stream2.cc
@@ -20,11 +20,14 @@
#include <utility>
#include "absl/algorithm/container.h"
+#include "absl/container/inlined_vector.h"
+#include "absl/functional/bind_front.h"
#include "absl/types/optional.h"
#include "api/array_view.h"
#include "api/crypto/frame_decryptor_interface.h"
#include "api/scoped_refptr.h"
#include "api/sequence_checker.h"
+#include "api/task_queue/task_queue_base.h"
#include "api/units/time_delta.h"
#include "api/video/encoded_image.h"
#include "api/video_codecs/h264_profile_level_id.h"
@@ -36,12 +39,17 @@
#include "call/rtx_receive_stream.h"
#include "common_video/include/incoming_video_stream.h"
#include "modules/video_coding/frame_buffer2.h"
+#include "modules/video_coding/frame_buffer3.h"
+#include "modules/video_coding/frame_helpers.h"
#include "modules/video_coding/include/video_codec_interface.h"
#include "modules/video_coding/include/video_coding_defines.h"
#include "modules/video_coding/include/video_error_codes.h"
+#include "modules/video_coding/inter_frame_delay.h"
+#include "modules/video_coding/jitter_estimator.h"
#include "modules/video_coding/timing.h"
#include "modules/video_coding/utility/vp8_header_parser.h"
#include "rtc_base/checks.h"
+#include "rtc_base/experiments/rtt_mult_experiment.h"
#include "rtc_base/location.h"
#include "rtc_base/logging.h"
#include "rtc_base/strings/string_builder.h"
@@ -50,13 +58,16 @@
#include "rtc_base/task_queue.h"
#include "rtc_base/task_utils/pending_task_safety_flag.h"
#include "rtc_base/task_utils/to_queued_task.h"
+#include "rtc_base/thread_annotations.h"
#include "rtc_base/time_utils.h"
#include "rtc_base/trace_event.h"
#include "system_wrappers/include/clock.h"
#include "system_wrappers/include/field_trial.h"
#include "video/call_stats2.h"
+#include "video/frame_decode_scheduler.h"
#include "video/frame_dumping_decoder.h"
#include "video/receive_statistics_proxy2.h"
+#include "video/video_receive_stream_timeout_tracker.h"
namespace webrtc {
@@ -259,7 +270,7 @@
timing_->set_render_delay(config_.render_delay_ms);
frame_buffer_ = FrameBufferProxy::CreateFromFieldTrial(
- clock_, call->worker_thread(), timing_.get(), &stats_proxy_,
+ clock_, call_->worker_thread(), timing_.get(), &stats_proxy_,
&decode_queue_, this, TimeDelta::Millis(max_wait_for_keyframe_ms_),
TimeDelta::Millis(max_wait_for_frame_ms_));
@@ -404,8 +415,8 @@
decode_queue_.PostTask([this] {
RTC_DCHECK_RUN_ON(&decode_queue_);
decoder_stopped_ = false;
- StartNextDecode();
});
+ frame_buffer_->StartNextDecode(true);
decoder_running_ = true;
{
@@ -646,6 +657,9 @@
RTC_DCHECK_RUN_ON(&worker_sequence_checker_);
// TODO(https://bugs.webrtc.org/9974): Consider removing this workaround.
+ // TODO(https://bugs.webrtc.org/13343): Remove this check when FrameBuffer3 is
+ // deployed. With FrameBuffer3, this case is properly handled and tested in
+ // the FrameBufferProxyTest.PausedStream unit test.
int64_t time_now_ms = clock_->TimeInMilliseconds();
if (last_complete_frame_time_ms_ > 0 &&
time_now_ms - last_complete_frame_time_ms_ > kInactiveStreamThresholdMs) {
@@ -730,8 +744,10 @@
return;
}
RTC_DCHECK_RUN_ON(&decode_queue_);
+ if (decoder_stopped_)
+ return;
HandleEncodedFrame(std::move(frame));
- StartNextDecode();
+ frame_buffer_->StartNextDecode(keyframe_required_);
}
void VideoReceiveStream2::OnDecodableFrameTimeout(TimeDelta wait_time) {
@@ -749,14 +765,10 @@
decode_queue_.PostTask([this] {
RTC_DCHECK_RUN_ON(&decode_queue_);
- StartNextDecode();
+ frame_buffer_->StartNextDecode(keyframe_required_);
});
}
-void VideoReceiveStream2::StartNextDecode() {
- frame_buffer_->StartNextDecode(keyframe_required_);
-}
-
void VideoReceiveStream2::HandleEncodedFrame(
std::unique_ptr<EncodedFrame> frame) {
// Running on `decode_queue_`.
diff --git a/video/video_receive_stream2.h b/video/video_receive_stream2.h
index 970e9bd..665faca 100644
--- a/video/video_receive_stream2.h
+++ b/video/video_receive_stream2.h
@@ -189,7 +189,6 @@
void OnDecodableFrameTimeout(TimeDelta wait_time) override;
void CreateAndRegisterExternalDecoder(const Decoder& decoder);
int64_t GetMaxWaitMs() const RTC_RUN_ON(decode_queue_);
- void StartNextDecode() RTC_RUN_ON(decode_queue_);
void HandleEncodedFrame(std::unique_ptr<EncodedFrame> frame)
RTC_RUN_ON(decode_queue_);
void HandleFrameBufferTimeout(int64_t now_ms, int64_t wait_ms)
diff --git a/video/video_receive_stream2_unittest.cc b/video/video_receive_stream2_unittest.cc
index 223523c..f3a5338 100644
--- a/video/video_receive_stream2_unittest.cc
+++ b/video/video_receive_stream2_unittest.cc
@@ -13,6 +13,7 @@
#include <algorithm>
#include <limits>
#include <memory>
+#include <tuple>
#include <utility>
#include <vector>
@@ -78,7 +79,8 @@
: task_queue_factory_(CreateDefaultTaskQueueFactory()),
h264_decoder_factory_(&mock_h264_video_decoder_),
config_(&mock_transport_, &h264_decoder_factory_),
- call_stats_(Clock::GetRealTimeClock(), loop_.task_queue()) {}
+ call_stats_(Clock::GetRealTimeClock(), loop_.task_queue()),
+ field_trials_("WebRTC-FrameBuffer3/Enabled/") {}
~VideoReceiveStream2Test() override {
if (video_receive_stream_)
video_receive_stream_->UnregisterFromTransport();
@@ -125,6 +127,7 @@
std::unique_ptr<webrtc::internal::VideoReceiveStream2> video_receive_stream_;
Clock* clock_;
VCMTiming* timing_;
+ const test::ScopedFieldTrials field_trials_;
};
TEST_F(VideoReceiveStream2Test, CreateFrameFromH264FmtpSpropAndIdr) {
@@ -476,7 +479,7 @@
}
class VideoReceiveStream2TestWithSimulatedClock
- : public ::testing::TestWithParam<int> {
+ : public ::testing::TestWithParam<std::tuple<int, bool>> {
public:
class FakeRenderer : public rtc::VideoSinkInterface<webrtc::VideoFrame> {
public:
@@ -524,7 +527,7 @@
VideoReceiveStream::Config config(transport, decoder_factory);
config.rtp.remote_ssrc = 1111;
config.rtp.local_ssrc = 2222;
- config.rtp.nack.rtp_history_ms = GetParam(); // rtx-time.
+ config.rtp.nack.rtp_history_ms = std::get<0>(GetParam()); // rtx-time.
config.renderer = renderer;
VideoReceiveStream::Decoder fake_decoder;
fake_decoder.payload_type = 99;
@@ -535,6 +538,9 @@
VideoReceiveStream2TestWithSimulatedClock()
: time_controller_(Timestamp::Millis(4711)),
+ field_trials_(std::get<1>(GetParam())
+ ? "WebRTC-FrameBuffer3/Enabled/"
+ : "WebRTC-FrameBuffer3/Disabled/"),
fake_decoder_factory_([this] {
return std::make_unique<FakeDecoder2>([this] { OnFrameDecoded(); });
}),
@@ -567,11 +573,15 @@
// This call will eventually end up in the Decoded method where the
// event is set.
video_receive_stream_.OnCompleteFrame(std::move(frame));
- event_->Wait(rtc::Event::kForever);
+ // FrameBuffer3 runs on the test sequence so flush to ensure that decoding
+ // happens.
+ loop_.Flush();
+ ASSERT_TRUE(event_->Wait(1000));
}
protected:
GlobalSimulatedTimeController time_controller_;
+ test::ScopedFieldTrials field_trials_;
test::RunLoop loop_;
test::FunctionVideoDecoderFactory fake_decoder_factory_;
MockTransport mock_transport_;
@@ -588,28 +598,37 @@
TEST_P(VideoReceiveStream2TestWithSimulatedClock,
RequestsKeyFramesUntilKeyFrameReceived) {
- auto tick = TimeDelta::Millis(GetParam() / 2);
- EXPECT_CALL(mock_transport_, SendRtcp).Times(1).WillOnce(Invoke([this]() {
- loop_.Quit();
- return 0;
- }));
+ auto tick = TimeDelta::Millis(std::get<0>(GetParam()) / 2);
+ bool sent_rtcp = false;
+ EXPECT_CALL(mock_transport_, SendRtcp)
+ .Times(1)
+ .WillOnce(Invoke([&sent_rtcp]() {
+ sent_rtcp = true;
+ return 0;
+ }));
video_receive_stream_.GenerateKeyFrame();
PassEncodedFrameAndWait(MakeFrame(VideoFrameType::kVideoFrameDelta, 0));
time_controller_.AdvanceTime(tick);
+ loop_.Flush();
PassEncodedFrameAndWait(MakeFrame(VideoFrameType::kVideoFrameDelta, 1));
- loop_.Run();
testing::Mock::VerifyAndClearExpectations(&mock_transport_);
+ EXPECT_TRUE(sent_rtcp);
// T+keyframetimeout: still no key frame received, expect key frame request
// sent again.
- EXPECT_CALL(mock_transport_, SendRtcp).Times(1).WillOnce(Invoke([this]() {
- loop_.Quit();
- return 0;
- }));
+ sent_rtcp = false;
+ EXPECT_CALL(mock_transport_, SendRtcp)
+ .Times(1)
+ .WillOnce(Invoke([&sent_rtcp]() {
+ sent_rtcp = true;
+ return 0;
+ }));
time_controller_.AdvanceTime(tick);
PassEncodedFrameAndWait(MakeFrame(VideoFrameType::kVideoFrameDelta, 2));
+ loop_.PostTask([this]() { loop_.Quit(); });
loop_.Run();
testing::Mock::VerifyAndClearExpectations(&mock_transport_);
+ EXPECT_TRUE(sent_rtcp);
// T+keyframetimeout: now send a key frame - we should not observe new key
// frame requests after this.
@@ -684,8 +703,10 @@
INSTANTIATE_TEST_SUITE_P(
RtxTime,
VideoReceiveStream2TestWithSimulatedClock,
- ::testing::Values(internal::VideoReceiveStream2::kMaxWaitForKeyFrameMs,
- 50 /*ms*/));
+ ::testing::Combine(
+ ::testing::Values(internal::VideoReceiveStream2::kMaxWaitForKeyFrameMs,
+ 50 /*ms*/),
+ ::testing::Bool()));
class VideoReceiveStream2TestWithLazyDecoderCreation : public ::testing::Test {
public:
diff --git a/video/video_receive_stream_timeout_tracker.cc b/video/video_receive_stream_timeout_tracker.cc
new file mode 100644
index 0000000..79e0e2c
--- /dev/null
+++ b/video/video_receive_stream_timeout_tracker.cc
@@ -0,0 +1,81 @@
+/*
+ * Copyright (c) 2022 The WebRTC project authors. All Rights Reserved.
+ *
+ * Use of this source code is governed by a BSD-style license
+ * that can be found in the LICENSE file in the root of the source
+ * tree. An additional intellectual property rights grant can be found
+ * in the file PATENTS. All contributing project authors may
+ * be found in the AUTHORS file in the root of the source tree.
+ */
+
+#include "video/video_receive_stream_timeout_tracker.h"
+
+#include <utility>
+
+namespace webrtc {
+
+VideoReceiveStreamTimeoutTracker::VideoReceiveStreamTimeoutTracker(
+ Clock* clock,
+ TaskQueueBase* const bookkeeping_queue,
+ const Timeouts& timeouts,
+ TimeoutCallback callback)
+ : clock_(clock),
+ bookkeeping_queue_(bookkeeping_queue),
+ timeouts_(timeouts),
+ callback_(std::move(callback)) {}
+
+VideoReceiveStreamTimeoutTracker::~VideoReceiveStreamTimeoutTracker() {
+ RTC_DCHECK(!timeout_task_.Running());
+}
+
+bool VideoReceiveStreamTimeoutTracker::Running() const {
+ return timeout_task_.Running();
+}
+
+void VideoReceiveStreamTimeoutTracker::Start(bool waiting_for_keyframe) {
+ RTC_DCHECK(!timeout_task_.Running());
+ waiting_for_keyframe_ = waiting_for_keyframe;
+ TimeDelta timeout_delay = TimeoutForNextFrame();
+ timeout_ = clock_->CurrentTime() + timeout_delay;
+ timeout_task_ = RepeatingTaskHandle::DelayedStart(
+ bookkeeping_queue_, timeout_delay, [this] {
+ RTC_DCHECK_RUN_ON(bookkeeping_queue_);
+ return HandleTimeoutTask();
+ });
+}
+
+void VideoReceiveStreamTimeoutTracker::Stop() {
+ timeout_task_.Stop();
+}
+
+void VideoReceiveStreamTimeoutTracker::SetWaitingForKeyframe() {
+ waiting_for_keyframe_ = true;
+ TimeDelta timeout_delay = TimeoutForNextFrame();
+ if (clock_->CurrentTime() + timeout_delay < timeout_) {
+ Stop();
+ Start(waiting_for_keyframe_);
+ }
+}
+
+void VideoReceiveStreamTimeoutTracker::OnEncodedFrameReleased() {
+ // If we were waiting for a keyframe, then it has just been released.
+ waiting_for_keyframe_ = false;
+ timeout_ = clock_->CurrentTime() + TimeoutForNextFrame();
+}
+
+TimeDelta VideoReceiveStreamTimeoutTracker::HandleTimeoutTask() {
+ Timestamp now = clock_->CurrentTime();
+ // `timeout_` is hit and we have timed out. Schedule the next timeout at
+ // the timeout delay.
+ if (now >= timeout_) {
+ TimeDelta timeout_delay = TimeoutForNextFrame();
+ timeout_ = now + timeout_delay;
+ callback_();
+ return timeout_delay;
+ }
+ // Otherwise, `timeout_` changed since we scheduled a timeout. Reschedule
+ // a timeout check.
+ return timeout_ - now;
+}
+
+} // namespace webrtc
diff --git a/video/video_receive_stream_timeout_tracker.h b/video/video_receive_stream_timeout_tracker.h
new file mode 100644
index 0000000..6993e13
--- /dev/null
+++ b/video/video_receive_stream_timeout_tracker.h
@@ -0,0 +1,65 @@
+/*
+ * Copyright (c) 2022 The WebRTC project authors. All Rights Reserved.
+ *
+ * Use of this source code is governed by a BSD-style license
+ * that can be found in the LICENSE file in the root of the source
+ * tree. An additional intellectual property rights grant can be found
+ * in the file PATENTS. All contributing project authors may
+ * be found in the AUTHORS file in the root of the source tree.
+ */
+
+#ifndef VIDEO_VIDEO_RECEIVE_STREAM_TIMEOUT_TRACKER_H_
+#define VIDEO_VIDEO_RECEIVE_STREAM_TIMEOUT_TRACKER_H_
+
+#include <functional>
+
+#include "api/task_queue/task_queue_base.h"
+#include "api/units/time_delta.h"
+#include "rtc_base/task_utils/repeating_task.h"
+#include "system_wrappers/include/clock.h"
+
+namespace webrtc {
+
+class VideoReceiveStreamTimeoutTracker {
+ public:
+ struct Timeouts {
+ TimeDelta max_wait_for_keyframe;
+ TimeDelta max_wait_for_frame;
+ };
+
+ using TimeoutCallback = std::function<void()>;
+ VideoReceiveStreamTimeoutTracker(Clock* clock,
+ TaskQueueBase* const bookkeeping_queue,
+ const Timeouts& timeouts,
+ TimeoutCallback callback);
+ ~VideoReceiveStreamTimeoutTracker();
+ VideoReceiveStreamTimeoutTracker(const VideoReceiveStreamTimeoutTracker&) =
+ delete;
+ VideoReceiveStreamTimeoutTracker& operator=(
+ const VideoReceiveStreamTimeoutTracker&) = delete;
+
+ bool Running() const;
+ void Start(bool waiting_for_keyframe);
+ void Stop();
+ void SetWaitingForKeyframe();
+ void OnEncodedFrameReleased();
+
+ private:
+ TimeDelta TimeoutForNextFrame() const {
+ return waiting_for_keyframe_ ? timeouts_.max_wait_for_keyframe
+ : timeouts_.max_wait_for_frame;
+ }
+ TimeDelta HandleTimeoutTask();
+
+ Clock* const clock_;
+ TaskQueueBase* const bookkeeping_queue_;
+ const Timeouts timeouts_;
+ const TimeoutCallback callback_;
+ RepeatingTaskHandle timeout_task_;
+
+ Timestamp timeout_ = Timestamp::MinusInfinity();
+ bool waiting_for_keyframe_;
+};
+} // namespace webrtc
+
+#endif // VIDEO_VIDEO_RECEIVE_STREAM_TIMEOUT_TRACKER_H_
diff --git a/video/video_receive_stream_timeout_tracker_unittest.cc b/video/video_receive_stream_timeout_tracker_unittest.cc
new file mode 100644
index 0000000..4aec4d3
--- /dev/null
+++ b/video/video_receive_stream_timeout_tracker_unittest.cc
@@ -0,0 +1,95 @@
+/*
+ * Copyright (c) 2022 The WebRTC project authors. All Rights Reserved.
+ *
+ * Use of this source code is governed by a BSD-style license
+ * that can be found in the LICENSE file in the root of the source
+ * tree. An additional intellectual property rights grant can be found
+ * in the file PATENTS. All contributing project authors may
+ * be found in the AUTHORS file in the root of the source tree.
+ */
+
+#include "video/video_receive_stream_timeout_tracker.h"
+
+#include <utility>
+
+#include "api/task_queue/task_queue_base.h"
+#include "rtc_base/task_queue.h"
+#include "test/gmock.h"
+#include "test/gtest.h"
+#include "test/time_controller/simulated_time_controller.h"
+
+namespace webrtc {
+
+namespace {
+
+constexpr auto kMaxWaitForKeyframe = TimeDelta::Millis(500);
+constexpr auto kMaxWaitForFrame = TimeDelta::Millis(1500);
+constexpr VideoReceiveStreamTimeoutTracker::Timeouts config = {
+ kMaxWaitForKeyframe, kMaxWaitForFrame};
+} // namespace
+
+class VideoReceiveStreamTimeoutTrackerTest : public ::testing::Test {
+ public:
+ VideoReceiveStreamTimeoutTrackerTest()
+ : time_controller_(Timestamp::Millis(2000)),
+ task_queue_(time_controller_.GetTaskQueueFactory()->CreateTaskQueue(
+ "scheduler",
+ TaskQueueFactory::Priority::NORMAL)),
+ timeout_tracker_(time_controller_.GetClock(),
+ task_queue_.Get(),
+ config,
+ [this] { OnTimeout(); }) {}
+
+ protected:
+ template <class Task>
+ void OnQueue(Task&& t) {
+ task_queue_.PostTask(std::forward<Task>(t));
+ time_controller_.AdvanceTime(TimeDelta::Zero());
+ }
+
+ GlobalSimulatedTimeController time_controller_;
+ rtc::TaskQueue task_queue_;
+ VideoReceiveStreamTimeoutTracker timeout_tracker_;
+ int timeouts_ = 0;
+
+ private:
+ void OnTimeout() { ++timeouts_; }
+};
+
+TEST_F(VideoReceiveStreamTimeoutTrackerTest, TimeoutAfterInitialPeriod) {
+ OnQueue([&] { timeout_tracker_.Start(true); });
+ time_controller_.AdvanceTime(kMaxWaitForKeyframe);
+ EXPECT_EQ(1, timeouts_);
+ OnQueue([&] { timeout_tracker_.Stop(); });
+}
+
+TEST_F(VideoReceiveStreamTimeoutTrackerTest, NoTimeoutAfterStop) {
+ OnQueue([&] { timeout_tracker_.Start(true); });
+ time_controller_.AdvanceTime(kMaxWaitForKeyframe / 2);
+ OnQueue([&] { timeout_tracker_.Stop(); });
+ time_controller_.AdvanceTime(kMaxWaitForKeyframe);
+ EXPECT_EQ(0, timeouts_);
+}
+
+TEST_F(VideoReceiveStreamTimeoutTrackerTest, TimeoutForDeltaFrame) {
+ OnQueue([&] { timeout_tracker_.Start(true); });
+ time_controller_.AdvanceTime(TimeDelta::Millis(5));
+ OnQueue([&] { timeout_tracker_.OnEncodedFrameReleased(); });
+ time_controller_.AdvanceTime(kMaxWaitForFrame);
+ EXPECT_EQ(1, timeouts_);
+
+ OnQueue([&] { timeout_tracker_.Stop(); });
+}
+
+TEST_F(VideoReceiveStreamTimeoutTrackerTest, TimeoutForKeyframeWhenForced) {
+ OnQueue([&] { timeout_tracker_.Start(true); });
+ time_controller_.AdvanceTime(TimeDelta::Millis(5));
+ OnQueue([&] { timeout_tracker_.OnEncodedFrameReleased(); });
+ OnQueue([&] { timeout_tracker_.SetWaitingForKeyframe(); });
+ time_controller_.AdvanceTime(kMaxWaitForKeyframe);
+ EXPECT_EQ(1, timeouts_);
+
+ OnQueue([&] { timeout_tracker_.Stop(); });
+}
+
+} // namespace webrtc